FastLeaderElection.tla 34 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408409410411412413414415416417418419420421422423424425426427428429430431432433434435436437438439440441442443444445446447448449450451452453454455456457458459460461462463464465466467468469470471472473474475476477478479480481482483484485486487488489490491492493494495496497498499500501502503504505506507508509510511512513514515516517518519520521522523524525526527528529530531532533534535536537538539540541542543544545546547548549550551552553
  1. (*
  2. * Licensed to the Apache Software Foundation (ASF) under one
  3. * or more contributor license agreements. See the NOTICE file
  4. * distributed with this work for additional information
  5. * regarding copyright ownership. The ASF licenses this file
  6. * to you under the Apache License, Version 2.0 (the
  7. * "License"); you may not use this file except in compliance
  8. * with the License. You may obtain a copy of the License at
  9. *
  10. * http://www.apache.org/licenses/LICENSE-2.0
  11. *
  12. * Unless required by applicable law or agreed to in writing, software
  13. * distributed under the License is distributed on an "AS IS" BASIS,
  14. * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
  15. * See the License for the specific language governing permissions and
  16. * limitations under the License.
  17. *)
  18. ------------------------- MODULE FastLeaderElection -------------------------
  19. \* This is the formal specification for Fast Leader Election in Zab protocol.
  20. (* Reference:
  21. FastLeaderElection.java, Vote.java, QuorumPeer.java in https://github.com/apache/zookeeper.
  22. Medeiros A. ZooKeeper's atomic broadcast protocol: Theory and practice[J]. Aalto University School of Science, 2012.
  23. *)
  24. EXTENDS Integers, FiniteSets, Sequences, Naturals, TLC
  25. -----------------------------------------------------------------------------
  26. \* The set of server identifiers
  27. CONSTANT Server
  28. \* Server states
  29. CONSTANTS LOOKING, FOLLOWING, LEADING
  30. (* NOTE: In spec, we do not discuss servers whose ServerState is OBSERVING.
  31. *)
  32. \* Message types
  33. CONSTANTS NOTIFICATION
  34. \* Timeout signal
  35. CONSTANT NONE
  36. -----------------------------------------------------------------------------
  37. Quorums == {Q \in SUBSET Server: Cardinality(Q)*2 > Cardinality(Server)}
  38. NullPoint == CHOOSE p: p \notin Server
  39. -----------------------------------------------------------------------------
  40. \* Server's state(LOOKING, FOLLOWING, LEADING).
  41. VARIABLE state
  42. VARIABLE history
  43. \* The epoch number of the last NEWLEADER packet accepted, used for comparing.
  44. VARIABLE currentEpoch
  45. \* The index and zxid of the last processed transaction in history.
  46. VARIABLE lastProcessed
  47. \* currentVote[i]: The server who i thinks is the current leader(id,zxid,peerEpoch,...).
  48. VARIABLE currentVote
  49. \* Election instance.(logicalClock in code)
  50. VARIABLE logicalClock
  51. \* The votes from the current leader election are stored in ReceiveVotes.
  52. VARIABLE receiveVotes
  53. (* The votes from previous leader elections, as well as the votes from the current leader election are
  54. stored in outofelection. Note that notifications in a LOOKING state are not stored in outofelection.
  55. Only FOLLOWING or LEADING notifications are stored in outofelection. *)
  56. VARIABLE outOfElection
  57. \* recvQueue[i]: The queue of received notifications or timeout signals in server i.
  58. VARIABLE recvQueue
  59. \* A variable to wait for new notifications, corresponding to line 1050 in FastLeaderElection.java.
  60. VARIABLE waitNotmsg
  61. \* leadingVoteSet[i]: The set of voters that follow i.
  62. VARIABLE leadingVoteSet
  63. (* The messages about election sent from one server to another.
  64. electionMsgs[i][j] means the input buffer of server j from server i. *)
  65. VARIABLE electionMsgs
  66. \* Set used for mapping Server to Integers, to compare ids from different servers.
  67. \* VARIABLE idTable
  68. serverVarsL == <<state, currentEpoch, lastProcessed, history>>
  69. electionVarsL == <<currentVote, logicalClock, receiveVotes, outOfElection, recvQueue, waitNotmsg>>
  70. leaderVarsL == <<leadingVoteSet>>
  71. varsL == <<serverVarsL, electionVarsL, leaderVarsL, electionMsgs>>
  72. -----------------------------------------------------------------------------
  73. \* Processing of electionMsgs
  74. BroadcastNotmsg(i, m) == electionMsgs' = [electionMsgs EXCEPT ![i] = [v \in Server |-> IF v /= i
  75. THEN Append(electionMsgs[i][v], m)
  76. ELSE electionMsgs[i][v]]]
  77. DiscardNotmsg(i, j) == electionMsgs' = [electionMsgs EXCEPT ![i][j] = IF electionMsgs[i][j] /= << >>
  78. THEN Tail(electionMsgs[i][j])
  79. ELSE << >>]
  80. ReplyNotmsg(i, j, m) == electionMsgs' = [electionMsgs EXCEPT ![i][j] = Append(electionMsgs[i][j], m),
  81. ![j][i] = Tail(electionMsgs[j][i])]
  82. -----------------------------------------------------------------------------
  83. \* Processing of recvQueue
  84. RECURSIVE RemoveNone(_)
  85. RemoveNone(seq) == CASE seq = << >> -> << >>
  86. [] seq /= << >> -> IF Head(seq).mtype = NONE THEN RemoveNone(Tail(seq))
  87. ELSE <<Head(seq)>> \o RemoveNone(Tail(seq))
  88. \* Processing of idTable and order comparing
  89. RECURSIVE InitializeIdTable(_)
  90. InitializeIdTable(Remaining) == IF Remaining = {} THEN {}
  91. ELSE LET chosen == CHOOSE i \in Remaining: TRUE
  92. re == Remaining \ {chosen}
  93. IN {<<chosen, Cardinality(Remaining)>>} \union InitializeIdTable(re)
  94. IdTable == InitializeIdTable(Server)
  95. \* FALSE: id1 < id2; TRUE: id1 > id2
  96. IdCompare(id1,id2) == LET item1 == CHOOSE item \in IdTable: item[1] = id1
  97. item2 == CHOOSE item \in IdTable: item[1] = id2
  98. IN item1[2] > item2[2]
  99. \* FALSE: zxid1 <= zxid2; TRUE: zxid1 > zxid2
  100. ZxidCompare(zxid1, zxid2) == \/ zxid1[1] > zxid2[1]
  101. \/ /\ zxid1[1] = zxid2[1]
  102. /\ zxid1[2] > zxid2[2]
  103. ZxidEqual(zxid1, zxid2) == zxid1[1] = zxid2[1] /\ zxid1[2] = zxid2[2]
  104. \* FALSE: vote1 <= vote2; TRUE: vote1 > vote2
  105. TotalOrderPredicate(vote1, vote2) == \/ vote1.proposedEpoch > vote2.proposedEpoch
  106. \/ /\ vote1.proposedEpoch = vote2.proposedEpoch
  107. /\ \/ ZxidCompare(vote1.proposedZxid, vote2.proposedZxid)
  108. \/ /\ ZxidEqual(vote1.proposedZxid, vote2.proposedZxid)
  109. /\ IdCompare(vote1.proposedLeader, vote2.proposedLeader)
  110. VoteEqual(vote1, round1, vote2, round2) == /\ vote1.proposedLeader = vote2.proposedLeader
  111. /\ ZxidEqual(vote1.proposedZxid, vote2.proposedZxid)
  112. /\ vote1.proposedEpoch = vote2.proposedEpoch
  113. /\ round1 = round2
  114. InitLastProcessed(i) == IF Len(history[i]) = 0 THEN [ index |-> 0,
  115. zxid |-> <<0, 0>> ]
  116. ELSE
  117. LET lastIndex == Len(history[i])
  118. entry == history[i][lastIndex]
  119. IN [ index |-> lastIndex,
  120. zxid |-> entry.zxid ]
  121. RECURSIVE InitAcksidInTxns(_,_)
  122. InitAcksidInTxns(txns, src) == IF Len(txns) = 0 THEN << >>
  123. ELSE LET newTxn == [ zxid |-> txns[1].zxid,
  124. value |-> txns[1].value,
  125. ackSid |-> {src},
  126. epoch |-> txns[1].epoch ]
  127. IN <<newTxn>> \o InitAcksidInTxns( Tail(txns), src)
  128. InitHistory(i) == LET newState == state'[i] IN
  129. IF newState = LEADING THEN InitAcksidInTxns(history[i], i)
  130. ELSE history[i]
  131. -----------------------------------------------------------------------------
  132. \* Processing of currentVote
  133. InitialVote == [proposedLeader |-> NullPoint,
  134. proposedZxid |-> <<0, 0>>,
  135. proposedEpoch |-> 0]
  136. SelfVote(i) == [proposedLeader |-> i,
  137. proposedZxid |-> lastProcessed[i].zxid,
  138. proposedEpoch |-> currentEpoch[i]]
  139. UpdateProposal(i, nid, nzxid, nepoch) == currentVote' = [currentVote EXCEPT ![i].proposedLeader = nid, \* no need to record state in LOOKING
  140. ![i].proposedZxid = nzxid,
  141. ![i].proposedEpoch = nepoch]
  142. -----------------------------------------------------------------------------
  143. \* Processing of receiveVotes and outOfElection
  144. RvClear(i) == receiveVotes' = [receiveVotes EXCEPT ![i] = [v \in Server |-> [vote |-> InitialVote,
  145. round |-> 0,
  146. state |-> LOOKING,
  147. version |-> 0]]]
  148. RvPut(i, id, mvote, mround, mstate) == receiveVotes' = CASE receiveVotes[i][id].round < mround -> [receiveVotes EXCEPT ![i][id].vote = mvote,
  149. ![i][id].round = mround,
  150. ![i][id].state = mstate,
  151. ![i][id].version = 1]
  152. [] receiveVotes[i][id].round = mround -> [receiveVotes EXCEPT ![i][id].vote = mvote,
  153. ![i][id].state = mstate,
  154. ![i][id].version = @ + 1]
  155. [] receiveVotes[i][id].round > mround -> receiveVotes
  156. Put(i, id, rcvset, mvote, mround, mstate) == CASE rcvset[id].round < mround -> [rcvset EXCEPT ![id].vote = mvote,
  157. ![id].round = mround,
  158. ![id].state = mstate,
  159. ![id].version = 1]
  160. [] rcvset[id].round = mround -> [rcvset EXCEPT ![id].vote = mvote,
  161. ![id].state = mstate,
  162. ![id].version = @ + 1]
  163. [] rcvset[id].round > mround -> rcvset
  164. RvClearAndPut(i, id, vote, round) == receiveVotes' = LET oneVote == [vote |-> vote,
  165. round |-> round,
  166. state |-> LOOKING,
  167. version |-> 1]
  168. IN [receiveVotes EXCEPT ![i] = [v \in Server |-> IF v = id THEN oneVote
  169. ELSE [vote |-> InitialVote,
  170. round |-> 0,
  171. state |-> LOOKING,
  172. version |-> 0]]]
  173. VoteSet(i, msource, rcvset, thisvote, thisround) == {msource} \union {s \in (Server \ {msource}): VoteEqual(rcvset[s].vote,
  174. rcvset[s].round,
  175. thisvote,
  176. thisround)}
  177. HasQuorums(i, msource, rcvset, thisvote, thisround) == LET Q == VoteSet(i, msource, rcvset, thisvote, thisround)
  178. IN IF Q \in Quorums THEN TRUE ELSE FALSE
  179. CheckLeader(i, votes, thisleader, thisround) == IF thisleader = i THEN (IF thisround = logicalClock[i] THEN TRUE ELSE FALSE)
  180. ELSE (IF votes[thisleader].vote.proposedLeader = NullPoint THEN FALSE
  181. ELSE (IF votes[thisleader].state = LEADING THEN TRUE
  182. ELSE FALSE))
  183. OoeClear(i) == outOfElection' = [outOfElection EXCEPT ![i] = [v \in Server |-> [vote |-> InitialVote,
  184. round |-> 0,
  185. state |-> LOOKING,
  186. version |-> 0]]]
  187. OoePut(i, id, mvote, mround, mstate) == outOfElection' = CASE outOfElection[i][id].round < mround -> [outOfElection EXCEPT ![i][id].vote = mvote,
  188. ![i][id].round = mround,
  189. ![i][id].state = mstate,
  190. ![i][id].version = 1]
  191. [] outOfElection[i][id].round = mround -> [outOfElection EXCEPT ![i][id].vote = mvote,
  192. ![i][id].state = mstate,
  193. ![i][id].version = @ + 1]
  194. [] outOfElection[i][id].round > mround -> outOfElection
  195. -----------------------------------------------------------------------------
  196. InitServerVarsL == /\ state = [s \in Server |-> LOOKING]
  197. /\ currentEpoch = [s \in Server |-> 0]
  198. /\ lastProcessed = [s \in Server |-> [index |-> 0,
  199. zxid |-> <<0, 0>>] ]
  200. /\ history = [s \in Server |-> << >>]
  201. InitElectionVarsL == /\ currentVote = [s \in Server |-> SelfVote(s)]
  202. /\ logicalClock = [s \in Server |-> 0]
  203. /\ receiveVotes = [s \in Server |-> [v \in Server |-> [vote |-> InitialVote,
  204. round |-> 0,
  205. state |-> LOOKING,
  206. version |-> 0]]]
  207. /\ outOfElection = [s \in Server |-> [v \in Server |-> [vote |-> InitialVote,
  208. round |-> 0,
  209. state |-> LOOKING,
  210. version |-> 0]]]
  211. /\ recvQueue = [s \in Server |-> << >>]
  212. /\ waitNotmsg = [s \in Server |-> FALSE]
  213. InitLeaderVarsL == leadingVoteSet = [s \in Server |-> {}]
  214. InitL == /\ InitServerVarsL
  215. /\ InitElectionVarsL
  216. /\ InitLeaderVarsL
  217. /\ electionMsgs = [s \in Server |-> [v \in Server |-> << >>]]
  218. \* /\ idTable = InitializeIdTable(Server)
  219. -----------------------------------------------------------------------------
  220. (* The beginning part of FLE's main function lookForLeader() *)
  221. ZabTimeout(i) ==
  222. /\ state[i] \in {LEADING, FOLLOWING}
  223. /\ state' = [state EXCEPT ![i] = LOOKING]
  224. /\ lastProcessed' = [lastProcessed EXCEPT ![i] = InitLastProcessed(i)]
  225. /\ logicalClock' = [logicalClock EXCEPT ![i] = logicalClock[i] + 1]
  226. /\ currentVote' = [currentVote EXCEPT ![i] = [proposedLeader |-> i,
  227. proposedZxid |-> lastProcessed'[i].zxid,
  228. proposedEpoch |-> currentEpoch[i]]]
  229. /\ receiveVotes' = [receiveVotes EXCEPT ![i] = [v \in Server |-> [vote |-> InitialVote,
  230. round |-> 0,
  231. state |-> LOOKING,
  232. version |-> 0]]]
  233. /\ outOfElection' = [outOfElection EXCEPT ![i] = [v \in Server |-> [vote |-> InitialVote,
  234. round |-> 0,
  235. state |-> LOOKING,
  236. version |-> 0]]]
  237. /\ recvQueue' = [recvQueue EXCEPT ![i] = << >>]
  238. /\ waitNotmsg' = [waitNotmsg EXCEPT ![i] = FALSE]
  239. /\ leadingVoteSet' = [leadingVoteSet EXCEPT ![i] = {}]
  240. /\ BroadcastNotmsg(i, [mtype |-> NOTIFICATION,
  241. msource |-> i,
  242. mstate |-> LOOKING,
  243. mround |-> logicalClock'[i],
  244. mvote |-> currentVote'[i]])
  245. /\ UNCHANGED <<currentEpoch, history>>
  246. (* Abstraction of WorkerReceiver.run() *)
  247. ReceiveNotmsg(i, j) ==
  248. /\ electionMsgs[j][i] /= << >>
  249. /\ LET notmsg == electionMsgs[j][i][1]
  250. toSend == [mtype |-> NOTIFICATION,
  251. msource |-> i,
  252. mstate |-> state[i],
  253. mround |-> logicalClock[i],
  254. mvote |-> currentVote[i]]
  255. IN \/ /\ state[i] = LOOKING
  256. /\ recvQueue' = [recvQueue EXCEPT ![i] = Append(RemoveNone(recvQueue[i]), notmsg)]
  257. /\ LET replyOk == /\ notmsg.mstate = LOOKING
  258. /\ notmsg.mround < logicalClock[i]
  259. IN
  260. \/ /\ replyOk
  261. /\ ReplyNotmsg(i, j, toSend)
  262. \/ /\ ~replyOk
  263. /\ DiscardNotmsg(j, i)
  264. \/ /\ state[i] \in {LEADING, FOLLOWING}
  265. /\ \/ \* Only reply when sender's state is LOOKING
  266. /\ notmsg.mstate = LOOKING
  267. /\ ReplyNotmsg(i, j, toSend)
  268. \/ \* sender's state and mine are both not LOOKING, just discard
  269. /\ notmsg.mstate /= LOOKING
  270. /\ DiscardNotmsg(j, i)
  271. /\ UNCHANGED recvQueue
  272. /\ UNCHANGED <<serverVarsL, currentVote, logicalClock, receiveVotes, outOfElection, waitNotmsg, leaderVarsL>>
  273. NotmsgTimeout(i) ==
  274. /\ state[i] = LOOKING
  275. /\ \A j \in Server: electionMsgs[j][i] = << >>
  276. /\ recvQueue[i] = << >>
  277. /\ recvQueue' = [recvQueue EXCEPT ![i] = Append(recvQueue[i], [mtype |-> NONE])]
  278. /\ UNCHANGED <<serverVarsL, currentVote, logicalClock, receiveVotes, outOfElection, waitNotmsg, leaderVarsL, electionMsgs>>
  279. -----------------------------------------------------------------------------
  280. \* Sub-action in HandleNotmsg
  281. ReceivedFollowingAndLeadingNotification(i, n) ==
  282. LET newVotes == Put(i, n.msource, receiveVotes[i], n.mvote, n.mround, n.mstate)
  283. voteSet1 == VoteSet(i, n.msource, newVotes, n.mvote, n.mround)
  284. hasQuorums1 == voteSet1 \in Quorums
  285. check1 == CheckLeader(i, newVotes, n.mvote.proposedLeader, n.mround)
  286. leaveOk1 == /\ n.mround = logicalClock[i]
  287. /\ hasQuorums1
  288. /\ check1 \* state and leadingVoteSet cannot be changed twice in the first '/\' and second '/\'.
  289. IN
  290. /\ \/ /\ n.mround = logicalClock[i]
  291. /\ receiveVotes' = [receiveVotes EXCEPT ![i] = newVotes]
  292. \/ /\ n.mround /= logicalClock[i]
  293. /\ UNCHANGED receiveVotes
  294. /\ \/ /\ leaveOk1
  295. \* /\ PrintT("leave with condition 1")
  296. /\ state' = [state EXCEPT ![i] = IF n.mvote.proposedLeader = i THEN LEADING ELSE FOLLOWING]
  297. /\ leadingVoteSet' = [leadingVoteSet EXCEPT ![i] = IF n.mvote.proposedLeader = i THEN voteSet1 ELSE @]
  298. /\ UpdateProposal(i, n.mvote.proposedLeader, n.mvote.proposedZxid, n.mvote.proposedEpoch)
  299. /\ UNCHANGED <<logicalClock, outOfElection>>
  300. \/ /\ ~leaveOk1
  301. /\ outOfElection' = [outOfElection EXCEPT ![i] = Put(i, n.msource, outOfElection[i], n.mvote,n.mround, n.mstate)]
  302. /\ LET voteSet2 == VoteSet(i, n.msource, outOfElection'[i], n.mvote, n.mround)
  303. hasQuorums2 == voteSet2 \in Quorums
  304. check2 == CheckLeader(i, outOfElection'[i], n.mvote.proposedLeader, n.mround)
  305. leaveOk2 == /\ hasQuorums2
  306. /\ check2
  307. IN
  308. \/ /\ leaveOk2
  309. \* /\ PrintT("leave with condition 2")
  310. /\ logicalClock' = [logicalClock EXCEPT ![i] = n.mround]
  311. /\ state' = [state EXCEPT ![i] = IF n.mvote.proposedLeader = i THEN LEADING ELSE FOLLOWING]
  312. /\ leadingVoteSet' = [leadingVoteSet EXCEPT ![i] = IF n.mvote.proposedLeader = i THEN voteSet2 ELSE @]
  313. /\ UpdateProposal(i, n.mvote.proposedLeader, n.mvote.proposedZxid, n.mvote.proposedEpoch)
  314. \/ /\ ~leaveOk2
  315. /\ LET leaveOk3 == /\ n.mstate = LEADING
  316. /\ n.mround = logicalClock[i]
  317. IN
  318. \/ /\ leaveOk3
  319. \* /\ PrintT("leave with condition 3")
  320. /\ state' = [state EXCEPT ![i] = IF n.mvote.proposedLeader = i THEN LEADING ELSE FOLLOWING]
  321. /\ UpdateProposal(i, n.mvote.proposedLeader, n.mvote.proposedZxid, n.mvote.proposedEpoch)
  322. \/ /\ ~leaveOk3
  323. /\ UNCHANGED <<state, currentVote>>
  324. /\ UNCHANGED <<logicalClock, leadingVoteSet>>
  325. (* Main part of lookForLeader() *)
  326. HandleNotmsg(i) ==
  327. /\ state[i] = LOOKING
  328. /\ \lnot waitNotmsg[i]
  329. /\ recvQueue[i] /= << >>
  330. /\ LET n == recvQueue[i][1]
  331. rawToSend == [mtype |-> NOTIFICATION,
  332. msource |-> i,
  333. mstate |-> LOOKING,
  334. mround |-> logicalClock[i],
  335. mvote |-> currentVote[i]]
  336. IN \/ /\ n.mtype = NONE
  337. /\ BroadcastNotmsg(i, rawToSend)
  338. /\ UNCHANGED <<history, logicalClock, currentVote, receiveVotes, waitNotmsg, outOfElection, state, leadingVoteSet>>
  339. \/ /\ n.mtype = NOTIFICATION
  340. /\ \/ /\ n.mstate = LOOKING
  341. /\ \/ \* n.round >= my round, then update data and receiveVotes.
  342. /\ n.mround >= logicalClock[i]
  343. /\ \/ \* n.round > my round, update round and decide new proposed leader.
  344. /\ n.mround > logicalClock[i]
  345. /\ logicalClock' = [logicalClock EXCEPT ![i] = n.mround] \* There should be RvClear, we will handle it in the following.
  346. /\ LET selfinfo == [proposedLeader |-> i,
  347. proposedZxid |-> lastProcessed[i].zxid,
  348. proposedEpoch |-> currentEpoch[i]]
  349. peerOk == TotalOrderPredicate(n.mvote, selfinfo)
  350. IN \/ /\ peerOk
  351. /\ UpdateProposal(i, n.mvote.proposedLeader, n.mvote.proposedZxid, n.mvote.proposedEpoch)
  352. \/ /\ ~peerOk
  353. /\ UpdateProposal(i, i, lastProcessed[i].zxid, currentEpoch[i])
  354. /\ BroadcastNotmsg(i, [mtype |-> NOTIFICATION,
  355. msource |-> i,
  356. mstate |-> LOOKING,
  357. mround |-> n.mround,
  358. mvote |-> currentVote'[i]])
  359. \/ \* n.round = my round & n.vote > my vote
  360. /\ n.mround = logicalClock[i]
  361. /\ LET peerOk == TotalOrderPredicate(n.mvote, currentVote[i])
  362. IN \/ /\ peerOk
  363. /\ UpdateProposal(i, n.mvote.proposedLeader, n.mvote.proposedZxid, n.mvote.proposedEpoch)
  364. /\ BroadcastNotmsg(i, [mtype |-> NOTIFICATION,
  365. msource |-> i,
  366. mstate |-> LOOKING,
  367. mround |-> logicalClock[i],
  368. mvote |-> n.mvote])
  369. \/ /\ ~peerOk
  370. /\ UNCHANGED <<currentVote, electionMsgs>>
  371. /\ UNCHANGED logicalClock
  372. /\ LET rcvsetModifiedTwice == n.mround > logicalClock[i]
  373. IN \/ /\ rcvsetModifiedTwice \* Since a variable cannot be changed more than once in one action, we handle receiveVotes here.
  374. /\ RvClearAndPut(i, n.msource, n.mvote, n.mround) \* clear + put
  375. \/ /\ ~rcvsetModifiedTwice
  376. /\ RvPut(i, n.msource, n.mvote, n.mround, n.mstate) \* put
  377. /\ LET hasQuorums == HasQuorums(i, i, receiveVotes'[i], currentVote'[i], n.mround)
  378. IN \/ /\ hasQuorums \* If hasQuorums, see action WaitNewNotmsg and WaitNewNotmsgEnd.
  379. /\ waitNotmsg' = [waitNotmsg EXCEPT ![i] = TRUE]
  380. \/ /\ ~hasQuorums
  381. /\ UNCHANGED waitNotmsg
  382. \/ \* n.round < my round, just discard it.
  383. /\ n.mround < logicalClock[i]
  384. /\ UNCHANGED <<logicalClock, currentVote, electionMsgs, receiveVotes, waitNotmsg>>
  385. /\ UNCHANGED <<state, history, outOfElection, leadingVoteSet>>
  386. \/ \* mainly contains receivedFollowingNotification(line 1146), receivedLeadingNotification(line 1185).
  387. /\ n.mstate \in {LEADING, FOLLOWING}
  388. /\ ReceivedFollowingAndLeadingNotification(i, n)
  389. /\ history' = [history EXCEPT ![i] = InitHistory(i) ]
  390. /\ UNCHANGED <<electionMsgs, waitNotmsg>>
  391. /\ recvQueue' = [recvQueue EXCEPT ![i] = Tail(recvQueue[i])]
  392. /\ UNCHANGED <<currentEpoch, lastProcessed>>
  393. \* On the premise that ReceiveVotes.HasQuorums = TRUE, corresponding to logic in LFE.java.
  394. WaitNewNotmsg(i) ==
  395. /\ state[i] = LOOKING
  396. /\ waitNotmsg[i] = TRUE
  397. /\ \/ /\ recvQueue[i] /= << >>
  398. /\ recvQueue[i][1].mtype = NOTIFICATION
  399. /\ LET n == recvQueue[i][1]
  400. peerOk == TotalOrderPredicate(n.mvote, currentVote[i])
  401. IN \/ /\ peerOk
  402. /\ waitNotmsg' = [waitNotmsg EXCEPT ![i] = FALSE]
  403. /\ recvQueue' = [recvQueue EXCEPT ![i] = Append(Tail(@), n)]
  404. \/ /\ ~peerOk
  405. /\ recvQueue' = [recvQueue EXCEPT ![i] = Tail(@)]
  406. /\ UNCHANGED waitNotmsg
  407. /\ UNCHANGED <<serverVarsL, currentVote, logicalClock, receiveVotes, outOfElection,
  408. leaderVarsL, electionMsgs>>
  409. \/ /\ \/ recvQueue[i] = << >>
  410. \/ /\ recvQueue[i] /= << >>
  411. /\ recvQueue[i][1].mtype = NONE
  412. /\ state' = [state EXCEPT ![i] = IF currentVote[i].proposedLeader = i THEN LEADING
  413. ELSE FOLLOWING ]
  414. /\ leadingVoteSet' = [leadingVoteSet EXCEPT ![i] =
  415. IF currentVote[i].proposedLeader = i
  416. THEN VoteSet(i, i, receiveVotes[i], currentVote[i],
  417. logicalClock[i])
  418. ELSE @]
  419. /\ history' = [history EXCEPT ![i] = InitHistory(i)]
  420. /\ UNCHANGED <<currentEpoch, lastProcessed, electionVarsL, electionMsgs>>
  421. -----------------------------------------------------------------------------
  422. (*Test - simulate modifying currentEpoch and lastProcessed.
  423. We want to reach violations to achieve some traces and see whether the whole state of system is advancing.
  424. The actions below are completely not equal to implementation in real,
  425. just simulate a process of leader updates state and followers get it. *)
  426. LeaderAdvanceEpoch(i) ==
  427. /\ state[i] = LEADING
  428. /\ currentEpoch' = [currentEpoch EXCEPT ![i] = @ + 1]
  429. /\ UNCHANGED <<state, lastProcessed, history, electionVarsL, leaderVarsL, electionMsgs>>
  430. FollowerUpdateEpoch(i, j) ==
  431. /\ state[i] = FOLLOWING
  432. /\ currentVote[i].proposedLeader = j
  433. /\ state[j] = LEADING
  434. /\ currentEpoch[i] < currentEpoch[j]
  435. /\ currentEpoch' = [currentEpoch EXCEPT ![i] = currentEpoch[j]]
  436. /\ UNCHANGED <<state, lastProcessed, history, electionVarsL, leaderVarsL, electionMsgs>>
  437. LeaderAdvanceZxid(i) ==
  438. /\ state[i] = LEADING
  439. /\ lastProcessed' = [lastProcessed EXCEPT ![i] = IF lastProcessed[i].zxid[1] = currentEpoch[i]
  440. THEN [ index |-> lastProcessed[i].index + 1,
  441. zxid |-> <<currentEpoch[i], lastProcessed[i].zxid[2] + 1>> ]
  442. ELSE [ index |-> lastProcessed[i].index + 1,
  443. zxid |-> <<currentEpoch[i], 1>> ] ]
  444. /\ history' = [history EXCEPT ![i] = Append(@, [zxid |-> lastProcessed'[i].zxid,
  445. value |-> NONE,
  446. ackSid |-> {},
  447. epoch |-> 0])]
  448. /\ UNCHANGED <<state, currentEpoch, electionVarsL, leaderVarsL, electionMsgs>>
  449. FollowerUpdateZxid(i, j) ==
  450. /\ state[i] = FOLLOWING
  451. /\ currentVote[i].proposedLeader = j
  452. /\ state[j] = LEADING
  453. /\ LET precede == \/ lastProcessed[i].zxid[1] < lastProcessed[j].zxid[1]
  454. \/ /\ lastProcessed[i].zxid[1] = lastProcessed[j].zxid[1]
  455. /\ lastProcessed[i].zxid[2] < lastProcessed[j].zxid[2]
  456. IN /\ precede
  457. /\ lastProcessed' = [lastProcessed EXCEPT ![i] = lastProcessed[j]]
  458. /\ history' = [history EXCEPT ![i] = history[j]]
  459. /\ UNCHANGED <<state, currentEpoch, electionVarsL, leaderVarsL, electionMsgs>>
  460. NextL ==
  461. \/ \E i \in Server: ZabTimeout(i)
  462. \/ \E i, j \in Server: ReceiveNotmsg(i, j)
  463. \/ \E i \in Server: NotmsgTimeout(i)
  464. \/ \E i \in Server: HandleNotmsg(i)
  465. \/ \E i \in Server: WaitNewNotmsg(i)
  466. \/ \E i \in Server: LeaderAdvanceEpoch(i)
  467. \/ \E i, j \in Server: FollowerUpdateEpoch(i, j)
  468. \/ \E i \in Server: LeaderAdvanceZxid(i)
  469. \/ \E i, j \in Server: FollowerUpdateZxid(i, j)
  470. SpecL == InitL /\ [][NextL]_varsL
  471. \* These invariants should be violated after running for minutes.
  472. ShouldBeTriggered1 == ~\E Q \in Quorums: /\ \A i \in Q: /\ state[i] \in {FOLLOWING, LEADING}
  473. /\ currentEpoch[i] > 3
  474. /\ logicalClock[i] > 2
  475. /\ currentVote[i].proposedLeader \in Q
  476. /\ \A i, j \in Q: currentVote[i].proposedLeader = currentVote[j].proposedLeader
  477. (*
  478. ShouldBeTriggered2 == ~\E Q \in Quorums: /\ \A i \in Q: /\ state[i] \in {FOLLOWING, LEADING}
  479. /\ currentEpoch[i] > 3
  480. /\ currentVote[i].proposedLeader \in Q
  481. /\ \A i, j \in Q: currentVote[i].proposedLeader = currentVote[j].proposedLeader*)
  482. =============================================================================
  483. \* Modification History
  484. \* Last modified Sat Jan 14 15:19:45 CST 2023 by huangbinyu
  485. \* Last modified Sun Nov 14 15:18:32 CST 2021 by Dell
  486. \* Created Fri Jun 18 20:23:47 CST 2021 by Dell