node.go 10 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375
  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. "reflect"
  18. "github.com/coreos/etcd/Godeps/_workspace/src/code.google.com/p/go.net/context"
  19. pb "github.com/coreos/etcd/raft/raftpb"
  20. )
  21. var (
  22. emptyState = pb.HardState{}
  23. // ErrStopped is returned by methods on Nodes that have been stopped.
  24. ErrStopped = errors.New("raft: stopped")
  25. )
  26. // SoftState provides state that is useful for logging and debugging.
  27. // The state is volatile and does not need to be persisted to the WAL.
  28. type SoftState struct {
  29. Lead uint64
  30. RaftState StateType
  31. Nodes []uint64
  32. }
  33. func (a *SoftState) equal(b *SoftState) bool {
  34. return reflect.DeepEqual(a, b)
  35. }
  36. // Ready encapsulates the entries and messages that are ready to read,
  37. // be saved to stable storage, committed or sent to other peers.
  38. // All fields in Ready are read-only.
  39. type Ready struct {
  40. // The current volatile state of a Node.
  41. // SoftState will be nil if there is no update.
  42. // It is not required to consume or store SoftState.
  43. *SoftState
  44. // The current state of a Node to be saved to stable storage BEFORE
  45. // Messages are sent.
  46. // HardState will be equal to empty state if there is no update.
  47. pb.HardState
  48. // Entries specifies entries to be saved to stable storage BEFORE
  49. // Messages are sent.
  50. Entries []pb.Entry
  51. // Snapshot specifies the snapshot to be saved to stable storage.
  52. Snapshot pb.Snapshot
  53. // CommittedEntries specifies entries to be committed to a
  54. // store/state-machine. These have previously been committed to stable
  55. // store.
  56. CommittedEntries []pb.Entry
  57. // Messages specifies outbound messages to be sent AFTER Entries are
  58. // committed to stable storage.
  59. Messages []pb.Message
  60. }
  61. type compact struct {
  62. index uint64
  63. nodes []uint64
  64. data []byte
  65. }
  66. func isHardStateEqual(a, b pb.HardState) bool {
  67. return a.Term == b.Term && a.Vote == b.Vote && a.Commit == b.Commit
  68. }
  69. // IsEmptyHardState returns true if the given HardState is empty.
  70. func IsEmptyHardState(st pb.HardState) bool {
  71. return isHardStateEqual(st, emptyState)
  72. }
  73. // IsEmptySnap returns true if the given Snapshot is empty.
  74. func IsEmptySnap(sp pb.Snapshot) bool {
  75. return sp.Index == 0
  76. }
  77. func (rd Ready) containsUpdates() bool {
  78. return rd.SoftState != nil || !IsEmptyHardState(rd.HardState) || !IsEmptySnap(rd.Snapshot) ||
  79. len(rd.Entries) > 0 || len(rd.CommittedEntries) > 0 || len(rd.Messages) > 0
  80. }
  81. // Node represents a node in a raft cluster.
  82. type Node interface {
  83. // Tick increments the internal logical clock for the Node by a single tick. Election
  84. // timeouts and heartbeat timeouts are in units of ticks.
  85. Tick()
  86. // Campaign causes the Node to transition to candidate state and start campaigning to become leader.
  87. Campaign(ctx context.Context) error
  88. // Propose proposes that data be appended to the log.
  89. Propose(ctx context.Context, data []byte) error
  90. // ProposeConfChange proposes config change.
  91. // At most one ConfChange can be in the process of going through consensus.
  92. // Application needs to call ApplyConfChange when applying EntryConfChange type entry.
  93. ProposeConfChange(ctx context.Context, cc pb.ConfChange) error
  94. // Step advances the state machine using the given message. ctx.Err() will be returned, if any.
  95. Step(ctx context.Context, msg pb.Message) error
  96. // Ready returns a channel that returns the current point-in-time state
  97. Ready() <-chan Ready
  98. // ApplyConfChange applies config change to the local node.
  99. // TODO: reject existing node when add node
  100. // TODO: reject non-existant node when remove node
  101. ApplyConfChange(cc pb.ConfChange)
  102. // Stop performs any necessary termination of the Node
  103. Stop()
  104. // Compact discards the entrire log up to the given index. It also
  105. // generates a raft snapshot containing the given nodes configuration
  106. // and the given snapshot data.
  107. // It is the caller's responsibility to ensure the given configuration
  108. // and snapshot data match the actual point-in-time configuration and snapshot
  109. // at the given index.
  110. Compact(index uint64, nodes []uint64, d []byte)
  111. }
  112. type Peer struct {
  113. ID uint64
  114. Context []byte
  115. }
  116. // StartNode returns a new Node given a unique raft id, a list of raft peers, and
  117. // the election and heartbeat timeouts in units of ticks.
  118. // It also builds ConfChangeAddNode entry for each peer and puts them at the head of the log.
  119. func StartNode(id uint64, peers []Peer, election, heartbeat int) Node {
  120. n := newNode()
  121. r := newRaft(id, nil, election, heartbeat)
  122. for _, peer := range peers {
  123. cc := pb.ConfChange{Type: pb.ConfChangeAddNode, NodeID: peer.ID, Context: peer.Context}
  124. d, err := cc.Marshal()
  125. if err != nil {
  126. panic("unexpected marshal error")
  127. }
  128. e := pb.Entry{Type: pb.EntryConfChange, Term: 1, Index: r.raftLog.lastIndex() + 1, Data: d}
  129. r.raftLog.append(r.raftLog.lastIndex(), e)
  130. }
  131. r.raftLog.committed = r.raftLog.lastIndex()
  132. go n.run(r)
  133. return &n
  134. }
  135. // RestartNode is identical to StartNode but takes an initial State and a slice
  136. // of entries. Generally this is used when restarting from a stable storage
  137. // log.
  138. func RestartNode(id uint64, election, heartbeat int, snapshot *pb.Snapshot, st pb.HardState, ents []pb.Entry) Node {
  139. n := newNode()
  140. r := newRaft(id, nil, election, heartbeat)
  141. if snapshot != nil {
  142. r.restore(*snapshot)
  143. }
  144. if !isHardStateEqual(st, emptyState) {
  145. r.loadState(st)
  146. }
  147. if len(ents) != 0 {
  148. r.loadEnts(ents)
  149. }
  150. go n.run(r)
  151. return &n
  152. }
  153. // node is the canonical implementation of the Node interface
  154. type node struct {
  155. propc chan pb.Message
  156. recvc chan pb.Message
  157. compactc chan compact
  158. confc chan pb.ConfChange
  159. readyc chan Ready
  160. tickc chan struct{}
  161. done chan struct{}
  162. }
  163. func newNode() node {
  164. return node{
  165. propc: make(chan pb.Message),
  166. recvc: make(chan pb.Message),
  167. compactc: make(chan compact),
  168. confc: make(chan pb.ConfChange),
  169. readyc: make(chan Ready),
  170. tickc: make(chan struct{}),
  171. done: make(chan struct{}),
  172. }
  173. }
  174. func (n *node) Stop() {
  175. close(n.done)
  176. }
  177. func (n *node) run(r *raft) {
  178. var propc chan pb.Message
  179. var readyc chan Ready
  180. lead := None
  181. prevSoftSt := r.softState()
  182. prevHardSt := r.HardState
  183. prevSnapi := r.raftLog.snapshot.Index
  184. for {
  185. rd := newReady(r, prevSoftSt, prevHardSt, prevSnapi)
  186. if rd.containsUpdates() {
  187. readyc = n.readyc
  188. } else {
  189. readyc = nil
  190. }
  191. if rd.SoftState != nil && lead != rd.SoftState.Lead {
  192. if r.hasLeader() {
  193. if lead == None {
  194. log.Printf("raft: elected leader %x at term %d", rd.SoftState.Lead, r.Term)
  195. } else {
  196. log.Printf("raft: leader changed from %x to %x at term %d", lead, rd.SoftState.Lead, r.Term)
  197. }
  198. propc = n.propc
  199. } else {
  200. log.Printf("raft: lost leader %x at term %d", lead, r.Term)
  201. propc = nil
  202. }
  203. lead = rd.SoftState.Lead
  204. }
  205. select {
  206. // TODO: maybe buffer the config propose if there exists one (the way
  207. // described in raft dissertation)
  208. // Currently it is dropped in Step silently.
  209. case m := <-propc:
  210. m.From = r.id
  211. r.Step(m)
  212. case m := <-n.recvc:
  213. r.Step(m) // raft never returns an error
  214. case c := <-n.compactc:
  215. r.compact(c.index, c.nodes, c.data)
  216. case cc := <-n.confc:
  217. if cc.NodeID == None {
  218. r.resetPendingConf()
  219. break
  220. }
  221. switch cc.Type {
  222. case pb.ConfChangeAddNode:
  223. r.addNode(cc.NodeID)
  224. case pb.ConfChangeRemoveNode:
  225. r.removeNode(cc.NodeID)
  226. default:
  227. panic("unexpected conf type")
  228. }
  229. case <-n.tickc:
  230. r.tick()
  231. case readyc <- rd:
  232. if rd.SoftState != nil {
  233. prevSoftSt = rd.SoftState
  234. }
  235. if !IsEmptyHardState(rd.HardState) {
  236. prevHardSt = rd.HardState
  237. }
  238. if !IsEmptySnap(rd.Snapshot) {
  239. prevSnapi = rd.Snapshot.Index
  240. }
  241. // TODO(yichengq): we assume that all committed config
  242. // entries will be applied to make things easy for now.
  243. // TODO(yichengq): it may have race because applied is set
  244. // before entries are applied.
  245. r.raftLog.resetNextEnts()
  246. r.raftLog.resetUnstable()
  247. r.msgs = nil
  248. case <-n.done:
  249. return
  250. }
  251. }
  252. }
  253. // Tick increments the internal logical clock for this Node. Election timeouts
  254. // and heartbeat timeouts are in units of ticks.
  255. func (n *node) Tick() {
  256. select {
  257. case n.tickc <- struct{}{}:
  258. case <-n.done:
  259. }
  260. }
  261. func (n *node) Campaign(ctx context.Context) error {
  262. return n.step(ctx, pb.Message{Type: pb.MsgHup})
  263. }
  264. func (n *node) Propose(ctx context.Context, data []byte) error {
  265. return n.step(ctx, pb.Message{Type: pb.MsgProp, Entries: []pb.Entry{{Data: data}}})
  266. }
  267. func (n *node) Step(ctx context.Context, m pb.Message) error {
  268. // ignore unexpected local messages receiving over network
  269. if m.Type == pb.MsgHup || m.Type == pb.MsgBeat {
  270. // TODO: return an error?
  271. return nil
  272. }
  273. return n.step(ctx, m)
  274. }
  275. func (n *node) ProposeConfChange(ctx context.Context, cc pb.ConfChange) error {
  276. data, err := cc.Marshal()
  277. if err != nil {
  278. return err
  279. }
  280. return n.Step(ctx, pb.Message{Type: pb.MsgProp, Entries: []pb.Entry{{Type: pb.EntryConfChange, Data: data}}})
  281. }
  282. // Step advances the state machine using msgs. The ctx.Err() will be returned,
  283. // if any.
  284. func (n *node) step(ctx context.Context, m pb.Message) error {
  285. ch := n.recvc
  286. if m.Type == pb.MsgProp {
  287. ch = n.propc
  288. }
  289. select {
  290. case ch <- m:
  291. return nil
  292. case <-ctx.Done():
  293. return ctx.Err()
  294. case <-n.done:
  295. return ErrStopped
  296. }
  297. }
  298. func (n *node) Ready() <-chan Ready {
  299. return n.readyc
  300. }
  301. func (n *node) ApplyConfChange(cc pb.ConfChange) {
  302. select {
  303. case n.confc <- cc:
  304. case <-n.done:
  305. }
  306. }
  307. func (n *node) Compact(index uint64, nodes []uint64, d []byte) {
  308. select {
  309. case n.compactc <- compact{index, nodes, d}:
  310. case <-n.done:
  311. }
  312. }
  313. func newReady(r *raft, prevSoftSt *SoftState, prevHardSt pb.HardState, prevSnapi uint64) Ready {
  314. rd := Ready{
  315. Entries: r.raftLog.unstableEnts(),
  316. CommittedEntries: r.raftLog.nextEnts(),
  317. Messages: r.msgs,
  318. }
  319. if softSt := r.softState(); !softSt.equal(prevSoftSt) {
  320. rd.SoftState = softSt
  321. }
  322. if !isHardStateEqual(r.HardState, prevHardSt) {
  323. rd.HardState = r.HardState
  324. }
  325. if prevSnapi != r.raftLog.snapshot.Index {
  326. rd.Snapshot = r.raftLog.snapshot
  327. }
  328. return rd
  329. }