rawnode.go 7.8 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242
  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 "go.etcd.io/etcd/raft/raftpb"
  18. "go.etcd.io/etcd/raft/tracker"
  19. )
  20. // ErrStepLocalMsg is returned when try to step a local raft message
  21. var ErrStepLocalMsg = errors.New("raft: cannot step raft local message")
  22. // ErrStepPeerNotFound is returned when try to step a response message
  23. // but there is no peer found in raft.prs for that node.
  24. var ErrStepPeerNotFound = errors.New("raft: cannot step as peer not found")
  25. // RawNode is a thread-unsafe Node.
  26. // The methods of this struct correspond to the methods of Node and are described
  27. // more fully there.
  28. type RawNode struct {
  29. raft *raft
  30. prevSoftSt *SoftState
  31. prevHardSt pb.HardState
  32. }
  33. // NewRawNode instantiates a RawNode from the given configuration.
  34. //
  35. // See Bootstrap() for bootstrapping an initial state; this replaces the former
  36. // 'peers' argument to this method (with identical behavior). However, It is
  37. // recommended that instead of calling Bootstrap, applications bootstrap their
  38. // state manually by setting up a Storage that has a first index > 1 and which
  39. // stores the desired ConfState as its InitialState.
  40. func NewRawNode(config *Config) (*RawNode, error) {
  41. if config.ID == 0 {
  42. panic("config.ID must not be zero")
  43. }
  44. r := newRaft(config)
  45. rn := &RawNode{
  46. raft: r,
  47. }
  48. rn.prevSoftSt = r.softState()
  49. rn.prevHardSt = r.hardState()
  50. return rn, nil
  51. }
  52. // Tick advances the internal logical clock by a single tick.
  53. func (rn *RawNode) Tick() {
  54. rn.raft.tick()
  55. }
  56. // TickQuiesced advances the internal logical clock by a single tick without
  57. // performing any other state machine processing. It allows the caller to avoid
  58. // periodic heartbeats and elections when all of the peers in a Raft group are
  59. // known to be at the same state. Expected usage is to periodically invoke Tick
  60. // or TickQuiesced depending on whether the group is "active" or "quiesced".
  61. //
  62. // WARNING: Be very careful about using this method as it subverts the Raft
  63. // state machine. You should probably be using Tick instead.
  64. func (rn *RawNode) TickQuiesced() {
  65. rn.raft.electionElapsed++
  66. }
  67. // Campaign causes this RawNode to transition to candidate state.
  68. func (rn *RawNode) Campaign() error {
  69. return rn.raft.Step(pb.Message{
  70. Type: pb.MsgHup,
  71. })
  72. }
  73. // Propose proposes data be appended to the raft log.
  74. func (rn *RawNode) Propose(data []byte) error {
  75. return rn.raft.Step(pb.Message{
  76. Type: pb.MsgProp,
  77. From: rn.raft.id,
  78. Entries: []pb.Entry{
  79. {Data: data},
  80. }})
  81. }
  82. // ProposeConfChange proposes a config change. See (Node).ProposeConfChange for
  83. // details.
  84. func (rn *RawNode) ProposeConfChange(cc pb.ConfChangeI) error {
  85. m, err := confChangeToMsg(cc)
  86. if err != nil {
  87. return err
  88. }
  89. return rn.raft.Step(m)
  90. }
  91. // ApplyConfChange applies a config change to the local node.
  92. func (rn *RawNode) ApplyConfChange(cc pb.ConfChangeI) *pb.ConfState {
  93. cs := rn.raft.applyConfChange(cc.AsV2())
  94. return &cs
  95. }
  96. // Step advances the state machine using the given message.
  97. func (rn *RawNode) Step(m pb.Message) error {
  98. // ignore unexpected local messages receiving over network
  99. if IsLocalMsg(m.Type) {
  100. return ErrStepLocalMsg
  101. }
  102. if pr := rn.raft.prs.Progress[m.From]; pr != nil || !IsResponseMsg(m.Type) {
  103. return rn.raft.Step(m)
  104. }
  105. return ErrStepPeerNotFound
  106. }
  107. // Ready returns the outstanding work that the application needs to handle. This
  108. // includes appending and applying entries or a snapshot, updating the HardState,
  109. // and sending messages. The returned Ready() *must* be handled and subsequently
  110. // passed back via Advance().
  111. func (rn *RawNode) Ready() Ready {
  112. rd := rn.readyWithoutAccept()
  113. rn.acceptReady(rd)
  114. return rd
  115. }
  116. // readyWithoutAccept returns a Ready. This is a read-only operation, i.e. there
  117. // is no obligation that the Ready must be handled.
  118. func (rn *RawNode) readyWithoutAccept() Ready {
  119. return newReady(rn.raft, rn.prevSoftSt, rn.prevHardSt)
  120. }
  121. // acceptReady is called when the consumer of the RawNode has decided to go
  122. // ahead and handle a Ready. Nothing must alter the state of the RawNode between
  123. // this call and the prior call to Ready().
  124. func (rn *RawNode) acceptReady(rd Ready) {
  125. if rd.SoftState != nil {
  126. rn.prevSoftSt = rd.SoftState
  127. }
  128. if len(rd.ReadStates) != 0 {
  129. rn.raft.readStates = nil
  130. }
  131. rn.raft.msgs = nil
  132. }
  133. // HasReady called when RawNode user need to check if any Ready pending.
  134. // Checking logic in this method should be consistent with Ready.containsUpdates().
  135. func (rn *RawNode) HasReady() bool {
  136. r := rn.raft
  137. if !r.softState().equal(rn.prevSoftSt) {
  138. return true
  139. }
  140. if hardSt := r.hardState(); !IsEmptyHardState(hardSt) && !isHardStateEqual(hardSt, rn.prevHardSt) {
  141. return true
  142. }
  143. if r.raftLog.unstable.snapshot != nil && !IsEmptySnap(*r.raftLog.unstable.snapshot) {
  144. return true
  145. }
  146. if len(r.msgs) > 0 || len(r.raftLog.unstableEntries()) > 0 || r.raftLog.hasNextEnts() {
  147. return true
  148. }
  149. if len(r.readStates) != 0 {
  150. return true
  151. }
  152. return false
  153. }
  154. // Advance notifies the RawNode that the application has applied and saved progress in the
  155. // last Ready results.
  156. func (rn *RawNode) Advance(rd Ready) {
  157. if !IsEmptyHardState(rd.HardState) {
  158. rn.prevHardSt = rd.HardState
  159. }
  160. rn.raft.advance(rd)
  161. }
  162. // Status returns the current status of the given group. This allocates, see
  163. // BasicStatus and WithProgress for allocation-friendlier choices.
  164. func (rn *RawNode) Status() Status {
  165. status := getStatus(rn.raft)
  166. return status
  167. }
  168. // BasicStatus returns a BasicStatus. Notably this does not contain the
  169. // Progress map; see WithProgress for an allocation-free way to inspect it.
  170. func (rn *RawNode) BasicStatus() BasicStatus {
  171. return getBasicStatus(rn.raft)
  172. }
  173. // ProgressType indicates the type of replica a Progress corresponds to.
  174. type ProgressType byte
  175. const (
  176. // ProgressTypePeer accompanies a Progress for a regular peer replica.
  177. ProgressTypePeer ProgressType = iota
  178. // ProgressTypeLearner accompanies a Progress for a learner replica.
  179. ProgressTypeLearner
  180. )
  181. // WithProgress is a helper to introspect the Progress for this node and its
  182. // peers.
  183. func (rn *RawNode) WithProgress(visitor func(id uint64, typ ProgressType, pr tracker.Progress)) {
  184. rn.raft.prs.Visit(func(id uint64, pr *tracker.Progress) {
  185. typ := ProgressTypePeer
  186. if pr.IsLearner {
  187. typ = ProgressTypeLearner
  188. }
  189. p := *pr
  190. p.Inflights = nil
  191. visitor(id, typ, p)
  192. })
  193. }
  194. // ReportUnreachable reports the given node is not reachable for the last send.
  195. func (rn *RawNode) ReportUnreachable(id uint64) {
  196. _ = rn.raft.Step(pb.Message{Type: pb.MsgUnreachable, From: id})
  197. }
  198. // ReportSnapshot reports the status of the sent snapshot.
  199. func (rn *RawNode) ReportSnapshot(id uint64, status SnapshotStatus) {
  200. rej := status == SnapshotFailure
  201. _ = rn.raft.Step(pb.Message{Type: pb.MsgSnapStatus, From: id, Reject: rej})
  202. }
  203. // TransferLeader tries to transfer leadership to the given transferee.
  204. func (rn *RawNode) TransferLeader(transferee uint64) {
  205. _ = rn.raft.Step(pb.Message{Type: pb.MsgTransferLeader, From: transferee})
  206. }
  207. // ReadIndex requests a read state. The read state will be set in ready.
  208. // Read State has a read index. Once the application advances further than the read
  209. // index, any linearizable read requests issued before the read request can be
  210. // processed safely. The read state will have the same rctx attached.
  211. func (rn *RawNode) ReadIndex(rctx []byte) {
  212. _ = rn.raft.Step(pb.Message{Type: pb.MsgReadIndex, Entries: []pb.Entry{{Data: rctx}}})
  213. }