| 123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322 |
- // Copyright 2015 The etcd Authors
- //
- // Licensed under the Apache License, Version 2.0 (the "License");
- // you may not use this file except in compliance with the License.
- // You may obtain a copy of the License at
- //
- // http://www.apache.org/licenses/LICENSE-2.0
- //
- // Unless required by applicable law or agreed to in writing, software
- // distributed under the License is distributed on an "AS IS" BASIS,
- // WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
- // See the License for the specific language governing permissions and
- // limitations under the License.
- package raft
- import (
- "errors"
- pb "go.etcd.io/etcd/raft/raftpb"
- "go.etcd.io/etcd/raft/tracker"
- )
- // ErrStepLocalMsg is returned when try to step a local raft message
- var ErrStepLocalMsg = errors.New("raft: cannot step raft local message")
- // ErrStepPeerNotFound is returned when try to step a response message
- // but there is no peer found in raft.prs for that node.
- var ErrStepPeerNotFound = errors.New("raft: cannot step as peer not found")
- // RawNode is a thread-unsafe Node.
- // The methods of this struct correspond to the methods of Node and are described
- // more fully there.
- type RawNode struct {
- raft *raft
- prevSoftSt *SoftState
- prevHardSt pb.HardState
- }
- // NewRawNode returns a new RawNode given configuration and a list of raft peers.
- func NewRawNode(config *Config, peers []Peer) (*RawNode, error) {
- if config.ID == 0 {
- panic("config.ID must not be zero")
- }
- r := newRaft(config)
- rn := &RawNode{
- raft: r,
- }
- if err := rn.init(peers); err != nil {
- return nil, err
- }
- return rn, nil
- }
- func (rn *RawNode) init(peers []Peer) error {
- r := rn.raft
- lastIndex, err := rn.raft.raftLog.storage.LastIndex()
- if err != nil {
- return err
- }
- // If the log is empty, this is a new RawNode (like StartNode); otherwise it's
- // restoring an existing RawNode (like RestartNode).
- // TODO(bdarnell): rethink RawNode initialization and whether the application needs
- // to be able to tell us when it expects the RawNode to exist.
- if lastIndex == 0 {
- rn.raft.becomeFollower(1, None)
- ents := make([]pb.Entry, len(peers))
- for i, peer := range peers {
- cc := pb.ConfChange{Type: pb.ConfChangeAddNode, NodeID: peer.ID, Context: peer.Context}
- data, err := cc.Marshal()
- if err != nil {
- return err
- }
- ents[i] = pb.Entry{Type: pb.EntryConfChange, Term: 1, Index: uint64(i + 1), Data: data}
- }
- rn.raft.raftLog.append(ents...)
- // Now apply them, mainly so that the application can call Campaign
- // immediately after StartNode in tests. Note that these nodes will
- // be added to raft twice: here and when the application's Ready
- // loop calls ApplyConfChange. The calls to addNode must come after
- // all calls to raftLog.append so progress.next is set after these
- // bootstrapping entries (it is an error if we try to append these
- // entries since they have already been committed).
- // We do not set raftLog.applied so the application will be able
- // to observe all conf changes via Ready.CommittedEntries.
- //
- // TODO(bdarnell): These entries are still unstable; do we need to preserve
- // the invariant that committed < unstable?
- r.raftLog.committed = uint64(len(ents))
- for _, peer := range peers {
- r.applyConfChange(pb.ConfChange{NodeID: peer.ID, Type: pb.ConfChangeAddNode})
- }
- }
- // Set the initial hard and soft states after performing all initialization.
- rn.prevSoftSt = r.softState()
- if lastIndex == 0 {
- rn.prevHardSt = emptyState
- } else {
- rn.prevHardSt = r.hardState()
- }
- return nil
- }
- // Tick advances the internal logical clock by a single tick.
- func (rn *RawNode) Tick() {
- rn.raft.tick()
- }
- // TickQuiesced advances the internal logical clock by a single tick without
- // performing any other state machine processing. It allows the caller to avoid
- // periodic heartbeats and elections when all of the peers in a Raft group are
- // known to be at the same state. Expected usage is to periodically invoke Tick
- // or TickQuiesced depending on whether the group is "active" or "quiesced".
- //
- // WARNING: Be very careful about using this method as it subverts the Raft
- // state machine. You should probably be using Tick instead.
- func (rn *RawNode) TickQuiesced() {
- rn.raft.electionElapsed++
- }
- // Campaign causes this RawNode to transition to candidate state.
- func (rn *RawNode) Campaign() error {
- return rn.raft.Step(pb.Message{
- Type: pb.MsgHup,
- })
- }
- // Propose proposes data be appended to the raft log.
- func (rn *RawNode) Propose(data []byte) error {
- return rn.raft.Step(pb.Message{
- Type: pb.MsgProp,
- From: rn.raft.id,
- Entries: []pb.Entry{
- {Data: data},
- }})
- }
- // ProposeConfChange proposes a config change.
- func (rn *RawNode) ProposeConfChange(cc pb.ConfChange) error {
- data, err := cc.Marshal()
- if err != nil {
- return err
- }
- return rn.raft.Step(pb.Message{
- Type: pb.MsgProp,
- Entries: []pb.Entry{
- {Type: pb.EntryConfChange, Data: data},
- },
- })
- }
- // ApplyConfChange applies a config change to the local node.
- func (rn *RawNode) ApplyConfChange(cc pb.ConfChange) *pb.ConfState {
- cs := rn.raft.applyConfChange(cc)
- return &cs
- }
- // Step advances the state machine using the given message.
- func (rn *RawNode) Step(m pb.Message) error {
- // ignore unexpected local messages receiving over network
- if IsLocalMsg(m.Type) {
- return ErrStepLocalMsg
- }
- if pr := rn.raft.prs.Progress[m.From]; pr != nil || !IsResponseMsg(m.Type) {
- return rn.raft.Step(m)
- }
- return ErrStepPeerNotFound
- }
- // Ready returns the outstanding work that the application needs to handle. This
- // includes appending and applying entries or a snapshot, updating the HardState,
- // and sending messages. Ready() is a read-only operation, that is, it does not
- // require the caller to actually handle the result. Typically, a caller will
- // want to handle the Ready and must pass the Ready to Advance *after* having
- // done so. While a Ready is being handled, the RawNode must not be used for
- // operations that may alter its state. For example, it is illegal to call
- // Ready, followed by Step, followed by Advance.
- func (rn *RawNode) Ready() Ready {
- rd := rn.newReady()
- return rd
- }
- func (rn *RawNode) newReady() Ready {
- return newReady(rn.raft, rn.prevSoftSt, rn.prevHardSt)
- }
- // acceptReady is called when the consumer of the RawNode has decided to go
- // ahead and handle a Ready. Nothing must alter the state of the RawNode between
- // this call and the prior call to Ready().
- func (rn *RawNode) acceptReady(rd Ready) {
- if rd.SoftState != nil {
- rn.prevSoftSt = rd.SoftState
- }
- if len(rd.ReadStates) != 0 {
- rn.raft.readStates = nil
- }
- rn.raft.msgs = nil
- }
- // commitReady is called when the consumer of the RawNode has successfully
- // handled a Ready (having previously called acceptReady).
- func (rn *RawNode) commitReady(rd Ready) {
- if !IsEmptyHardState(rd.HardState) {
- rn.prevHardSt = rd.HardState
- }
- // If entries were applied (or a snapshot), update our cursor for
- // the next Ready. Note that if the current HardState contains a
- // new Commit index, this does not mean that we're also applying
- // all of the new entries due to commit pagination by size.
- if index := rd.appliedCursor(); index > 0 {
- rn.raft.raftLog.appliedTo(index)
- }
- rn.raft.reduceUncommittedSize(rd.CommittedEntries)
- if len(rd.Entries) > 0 {
- e := rd.Entries[len(rd.Entries)-1]
- rn.raft.raftLog.stableTo(e.Index, e.Term)
- }
- if !IsEmptySnap(rd.Snapshot) {
- rn.raft.raftLog.stableSnapTo(rd.Snapshot.Metadata.Index)
- }
- }
- // HasReady called when RawNode user need to check if any Ready pending.
- // Checking logic in this method should be consistent with Ready.containsUpdates().
- func (rn *RawNode) HasReady() bool {
- r := rn.raft
- if !r.softState().equal(rn.prevSoftSt) {
- return true
- }
- if hardSt := r.hardState(); !IsEmptyHardState(hardSt) && !isHardStateEqual(hardSt, rn.prevHardSt) {
- return true
- }
- if r.raftLog.unstable.snapshot != nil && !IsEmptySnap(*r.raftLog.unstable.snapshot) {
- return true
- }
- if len(r.msgs) > 0 || len(r.raftLog.unstableEntries()) > 0 || r.raftLog.hasNextEnts() {
- return true
- }
- if len(r.readStates) != 0 {
- return true
- }
- return false
- }
- // Advance notifies the RawNode that the application has applied and saved progress in the
- // last Ready results.
- func (rn *RawNode) Advance(rd Ready) {
- // Advance combines accept and commit. Callers can't mutate the RawNode
- // between the call to Ready and the matching call to Advance, or the work
- // done in acceptReady will clobber potentially newer data that has not been
- // emitted in a Ready yet.
- rn.acceptReady(rd)
- rn.commitReady(rd)
- }
- // Status returns the current status of the given group. This allocates, see
- // BasicStatus and WithProgress for allocation-friendlier choices.
- func (rn *RawNode) Status() Status {
- status := getStatus(rn.raft)
- return status
- }
- // BasicStatus returns a BasicStatus. Notably this does not contain the
- // Progress map; see WithProgress for an allocation-free way to inspect it.
- func (rn *RawNode) BasicStatus() BasicStatus {
- return getBasicStatus(rn.raft)
- }
- // ProgressType indicates the type of replica a Progress corresponds to.
- type ProgressType byte
- const (
- // ProgressTypePeer accompanies a Progress for a regular peer replica.
- ProgressTypePeer ProgressType = iota
- // ProgressTypeLearner accompanies a Progress for a learner replica.
- ProgressTypeLearner
- )
- // WithProgress is a helper to introspect the Progress for this node and its
- // peers.
- func (rn *RawNode) WithProgress(visitor func(id uint64, typ ProgressType, pr tracker.Progress)) {
- rn.raft.prs.Visit(func(id uint64, pr *tracker.Progress) {
- typ := ProgressTypePeer
- if pr.IsLearner {
- typ = ProgressTypeLearner
- }
- p := *pr
- p.Inflights = nil
- visitor(id, typ, p)
- })
- }
- // ReportUnreachable reports the given node is not reachable for the last send.
- func (rn *RawNode) ReportUnreachable(id uint64) {
- _ = rn.raft.Step(pb.Message{Type: pb.MsgUnreachable, From: id})
- }
- // ReportSnapshot reports the status of the sent snapshot.
- func (rn *RawNode) ReportSnapshot(id uint64, status SnapshotStatus) {
- rej := status == SnapshotFailure
- _ = rn.raft.Step(pb.Message{Type: pb.MsgSnapStatus, From: id, Reject: rej})
- }
- // TransferLeader tries to transfer leadership to the given transferee.
- func (rn *RawNode) TransferLeader(transferee uint64) {
- _ = rn.raft.Step(pb.Message{Type: pb.MsgTransferLeader, From: transferee})
- }
- // ReadIndex requests a read state. The read state will be set in ready.
- // Read State has a read index. Once the application advances further than the read
- // index, any linearizable read requests issued before the read request can be
- // processed safely. The read state will have the same rctx attached.
- func (rn *RawNode) ReadIndex(rctx []byte) {
- _ = rn.raft.Step(pb.Message{Type: pb.MsgReadIndex, Entries: []pb.Entry{{Data: rctx}}})
- }
|