node.go 12 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408409410411412413414415416417418419420421422423424425426427428429430431432433434435
  1. /*
  2. Copyright 2014 CoreOS, Inc.
  3. Licensed under the Apache License, Version 2.0 (the "License");
  4. you may not use this file except in compliance with the License.
  5. You may obtain a copy of the License at
  6. http://www.apache.org/licenses/LICENSE-2.0
  7. Unless required by applicable law or agreed to in writing, software
  8. distributed under the License is distributed on an "AS IS" BASIS,
  9. WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
  10. See the License for the specific language governing permissions and
  11. limitations under the License.
  12. */
  13. package raft
  14. import (
  15. "errors"
  16. "log"
  17. "github.com/coreos/etcd/Godeps/_workspace/src/golang.org/x/net/context"
  18. pb "github.com/coreos/etcd/raft/raftpb"
  19. )
  20. var (
  21. emptyState = pb.HardState{}
  22. // ErrStopped is returned by methods on Nodes that have been stopped.
  23. ErrStopped = errors.New("raft: stopped")
  24. )
  25. // SoftState provides state that is useful for logging and debugging.
  26. // The state is volatile and does not need to be persisted to the WAL.
  27. type SoftState struct {
  28. Lead uint64
  29. RaftState StateType
  30. }
  31. func (a *SoftState) equal(b *SoftState) bool {
  32. return a.Lead == b.Lead && a.RaftState == b.RaftState
  33. }
  34. // Ready encapsulates the entries and messages that are ready to read,
  35. // be saved to stable storage, committed or sent to other peers.
  36. // All fields in Ready are read-only.
  37. type Ready struct {
  38. // The current volatile state of a Node.
  39. // SoftState will be nil if there is no update.
  40. // It is not required to consume or store SoftState.
  41. *SoftState
  42. // The current state of a Node to be saved to stable storage BEFORE
  43. // Messages are sent.
  44. // HardState will be equal to empty state if there is no update.
  45. pb.HardState
  46. // Entries specifies entries to be saved to stable storage BEFORE
  47. // Messages are sent.
  48. Entries []pb.Entry
  49. // Snapshot specifies the snapshot to be saved to stable storage.
  50. Snapshot pb.Snapshot
  51. // CommittedEntries specifies entries to be committed to a
  52. // store/state-machine. These have previously been committed to stable
  53. // store.
  54. CommittedEntries []pb.Entry
  55. // Messages specifies outbound messages to be sent AFTER Entries are
  56. // committed to stable storage.
  57. Messages []pb.Message
  58. }
  59. func isHardStateEqual(a, b pb.HardState) bool {
  60. return a.Term == b.Term && a.Vote == b.Vote && a.Commit == b.Commit
  61. }
  62. // IsEmptyHardState returns true if the given HardState is empty.
  63. func IsEmptyHardState(st pb.HardState) bool {
  64. return isHardStateEqual(st, emptyState)
  65. }
  66. // IsEmptySnap returns true if the given Snapshot is empty.
  67. func IsEmptySnap(sp pb.Snapshot) bool {
  68. return sp.Metadata.Index == 0
  69. }
  70. func (rd Ready) containsUpdates() bool {
  71. return rd.SoftState != nil || !IsEmptyHardState(rd.HardState) ||
  72. !IsEmptySnap(rd.Snapshot) || len(rd.Entries) > 0 ||
  73. len(rd.CommittedEntries) > 0 || len(rd.Messages) > 0
  74. }
  75. // Node represents a node in a raft cluster.
  76. type Node interface {
  77. // Tick increments the internal logical clock for the Node by a single tick. Election
  78. // timeouts and heartbeat timeouts are in units of ticks.
  79. Tick()
  80. // Campaign causes the Node to transition to candidate state and start campaigning to become leader.
  81. Campaign(ctx context.Context) error
  82. // Propose proposes that data be appended to the log.
  83. Propose(ctx context.Context, data []byte) error
  84. // ProposeConfChange proposes config change.
  85. // At most one ConfChange can be in the process of going through consensus.
  86. // Application needs to call ApplyConfChange when applying EntryConfChange type entry.
  87. ProposeConfChange(ctx context.Context, cc pb.ConfChange) error
  88. // Step advances the state machine using the given message. ctx.Err() will be returned, if any.
  89. Step(ctx context.Context, msg pb.Message) error
  90. // Ready returns a channel that returns the current point-in-time state
  91. // Users of the Node must call Advance after applying the state returned by Ready
  92. Ready() <-chan Ready
  93. // Advance notifies the Node that the application has applied and saved progress up to the last Ready.
  94. // It prepares the node to return the next available Ready.
  95. Advance()
  96. // ApplyConfChange applies config change to the local node.
  97. // Returns an opaque ConfState protobuf which must be recorded
  98. // in snapshots. Will never return nil; it returns a pointer only
  99. // to match MemoryStorage.Compact.
  100. ApplyConfChange(cc pb.ConfChange) *pb.ConfState
  101. // Status returns the current status of the raft state machine.
  102. Status() Status
  103. // Stop performs any necessary termination of the Node
  104. Stop()
  105. }
  106. type Peer struct {
  107. ID uint64
  108. Context []byte
  109. }
  110. // StartNode returns a new Node given a unique raft id, a list of raft peers, and
  111. // the election and heartbeat timeouts in units of ticks.
  112. // It appends a ConfChangeAddNode entry for each given peer to the initial log.
  113. func StartNode(id uint64, peers []Peer, election, heartbeat int, storage Storage) Node {
  114. n := newNode()
  115. r := newRaft(id, nil, election, heartbeat, storage)
  116. // become the follower at term 1 and apply initial configuration
  117. // entires of term 1
  118. r.becomeFollower(1, None)
  119. for _, peer := range peers {
  120. cc := pb.ConfChange{Type: pb.ConfChangeAddNode, NodeID: peer.ID, Context: peer.Context}
  121. d, err := cc.Marshal()
  122. if err != nil {
  123. panic("unexpected marshal error")
  124. }
  125. e := pb.Entry{Type: pb.EntryConfChange, Term: 1, Index: r.raftLog.lastIndex() + 1, Data: d}
  126. r.raftLog.append(e)
  127. }
  128. // Mark these initial entries as committed.
  129. // TODO(bdarnell): These entries are still unstable; do we need to preserve
  130. // the invariant that committed < unstable?
  131. r.raftLog.committed = r.raftLog.lastIndex()
  132. r.Commit = r.raftLog.committed
  133. // Now apply them, mainly so that the application can call Campaign
  134. // immediately after StartNode in tests. Note that these nodes will
  135. // be added to raft twice: here and when the application's Ready
  136. // loop calls ApplyConfChange. The calls to addNode must come after
  137. // all calls to raftLog.append so progress.next is set after these
  138. // bootstrapping entries (it is an error if we try to append these
  139. // entries since they have already been committed).
  140. // We do not set raftLog.applied so the application will be able
  141. // to observe all conf changes via Ready.CommittedEntries.
  142. for _, peer := range peers {
  143. r.addNode(peer.ID)
  144. }
  145. go n.run(r)
  146. return &n
  147. }
  148. // RestartNode is identical to StartNode but does not take a list of peers.
  149. // The current membership of the cluster will be restored from the Storage.
  150. func RestartNode(id uint64, election, heartbeat int, storage Storage) Node {
  151. n := newNode()
  152. r := newRaft(id, nil, election, heartbeat, storage)
  153. go n.run(r)
  154. return &n
  155. }
  156. // node is the canonical implementation of the Node interface
  157. type node struct {
  158. propc chan pb.Message
  159. recvc chan pb.Message
  160. confc chan pb.ConfChange
  161. confstatec chan pb.ConfState
  162. readyc chan Ready
  163. advancec chan struct{}
  164. tickc chan struct{}
  165. done chan struct{}
  166. stop chan struct{}
  167. status chan Status
  168. }
  169. func newNode() node {
  170. return node{
  171. propc: make(chan pb.Message),
  172. recvc: make(chan pb.Message),
  173. confc: make(chan pb.ConfChange),
  174. confstatec: make(chan pb.ConfState),
  175. readyc: make(chan Ready),
  176. advancec: make(chan struct{}),
  177. tickc: make(chan struct{}),
  178. done: make(chan struct{}),
  179. stop: make(chan struct{}),
  180. status: make(chan Status),
  181. }
  182. }
  183. func (n *node) Stop() {
  184. select {
  185. case n.stop <- struct{}{}:
  186. // Not already stopped, so trigger it
  187. case <-n.done:
  188. // Node has already been stopped - no need to do anything
  189. return
  190. }
  191. // Block until the stop has been acknowledged by run()
  192. <-n.done
  193. }
  194. func (n *node) run(r *raft) {
  195. var propc chan pb.Message
  196. var readyc chan Ready
  197. var advancec chan struct{}
  198. var prevLastUnstablei, prevLastUnstablet uint64
  199. var havePrevLastUnstablei bool
  200. var prevSnapi uint64
  201. var rd Ready
  202. lead := None
  203. prevSoftSt := r.softState()
  204. prevHardSt := r.HardState
  205. status := &Status{ID: r.id}
  206. for {
  207. status.update(r)
  208. if advancec != nil {
  209. readyc = nil
  210. } else {
  211. rd = newReady(r, prevSoftSt, prevHardSt)
  212. if rd.containsUpdates() {
  213. readyc = n.readyc
  214. } else {
  215. readyc = nil
  216. }
  217. }
  218. if lead != r.lead {
  219. if r.hasLeader() {
  220. if lead == None {
  221. log.Printf("raft.node: %x elected leader %x at term %d", r.id, r.lead, r.Term)
  222. } else {
  223. log.Printf("raft.node: %x changed leader from %x to %x at term %d", r.id, lead, r.lead, r.Term)
  224. }
  225. propc = n.propc
  226. } else {
  227. log.Printf("raft.node: %x lost leader %x at term %d", r.id, lead, r.Term)
  228. propc = nil
  229. }
  230. lead = r.lead
  231. }
  232. select {
  233. // TODO: maybe buffer the config propose if there exists one (the way
  234. // described in raft dissertation)
  235. // Currently it is dropped in Step silently.
  236. case m := <-propc:
  237. m.From = r.id
  238. r.Step(m)
  239. case m := <-n.recvc:
  240. // filter out response message from unknow From.
  241. if _, ok := r.prs[m.From]; ok || !IsResponseMsg(m) {
  242. r.Step(m) // raft never returns an error
  243. }
  244. case cc := <-n.confc:
  245. if cc.NodeID == None {
  246. r.resetPendingConf()
  247. select {
  248. case n.confstatec <- pb.ConfState{Nodes: r.nodes()}:
  249. case <-n.done:
  250. }
  251. break
  252. }
  253. switch cc.Type {
  254. case pb.ConfChangeAddNode:
  255. r.addNode(cc.NodeID)
  256. case pb.ConfChangeRemoveNode:
  257. // block incoming proposal when local node is
  258. // removed
  259. if cc.NodeID == r.id {
  260. n.propc = nil
  261. }
  262. r.removeNode(cc.NodeID)
  263. case pb.ConfChangeUpdateNode:
  264. r.resetPendingConf()
  265. default:
  266. panic("unexpected conf type")
  267. }
  268. select {
  269. case n.confstatec <- pb.ConfState{Nodes: r.nodes()}:
  270. case <-n.done:
  271. }
  272. case <-n.tickc:
  273. r.tick()
  274. case readyc <- rd:
  275. if rd.SoftState != nil {
  276. prevSoftSt = rd.SoftState
  277. }
  278. if len(rd.Entries) > 0 {
  279. prevLastUnstablei = rd.Entries[len(rd.Entries)-1].Index
  280. prevLastUnstablet = rd.Entries[len(rd.Entries)-1].Term
  281. havePrevLastUnstablei = true
  282. }
  283. if !IsEmptyHardState(rd.HardState) {
  284. prevHardSt = rd.HardState
  285. }
  286. if !IsEmptySnap(rd.Snapshot) {
  287. prevSnapi = rd.Snapshot.Metadata.Index
  288. }
  289. r.msgs = nil
  290. advancec = n.advancec
  291. case <-advancec:
  292. if prevHardSt.Commit != 0 {
  293. r.raftLog.appliedTo(prevHardSt.Commit)
  294. }
  295. if havePrevLastUnstablei {
  296. r.raftLog.stableTo(prevLastUnstablei, prevLastUnstablet)
  297. havePrevLastUnstablei = false
  298. }
  299. r.raftLog.stableSnapTo(prevSnapi)
  300. advancec = nil
  301. case n.status <- status.get():
  302. case <-n.stop:
  303. close(n.done)
  304. return
  305. }
  306. }
  307. }
  308. // Tick increments the internal logical clock for this Node. Election timeouts
  309. // and heartbeat timeouts are in units of ticks.
  310. func (n *node) Tick() {
  311. select {
  312. case n.tickc <- struct{}{}:
  313. case <-n.done:
  314. }
  315. }
  316. func (n *node) Campaign(ctx context.Context) error { return n.step(ctx, pb.Message{Type: pb.MsgHup}) }
  317. func (n *node) Propose(ctx context.Context, data []byte) error {
  318. return n.step(ctx, pb.Message{Type: pb.MsgProp, Entries: []pb.Entry{{Data: data}}})
  319. }
  320. func (n *node) Step(ctx context.Context, m pb.Message) error {
  321. // ignore unexpected local messages receiving over network
  322. if IsLocalMsg(m) {
  323. // TODO: return an error?
  324. return nil
  325. }
  326. return n.step(ctx, m)
  327. }
  328. func (n *node) ProposeConfChange(ctx context.Context, cc pb.ConfChange) error {
  329. data, err := cc.Marshal()
  330. if err != nil {
  331. return err
  332. }
  333. return n.Step(ctx, pb.Message{Type: pb.MsgProp, Entries: []pb.Entry{{Type: pb.EntryConfChange, Data: data}}})
  334. }
  335. // Step advances the state machine using msgs. The ctx.Err() will be returned,
  336. // if any.
  337. func (n *node) step(ctx context.Context, m pb.Message) error {
  338. ch := n.recvc
  339. if m.Type == pb.MsgProp {
  340. ch = n.propc
  341. }
  342. select {
  343. case ch <- m:
  344. return nil
  345. case <-ctx.Done():
  346. return ctx.Err()
  347. case <-n.done:
  348. return ErrStopped
  349. }
  350. }
  351. func (n *node) Ready() <-chan Ready { return n.readyc }
  352. func (n *node) Advance() {
  353. select {
  354. case n.advancec <- struct{}{}:
  355. case <-n.done:
  356. }
  357. }
  358. func (n *node) ApplyConfChange(cc pb.ConfChange) *pb.ConfState {
  359. var cs pb.ConfState
  360. select {
  361. case n.confc <- cc:
  362. case <-n.done:
  363. }
  364. select {
  365. case cs = <-n.confstatec:
  366. case <-n.done:
  367. }
  368. return &cs
  369. }
  370. func (n *node) Status() Status { return <-n.status }
  371. func newReady(r *raft, prevSoftSt *SoftState, prevHardSt pb.HardState) Ready {
  372. rd := Ready{
  373. Entries: r.raftLog.unstableEntries(),
  374. CommittedEntries: r.raftLog.nextEnts(),
  375. Messages: r.msgs,
  376. }
  377. if softSt := r.softState(); !softSt.equal(prevSoftSt) {
  378. rd.SoftState = softSt
  379. }
  380. if !isHardStateEqual(r.HardState, prevHardSt) {
  381. rd.HardState = r.HardState
  382. }
  383. if r.raftLog.unstable.snapshot != nil {
  384. rd.Snapshot = *r.raftLog.unstable.snapshot
  385. }
  386. return rd
  387. }