rawnode.go 7.0 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238
  1. // Copyright 2015 The etcd Authors
  2. //
  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. //
  7. // http://www.apache.org/licenses/LICENSE-2.0
  8. //
  9. // Unless required by applicable law or agreed to in writing, software
  10. // distributed under the License is distributed on an "AS IS" BASIS,
  11. // WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
  12. // See the License for the specific language governing permissions and
  13. // limitations under the License.
  14. package raft
  15. import (
  16. "errors"
  17. pb "github.com/coreos/etcd/raft/raftpb"
  18. )
  19. // ErrStepLocalMsg is returned when try to step a local raft message
  20. var ErrStepLocalMsg = errors.New("raft: cannot step raft local message")
  21. // ErrStepPeerNotFound is returned when try to step a response message
  22. // but there is no peer found in raft.prs for that node.
  23. var ErrStepPeerNotFound = errors.New("raft: cannot step as peer not found")
  24. // RawNode is a thread-unsafe Node.
  25. // The methods of this struct correspond to the methods of Node and are described
  26. // more fully there.
  27. type RawNode struct {
  28. raft *raft
  29. prevSoftSt *SoftState
  30. prevHardSt pb.HardState
  31. }
  32. func (rn *RawNode) newReady() Ready {
  33. return newReady(rn.raft, rn.prevSoftSt, rn.prevHardSt)
  34. }
  35. func (rn *RawNode) commitReady(rd Ready) {
  36. if rd.SoftState != nil {
  37. rn.prevSoftSt = rd.SoftState
  38. }
  39. if !IsEmptyHardState(rd.HardState) {
  40. rn.prevHardSt = rd.HardState
  41. }
  42. if rn.prevHardSt.Commit != 0 {
  43. // In most cases, prevHardSt and rd.HardState will be the same
  44. // because when there are new entries to apply we just sent a
  45. // HardState with an updated Commit value. However, on initial
  46. // startup the two are different because we don't send a HardState
  47. // until something changes, but we do send any un-applied but
  48. // committed entries (and previously-committed entries may be
  49. // incorporated into the snapshot, even if rd.CommittedEntries is
  50. // empty). Therefore we mark all committed entries as applied
  51. // whether they were included in rd.HardState or not.
  52. rn.raft.raftLog.appliedTo(rn.prevHardSt.Commit)
  53. }
  54. if len(rd.Entries) > 0 {
  55. e := rd.Entries[len(rd.Entries)-1]
  56. rn.raft.raftLog.stableTo(e.Index, e.Term)
  57. }
  58. if !IsEmptySnap(rd.Snapshot) {
  59. rn.raft.raftLog.stableSnapTo(rd.Snapshot.Metadata.Index)
  60. }
  61. }
  62. // NewRawNode returns a new RawNode given configuration and a list of raft peers.
  63. func NewRawNode(config *Config, peers []Peer) (*RawNode, error) {
  64. if config.ID == 0 {
  65. panic("config.ID must not be zero")
  66. }
  67. r := newRaft(config)
  68. rn := &RawNode{
  69. raft: r,
  70. }
  71. lastIndex, err := config.Storage.LastIndex()
  72. if err != nil {
  73. panic(err) // TODO(bdarnell)
  74. }
  75. // If the log is empty, this is a new RawNode (like StartNode); otherwise it's
  76. // restoring an existing RawNode (like RestartNode).
  77. // TODO(bdarnell): rethink RawNode initialization and whether the application needs
  78. // to be able to tell us when it expects the RawNode to exist.
  79. if lastIndex == 0 {
  80. r.becomeFollower(1, None)
  81. ents := make([]pb.Entry, len(peers))
  82. for i, peer := range peers {
  83. cc := pb.ConfChange{Type: pb.ConfChangeAddNode, NodeID: peer.ID, Context: peer.Context}
  84. data, err := cc.Marshal()
  85. if err != nil {
  86. panic("unexpected marshal error")
  87. }
  88. ents[i] = pb.Entry{Type: pb.EntryConfChange, Term: 1, Index: uint64(i + 1), Data: data}
  89. }
  90. r.raftLog.append(ents...)
  91. r.raftLog.committed = uint64(len(ents))
  92. for _, peer := range peers {
  93. r.addNode(peer.ID)
  94. }
  95. }
  96. // Set the initial hard and soft states after performing all initialization.
  97. rn.prevSoftSt = r.softState()
  98. if lastIndex == 0 {
  99. rn.prevHardSt = emptyState
  100. } else {
  101. rn.prevHardSt = r.hardState()
  102. }
  103. return rn, nil
  104. }
  105. // Tick advances the internal logical clock by a single tick.
  106. func (rn *RawNode) Tick() {
  107. rn.raft.tick()
  108. }
  109. // Campaign causes this RawNode to transition to candidate state.
  110. func (rn *RawNode) Campaign() error {
  111. return rn.raft.Step(pb.Message{
  112. Type: pb.MsgHup,
  113. })
  114. }
  115. // Propose proposes data be appended to the raft log.
  116. func (rn *RawNode) Propose(data []byte) error {
  117. return rn.raft.Step(pb.Message{
  118. Type: pb.MsgProp,
  119. From: rn.raft.id,
  120. Entries: []pb.Entry{
  121. {Data: data},
  122. }})
  123. }
  124. // ProposeConfChange proposes a config change.
  125. func (rn *RawNode) ProposeConfChange(cc pb.ConfChange) error {
  126. data, err := cc.Marshal()
  127. if err != nil {
  128. return err
  129. }
  130. return rn.raft.Step(pb.Message{
  131. Type: pb.MsgProp,
  132. Entries: []pb.Entry{
  133. {Type: pb.EntryConfChange, Data: data},
  134. },
  135. })
  136. }
  137. // ApplyConfChange applies a config change to the local node.
  138. func (rn *RawNode) ApplyConfChange(cc pb.ConfChange) *pb.ConfState {
  139. if cc.NodeID == None {
  140. rn.raft.resetPendingConf()
  141. return &pb.ConfState{Nodes: rn.raft.nodes()}
  142. }
  143. switch cc.Type {
  144. case pb.ConfChangeAddNode:
  145. rn.raft.addNode(cc.NodeID)
  146. case pb.ConfChangeRemoveNode:
  147. rn.raft.removeNode(cc.NodeID)
  148. case pb.ConfChangeUpdateNode:
  149. rn.raft.resetPendingConf()
  150. default:
  151. panic("unexpected conf type")
  152. }
  153. return &pb.ConfState{Nodes: rn.raft.nodes()}
  154. }
  155. // Step advances the state machine using the given message.
  156. func (rn *RawNode) Step(m pb.Message) error {
  157. // ignore unexpected local messages receiving over network
  158. if IsLocalMsg(m.Type) {
  159. return ErrStepLocalMsg
  160. }
  161. if _, ok := rn.raft.prs[m.From]; ok || !IsResponseMsg(m.Type) {
  162. return rn.raft.Step(m)
  163. }
  164. return ErrStepPeerNotFound
  165. }
  166. // Ready returns the current point-in-time state of this RawNode.
  167. func (rn *RawNode) Ready() Ready {
  168. rd := rn.newReady()
  169. rn.raft.msgs = nil
  170. return rd
  171. }
  172. // HasReady called when RawNode user need to check if any Ready pending.
  173. // Checking logic in this method should be consistent with Ready.containsUpdates().
  174. func (rn *RawNode) HasReady() bool {
  175. r := rn.raft
  176. if !r.softState().equal(rn.prevSoftSt) {
  177. return true
  178. }
  179. if hardSt := r.hardState(); !IsEmptyHardState(hardSt) && !isHardStateEqual(hardSt, rn.prevHardSt) {
  180. return true
  181. }
  182. if r.raftLog.unstable.snapshot != nil && !IsEmptySnap(*r.raftLog.unstable.snapshot) {
  183. return true
  184. }
  185. if len(r.msgs) > 0 || len(r.raftLog.unstableEntries()) > 0 || r.raftLog.hasNextEnts() {
  186. return true
  187. }
  188. return false
  189. }
  190. // Advance notifies the RawNode that the application has applied and saved progress in the
  191. // last Ready results.
  192. func (rn *RawNode) Advance(rd Ready) {
  193. rn.commitReady(rd)
  194. }
  195. // Status returns the current status of the given group.
  196. func (rn *RawNode) Status() *Status {
  197. status := getStatus(rn.raft)
  198. return &status
  199. }
  200. // ReportUnreachable reports the given node is not reachable for the last send.
  201. func (rn *RawNode) ReportUnreachable(id uint64) {
  202. _ = rn.raft.Step(pb.Message{Type: pb.MsgUnreachable, From: id})
  203. }
  204. // ReportSnapshot reports the status of the sent snapshot.
  205. func (rn *RawNode) ReportSnapshot(id uint64, status SnapshotStatus) {
  206. rej := status == SnapshotFailure
  207. _ = rn.raft.Step(pb.Message{Type: pb.MsgSnapStatus, From: id, Reject: rej})
  208. }
  209. // TransferLeader tries to transfer leadership to the given transferee.
  210. func (rn *RawNode) TransferLeader(transferee uint64) {
  211. _ = rn.raft.Step(pb.Message{Type: pb.MsgTransferLeader, From: transferee})
  212. }