Bug 6659: Fix intermittent PartitionedCandidateOnStartupElectionScenarioTest failure
[controller.git] / opendaylight / md-sal / sal-akka-raft / src / test / java / org / opendaylight / controller / cluster / raft / behaviors / PartitionedCandidateOnStartupElectionScenarioTest.java
1 /*
2  * Copyright (c) 2015 Brocade Communications Systems, Inc. and others.  All rights reserved.
3  *
4  * This program and the accompanying materials are made available under the
5  * terms of the Eclipse Public License v1.0 which accompanies this distribution,
6  * and is available at http://www.eclipse.org/legal/epl-v10.html
7  */
8 package org.opendaylight.controller.cluster.raft.behaviors;
9
10 import static org.junit.Assert.assertEquals;
11 import akka.actor.ActorRef;
12 import com.google.common.collect.ImmutableMap;
13 import org.junit.Test;
14 import org.opendaylight.controller.cluster.raft.DefaultConfigParamsImpl;
15 import org.opendaylight.controller.cluster.raft.MockRaftActorContext.MockPayload;
16 import org.opendaylight.controller.cluster.raft.MockRaftActorContext.MockReplicatedLogEntry;
17 import org.opendaylight.controller.cluster.raft.MockRaftActorContext.SimpleReplicatedLog;
18 import org.opendaylight.controller.cluster.raft.RaftState;
19 import org.opendaylight.controller.cluster.raft.base.messages.ElectionTimeout;
20 import org.opendaylight.controller.cluster.raft.base.messages.TimeoutNow;
21 import org.opendaylight.controller.cluster.raft.messages.RequestVote;
22 import org.opendaylight.controller.cluster.raft.messages.RequestVoteReply;
23
24 /**
25  * A leader election scenario test that partitions a candidate when trying to join a cluster on startup.
26  *
27  * @author Thomas Pantelis
28  */
29 public class PartitionedCandidateOnStartupElectionScenarioTest extends AbstractLeaderElectionScenarioTest {
30
31     private final int numCandidateElections = 5;
32     private long candidateElectionTerm;
33
34     @Test
35     public void runTest() throws Exception {
36         testLog.info("PartitionedCandidateOnStartupElectionScenarioTest starting");
37
38         setupInitialMember1AndMember2Behaviors();
39
40         setupPartitionedCandidateMember3AndSendElectionTimeouts();
41
42         resolvePartitionAndSendElectionTimeoutsToCandidateMember3();
43
44         sendElectionTimeoutToFollowerMember1();
45
46         testLog.info("PartitionedCandidateOnStartupElectionScenarioTest ending");
47     }
48
49     private void sendElectionTimeoutToFollowerMember1() throws Exception {
50         testLog.info("sendElectionTimeoutToFollowerMember1 starting");
51
52         // At this point we have no leader. Candidate member 3 would continue to start new elections
53         // but wouldn't be granted a vote. One of the 2 followers would eventually time out from
54         // not having received a heartbeat from a leader and switch to candidate and start a new
55         // election. We'll simulate that here by sending an ElectionTimeout to member 1.
56
57         member1Actor.clear();
58         member1Actor.expectMessageClass(RequestVoteReply.class, 1);
59         member2Actor.clear();
60         member2Actor.expectMessageClass(RequestVote.class, 1);
61         member3Actor.clear();
62         member3Actor.expectMessageClass(RequestVote.class, 1);
63         member3Actor.expectBehaviorStateChange();
64
65         member1ActorRef.tell(TimeoutNow.INSTANCE, ActorRef.noSender());
66
67         member2Actor.waitForExpectedMessages(RequestVote.class);
68         member3Actor.waitForExpectedMessages(RequestVote.class);
69
70         // The RequestVoteReply should come from Follower member 2 and the vote should be granted
71         // since member 2's last term and index matches member 1's.
72
73         member1Actor.waitForExpectedMessages(RequestVoteReply.class);
74
75         RequestVoteReply requestVoteReply = member1Actor.getCapturedMessage(RequestVoteReply.class);
76         assertEquals("getTerm", member1Context.getTermInformation().getCurrentTerm(), requestVoteReply.getTerm());
77         assertEquals("isVoteGranted", true, requestVoteReply.isVoteGranted());
78
79         // Candidate member 3 should change to follower as its term should be less than the
80         // RequestVote term (member 1 started a new term higher than the other member's terms).
81
82         member3Actor.waitForBehaviorStateChange();
83
84         verifyBehaviorState("member 1", member1Actor, RaftState.Leader);
85         verifyBehaviorState("member 2", member2Actor, RaftState.Follower);
86         verifyBehaviorState("member 3", member3Actor, RaftState.Follower);
87
88         // newTerm should be 10.
89
90         long newTerm = candidateElectionTerm + 1;
91         assertEquals("member 1 election term", newTerm, member1Context.getTermInformation().getCurrentTerm());
92         assertEquals("member 2 election term", newTerm, member2Context.getTermInformation().getCurrentTerm());
93         assertEquals("member 3 election term", newTerm, member3Context.getTermInformation().getCurrentTerm());
94
95         testLog.info("sendElectionTimeoutToFollowerMember1 ending");
96     }
97
98     private void resolvePartitionAndSendElectionTimeoutsToCandidateMember3() throws Exception {
99         testLog.info("resolvePartitionAndSendElectionTimeoutsToCandidateMember3 starting");
100
101         // Now send a couple more ElectionTimeouts to Candidate member 3 with the partition resolved.
102         //
103         // On the first RequestVote, Leader member 1 should switch to Follower as its term (s) is less than
104         // the RequestVote's term (8) from member 3. No RequestVoteReply should be sent by member 1.
105         // Follower member 2 should update its term since it less than the RequestVote's term and
106         // should return a RequestVoteReply but should not grant the vote as its last term and index
107         // is greater than the RequestVote's lastLogTerm and lastLogIndex, ie member 2's log is later
108         // or more up to date than member 3's.
109         //
110         // On the second RequestVote, both member 1 and 2 are followers so they should update their
111         // term and return a RequestVoteReply but should not grant the vote.
112
113         candidateElectionTerm += 2;
114         for(int i = 0; i < 2; i++) {
115             member1Actor.clear();
116             member1Actor.expectMessageClass(RequestVote.class, 1);
117             member2Actor.clear();
118             member2Actor.expectMessageClass(RequestVote.class, 1);
119             member3Actor.clear();
120             member3Actor.expectMessageClass(RequestVoteReply.class, 1);
121
122             member3ActorRef.tell(ElectionTimeout.INSTANCE, ActorRef.noSender());
123
124             member1Actor.waitForExpectedMessages(RequestVote.class);
125             member2Actor.waitForExpectedMessages(RequestVote.class);
126
127             member3Actor.waitForExpectedMessages(RequestVoteReply.class);
128
129             RequestVoteReply requestVoteReply = member3Actor.getCapturedMessage(RequestVoteReply.class);
130             assertEquals("getTerm", member3Context.getTermInformation().getCurrentTerm(), requestVoteReply.getTerm());
131             assertEquals("isVoteGranted", false, requestVoteReply.isVoteGranted());
132         }
133
134         verifyBehaviorState("member 1", member1Actor, RaftState.Follower);
135         verifyBehaviorState("member 2", member2Actor, RaftState.Follower);
136         verifyBehaviorState("member 3", member3Actor, RaftState.Candidate);
137
138         // Even though member 3 didn't get voted for, member 1 and 2 should have updated their term
139         // to member 3's.
140
141         assertEquals("member 1 election term", candidateElectionTerm,
142                 member1Context.getTermInformation().getCurrentTerm());
143         assertEquals("member 2 election term", candidateElectionTerm,
144                 member2Context.getTermInformation().getCurrentTerm());
145         assertEquals("member 3 election term", candidateElectionTerm,
146                 member3Context.getTermInformation().getCurrentTerm());
147
148         testLog.info("resolvePartitionAndSendElectionTimeoutsToCandidateMember3 ending");
149     }
150
151     private void setupPartitionedCandidateMember3AndSendElectionTimeouts() {
152         testLog.info("setupPartitionedCandidateMember3AndSendElectionTimeouts starting");
153
154         // Create member 3's behavior initially as a Candidate.
155
156         member3Context = newRaftActorContext("member3", member3ActorRef,
157                 ImmutableMap.<String,String>builder().
158                     put("member1", member1ActorRef.path().toString()).
159                     put("member2", member2ActorRef.path().toString()).build());
160
161         DefaultConfigParamsImpl member3ConfigParams = newConfigParams();
162         member3Context.setConfigParams(member3ConfigParams);
163
164         // Initialize the ReplicatedLog and election term info for Candidate member 3. The current term
165         // will be 2 and the last term will be 1 so it is behind the leader's log.
166
167         SimpleReplicatedLog candidateReplicatedLog = new SimpleReplicatedLog();
168         candidateReplicatedLog.append(new MockReplicatedLogEntry(2, 0, new MockPayload("")));
169
170         member3Context.setReplicatedLog(candidateReplicatedLog);
171         member3Context.setCommitIndex(candidateReplicatedLog.lastIndex());
172         member3Context.setLastApplied(candidateReplicatedLog.lastIndex());
173         member3Context.getTermInformation().update(2, member1Context.getId());
174
175         // The member 3 Candidate will start a new term and send RequestVotes. However it will be
176         // partitioned from the cluster by having member 1 and 2 drop its RequestVote messages.
177
178         candidateElectionTerm = member3Context.getTermInformation().getCurrentTerm() + numCandidateElections;
179
180         member1Actor.dropMessagesToBehavior(RequestVote.class, numCandidateElections);
181
182         member2Actor.dropMessagesToBehavior(RequestVote.class, numCandidateElections);
183
184         Candidate member3Behavior = new Candidate(member3Context);
185         member3Actor.behavior = member3Behavior;
186         member3Context.setCurrentBehavior(member3Behavior);
187
188         // Send several additional ElectionTimeouts to Candidate member 3. Each ElectionTimeout will
189         // start a new term so Candidate member 3's current term will be greater than the leader's
190         // current term.
191
192         for(int i = 0; i < numCandidateElections - 1; i++) {
193             member3ActorRef.tell(ElectionTimeout.INSTANCE, ActorRef.noSender());
194         }
195
196         member1Actor.waitForExpectedMessages(RequestVote.class);
197         member2Actor.waitForExpectedMessages(RequestVote.class);
198
199         verifyBehaviorState("member 1", member1Actor, RaftState.Leader);
200         verifyBehaviorState("member 2", member2Actor, RaftState.Follower);
201         verifyBehaviorState("member 3", member3Actor, RaftState.Candidate);
202
203         assertEquals("member 1 election term", 3, member1Context.getTermInformation().getCurrentTerm());
204         assertEquals("member 2 election term", 3, member2Context.getTermInformation().getCurrentTerm());
205         assertEquals("member 3 election term", candidateElectionTerm,
206                 member3Context.getTermInformation().getCurrentTerm());
207
208         testLog.info("setupPartitionedCandidateMember3AndSendElectionTimeouts ending");
209     }
210
211     private void setupInitialMember1AndMember2Behaviors() throws Exception {
212         testLog.info("setupInitialMember1AndMember2Behaviors starting");
213
214         // Initialize the ReplicatedLog and election term info for member 1 and 2. The current term
215         // will be 3 and the last term will be 2.
216
217         SimpleReplicatedLog replicatedLog = new SimpleReplicatedLog();
218         replicatedLog.append(new MockReplicatedLogEntry(2, 0, new MockPayload("")));
219         replicatedLog.append(new MockReplicatedLogEntry(3, 1, new MockPayload("")));
220
221         // Create member 2's behavior as Follower.
222
223         member2Context = newRaftActorContext("member2", member2ActorRef,
224                 ImmutableMap.<String,String>builder().
225                     put("member1", member1ActorRef.path().toString()).
226                     put("member3", member3ActorRef.path().toString()).build());
227
228         DefaultConfigParamsImpl member2ConfigParams = newConfigParams();
229         member2Context.setConfigParams(member2ConfigParams);
230
231         member2Context.setReplicatedLog(replicatedLog);
232         member2Context.setCommitIndex(replicatedLog.lastIndex());
233         member2Context.setLastApplied(replicatedLog.lastIndex());
234         member2Context.getTermInformation().update(3, "member1");
235
236         member2Actor.behavior = new Follower(member2Context);
237         member2Context.setCurrentBehavior(member2Actor.behavior);
238
239         // Create member 1's behavior as Leader.
240
241         member1Context = newRaftActorContext("member1", member1ActorRef,
242                 ImmutableMap.<String,String>builder().
243                     put("member2", member2ActorRef.path().toString()).
244                     put("member3", member3ActorRef.path().toString()).build());
245
246         DefaultConfigParamsImpl member1ConfigParams = newConfigParams();
247         member1Context.setConfigParams(member1ConfigParams);
248
249         member1Context.setReplicatedLog(replicatedLog);
250         member1Context.setCommitIndex(replicatedLog.lastIndex());
251         member1Context.setLastApplied(replicatedLog.lastIndex());
252         member1Context.getTermInformation().update(3, "member1");
253
254         initializeLeaderBehavior(member1Actor, member1Context, 1);
255
256         member2Actor.clear();
257         member3Actor.clear();
258
259         testLog.info("setupInitialMember1AndMember2Behaviors ending");
260
261     }
262 }