X-Git-Url: https://git.opendaylight.org/gerrit/gitweb?p=controller.git;a=blobdiff_plain;f=opendaylight%2Fmd-sal%2Fsal-akka-raft%2Fsrc%2Fmain%2Fjava%2Forg%2Fopendaylight%2Fcontroller%2Fcluster%2Fraft%2FRaftActor.java;h=1738cc5fe001ab8b30cfd105f67dc4d9adcc8200;hp=1c30fe23175b5af62ffe808d9428c3361688a4b0;hb=0ed66e3d67fe6b021844c4c3def8583e80a2e934;hpb=1d643894797401ebec8e2242c234779675ca37c3 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 1c30fe2317..1738cc5fe0 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 @@ -1,5 +1,6 @@ /* * Copyright (c) 2014 Cisco Systems, Inc. and others. All rights reserved. + * 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, @@ -11,22 +12,18 @@ package org.opendaylight.controller.cluster.raft; import akka.actor.ActorRef; import akka.actor.ActorSelection; import akka.japi.Procedure; -import akka.persistence.RecoveryCompleted; -import akka.persistence.SaveSnapshotFailure; -import akka.persistence.SaveSnapshotSuccess; -import akka.persistence.SnapshotOffer; import akka.persistence.SnapshotSelectionCriteria; import com.google.common.annotations.VisibleForTesting; import com.google.common.base.Objects; import com.google.common.base.Optional; -import com.google.common.base.Stopwatch; -import com.google.common.collect.ImmutableMap; import com.google.common.collect.Lists; import java.io.Serializable; import java.util.Collection; +import java.util.HashMap; import java.util.List; import java.util.Map; import java.util.concurrent.TimeUnit; +import javax.annotation.Nonnull; import org.apache.commons.lang3.time.DurationFormatUtils; import org.opendaylight.controller.cluster.DataPersistenceProvider; import org.opendaylight.controller.cluster.DelegatingPersistentDataProvider; @@ -36,11 +33,7 @@ import org.opendaylight.controller.cluster.common.actor.AbstractUntypedPersisten import org.opendaylight.controller.cluster.notifications.LeaderStateChanged; import org.opendaylight.controller.cluster.notifications.RoleChanged; import org.opendaylight.controller.cluster.raft.base.messages.ApplyJournalEntries; -import org.opendaylight.controller.cluster.raft.base.messages.ApplyLogEntries; -import org.opendaylight.controller.cluster.raft.base.messages.ApplySnapshot; import org.opendaylight.controller.cluster.raft.base.messages.ApplyState; -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.DelegatingRaftActorBehavior; @@ -99,8 +92,6 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { private static final long APPLY_STATE_DELAY_THRESHOLD_IN_NANOS = TimeUnit.MILLISECONDS.toNanos(50L); // 50 millis - private static final String COMMIT_SNAPSHOT = "commit_snapshot"; - protected final Logger LOG = LoggerFactory.getLogger(getClass()); /** @@ -117,11 +108,9 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { private final DelegatingPersistentDataProvider delegatingPersistenceProvider = new DelegatingPersistentDataProvider(null); - private final Procedure createSnapshotProcedure = new CreateSnapshotProcedure(); - - private Stopwatch recoveryTimer; + private RaftActorRecoverySupport raftRecovery; - private int currentRecoveryBatchCount; + private RaftActorSnapshotMessageSupport snapshotSupport; private final BehaviorStateHolder reusableBehaviorStateHolder = new BehaviorStateHolder(); @@ -135,15 +124,10 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { context = new RaftActorContextImpl(this.getSelf(), this.getContext(), id, new ElectionTermImpl(delegatingPersistenceProvider, id, LOG), -1, -1, peerAddresses, - (configParams.isPresent() ? configParams.get(): new DefaultConfigParamsImpl()), LOG); - - context.setReplicatedLog(ReplicatedLogImpl.newInstance(context, delegatingPersistenceProvider, currentBehavior)); - } + (configParams.isPresent() ? configParams.get(): new DefaultConfigParamsImpl()), + delegatingPersistenceProvider, LOG); - private void initRecoveryTimer() { - if(recoveryTimer == null) { - recoveryTimer = Stopwatch.createStarted(); - } + context.setReplicatedLog(ReplicatedLogImpl.newInstance(context, currentBehavior)); } @Override @@ -152,11 +136,13 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { context.getConfigParams().getJournalRecoveryLogBatchSize()); super.preStart(); + + snapshotSupport = newRaftActorSnapshotMessageSupport(); } @Override public void postStop() { - if(currentBehavior != null) { + if(currentBehavior.getDelegate() != null) { try { currentBehavior.close(); } catch (Exception e) { @@ -169,134 +155,31 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { @Override public void handleRecover(Object message) { - if(persistence().isRecoveryApplicable()) { - if (message instanceof SnapshotOffer) { - onRecoveredSnapshot((SnapshotOffer) message); - } else if (message instanceof ReplicatedLogEntry) { - onRecoveredJournalLogEntry((ReplicatedLogEntry) message); - } else if (message instanceof ApplyLogEntries) { - // Handle this message for backwards compatibility with pre-Lithium versions. - onRecoveredApplyLogEntries(((ApplyLogEntries) message).getToIndex()); - } else if (message instanceof ApplyJournalEntries) { - onRecoveredApplyLogEntries(((ApplyJournalEntries) message).getToIndex()); - } else if (message instanceof DeleteEntries) { - replicatedLog().removeFrom(((DeleteEntries) message).getFromIndex()); - } else if (message instanceof UpdateElectionTerm) { - context.getTermInformation().update(((UpdateElectionTerm) message).getCurrentTerm(), - ((UpdateElectionTerm) message).getVotedFor()); - } else if (message instanceof RecoveryCompleted) { - onRecoveryCompletedMessage(); - } - } else { - if (message instanceof RecoveryCompleted) { + if(raftRecovery == null) { + raftRecovery = newRaftActorRecoverySupport(); + } + + boolean recoveryComplete = raftRecovery.handleRecoveryMessage(message); + if(recoveryComplete) { + if(!persistence().isRecoveryApplicable()) { // Delete all the messages from the akka journal so that we do not end up with consistency issues // Note I am not using the dataPersistenceProvider and directly using the akka api here deleteMessages(lastSequenceNr()); // Delete all the akka snapshots as they will not be needed deleteSnapshots(new SnapshotSelectionCriteria(scala.Long.MaxValue(), scala.Long.MaxValue())); - - onRecoveryComplete(); - - initializeBehavior(); } - } - } - - private void onRecoveredSnapshot(SnapshotOffer offer) { - if(LOG.isDebugEnabled()) { - LOG.debug("{}: SnapshotOffer called..", persistenceId()); - } - - initRecoveryTimer(); - Snapshot snapshot = (Snapshot) offer.snapshot(); + onRecoveryComplete(); - // 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 + initializeBehavior(); - context.setReplicatedLog(ReplicatedLogImpl.newInstance(snapshot, context, delegatingPersistenceProvider, - currentBehavior)); - context.setLastApplied(snapshot.getLastAppliedIndex()); - context.setCommitIndex(snapshot.getLastAppliedIndex()); - - Stopwatch timer = Stopwatch.createStarted(); - - // Apply the snapshot to the actors state - applyRecoverySnapshot(snapshot.getState()); - - timer.stop(); - LOG.info("Recovery snapshot applied for {} in {}: snapshotIndex={}, snapshotTerm={}, journal-size=" + - replicatedLog().size(), persistenceId(), timer.toString(), - replicatedLog().getSnapshotIndex(), replicatedLog().getSnapshotTerm()); - } - - private void onRecoveredJournalLogEntry(ReplicatedLogEntry logEntry) { - if(LOG.isDebugEnabled()) { - LOG.debug("{}: Received ReplicatedLogEntry for recovery: {}", persistenceId(), logEntry.getIndex()); + raftRecovery = null; } - - replicatedLog().append(logEntry); - } - - private void onRecoveredApplyLogEntries(long toIndex) { - if(LOG.isDebugEnabled()) { - LOG.debug("{}: Received ApplyLogEntries for recovery, applying to state: {} to {}", - persistenceId(), context.getLastApplied() + 1, toIndex); - } - - for (long i = context.getLastApplied() + 1; i <= toIndex; i++) { - batchRecoveredLogEntry(replicatedLog().get(i)); - } - - context.setLastApplied(toIndex); - context.setCommitIndex(toIndex); } - private void batchRecoveredLogEntry(ReplicatedLogEntry logEntry) { - initRecoveryTimer(); - - int batchSize = context.getConfigParams().getJournalRecoveryLogBatchSize(); - if(currentRecoveryBatchCount == 0) { - startLogRecoveryBatch(batchSize); - } - - appendRecoveredLogEntry(logEntry.getData()); - - if(++currentRecoveryBatchCount >= batchSize) { - endCurrentLogRecoveryBatch(); - } - } - - private void endCurrentLogRecoveryBatch() { - applyCurrentLogRecoveryBatch(); - currentRecoveryBatchCount = 0; - } - - private void onRecoveryCompletedMessage() { - if(currentRecoveryBatchCount > 0) { - endCurrentLogRecoveryBatch(); - } - - onRecoveryComplete(); - - String recoveryTime = ""; - if(recoveryTimer != null) { - recoveryTimer.stop(); - recoveryTime = " in " + recoveryTimer.toString(); - recoveryTimer = null; - } - - LOG.info( - "Recovery completed" + recoveryTime + " - Switching actor to Follower - " + - "Persistence Id = " + persistenceId() + - " Last index in log={}, snapshotIndex={}, snapshotTerm={}, " + - "journal-size={}", - replicatedLog().lastIndex(), replicatedLog().getSnapshotIndex(), - replicatedLog().getSnapshotTerm(), replicatedLog().size()); - - initializeBehavior(); + protected RaftActorRecoverySupport newRaftActorRecoverySupport() { + return new RaftActorRecoverySupport(context, currentBehavior, getRaftActorRecoveryCohort()); } protected void initializeBehavior(){ @@ -309,7 +192,8 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { handleBehaviorChange(reusableBehaviorStateHolder, getCurrentBehavior()); } - @Override public void handleCommand(Object message) { + @Override + public void handleCommand(Object message) { if (message instanceof ApplyState){ ApplyState applyState = (ApplyState) message; @@ -336,57 +220,14 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { persistence().persist(applyEntries, NoopProcedure.instance()); - } else if(message instanceof ApplySnapshot ) { - Snapshot snapshot = ((ApplySnapshot) message).getSnapshot(); - - if(LOG.isDebugEnabled()) { - LOG.debug("{}: ApplySnapshot called on Follower Actor " + - "snapshotIndex:{}, snapshotTerm:{}", persistenceId(), snapshot.getLastAppliedIndex(), - snapshot.getLastAppliedTerm() - ); - } - - applySnapshot(snapshot.getState()); - - //clears the followers log, sets the snapshot index to ensure adjusted-index works - context.setReplicatedLog(ReplicatedLogImpl.newInstance(snapshot, context, delegatingPersistenceProvider, - currentBehavior)); - context.setLastApplied(snapshot.getLastAppliedIndex()); - } else if (message instanceof FindLeader) { getSender().tell( new FindLeaderReply(getLeaderAddress()), getSelf() ); - - } else if (message instanceof SaveSnapshotSuccess) { - SaveSnapshotSuccess success = (SaveSnapshotSuccess) message; - LOG.info("{}: SaveSnapshotSuccess received for snapshot", persistenceId()); - - long sequenceNumber = success.metadata().sequenceNr(); - - commitSnapshot(sequenceNumber); - - } else if (message instanceof SaveSnapshotFailure) { - SaveSnapshotFailure saveSnapshotFailure = (SaveSnapshotFailure) message; - - LOG.error("{}: SaveSnapshotFailure received for snapshot Cause:", - persistenceId(), saveSnapshotFailure.cause()); - - context.getSnapshotManager().rollback(); - - } else if (message instanceof CaptureSnapshot) { - LOG.debug("{}: CaptureSnapshot received by actor: {}", persistenceId(), message); - - context.getSnapshotManager().create(createSnapshotProcedure); - - } else if (message instanceof CaptureSnapshotReply) { - handleCaptureSnapshotReply(((CaptureSnapshotReply) message).getSnapshot()); } else if(message instanceof GetOnDemandRaftState) { onGetOnDemandRaftStats(); - } else if (message.equals(COMMIT_SNAPSHOT)) { - commitSnapshot(-1); - } else { + } else if(!snapshotSupport.handleSnapshotMessage(message)) { reusableBehaviorStateHolder.init(getCurrentBehavior()); setCurrentBehavior(currentBehavior.handleMessage(getSender(), message)); @@ -395,6 +236,11 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { } } + protected RaftActorSnapshotMessageSupport newRaftActorSnapshotMessageSupport() { + return new RaftActorSnapshotMessageSupport(context, currentBehavior, + getRaftActorSnapshotCohort()); + } + private void onGetOnDemandRaftStats() { // Debugging message to retrieve raft stats. @@ -413,7 +259,7 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { .snapshotIndex(replicatedLog().getSnapshotIndex()) .snapshotTerm(replicatedLog().getSnapshotTerm()) .votedFor(context.getTermInformation().getVotedFor()) - .peerAddresses(ImmutableMap.copyOf(context.getPeerAddresses())); + .peerAddresses(new HashMap<>(context.getPeerAddresses())); ReplicatedLogEntry lastLogEntry = getLastLogEntry(); if (lastLogEntry != null) { @@ -621,7 +467,7 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { // Make saving Snapshot successful // Committing the snapshot here would end up calling commit in the creating state which would // be a state violation. That's why now we send a message to commit the snapshot. - self().tell(COMMIT_SNAPSHOT, self()); + self().tell(RaftActorSnapshotMessageSupport.COMMIT_SNAPSHOT, self()); } }); } @@ -645,10 +491,6 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { context.setPeerAddress(peerId, peerAddress); } - protected void commitSnapshot(long sequenceNumber) { - context.getSnapshotManager().commit(persistence(), sequenceNumber); - } - /** * The applyState method will be called by the RaftActor when some data * needs to be applied to the actor's state @@ -670,31 +512,10 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { Object data); /** - * This method is called during recovery at the start of a batch of state entries. Derived - * classes should perform any initialization needed to start a batch. - */ - protected abstract void startLogRecoveryBatch(int maxBatchSize); - - /** - * This method is called during recovery to append state data to the current batch. This method - * is called 1 or more times after {@link #startLogRecoveryBatch}. - * - * @param data the state data - */ - protected abstract void appendRecoveredLogEntry(Payload data); - - /** - * This method is called during recovery to reconstruct the state of the actor. - * - * @param snapshotBytes A snapshot of the state of the actor - */ - protected abstract void applyRecoverySnapshot(byte[] snapshotBytes); - - /** - * This method is called during recovery at the end of a batch to apply the current batched - * log entries. This method is called after {@link #appendRecoveredLogEntry}. + * Returns the RaftActorRecoveryCohort to participate in persistence recovery. */ - protected abstract void applyCurrentLogRecoveryBatch(); + @Nonnull + protected abstract RaftActorRecoveryCohort getRaftActorRecoveryCohort(); /** * This method is called when recovery is complete. @@ -702,24 +523,10 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { protected abstract void onRecoveryComplete(); /** - * This method will be called by the RaftActor when a snapshot needs to be - * created. The derived actor should respond with its current state. - *

- * During recovery the state that is returned by the derived actor will - * be passed back to it by calling the applySnapshot method - * - * @return The current state of the actor + * Returns the RaftActorSnapshotCohort to participate in persistence recovery. */ - protected abstract void createSnapshot(); - - /** - * This method can be called at any other point during normal - * operations when the derived actor is out of sync with it's peers - * and the only way to bring it in sync is by applying a snapshot - * - * @param snapshotBytes A snapshot of the state of the actor - */ - protected abstract void applySnapshot(byte[] snapshotBytes); + @Nonnull + protected abstract RaftActorSnapshotCohort getRaftActorSnapshotCohort(); /** * This method will be called by the RaftActor when the state of the @@ -753,16 +560,16 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { return peerAddress; } - private void handleCaptureSnapshotReply(byte[] snapshotBytes) { - LOG.debug("{}: CaptureSnapshotReply received by actor: snapshot size {}", persistenceId(), snapshotBytes.length); - - context.getSnapshotManager().persist(persistence(), snapshotBytes, currentBehavior, context.getTotalMemory()); - } - protected boolean hasFollowers(){ return getRaftActorContext().hasFollowers(); } + /** + * @deprecated Deprecated in favor of {@link org.opendaylight.controller.cluster.raft.base.messages.DeleteEntriesTest} + * whose type for fromIndex is long instead of int. This class was kept for backwards + * compatibility with Helium. + */ + @Deprecated static class DeleteEntries implements Serializable { private static final long serialVersionUID = 1L; private final int fromIndex; @@ -795,14 +602,6 @@ public abstract class RaftActor extends AbstractUntypedPersistentActor { } } - private class CreateSnapshotProcedure implements Procedure { - - @Override - public void apply(Void aVoid) throws Exception { - createSnapshot(); - } - } - private static class BehaviorStateHolder { private RaftActorBehavior behavior; private String leaderId;