From 900825c10932c82d66dcaaaabdedf3cb0f22928f Mon Sep 17 00:00:00 2001 From: Tom Pantelis Date: Thu, 26 Mar 2015 01:25:11 -0400 Subject: [PATCH] Refactor ReplicatedLogImpl to separate class To reduce the size of the RaftActor class for improved readability. Change-Id: I845cfeb4bc48f0c5eb96ba2cc0a925d9ce5fa416 Signed-off-by: Tom Pantelis --- .../controller/cluster/raft/RaftActor.java | 251 +++++------------- .../cluster/raft/RaftActorContext.java | 9 + .../cluster/raft/RaftActorContextImpl.java | 31 ++- .../cluster/raft/ReplicatedLog.java | 3 + .../cluster/raft/ReplicatedLogImpl.java | 140 ++++++++++ .../DelegatingRaftActorBehavior.java | 58 ++++ .../AbstractRaftActorIntegrationTest.java | 19 +- .../raft/AbstractReplicatedLogImplTest.java | 5 + .../cluster/raft/MockRaftActorContext.java | 29 ++ 9 files changed, 346 insertions(+), 199 deletions(-) create mode 100644 opendaylight/md-sal/sal-akka-raft/src/main/java/org/opendaylight/controller/cluster/raft/ReplicatedLogImpl.java create mode 100644 opendaylight/md-sal/sal-akka-raft/src/main/java/org/opendaylight/controller/cluster/raft/behaviors/DelegatingRaftActorBehavior.java diff --git a/opendaylight/md-sal/sal-akka-raft/src/main/java/org/opendaylight/controller/cluster/raft/RaftActor.java b/opendaylight/md-sal/sal-akka-raft/src/main/java/org/opendaylight/controller/cluster/raft/RaftActor.java index a13b6ff95a..1c30fe2317 100644 --- a/opendaylight/md-sal/sal-akka-raft/src/main/java/org/opendaylight/controller/cluster/raft/RaftActor.java +++ b/opendaylight/md-sal/sal-akka-raft/src/main/java/org/opendaylight/controller/cluster/raft/RaftActor.java @@ -43,7 +43,7 @@ import org.opendaylight.controller.cluster.raft.base.messages.CaptureSnapshot; import org.opendaylight.controller.cluster.raft.base.messages.CaptureSnapshotReply; import org.opendaylight.controller.cluster.raft.base.messages.Replicate; import org.opendaylight.controller.cluster.raft.behaviors.AbstractLeader; -import org.opendaylight.controller.cluster.raft.behaviors.AbstractRaftActorBehavior; +import org.opendaylight.controller.cluster.raft.behaviors.DelegatingRaftActorBehavior; import org.opendaylight.controller.cluster.raft.behaviors.Follower; import org.opendaylight.controller.cluster.raft.behaviors.RaftActorBehavior; import org.opendaylight.controller.cluster.raft.client.messages.FindLeader; @@ -107,7 +107,7 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { * The current state determines the current behavior of a RaftActor * A Raft Actor always starts off in the Follower State */ - private RaftActorBehavior currentBehavior; + private final DelegatingRaftActorBehavior currentBehavior = new DelegatingRaftActorBehavior(); /** * This context should NOT be passed directly to any other actor it is @@ -119,11 +119,6 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { private final Procedure createSnapshotProcedure = new CreateSnapshotProcedure(); - /** - * The in-memory journal - */ - private ReplicatedLogImpl replicatedLog = new ReplicatedLogImpl(); - private Stopwatch recoveryTimer; private int currentRecoveryBatchCount; @@ -139,9 +134,10 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { context = new RaftActorContextImpl(this.getSelf(), this.getContext(), id, new ElectionTermImpl(delegatingPersistenceProvider, id, LOG), - -1, -1, replicatedLog, peerAddresses, - (configParams.isPresent() ? configParams.get(): new DefaultConfigParamsImpl()), - LOG); + -1, -1, peerAddresses, + (configParams.isPresent() ? configParams.get(): new DefaultConfigParamsImpl()), LOG); + + context.setReplicatedLog(ReplicatedLogImpl.newInstance(context, delegatingPersistenceProvider, currentBehavior)); } private void initRecoveryTimer() { @@ -184,7 +180,7 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { } else if (message instanceof ApplyJournalEntries) { onRecoveredApplyLogEntries(((ApplyJournalEntries) message).getToIndex()); } else if (message instanceof DeleteEntries) { - replicatedLog.removeFrom(((DeleteEntries) message).getFromIndex()); + replicatedLog().removeFrom(((DeleteEntries) message).getFromIndex()); } else if (message instanceof UpdateElectionTerm) { context.getTermInformation().update(((UpdateElectionTerm) message).getCurrentTerm(), ((UpdateElectionTerm) message).getVotedFor()); @@ -219,9 +215,9 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { // Create a replicated log with the snapshot information // The replicated log can be used later on to retrieve this snapshot // when we need to install it on a peer - replicatedLog = new ReplicatedLogImpl(snapshot); - context.setReplicatedLog(replicatedLog); + context.setReplicatedLog(ReplicatedLogImpl.newInstance(snapshot, context, delegatingPersistenceProvider, + currentBehavior)); context.setLastApplied(snapshot.getLastAppliedIndex()); context.setCommitIndex(snapshot.getLastAppliedIndex()); @@ -232,8 +228,8 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { timer.stop(); LOG.info("Recovery snapshot applied for {} in {}: snapshotIndex={}, snapshotTerm={}, journal-size=" + - replicatedLog.size(), persistenceId(), timer.toString(), - replicatedLog.getSnapshotIndex(), replicatedLog.getSnapshotTerm()); + replicatedLog().size(), persistenceId(), timer.toString(), + replicatedLog().getSnapshotIndex(), replicatedLog().getSnapshotTerm()); } private void onRecoveredJournalLogEntry(ReplicatedLogEntry logEntry) { @@ -241,7 +237,7 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { LOG.debug("{}: Received ReplicatedLogEntry for recovery: {}", persistenceId(), logEntry.getIndex()); } - replicatedLog.append(logEntry); + replicatedLog().append(logEntry); } private void onRecoveredApplyLogEntries(long toIndex) { @@ -251,7 +247,7 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { } for (long i = context.getLastApplied() + 1; i <= toIndex; i++) { - batchRecoveredLogEntry(replicatedLog.get(i)); + batchRecoveredLogEntry(replicatedLog().get(i)); } context.setLastApplied(toIndex); @@ -297,8 +293,8 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { "Persistence Id = " + persistenceId() + " Last index in log={}, snapshotIndex={}, snapshotTerm={}, " + "journal-size={}", - replicatedLog.lastIndex(), replicatedLog.getSnapshotIndex(), - replicatedLog.getSnapshotTerm(), replicatedLog.size()); + replicatedLog().lastIndex(), replicatedLog().getSnapshotIndex(), + replicatedLog().getSnapshotTerm(), replicatedLog().size()); initializeBehavior(); } @@ -308,9 +304,9 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { } protected void changeCurrentBehavior(RaftActorBehavior newBehavior){ - reusableBehaviorStateHolder.init(currentBehavior); - currentBehavior = newBehavior; - handleBehaviorChange(reusableBehaviorStateHolder, currentBehavior); + reusableBehaviorStateHolder.init(getCurrentBehavior()); + setCurrentBehavior(newBehavior); + handleBehaviorChange(reusableBehaviorStateHolder, getCurrentBehavior()); } @Override public void handleCommand(Object message) { @@ -353,8 +349,8 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { applySnapshot(snapshot.getState()); //clears the followers log, sets the snapshot index to ensure adjusted-index works - replicatedLog = new ReplicatedLogImpl(snapshot); - context.setReplicatedLog(replicatedLog); + context.setReplicatedLog(ReplicatedLogImpl.newInstance(snapshot, context, delegatingPersistenceProvider, + currentBehavior)); context.setLastApplied(snapshot.getLastAppliedIndex()); } else if (message instanceof FindLeader) { @@ -391,11 +387,11 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { } else if (message.equals(COMMIT_SNAPSHOT)) { commitSnapshot(-1); } else { - reusableBehaviorStateHolder.init(currentBehavior); + reusableBehaviorStateHolder.init(getCurrentBehavior()); - currentBehavior = currentBehavior.handleMessage(getSender(), message); + setCurrentBehavior(currentBehavior.handleMessage(getSender(), message)); - handleBehaviorChange(reusableBehaviorStateHolder, currentBehavior); + handleBehaviorChange(reusableBehaviorStateHolder, getCurrentBehavior()); } } @@ -405,17 +401,17 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { OnDemandRaftState.Builder builder = OnDemandRaftState.builder() .commitIndex(context.getCommitIndex()) .currentTerm(context.getTermInformation().getCurrentTerm()) - .inMemoryJournalDataSize(replicatedLog.dataSize()) - .inMemoryJournalLogSize(replicatedLog.size()) + .inMemoryJournalDataSize(replicatedLog().dataSize()) + .inMemoryJournalLogSize(replicatedLog().size()) .isSnapshotCaptureInitiated(context.getSnapshotManager().isCapturing()) .lastApplied(context.getLastApplied()) - .lastIndex(replicatedLog.lastIndex()) - .lastTerm(replicatedLog.lastTerm()) + .lastIndex(replicatedLog().lastIndex()) + .lastTerm(replicatedLog().lastTerm()) .leader(getLeaderId()) .raftState(currentBehavior.state().toString()) .replicatedToAllIndex(currentBehavior.getReplicatedToAllIndex()) - .snapshotIndex(replicatedLog.getSnapshotIndex()) - .snapshotTerm(replicatedLog.getSnapshotTerm()) + .snapshotIndex(replicatedLog().getSnapshotIndex()) + .snapshotTerm(replicatedLog().getSnapshotTerm()) .votedFor(context.getTermInformation().getVotedFor()) .peerAddresses(ImmutableMap.copyOf(context.getPeerAddresses())); @@ -425,8 +421,8 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { builder.lastLogTerm(lastLogEntry.getTerm()); } - if(currentBehavior instanceof AbstractLeader) { - AbstractLeader leader = (AbstractLeader)currentBehavior; + if(getCurrentBehavior() instanceof AbstractLeader) { + AbstractLeader leader = (AbstractLeader)getCurrentBehavior(); Collection followerIds = leader.getFollowerIds(); List followerInfoList = Lists.newArrayListWithCapacity(followerIds.size()); for(String id: followerIds) { @@ -490,39 +486,49 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { final RaftActorContext raftContext = getRaftActorContext(); - replicatedLog - .appendAndPersist(replicatedLogEntry, new Procedure() { - @Override - public void apply(ReplicatedLogEntry replicatedLogEntry) throws Exception { - if(!hasFollowers()){ - // Increment the Commit Index and the Last Applied values - raftContext.setCommitIndex(replicatedLogEntry.getIndex()); - raftContext.setLastApplied(replicatedLogEntry.getIndex()); + replicatedLog().appendAndPersist(replicatedLogEntry, new Procedure() { + @Override + public void apply(ReplicatedLogEntry replicatedLogEntry) throws Exception { + if(!hasFollowers()){ + // Increment the Commit Index and the Last Applied values + raftContext.setCommitIndex(replicatedLogEntry.getIndex()); + raftContext.setLastApplied(replicatedLogEntry.getIndex()); - // Apply the state immediately - applyState(clientActor, identifier, data); + // Apply the state immediately + applyState(clientActor, identifier, data); - // Send a ApplyJournalEntries message so that we write the fact that we applied - // the state to durable storage - self().tell(new ApplyJournalEntries(replicatedLogEntry.getIndex()), self()); + // Send a ApplyJournalEntries message so that we write the fact that we applied + // the state to durable storage + self().tell(new ApplyJournalEntries(replicatedLogEntry.getIndex()), self()); - context.getSnapshotManager().trimLog(context.getLastApplied(), currentBehavior); + context.getSnapshotManager().trimLog(context.getLastApplied(), currentBehavior); - } else if (clientActor != null) { - // Send message for replication - currentBehavior.handleMessage(getSelf(), - new Replicate(clientActor, identifier, - replicatedLogEntry) - ); - } + } else if (clientActor != null) { + // Send message for replication + currentBehavior.handleMessage(getSelf(), + new Replicate(clientActor, identifier, replicatedLogEntry)); + } + } + }); + } - } - }); } + private ReplicatedLog replicatedLog() { + return context.getReplicatedLog(); + } protected String getId() { return context.getId(); } + @VisibleForTesting + void setCurrentBehavior(RaftActorBehavior behavior) { + currentBehavior.setDelegate(behavior); + } + + protected RaftActorBehavior getCurrentBehavior() { + return currentBehavior.getDelegate(); + } + /** * Derived actors can call the isLeader method to check if the current * RaftActor is the Leader or not @@ -563,7 +569,7 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { } protected ReplicatedLogEntry getLastLogEntry() { - return replicatedLog.last(); + return replicatedLog().last(); } protected Long getCurrentTerm(){ @@ -750,125 +756,11 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { private void handleCaptureSnapshotReply(byte[] snapshotBytes) { LOG.debug("{}: CaptureSnapshotReply received by actor: snapshot size {}", persistenceId(), snapshotBytes.length); - context.getSnapshotManager().persist(persistence(), snapshotBytes, currentBehavior, getTotalMemory()); - } - - protected long getTotalMemory() { - return Runtime.getRuntime().totalMemory(); + context.getSnapshotManager().persist(persistence(), snapshotBytes, currentBehavior, context.getTotalMemory()); } protected boolean hasFollowers(){ - return getRaftActorContext().getPeerAddresses().keySet().size() > 0; - } - - private class ReplicatedLogImpl extends AbstractReplicatedLogImpl { - private static final int DATA_SIZE_DIVIDER = 5; - private long dataSizeSinceLastSnapshot = 0L; - - - public ReplicatedLogImpl(Snapshot snapshot) { - super(snapshot.getLastAppliedIndex(), snapshot.getLastAppliedTerm(), - snapshot.getUnAppliedEntries()); - } - - public ReplicatedLogImpl() { - super(); - } - - @Override public void removeFromAndPersist(long logEntryIndex) { - int adjustedIndex = adjustedIndex(logEntryIndex); - - if (adjustedIndex < 0) { - return; - } - - // FIXME: Maybe this should be done after the command is saved - journal.subList(adjustedIndex , journal.size()).clear(); - - persistence().persist(new DeleteEntries(adjustedIndex), new Procedure() { - - @Override - public void apply(DeleteEntries param) - throws Exception { - //FIXME : Doing nothing for now - dataSize = 0; - for (ReplicatedLogEntry entry : journal) { - dataSize += entry.size(); - } - } - }); - } - - @Override public void appendAndPersist( - final ReplicatedLogEntry replicatedLogEntry) { - appendAndPersist(replicatedLogEntry, null); - } - - public void appendAndPersist( - final ReplicatedLogEntry replicatedLogEntry, - final Procedure callback) { - - if(LOG.isDebugEnabled()) { - LOG.debug("{}: Append log entry and persist {} ", persistenceId(), replicatedLogEntry); - } - - // FIXME : By adding the replicated log entry to the in-memory journal we are not truly ensuring durability of the logs - journal.add(replicatedLogEntry); - - // When persisting events with persist it is guaranteed that the - // persistent actor will not receive further commands between the - // persist call and the execution(s) of the associated event - // handler. This also holds for multiple persist calls in context - // of a single command. - persistence().persist(replicatedLogEntry, - new Procedure() { - @Override - public void apply(ReplicatedLogEntry evt) throws Exception { - int logEntrySize = replicatedLogEntry.size(); - - dataSize += logEntrySize; - long dataSizeForCheck = dataSize; - - dataSizeSinceLastSnapshot += logEntrySize; - - if (!hasFollowers()) { - // When we do not have followers we do not maintain an in-memory log - // due to this the journalSize will never become anything close to the - // snapshot batch count. In fact will mostly be 1. - // Similarly since the journal's dataSize depends on the entries in the - // journal the journal's dataSize will never reach a value close to the - // memory threshold. - // By maintaining the dataSize outside the journal we are tracking essentially - // what we have written to the disk however since we no longer are in - // need of doing a snapshot just for the sake of freeing up memory we adjust - // the real size of data by the DATA_SIZE_DIVIDER so that we do not snapshot as often - // as if we were maintaining a real snapshot - dataSizeForCheck = dataSizeSinceLastSnapshot / DATA_SIZE_DIVIDER; - } - long journalSize = replicatedLogEntry.getIndex() + 1; - long dataThreshold = getTotalMemory() * - context.getConfigParams().getSnapshotDataThresholdPercentage() / 100; - - if ((journalSize % context.getConfigParams().getSnapshotBatchCount() == 0 - || dataSizeForCheck > dataThreshold)) { - - boolean started = context.getSnapshotManager().capture(replicatedLogEntry, - currentBehavior.getReplicatedToAllIndex()); - - if(started){ - dataSizeSinceLastSnapshot = 0; - } - - } - - if (callback != null){ - callback.apply(replicatedLogEntry); - } - } - } - ); - } - + return getRaftActorContext().hasFollowers(); } static class DeleteEntries implements Serializable { @@ -911,15 +803,6 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { } } - @VisibleForTesting - void setCurrentBehavior(AbstractRaftActorBehavior behavior) { - currentBehavior = behavior; - } - - protected RaftActorBehavior getCurrentBehavior() { - return currentBehavior; - } - private static class BehaviorStateHolder { private RaftActorBehavior behavior; private String leaderId; diff --git a/opendaylight/md-sal/sal-akka-raft/src/main/java/org/opendaylight/controller/cluster/raft/RaftActorContext.java b/opendaylight/md-sal/sal-akka-raft/src/main/java/org/opendaylight/controller/cluster/raft/RaftActorContext.java index 2e7eb5eb3a..9f4b7cb482 100644 --- a/opendaylight/md-sal/sal-akka-raft/src/main/java/org/opendaylight/controller/cluster/raft/RaftActorContext.java +++ b/opendaylight/md-sal/sal-akka-raft/src/main/java/org/opendaylight/controller/cluster/raft/RaftActorContext.java @@ -12,6 +12,8 @@ import akka.actor.ActorRef; import akka.actor.ActorSelection; import akka.actor.ActorSystem; import akka.actor.Props; +import com.google.common.annotations.VisibleForTesting; +import com.google.common.base.Supplier; import java.util.Map; import org.slf4j.Logger; @@ -168,4 +170,11 @@ public interface RaftActorContext { SnapshotManager getSnapshotManager(); + boolean hasFollowers(); + + long getTotalMemory(); + + @VisibleForTesting + void setTotalMemoryRetriever(Supplier retriever); + } diff --git a/opendaylight/md-sal/sal-akka-raft/src/main/java/org/opendaylight/controller/cluster/raft/RaftActorContextImpl.java b/opendaylight/md-sal/sal-akka-raft/src/main/java/org/opendaylight/controller/cluster/raft/RaftActorContextImpl.java index eb059d60fb..684845c270 100644 --- a/opendaylight/md-sal/sal-akka-raft/src/main/java/org/opendaylight/controller/cluster/raft/RaftActorContextImpl.java +++ b/opendaylight/md-sal/sal-akka-raft/src/main/java/org/opendaylight/controller/cluster/raft/RaftActorContextImpl.java @@ -14,6 +14,8 @@ import akka.actor.ActorSelection; import akka.actor.ActorSystem; import akka.actor.Props; import akka.actor.UntypedActorContext; +import com.google.common.annotations.VisibleForTesting; +import com.google.common.base.Supplier; import java.util.Map; import org.slf4j.Logger; @@ -39,25 +41,22 @@ public class RaftActorContextImpl implements RaftActorContext { private ConfigParams configParams; - private boolean snapshotCaptureInitiated; + @VisibleForTesting + private Supplier totalMemoryRetriever; // Snapshot manager will need to be created on demand as it needs raft actor context which cannot // be passed to it in the constructor private SnapshotManager snapshotManager; - public RaftActorContextImpl(ActorRef actor, UntypedActorContext context, - String id, - ElectionTerm termInformation, long commitIndex, - long lastApplied, ReplicatedLog replicatedLog, - Map peerAddresses, ConfigParams configParams, - Logger logger) { + public RaftActorContextImpl(ActorRef actor, UntypedActorContext context, String id, + ElectionTerm termInformation, long commitIndex, long lastApplied, Map peerAddresses, + ConfigParams configParams, Logger logger) { this.actor = actor; this.context = context; this.id = id; this.termInformation = termInformation; this.commitIndex = commitIndex; this.lastApplied = lastApplied; - this.replicatedLog = replicatedLog; this.peerAddresses = peerAddresses; this.configParams = configParams; this.LOG = logger; @@ -161,10 +160,26 @@ public class RaftActorContextImpl implements RaftActorContext { peerAddresses.put(peerId, peerAddress); } + @Override public SnapshotManager getSnapshotManager() { if(snapshotManager == null){ snapshotManager = new SnapshotManager(this, LOG); } return snapshotManager; } + + @Override + public long getTotalMemory() { + return totalMemoryRetriever != null ? totalMemoryRetriever.get() : Runtime.getRuntime().totalMemory(); + } + + @Override + public void setTotalMemoryRetriever(Supplier retriever) { + totalMemoryRetriever = retriever; + } + + @Override + public boolean hasFollowers() { + return getPeerAddresses().keySet().size() > 0; + } } diff --git a/opendaylight/md-sal/sal-akka-raft/src/main/java/org/opendaylight/controller/cluster/raft/ReplicatedLog.java b/opendaylight/md-sal/sal-akka-raft/src/main/java/org/opendaylight/controller/cluster/raft/ReplicatedLog.java index 82d0839bee..3e4d727c71 100644 --- a/opendaylight/md-sal/sal-akka-raft/src/main/java/org/opendaylight/controller/cluster/raft/ReplicatedLog.java +++ b/opendaylight/md-sal/sal-akka-raft/src/main/java/org/opendaylight/controller/cluster/raft/ReplicatedLog.java @@ -8,6 +8,7 @@ package org.opendaylight.controller.cluster.raft; +import akka.japi.Procedure; import java.util.List; /** @@ -85,6 +86,8 @@ public interface ReplicatedLog { */ void appendAndPersist(final ReplicatedLogEntry replicatedLogEntry); + void appendAndPersist(ReplicatedLogEntry replicatedLogEntry, Procedure callback); + /** * * @param index the index of the log entry diff --git a/opendaylight/md-sal/sal-akka-raft/src/main/java/org/opendaylight/controller/cluster/raft/ReplicatedLogImpl.java b/opendaylight/md-sal/sal-akka-raft/src/main/java/org/opendaylight/controller/cluster/raft/ReplicatedLogImpl.java new file mode 100644 index 0000000000..fdb6305381 --- /dev/null +++ b/opendaylight/md-sal/sal-akka-raft/src/main/java/org/opendaylight/controller/cluster/raft/ReplicatedLogImpl.java @@ -0,0 +1,140 @@ +/* + * Copyright (c) 2015 Cisco Systems, Inc. and others. All rights reserved. + * + * This program and the accompanying materials are made available under the + * terms of the Eclipse Public License v1.0 which accompanies this distribution, + * and is available at http://www.eclipse.org/legal/epl-v10.html + */ +package org.opendaylight.controller.cluster.raft; + +import akka.japi.Procedure; +import java.util.Collections; +import java.util.List; +import org.opendaylight.controller.cluster.DataPersistenceProvider; +import org.opendaylight.controller.cluster.raft.RaftActor.DeleteEntries; +import org.opendaylight.controller.cluster.raft.behaviors.RaftActorBehavior; + +/** + * Implementation of ReplicatedLog used by the RaftActor. + */ +class ReplicatedLogImpl extends AbstractReplicatedLogImpl { + private static final int DATA_SIZE_DIVIDER = 5; + + private long dataSizeSinceLastSnapshot = 0L; + private final RaftActorContext context; + private final DataPersistenceProvider persistence; + private final RaftActorBehavior currentBehavior; + + private final Procedure deleteProcedure = new Procedure() { + @Override + public void apply(DeleteEntries param) { + dataSize = 0; + for (ReplicatedLogEntry entry : journal) { + dataSize += entry.size(); + } + } + }; + + static ReplicatedLog newInstance(Snapshot snapshot, RaftActorContext context, + DataPersistenceProvider persistence, RaftActorBehavior currentBehavior) { + return new ReplicatedLogImpl(snapshot.getLastAppliedIndex(), snapshot.getLastAppliedTerm(), + snapshot.getUnAppliedEntries(), context, persistence, currentBehavior); + } + + static ReplicatedLog newInstance(RaftActorContext context, + DataPersistenceProvider persistence, RaftActorBehavior currentBehavior) { + return new ReplicatedLogImpl(-1L, -1L, Collections.emptyList(), context, + persistence, currentBehavior); + } + + private ReplicatedLogImpl(long snapshotIndex, long snapshotTerm, List unAppliedEntries, + RaftActorContext context, DataPersistenceProvider persistence, RaftActorBehavior currentBehavior) { + super(snapshotIndex, snapshotTerm, unAppliedEntries); + this.context = context; + this.persistence = persistence; + this.currentBehavior = currentBehavior; + } + + @Override + public void removeFromAndPersist(long logEntryIndex) { + int adjustedIndex = adjustedIndex(logEntryIndex); + + if (adjustedIndex < 0) { + return; + } + + // FIXME: Maybe this should be done after the command is saved + journal.subList(adjustedIndex , journal.size()).clear(); + + persistence.persist(new DeleteEntries(adjustedIndex), deleteProcedure); + } + + @Override + public void appendAndPersist(final ReplicatedLogEntry replicatedLogEntry) { + appendAndPersist(replicatedLogEntry, null); + } + + @Override + public void appendAndPersist(final ReplicatedLogEntry replicatedLogEntry, + final Procedure callback) { + + if(context.getLogger().isDebugEnabled()) { + context.getLogger().debug("{}: Append log entry and persist {} ", context.getId(), replicatedLogEntry); + } + + // FIXME : By adding the replicated log entry to the in-memory journal we are not truly ensuring durability of the logs + journal.add(replicatedLogEntry); + + // When persisting events with persist it is guaranteed that the + // persistent actor will not receive further commands between the + // persist call and the execution(s) of the associated event + // handler. This also holds for multiple persist calls in context + // of a single command. + persistence.persist(replicatedLogEntry, + new Procedure() { + @Override + public void apply(ReplicatedLogEntry evt) throws Exception { + int logEntrySize = replicatedLogEntry.size(); + + dataSize += logEntrySize; + long dataSizeForCheck = dataSize; + + dataSizeSinceLastSnapshot += logEntrySize; + + if (!context.hasFollowers()) { + // When we do not have followers we do not maintain an in-memory log + // due to this the journalSize will never become anything close to the + // snapshot batch count. In fact will mostly be 1. + // Similarly since the journal's dataSize depends on the entries in the + // journal the journal's dataSize will never reach a value close to the + // memory threshold. + // By maintaining the dataSize outside the journal we are tracking essentially + // what we have written to the disk however since we no longer are in + // need of doing a snapshot just for the sake of freeing up memory we adjust + // the real size of data by the DATA_SIZE_DIVIDER so that we do not snapshot as often + // as if we were maintaining a real snapshot + dataSizeForCheck = dataSizeSinceLastSnapshot / DATA_SIZE_DIVIDER; + } + long journalSize = replicatedLogEntry.getIndex() + 1; + long dataThreshold = context.getTotalMemory() * + context.getConfigParams().getSnapshotDataThresholdPercentage() / 100; + + if ((journalSize % context.getConfigParams().getSnapshotBatchCount() == 0 + || dataSizeForCheck > dataThreshold)) { + + boolean started = context.getSnapshotManager().capture(replicatedLogEntry, + currentBehavior.getReplicatedToAllIndex()); + + if(started){ + dataSizeSinceLastSnapshot = 0; + } + } + + if (callback != null){ + callback.apply(replicatedLogEntry); + } + } + } + ); + } +} \ No newline at end of file diff --git a/opendaylight/md-sal/sal-akka-raft/src/main/java/org/opendaylight/controller/cluster/raft/behaviors/DelegatingRaftActorBehavior.java b/opendaylight/md-sal/sal-akka-raft/src/main/java/org/opendaylight/controller/cluster/raft/behaviors/DelegatingRaftActorBehavior.java new file mode 100644 index 0000000000..776dae73fa --- /dev/null +++ b/opendaylight/md-sal/sal-akka-raft/src/main/java/org/opendaylight/controller/cluster/raft/behaviors/DelegatingRaftActorBehavior.java @@ -0,0 +1,58 @@ +/* + * Copyright (c) 2015 Brocade Communications Systems, Inc. and others. All rights reserved. + * + * This program and the accompanying materials are made available under the + * terms of the Eclipse Public License v1.0 which accompanies this distribution, + * and is available at http://www.eclipse.org/legal/epl-v10.html + */ +package org.opendaylight.controller.cluster.raft.behaviors; + +import akka.actor.ActorRef; +import org.opendaylight.controller.cluster.raft.RaftState; + +/** + * A RaftActorBehavior implementation that delegates to another implementation. + * + * @author Thomas Pantelis + */ +public class DelegatingRaftActorBehavior implements RaftActorBehavior { + private RaftActorBehavior delegate; + + public RaftActorBehavior getDelegate() { + return delegate; + } + + public void setDelegate(RaftActorBehavior delegate) { + this.delegate = delegate; + } + + @Override + public void close() throws Exception { + delegate.close(); + } + + @Override + public RaftActorBehavior handleMessage(ActorRef sender, Object message) { + return delegate.handleMessage(sender, message); + } + + @Override + public RaftState state() { + return delegate.state(); + } + + @Override + public String getLeaderId() { + return delegate.getLeaderId(); + } + + @Override + public void setReplicatedToAllIndex(long replicatedToAllIndex) { + delegate.setReplicatedToAllIndex(replicatedToAllIndex); + } + + @Override + public long getReplicatedToAllIndex() { + return delegate.getReplicatedToAllIndex(); + } +} diff --git a/opendaylight/md-sal/sal-akka-raft/src/test/java/org/opendaylight/controller/cluster/raft/AbstractRaftActorIntegrationTest.java b/opendaylight/md-sal/sal-akka-raft/src/test/java/org/opendaylight/controller/cluster/raft/AbstractRaftActorIntegrationTest.java index dfaa8d55f6..b910313b09 100644 --- a/opendaylight/md-sal/sal-akka-raft/src/test/java/org/opendaylight/controller/cluster/raft/AbstractRaftActorIntegrationTest.java +++ b/opendaylight/md-sal/sal-akka-raft/src/test/java/org/opendaylight/controller/cluster/raft/AbstractRaftActorIntegrationTest.java @@ -18,6 +18,7 @@ import akka.testkit.JavaTestKit; import akka.testkit.TestActorRef; import com.google.common.base.Optional; import com.google.common.base.Predicate; +import com.google.common.base.Supplier; import com.google.common.collect.ImmutableMap; import java.util.Collections; import java.util.List; @@ -52,7 +53,6 @@ public abstract class AbstractRaftActorIntegrationTest extends AbstractActorTest private final TestActorRef collectorActor; private final Map, Boolean> dropMessages = new ConcurrentHashMap<>(); private volatile byte[] snapshot; - private volatile long mockTotalMemory; private TestRaftActor(String id, Map peerAddresses, ConfigParams config, TestActorRef collectorActor) { @@ -74,13 +74,18 @@ public abstract class AbstractRaftActorIntegrationTest extends AbstractActorTest dropMessages.remove(msgClass); } - void setMockTotalMemory(long mockTotalMemory) { - this.mockTotalMemory = mockTotalMemory; - } + void setMockTotalMemory(final long mockTotalMemory) { + if(mockTotalMemory > 0) { + getRaftActorContext().setTotalMemoryRetriever(new Supplier() { + @Override + public Long get() { + return mockTotalMemory; + } - @Override - protected long getTotalMemory() { - return mockTotalMemory > 0 ? mockTotalMemory : super.getTotalMemory(); + }); + } else { + getRaftActorContext().setTotalMemoryRetriever(null); + } } @Override diff --git a/opendaylight/md-sal/sal-akka-raft/src/test/java/org/opendaylight/controller/cluster/raft/AbstractReplicatedLogImplTest.java b/opendaylight/md-sal/sal-akka-raft/src/test/java/org/opendaylight/controller/cluster/raft/AbstractReplicatedLogImplTest.java index 885c3ab109..8fdb7ea226 100644 --- a/opendaylight/md-sal/sal-akka-raft/src/test/java/org/opendaylight/controller/cluster/raft/AbstractReplicatedLogImplTest.java +++ b/opendaylight/md-sal/sal-akka-raft/src/test/java/org/opendaylight/controller/cluster/raft/AbstractReplicatedLogImplTest.java @@ -11,6 +11,7 @@ import static org.junit.Assert.assertEquals; import static org.junit.Assert.assertFalse; import static org.junit.Assert.assertNull; import static org.junit.Assert.assertTrue; +import akka.japi.Procedure; import java.util.HashMap; import java.util.List; import java.util.Map; @@ -220,5 +221,9 @@ public class AbstractReplicatedLogImplTest { public List getEntriesTill(final int index) { return journal.subList(0, index); } + + @Override + public void appendAndPersist(ReplicatedLogEntry replicatedLogEntry, Procedure callback) { + } } } diff --git a/opendaylight/md-sal/sal-akka-raft/src/test/java/org/opendaylight/controller/cluster/raft/MockRaftActorContext.java b/opendaylight/md-sal/sal-akka-raft/src/test/java/org/opendaylight/controller/cluster/raft/MockRaftActorContext.java index 53cca23741..63f0df2f8c 100644 --- a/opendaylight/md-sal/sal-akka-raft/src/test/java/org/opendaylight/controller/cluster/raft/MockRaftActorContext.java +++ b/opendaylight/md-sal/sal-akka-raft/src/test/java/org/opendaylight/controller/cluster/raft/MockRaftActorContext.java @@ -12,7 +12,9 @@ import akka.actor.ActorRef; import akka.actor.ActorSelection; import akka.actor.ActorSystem; import akka.actor.Props; +import akka.japi.Procedure; import com.google.common.base.Preconditions; +import com.google.common.base.Supplier; import com.google.protobuf.GeneratedMessage; import java.io.Serializable; import java.util.HashMap; @@ -203,6 +205,20 @@ public class MockRaftActorContext implements RaftActorContext { this.configParams = configParams; } + @Override + public long getTotalMemory() { + return Runtime.getRuntime().totalMemory(); + } + + @Override + public void setTotalMemoryRetriever(Supplier retriever) { + } + + @Override + public boolean hasFollowers() { + return getPeerAddresses().keySet().size() > 0; + } + public static class SimpleReplicatedLog extends AbstractReplicatedLogImpl { @Override public void appendAndPersist( ReplicatedLogEntry replicatedLogEntry) { @@ -217,6 +233,19 @@ public class MockRaftActorContext implements RaftActorContext { @Override public void removeFromAndPersist(long index) { removeFrom(index); } + + @Override + public void appendAndPersist(ReplicatedLogEntry replicatedLogEntry, Procedure callback) { + append(replicatedLogEntry); + + if(callback != null) { + try { + callback.apply(replicatedLogEntry); + } catch (Exception e) { + e.printStackTrace(); + } + } + } } public static class MockPayload extends Payload implements Serializable { -- 2.36.6