BUG-5626: do not allow overriding of RaftActor.handleCommand()
[controller.git] / opendaylight / md-sal / sal-akka-raft / src / main / java / org / opendaylight / controller / cluster / raft / behaviors / AbstractRaftActorBehavior.java
index fd80d66ebfbc274aa676d098afba452547d7688a..ab09ba3967955bad54139dd92c7c6bc977714d2c 100644 (file)
@@ -10,6 +10,7 @@ package org.opendaylight.controller.cluster.raft.behaviors;
 
 import akka.actor.ActorRef;
 import akka.actor.Cancellable;
+import com.google.common.base.Preconditions;
 import java.util.Random;
 import java.util.concurrent.TimeUnit;
 import org.opendaylight.controller.cluster.raft.ClientRequestTracker;
@@ -38,9 +39,6 @@ import scala.concurrent.duration.FiniteDuration;
  * set currentTerm = T, convert to follower (ยง5.1)
  */
 public abstract class AbstractRaftActorBehavior implements RaftActorBehavior {
-
-    protected static final ElectionTimeout ELECTION_TIMEOUT = new ElectionTimeout();
-
     /**
      * Information about the RaftActor whose behavior this class represents
      */
@@ -56,33 +54,41 @@ public abstract class AbstractRaftActorBehavior implements RaftActorBehavior {
      */
     private Cancellable electionCancel = null;
 
-    /**
-     *
-     */
-    protected String leaderId = null;
-
-    private short leaderPayloadVersion = -1;
-
     private long replicatedToAllIndex = -1;
 
     private final String logName;
 
     private final RaftState state;
 
-    protected AbstractRaftActorBehavior(RaftActorContext context, RaftState state) {
-        this.context = context;
-        this.state = state;
+    AbstractRaftActorBehavior(final RaftActorContext context, final RaftState state) {
+        this.context = Preconditions.checkNotNull(context);
+        this.state = Preconditions.checkNotNull(state);
         this.LOG = context.getLogger();
 
         logName = String.format("%s (%s)", context.getId(), state);
     }
 
+    public static RaftActorBehavior createBehavior(final RaftActorContext context, final RaftState state) {
+        switch (state) {
+            case Candidate:
+                return new Candidate(context);
+            case Follower:
+                return new Follower(context);
+            case IsolatedLeader:
+                return new IsolatedLeader(context);
+            case Leader:
+                return new Leader(context);
+            default:
+                throw new IllegalArgumentException("Unhandled state " + state);
+        }
+    }
+
     @Override
-    public RaftState state() {
+    public final RaftState state() {
         return state;
     }
 
-    public String logName() {
+    protected final String logName() {
         return logName;
     }
 
@@ -252,6 +258,10 @@ public abstract class AbstractRaftActorBehavior implements RaftActorBehavior {
         }
     }
 
+    protected boolean canStartElection() {
+        return context.getRaftPolicy().automaticElectionsEnabled() && context.isVotingMember();
+    }
+
     /**
      * schedule a new election
      *
@@ -260,12 +270,11 @@ public abstract class AbstractRaftActorBehavior implements RaftActorBehavior {
     protected void scheduleElection(FiniteDuration interval) {
         stopElection();
 
-        // Schedule an election. When the scheduler triggers an ElectionTimeout
-        // message is sent to itself
-        electionCancel =
-            context.getActorSystem().scheduler().scheduleOnce(interval,
-                context.getActor(), ELECTION_TIMEOUT,
-                context.getActorSystem().dispatcher(), context.getActor());
+        if(canStartElection()) {
+            // Schedule an election. When the scheduler triggers an ElectionTimeout message is sent to itself
+            electionCancel = context.getActorSystem().scheduler().scheduleOnce(interval, context.getActor(),
+                    ElectionTimeout.INSTANCE, context.getActorSystem().dispatcher(), context.getActor());
+        }
     }
 
     /**
@@ -408,34 +417,34 @@ public abstract class AbstractRaftActorBehavior implements RaftActorBehavior {
             return requestVote(sender, (RequestVote) message);
         } else if (message instanceof RequestVoteReply) {
             return handleRequestVoteReply(sender, (RequestVoteReply) message);
+        } else {
+            return null;
         }
-        return this;
-    }
-
-    @Override public String getLeaderId() {
-        return leaderId;
     }
 
     @Override
-    public short getLeaderPayloadVersion() {
-        return leaderPayloadVersion;
+    public RaftActorBehavior switchBehavior(RaftActorBehavior behavior) {
+        return internalSwitchBehavior(behavior);
     }
 
-    public void setLeaderPayloadVersion(short leaderPayloadVersion) {
-        this.leaderPayloadVersion = leaderPayloadVersion;
+    protected RaftActorBehavior internalSwitchBehavior(RaftState newState) {
+        if(context.getRaftPolicy().automaticElectionsEnabled()){
+            return internalSwitchBehavior(createBehavior(context, newState));
+        }
+        return this;
     }
 
-    protected RaftActorBehavior switchBehavior(RaftActorBehavior behavior) {
-        LOG.info("{} :- Switching from behavior {} to {}", logName(), this.state(), behavior.state());
+    private RaftActorBehavior internalSwitchBehavior(RaftActorBehavior newBehavior) {
+        LOG.info("{} :- Switching from behavior {} to {}", logName(), this.state(), newBehavior.state());
         try {
             close();
         } catch (Exception e) {
             LOG.error("{}: Failed to close behavior : {}", logName(), this.state(), e);
         }
-
-        return behavior;
+        return newBehavior;
     }
 
+
     protected int getMajorityVoteCount(int numPeers) {
         // Votes are required from a majority of the peers including self.
         // The numMajority field therefore stores a calculated value
@@ -467,7 +476,7 @@ public abstract class AbstractRaftActorBehavior implements RaftActorBehavior {
      * @param snapshotCapturedIndex
      */
     protected void performSnapshotWithoutCapture(final long snapshotCapturedIndex) {
-        long actualIndex = context.getSnapshotManager().trimLog(snapshotCapturedIndex, this);
+        long actualIndex = context.getSnapshotManager().trimLog(snapshotCapturedIndex);
 
         if(actualIndex != -1){
             setReplicatedToAllIndex(actualIndex);
@@ -477,5 +486,4 @@ public abstract class AbstractRaftActorBehavior implements RaftActorBehavior {
     protected String getId(){
         return context.getId();
     }
-
 }