25710880dd8ab0c5c7b03e31f8d1fa6fd048f5d5
[controller.git] / opendaylight / md-sal / sal-akka-raft / src / main / java / org / opendaylight / controller / cluster / raft / FollowerLogInformationImpl.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
9 package org.opendaylight.controller.cluster.raft;
10
11 import com.google.common.base.Preconditions;
12 import com.google.common.base.Stopwatch;
13 import java.util.concurrent.TimeUnit;
14 import javax.annotation.Nonnull;
15 import javax.annotation.Nullable;
16 import org.opendaylight.controller.cluster.raft.behaviors.LeaderInstallSnapshotState;
17
18 /**
19  * Implementation of the FollowerLogInformation interface.
20  *
21  * @author Moiz Raja
22  * @author Thomas Pantelis
23  */
24 public class FollowerLogInformationImpl implements FollowerLogInformation {
25     private final Stopwatch stopwatch = Stopwatch.createUnstarted();
26
27     private final RaftActorContext context;
28
29     private long nextIndex;
30
31     private long matchIndex;
32
33     private long lastReplicatedIndex = -1L;
34
35     private final Stopwatch lastReplicatedStopwatch = Stopwatch.createUnstarted();
36
37     private short payloadVersion = -1;
38
39     // Assume the HELIUM_VERSION version initially for backwards compatibility until we obtain the follower's
40     // actual version via AppendEntriesReply. Although we no longer support the Helium version, a pre-Boron
41     // follower will not have the version field in AppendEntriesReply so it will be set to 0 which is
42     // HELIUM_VERSION.
43     private short raftVersion = RaftVersions.HELIUM_VERSION;
44
45     private final PeerInfo peerInfo;
46
47     private LeaderInstallSnapshotState installSnapshotState;
48
49     private long slicedLogEntryIndex = NO_INDEX;
50
51     /**
52      * Constructs an instance.
53      *
54      * @param peerInfo the associated PeerInfo of the follower.
55      * @param matchIndex the initial match index.
56      * @param context the RaftActorContext.
57      */
58     public FollowerLogInformationImpl(final PeerInfo peerInfo, final long matchIndex, final RaftActorContext context) {
59         this.nextIndex = context.getCommitIndex();
60         this.matchIndex = matchIndex;
61         this.context = context;
62         this.peerInfo = Preconditions.checkNotNull(peerInfo);
63     }
64
65     @Override
66     public long incrNextIndex() {
67         return nextIndex++;
68     }
69
70     @Override
71     public boolean decrNextIndex() {
72         if (nextIndex >= 0) {
73             nextIndex--;
74             return true;
75         }
76
77         return false;
78     }
79
80     @Override
81     @SuppressWarnings("checkstyle:hiddenField")
82     public boolean setNextIndex(final long nextIndex) {
83         if (this.nextIndex != nextIndex) {
84             this.nextIndex = nextIndex;
85             return true;
86         }
87
88         return false;
89     }
90
91     @Override
92     public long incrMatchIndex() {
93         return matchIndex++;
94     }
95
96     @Override
97     @SuppressWarnings("checkstyle:hiddenField")
98     public boolean setMatchIndex(final long matchIndex) {
99         // If the new match index is the index of the entry currently being sliced, then we know slicing is complete
100         // and the follower received the entry and responded so clear the slicedLogEntryIndex
101         if (isLogEntrySlicingInProgress() && slicedLogEntryIndex == matchIndex) {
102             slicedLogEntryIndex = NO_INDEX;
103         }
104
105         if (this.matchIndex != matchIndex) {
106             this.matchIndex = matchIndex;
107             return true;
108         }
109
110         return false;
111     }
112
113     @Override
114     public String getId() {
115         return peerInfo.getId();
116     }
117
118     @Override
119     public long getNextIndex() {
120         return nextIndex;
121     }
122
123     @Override
124     public long getMatchIndex() {
125         return matchIndex;
126     }
127
128     @Override
129     public boolean isFollowerActive() {
130         if (peerInfo.getVotingState() == VotingState.VOTING_NOT_INITIALIZED) {
131             return false;
132         }
133
134         long elapsed = stopwatch.elapsed(TimeUnit.MILLISECONDS);
135         return stopwatch.isRunning()
136                 && elapsed <= context.getConfigParams().getElectionTimeOutInterval().toMillis();
137     }
138
139     @Override
140     public void markFollowerActive() {
141         if (stopwatch.isRunning()) {
142             stopwatch.reset();
143         }
144         stopwatch.start();
145     }
146
147     @Override
148     public void markFollowerInActive() {
149         if (stopwatch.isRunning()) {
150             stopwatch.stop();
151         }
152     }
153
154     @Override
155     public long timeSinceLastActivity() {
156         return stopwatch.elapsed(TimeUnit.MILLISECONDS);
157     }
158
159     @Override
160     public boolean okToReplicate() {
161         if (peerInfo.getVotingState() == VotingState.VOTING_NOT_INITIALIZED) {
162             return false;
163         }
164
165         // Return false if we are trying to send duplicate data before the heartbeat interval
166         if (getNextIndex() == lastReplicatedIndex && lastReplicatedStopwatch.elapsed(TimeUnit.MILLISECONDS)
167                 < context.getConfigParams().getHeartBeatInterval().toMillis()) {
168             return false;
169         }
170
171         resetLastReplicated();
172         return true;
173     }
174
175     private void resetLastReplicated() {
176         lastReplicatedIndex = getNextIndex();
177         if (lastReplicatedStopwatch.isRunning()) {
178             lastReplicatedStopwatch.reset();
179         }
180         lastReplicatedStopwatch.start();
181     }
182
183     @Override
184     public short getPayloadVersion() {
185         return payloadVersion;
186     }
187
188     @Override
189     public void setPayloadVersion(final short payloadVersion) {
190         this.payloadVersion = payloadVersion;
191     }
192
193     @Override
194     public short getRaftVersion() {
195         return raftVersion;
196     }
197
198     @Override
199     public void setRaftVersion(final short raftVersion) {
200         this.raftVersion = raftVersion;
201     }
202
203     @Override
204     @Nullable
205     public LeaderInstallSnapshotState getInstallSnapshotState() {
206         return installSnapshotState;
207     }
208
209     @Override
210     public void setLeaderInstallSnapshotState(@Nonnull final LeaderInstallSnapshotState state) {
211         if (this.installSnapshotState == null) {
212             this.installSnapshotState = Preconditions.checkNotNull(state);
213         }
214     }
215
216     @Override
217     public void clearLeaderInstallSnapshotState() {
218         Preconditions.checkState(installSnapshotState != null);
219         installSnapshotState.close();
220         installSnapshotState = null;
221     }
222
223     @Override
224     public void setSlicedLogEntryIndex(final long index) {
225         slicedLogEntryIndex  = index;
226     }
227
228     @Override
229     public boolean isLogEntrySlicingInProgress() {
230         return slicedLogEntryIndex != NO_INDEX;
231     }
232
233     @Override
234     public String toString() {
235         return "FollowerLogInformationImpl [id=" + getId() + ", nextIndex=" + nextIndex + ", matchIndex=" + matchIndex
236                 + ", lastReplicatedIndex=" + lastReplicatedIndex + ", votingState=" + peerInfo.getVotingState()
237                 + ", stopwatch=" + stopwatch.elapsed(TimeUnit.MILLISECONDS) + ", followerTimeoutMillis="
238                 + context.getConfigParams().getElectionTimeOutInterval().toMillis() + "]";
239     }
240 }