BUG 4212 : Follower should not reschedule election timeout in certain cases. 00/26100/4
authorMoiz Raja <moraja@cisco.com>
Thu, 27 Aug 2015 17:34:36 +0000 (10:34 -0700)
committerGerrit Code Review <gerrit@opendaylight.org>
Thu, 27 Aug 2015 22:01:53 +0000 (22:01 +0000)
Before:
Follower rescheduled election whenever it received any message

Now:
Followe reschedules election only if
    - The message received is a RaftRPC message
    - If the RaftRPC message is a RequestVote then only reschedule
      if vote is granted

Change-Id: Ia59c65e4896d72dfc49e86e59b6a9e9331a945ca
Signed-off-by: Moiz Raja <moraja@cisco.com>
opendaylight/md-sal/sal-akka-raft/src/main/java/org/opendaylight/controller/cluster/raft/behaviors/AbstractRaftActorBehavior.java
opendaylight/md-sal/sal-akka-raft/src/main/java/org/opendaylight/controller/cluster/raft/behaviors/Follower.java
opendaylight/md-sal/sal-akka-raft/src/test/java/org/opendaylight/controller/cluster/raft/behaviors/FollowerTest.java

index 8692e9948a4def892062a0b21d6129cd0f29c35c..6ef4914477bee8fc2f7a50a78f66fdfd8be2f1d0 100644 (file)
@@ -168,6 +168,22 @@ public abstract class AbstractRaftActorBehavior implements RaftActorBehavior {
 
         LOG.debug("{}: In requestVote:  {}", logName(), requestVote);
 
+        boolean grantVote = canGrantVote(requestVote);
+
+        if(grantVote) {
+            context.getTermInformation().updateAndPersist(requestVote.getTerm(), requestVote.getCandidateId());
+        }
+
+        RequestVoteReply reply = new RequestVoteReply(currentTerm(), grantVote);
+
+        LOG.debug("{}: requestVote returning: {}", logName(), reply);
+
+        sender.tell(reply, actor());
+
+        return this;
+    }
+
+    protected boolean canGrantVote(RequestVote requestVote){
         boolean grantVote = false;
 
         //  Reply false if term < currentTerm (§5.1)
@@ -177,7 +193,7 @@ public abstract class AbstractRaftActorBehavior implements RaftActorBehavior {
             // If votedFor is null or candidateId, and candidate’s log is at
             // least as up-to-date as receiver’s log, grant vote (§5.2, §5.4)
         } else if (votedFor() == null || votedFor()
-            .equals(requestVote.getCandidateId())) {
+                .equals(requestVote.getCandidateId())) {
 
             boolean candidateLatest = false;
 
@@ -191,24 +207,15 @@ public abstract class AbstractRaftActorBehavior implements RaftActorBehavior {
             if (requestVote.getLastLogTerm() > lastTerm()) {
                 candidateLatest = true;
             } else if ((requestVote.getLastLogTerm() == lastTerm())
-                && requestVote.getLastLogIndex() >= lastIndex()) {
+                    && requestVote.getLastLogIndex() >= lastIndex()) {
                 candidateLatest = true;
             }
 
             if (candidateLatest) {
                 grantVote = true;
-                context.getTermInformation().updateAndPersist(requestVote.getTerm(),
-                    requestVote.getCandidateId());
             }
         }
-
-        RequestVoteReply reply = new RequestVoteReply(currentTerm(), grantVote);
-
-        LOG.debug("{}: requestVote returning: {}", logName(), reply);
-
-        sender.tell(reply, actor());
-
-        return this;
+        return grantVote;
     }
 
     /**
index c1d261c56192cb0b1bbe6905b77faa7d1d8fad0c..d4755e63586f564cd71d106529cef910074b8cba 100644 (file)
@@ -23,6 +23,7 @@ import org.opendaylight.controller.cluster.raft.messages.AppendEntriesReply;
 import org.opendaylight.controller.cluster.raft.messages.InstallSnapshot;
 import org.opendaylight.controller.cluster.raft.messages.InstallSnapshotReply;
 import org.opendaylight.controller.cluster.raft.messages.RaftRPC;
+import org.opendaylight.controller.cluster.raft.messages.RequestVote;
 import org.opendaylight.controller.cluster.raft.messages.RequestVoteReply;
 
 /**
@@ -37,8 +38,6 @@ import org.opendaylight.controller.cluster.raft.messages.RequestVoteReply;
  */
 public class Follower extends AbstractRaftActorBehavior {
 
-
-
     private SnapshotTracker snapshotTracker = null;
 
     private final InitialSyncStatusTracker initialSyncStatusTracker;
@@ -321,7 +320,9 @@ public class Follower extends AbstractRaftActorBehavior {
             handleInstallSnapshot(sender, installSnapshot);
         }
 
-        scheduleElection(electionDuration());
+        if(message instanceof RaftRPC && (!(message instanceof RequestVote) || (canGrantVote((RequestVote) message)))){
+            scheduleElection(electionDuration());
+        }
 
         return super.handleMessage(sender, message);
     }
index 2cb0d7ce6546ad93d6cb71196e0bf020a4450fb3..62b2112d79f850aa62910300a0606fb9992da6fa 100644 (file)
@@ -64,7 +64,7 @@ public class FollowerTest extends AbstractRaftActorBehaviorTest {
 
     @Override
     protected RaftActorBehavior createBehavior(RaftActorContext actorContext) {
-        return new Follower(actorContext);
+        return new TestFollower(actorContext);
     }
 
     @Override
@@ -79,6 +79,13 @@ public class FollowerTest extends AbstractRaftActorBehaviorTest {
         return context;
     }
 
+    private int getElectionTimeoutCount(RaftActorBehavior follower){
+        if(follower instanceof TestFollower){
+            return ((TestFollower) follower).getElectionTimeoutCount();
+        }
+        return -1;
+    }
+
     @Test
     public void testThatAnElectionTimeoutIsTriggered(){
         MockRaftActorContext actorContext = createActorContext();
@@ -115,6 +122,7 @@ public class FollowerTest extends AbstractRaftActorBehaviorTest {
 
         assertEquals("isVoteGranted", true, reply.isVoteGranted());
         assertEquals("getTerm", term, reply.getTerm());
+        assertEquals("schedule election", 1, getElectionTimeoutCount(follower));
     }
 
     @Test
@@ -132,6 +140,7 @@ public class FollowerTest extends AbstractRaftActorBehaviorTest {
         RequestVoteReply reply = MessageCollectorActor.expectFirstMatching(leaderActor, RequestVoteReply.class);
 
         assertEquals("isVoteGranted", false, reply.isVoteGranted());
+        assertEquals("schedule election", 0, getElectionTimeoutCount(follower));
     }
 
 
@@ -190,7 +199,7 @@ public class FollowerTest extends AbstractRaftActorBehaviorTest {
 
         MockRaftActorContext context = createActorContext();
         context.getReplicatedLog().clear(0,2);
-        context.getReplicatedLog().append(newReplicatedLogEntry(1,100, "bar"));
+        context.getReplicatedLog().append(newReplicatedLogEntry(1, 100, "bar"));
         context.getReplicatedLog().setSnapshotIndex(99);
 
         List<ReplicatedLogEntry> entries = Arrays.asList(
@@ -903,6 +912,26 @@ public class FollowerTest extends AbstractRaftActorBehaviorTest {
         MessageCollectorActor.assertNoneMatching(followerActor, ElectionTimeout.class, 500);
     }
 
+    @Test
+    public void testElectionScheduledWhenAnyRaftRPCReceived(){
+        MockRaftActorContext context = createActorContext();
+        follower = createBehavior(context);
+        follower.handleMessage(leaderActor, new RaftRPC() {
+            @Override
+            public long getTerm() {
+                return 100;
+            }
+        });
+        assertEquals("schedule election", 1, getElectionTimeoutCount(follower));
+    }
+
+    @Test
+    public void testElectionNotScheduledWhenNonRaftRPCMessageReceived(){
+        MockRaftActorContext context = createActorContext();
+        follower = createBehavior(context);
+        follower.handleMessage(leaderActor, "non-raft-rpc");
+        assertEquals("schedule election", 0, getElectionTimeoutCount(follower));
+    }
 
     public ByteString getNextChunk (ByteString bs, int offset, int chunkSize){
         int snapshotLength = bs.size();
@@ -961,4 +990,23 @@ public class FollowerTest extends AbstractRaftActorBehaviorTest {
         AppendEntriesReply reply = MessageCollectorActor.expectFirstMatching(replyActor, AppendEntriesReply.class);
         assertEquals("isSuccess", true, reply.isSuccess());
     }
+
+    private static class TestFollower extends Follower {
+
+        int electionTimeoutCount = 0;
+
+        public TestFollower(RaftActorContext context) {
+            super(context);
+        }
+
+        @Override
+        protected void scheduleElection(FiniteDuration interval) {
+            electionTimeoutCount++;
+            super.scheduleElection(interval);
+        }
+
+        public int getElectionTimeoutCount() {
+            return electionTimeoutCount;
+        }
+    }
 }