Simplify isolated leader check
[controller.git] / opendaylight / md-sal / sal-akka-raft / src / main / java / org / opendaylight / controller / cluster / raft / behaviors / Leader.java
1 /*
2  * Copyright (c) 2014 Cisco 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 akka.actor.ActorRef;
11 import akka.actor.ActorSelection;
12 import com.google.common.annotations.VisibleForTesting;
13 import com.google.common.base.Preconditions;
14 import com.google.common.base.Stopwatch;
15 import java.util.concurrent.TimeUnit;
16 import javax.annotation.Nonnull;
17 import javax.annotation.Nullable;
18 import org.opendaylight.controller.cluster.raft.FollowerLogInformation;
19 import org.opendaylight.controller.cluster.raft.RaftActorContext;
20 import org.opendaylight.controller.cluster.raft.RaftActorLeadershipTransferCohort;
21 import org.opendaylight.controller.cluster.raft.RaftState;
22 import org.opendaylight.controller.cluster.raft.base.messages.ElectionTimeout;
23 import org.opendaylight.controller.cluster.raft.messages.AppendEntriesReply;
24
25 /**
26  * The behavior of a RaftActor when it is in the Leader state
27  * <p/>
28  * Leaders:
29  * <ul>
30  * <li> Upon election: send initial empty AppendEntries RPCs
31  * (heartbeat) to each server; repeat during idle periods to
32  * prevent election timeouts (§5.2)
33  * <li> If command received from client: append entry to local log,
34  * respond after entry applied to state machine (§5.3)
35  * <li> If last log index ≥ nextIndex for a follower: send
36  * AppendEntries RPC with log entries starting at nextIndex
37  * <ul>
38  * <li> If successful: update nextIndex and matchIndex for
39  * follower (§5.3)
40  * <li> If AppendEntries fails because of log inconsistency:
41  * decrement nextIndex and retry (§5.3)
42  * </ul>
43  * <li> If there exists an N such that N > commitIndex, a majority
44  * of matchIndex[i] ≥ N, and log[N].term == currentTerm:
45  * set commitIndex = N (§5.3, §5.4).
46  */
47 public class Leader extends AbstractLeader {
48     /**
49      * Internal message sent to periodically check if this leader has become isolated and should transition
50      * to {@link IsolatedLeader}.
51      */
52     @VisibleForTesting
53     static final Object ISOLATED_LEADER_CHECK = new Object();
54
55     private final Stopwatch isolatedLeaderCheck;
56     private @Nullable LeadershipTransferContext leadershipTransferContext;
57
58     public Leader(RaftActorContext context) {
59         super(context, RaftState.Leader);
60         isolatedLeaderCheck = Stopwatch.createStarted();
61     }
62
63     @Override
64     public RaftActorBehavior handleMessage(ActorRef sender, Object originalMessage) {
65         Preconditions.checkNotNull(sender, "sender should not be null");
66
67         if (ISOLATED_LEADER_CHECK.equals(originalMessage) && isLeaderIsolated()) {
68             LOG.warn("{}: At least {} followers need to be active, Switching {} from Leader to IsolatedLeader",
69                 context.getId(), getMinIsolatedLeaderPeerCount(), leaderId);
70
71             return internalSwitchBehavior(RaftState.IsolatedLeader);
72         }
73
74         return super.handleMessage(sender, originalMessage);
75     }
76
77     @Override
78     protected void beforeSendHeartbeat(){
79         if(isolatedLeaderCheck.elapsed(TimeUnit.MILLISECONDS) > context.getConfigParams().getIsolatedCheckIntervalInMillis()){
80             context.getActor().tell(ISOLATED_LEADER_CHECK, context.getActor());
81             isolatedLeaderCheck.reset().start();
82         }
83
84         if(leadershipTransferContext != null && leadershipTransferContext.isExpired(
85                 context.getConfigParams().getElectionTimeOutInterval().toMillis())) {
86             LOG.debug("{}: Leadership transfer expired", logName());
87             leadershipTransferContext = null;
88         }
89     }
90
91     @Override
92     protected RaftActorBehavior handleAppendEntriesReply(ActorRef sender, AppendEntriesReply appendEntriesReply) {
93         RaftActorBehavior returnBehavior = super.handleAppendEntriesReply(sender, appendEntriesReply);
94         tryToCompleteLeadershipTransfer(appendEntriesReply.getFollowerId());
95         return returnBehavior;
96     }
97
98     /**
99      * Attempts to transfer leadership to a follower as per the raft paper (§3.10) as follows:
100      * <ul>
101      * <li>Start a timer (Stopwatch).</li>
102      * <li>Send an initial AppendEntries heartbeat to all followers.</li>
103      * <li>On AppendEntriesReply, check if the follower's new match Index matches the leader's last index</li>
104      * <li>If it matches, </li>
105      *   <ul>
106      *   <li>Send an additional AppendEntries to ensure the follower has applied all its log entries to its state.</li>
107      *   <li>Send an ElectionTimeout to the follower to immediately start an election.</li>
108      *   <li>Notify {@link RaftActorLeadershipTransferCohort#transferComplete}.</li>
109      *   </ul>
110      * <li>Otherwise if the election time out period elapses, notify
111      *     {@link RaftActorLeadershipTransferCohort#abortTtransfer}.</li>
112      * </ul>
113      *
114      * @param leadershipTransferCohort
115      */
116     public void transferLeadership(@Nonnull RaftActorLeadershipTransferCohort leadershipTransferCohort) {
117         LOG.debug("{}: Attempting to transfer leadership", logName());
118
119         leadershipTransferContext = new LeadershipTransferContext(leadershipTransferCohort);
120
121         // Send an immediate heart beat to the followers.
122         sendAppendEntries(0, false);
123     }
124
125     private void tryToCompleteLeadershipTransfer(String followerId) {
126         if(leadershipTransferContext == null) {
127             return;
128         }
129
130         FollowerLogInformation followerInfo = getFollower(followerId);
131         if(followerInfo == null) {
132             return;
133         }
134
135         long lastIndex = context.getReplicatedLog().lastIndex();
136         boolean isVoting = context.getPeerInfo(followerId).isVoting();
137
138         LOG.debug("{}: tryToCompleteLeadershipTransfer: followerId: {}, matchIndex: {}, lastIndex: {}, isVoting: {}",
139                 logName(), followerId, followerInfo.getMatchIndex(), lastIndex, isVoting);
140
141         if(isVoting && followerInfo.getMatchIndex() == lastIndex) {
142             LOG.debug("{}: Follower's log matches - sending ElectionTimeout", logName());
143
144             // We can't be sure if the follower has applied all its log entries to its state so send an
145             // additional AppendEntries with the latest commit index.
146             sendAppendEntries(0, false);
147
148             // Now send an ElectionTimeout to the matching follower to immediately start an election.
149             ActorSelection followerActor = context.getPeerActorSelection(followerId);
150             followerActor.tell(ElectionTimeout.INSTANCE, context.getActor());
151
152             LOG.debug("{}: Leader transfer complete", logName());
153
154             leadershipTransferContext.transferCohort.transferComplete();
155             leadershipTransferContext = null;
156         }
157     }
158
159     @Override
160     public void close() {
161         if(leadershipTransferContext != null) {
162             leadershipTransferContext.transferCohort.abortTransfer();
163         }
164
165         super.close();
166     }
167
168     @VisibleForTesting
169     void markFollowerActive(String followerId) {
170         getFollower(followerId).markFollowerActive();
171     }
172
173     @VisibleForTesting
174     void markFollowerInActive(String followerId) {
175         getFollower(followerId).markFollowerInActive();
176     }
177
178     private static class LeadershipTransferContext {
179         RaftActorLeadershipTransferCohort transferCohort;
180         Stopwatch timer = Stopwatch.createStarted();
181
182         LeadershipTransferContext(RaftActorLeadershipTransferCohort transferCohort) {
183             this.transferCohort = transferCohort;
184         }
185
186         boolean isExpired(long timeout) {
187             if(timer.elapsed(TimeUnit.MILLISECONDS) >= timeout) {
188                 transferCohort.abortTransfer();
189                 return true;
190             }
191
192             return false;
193         }
194     }
195 }