Browse Source

extract some of the consensus types into ./types

so they can be used in rpc/core/types/responses.go.

```
So, it seems like we could use the actual structs here, but we don't want to have to import consensus to get them, as then clients are importing too much crap. So probably we should move some types from consensus into consensus/types so we can import.

Will these raw messages be identical to:

type ResultDumpConsensusState struct {
  RoundState cstypes.RoundState
  PeerRoundStates map[string]cstypes.PeerRoundState
}
```
https://github.com/tendermint/tendermint/pull/724#discussion_r143598193
pull/724/head
Anton Kaliaev 7 years ago
parent
commit
dc0e8de9b0
No known key found for this signature in database GPG Key ID: 7B6881D965918214
11 changed files with 283 additions and 271 deletions
  1. +2
    -1
      consensus/common_test.go
  2. +15
    -62
      consensus/reactor.go
  3. +51
    -168
      consensus/state.go
  4. +16
    -15
      consensus/state_test.go
  5. +1
    -1
      consensus/types/height_vote_set.go
  6. +5
    -2
      consensus/types/height_vote_set_test.go
  7. +57
    -0
      consensus/types/reactor.go
  8. +126
    -0
      consensus/types/state.go
  9. +5
    -18
      rpc/core/consensus.go
  10. +2
    -1
      rpc/core/pipe.go
  11. +3
    -3
      rpc/core/types/responses.go

+ 2
- 1
consensus/common_test.go View File

@ -15,6 +15,7 @@ import (
abci "github.com/tendermint/abci/types" abci "github.com/tendermint/abci/types"
bc "github.com/tendermint/tendermint/blockchain" bc "github.com/tendermint/tendermint/blockchain"
cfg "github.com/tendermint/tendermint/config" cfg "github.com/tendermint/tendermint/config"
cstypes "github.com/tendermint/tendermint/consensus/types"
mempl "github.com/tendermint/tendermint/mempool" mempl "github.com/tendermint/tendermint/mempool"
"github.com/tendermint/tendermint/p2p" "github.com/tendermint/tendermint/p2p"
sm "github.com/tendermint/tendermint/state" sm "github.com/tendermint/tendermint/state"
@ -456,7 +457,7 @@ func (m *mockTicker) ScheduleTimeout(ti timeoutInfo) {
if m.onlyOnce && m.fired { if m.onlyOnce && m.fired {
return return
} }
if ti.Step == RoundStepNewHeight {
if ti.Step == cstypes.RoundStepNewHeight {
m.c <- ti m.c <- ti
m.fired = true m.fired = true
} }


+ 15
- 62
consensus/reactor.go View File

@ -12,6 +12,7 @@ import (
cmn "github.com/tendermint/tmlibs/common" cmn "github.com/tendermint/tmlibs/common"
"github.com/tendermint/tmlibs/log" "github.com/tendermint/tmlibs/log"
cstypes "github.com/tendermint/tendermint/consensus/types"
"github.com/tendermint/tendermint/p2p" "github.com/tendermint/tendermint/p2p"
sm "github.com/tendermint/tendermint/state" sm "github.com/tendermint/tendermint/state"
"github.com/tendermint/tendermint/types" "github.com/tendermint/tendermint/types"
@ -321,7 +322,7 @@ func (conR *ConsensusReactor) FastSync() bool {
func (conR *ConsensusReactor) registerEventCallbacks() { func (conR *ConsensusReactor) registerEventCallbacks() {
types.AddListenerForEvent(conR.evsw, "conR", types.EventStringNewRoundStep(), func(data types.TMEventData) { types.AddListenerForEvent(conR.evsw, "conR", types.EventStringNewRoundStep(), func(data types.TMEventData) {
rs := data.Unwrap().(types.EventDataRoundState).RoundState.(*RoundState)
rs := data.Unwrap().(types.EventDataRoundState).RoundState.(*cstypes.RoundState)
conR.broadcastNewRoundStep(rs) conR.broadcastNewRoundStep(rs)
}) })
@ -344,7 +345,7 @@ func (conR *ConsensusReactor) broadcastProposalHeartbeatMessage(heartbeat types.
conR.Switch.Broadcast(StateChannel, struct{ ConsensusMessage }{msg}) conR.Switch.Broadcast(StateChannel, struct{ ConsensusMessage }{msg})
} }
func (conR *ConsensusReactor) broadcastNewRoundStep(rs *RoundState) {
func (conR *ConsensusReactor) broadcastNewRoundStep(rs *cstypes.RoundState) {
nrsMsg, csMsg := makeRoundStepMessages(rs) nrsMsg, csMsg := makeRoundStepMessages(rs)
if nrsMsg != nil { if nrsMsg != nil {
@ -381,7 +382,7 @@ func (conR *ConsensusReactor) broadcastHasVoteMessage(vote *types.Vote) {
*/ */
} }
func makeRoundStepMessages(rs *RoundState) (nrsMsg *NewRoundStepMessage, csMsg *CommitStepMessage) {
func makeRoundStepMessages(rs *cstypes.RoundState) (nrsMsg *NewRoundStepMessage, csMsg *CommitStepMessage) {
nrsMsg = &NewRoundStepMessage{ nrsMsg = &NewRoundStepMessage{
Height: rs.Height, Height: rs.Height,
Round: rs.Round, Round: rs.Round,
@ -389,7 +390,7 @@ func makeRoundStepMessages(rs *RoundState) (nrsMsg *NewRoundStepMessage, csMsg *
SecondsSinceStartTime: int(time.Since(rs.StartTime).Seconds()), SecondsSinceStartTime: int(time.Since(rs.StartTime).Seconds()),
LastCommitRound: rs.LastCommit.Round(), LastCommitRound: rs.LastCommit.Round(),
} }
if rs.Step == RoundStepCommit {
if rs.Step == cstypes.RoundStepCommit {
csMsg = &CommitStepMessage{ csMsg = &CommitStepMessage{
Height: rs.Height, Height: rs.Height,
BlockPartsHeader: rs.ProposalBlockParts.Header(), BlockPartsHeader: rs.ProposalBlockParts.Header(),
@ -491,8 +492,8 @@ OUTER_LOOP:
} }
} }
func (conR *ConsensusReactor) gossipDataForCatchup(logger log.Logger, rs *RoundState,
prs *PeerRoundState, ps *PeerState, peer p2p.Peer) {
func (conR *ConsensusReactor) gossipDataForCatchup(logger log.Logger, rs *cstypes.RoundState,
prs *cstypes.PeerRoundState, ps *PeerState, peer p2p.Peer) {
if index, ok := prs.ProposalBlockParts.Not().PickRandom(); ok { if index, ok := prs.ProposalBlockParts.Not().PickRandom(); ok {
// Ensure that the peer's PartSetHeader is correct // Ensure that the peer's PartSetHeader is correct
@ -606,24 +607,24 @@ OUTER_LOOP:
} }
} }
func (conR *ConsensusReactor) gossipVotesForHeight(logger log.Logger, rs *RoundState, prs *PeerRoundState, ps *PeerState) bool {
func (conR *ConsensusReactor) gossipVotesForHeight(logger log.Logger, rs *cstypes.RoundState, prs *cstypes.PeerRoundState, ps *PeerState) bool {
// If there are lastCommits to send... // If there are lastCommits to send...
if prs.Step == RoundStepNewHeight {
if prs.Step == cstypes.RoundStepNewHeight {
if ps.PickSendVote(rs.LastCommit) { if ps.PickSendVote(rs.LastCommit) {
logger.Debug("Picked rs.LastCommit to send") logger.Debug("Picked rs.LastCommit to send")
return true return true
} }
} }
// If there are prevotes to send... // If there are prevotes to send...
if prs.Step <= RoundStepPrevote && prs.Round != -1 && prs.Round <= rs.Round {
if prs.Step <= cstypes.RoundStepPrevote && prs.Round != -1 && prs.Round <= rs.Round {
if ps.PickSendVote(rs.Votes.Prevotes(prs.Round)) { if ps.PickSendVote(rs.Votes.Prevotes(prs.Round)) {
logger.Debug("Picked rs.Prevotes(prs.Round) to send", "round", prs.Round) logger.Debug("Picked rs.Prevotes(prs.Round) to send", "round", prs.Round)
return true return true
} }
} }
// If there are precommits to send... // If there are precommits to send...
if prs.Step <= RoundStepPrecommit && prs.Round != -1 && prs.Round <= rs.Round {
if prs.Step <= cstypes.RoundStepPrecommit && prs.Round != -1 && prs.Round <= rs.Round {
if ps.PickSendVote(rs.Votes.Precommits(prs.Round)) { if ps.PickSendVote(rs.Votes.Precommits(prs.Round)) {
logger.Debug("Picked rs.Precommits(prs.Round) to send", "round", prs.Round) logger.Debug("Picked rs.Precommits(prs.Round) to send", "round", prs.Round)
return true return true
@ -752,54 +753,6 @@ func (conR *ConsensusReactor) StringIndented(indent string) string {
//----------------------------------------------------------------------------- //-----------------------------------------------------------------------------
// PeerRoundState contains the known state of a peer.
// NOTE: Read-only when returned by PeerState.GetRoundState().
type PeerRoundState struct {
Height int // Height peer is at
Round int // Round peer is at, -1 if unknown.
Step RoundStepType // Step peer is at
StartTime time.Time // Estimated start of round 0 at this height
Proposal bool // True if peer has proposal for this round
ProposalBlockPartsHeader types.PartSetHeader //
ProposalBlockParts *cmn.BitArray //
ProposalPOLRound int // Proposal's POL round. -1 if none.
ProposalPOL *cmn.BitArray // nil until ProposalPOLMessage received.
Prevotes *cmn.BitArray // All votes peer has for this round
Precommits *cmn.BitArray // All precommits peer has for this round
LastCommitRound int // Round of commit for last height. -1 if none.
LastCommit *cmn.BitArray // All commit precommits of commit for last height.
CatchupCommitRound int // Round that we have commit for. Not necessarily unique. -1 if none.
CatchupCommit *cmn.BitArray // All commit precommits peer has for this height & CatchupCommitRound
}
// String returns a string representation of the PeerRoundState
func (prs PeerRoundState) String() string {
return prs.StringIndented("")
}
// StringIndented returns a string representation of the PeerRoundState
func (prs PeerRoundState) StringIndented(indent string) string {
return fmt.Sprintf(`PeerRoundState{
%s %v/%v/%v @%v
%s Proposal %v -> %v
%s POL %v (round %v)
%s Prevotes %v
%s Precommits %v
%s LastCommit %v (round %v)
%s Catchup %v (round %v)
%s}`,
indent, prs.Height, prs.Round, prs.Step, prs.StartTime,
indent, prs.ProposalBlockPartsHeader, prs.ProposalBlockParts,
indent, prs.ProposalPOL, prs.ProposalPOLRound,
indent, prs.Prevotes,
indent, prs.Precommits,
indent, prs.LastCommit, prs.LastCommitRound,
indent, prs.CatchupCommit, prs.CatchupCommitRound,
indent)
}
//-----------------------------------------------------------------------------
var ( var (
ErrPeerStateHeightRegression = errors.New("Error peer state height regression") ErrPeerStateHeightRegression = errors.New("Error peer state height regression")
ErrPeerStateInvalidStartTime = errors.New("Error peer state invalid startTime") ErrPeerStateInvalidStartTime = errors.New("Error peer state invalid startTime")
@ -812,7 +765,7 @@ type PeerState struct {
logger log.Logger logger log.Logger
mtx sync.Mutex mtx sync.Mutex
PeerRoundState
cstypes.PeerRoundState
} }
// NewPeerState returns a new PeerState for the given Peer // NewPeerState returns a new PeerState for the given Peer
@ -820,7 +773,7 @@ func NewPeerState(peer p2p.Peer) *PeerState {
return &PeerState{ return &PeerState{
Peer: peer, Peer: peer,
logger: log.NewNopLogger(), logger: log.NewNopLogger(),
PeerRoundState: PeerRoundState{
PeerRoundState: cstypes.PeerRoundState{
Round: -1, Round: -1,
ProposalPOLRound: -1, ProposalPOLRound: -1,
LastCommitRound: -1, LastCommitRound: -1,
@ -836,7 +789,7 @@ func (ps *PeerState) SetLogger(logger log.Logger) *PeerState {
// GetRoundState returns an atomic snapshot of the PeerRoundState. // GetRoundState returns an atomic snapshot of the PeerRoundState.
// There's no point in mutating it since it won't change PeerState. // There's no point in mutating it since it won't change PeerState.
func (ps *PeerState) GetRoundState() *PeerRoundState {
func (ps *PeerState) GetRoundState() *cstypes.PeerRoundState {
ps.mtx.Lock() ps.mtx.Lock()
defer ps.mtx.Unlock() defer ps.mtx.Unlock()
@ -1227,7 +1180,7 @@ func DecodeMessage(bz []byte) (msgType byte, msg ConsensusMessage, err error) {
type NewRoundStepMessage struct { type NewRoundStepMessage struct {
Height int Height int
Round int Round int
Step RoundStepType
Step cstypes.RoundStepType
SecondsSinceStartTime int SecondsSinceStartTime int
LastCommitRound int LastCommitRound int
} }


+ 51
- 168
consensus/state.go View File

@ -17,6 +17,7 @@ import (
"github.com/tendermint/tmlibs/log" "github.com/tendermint/tmlibs/log"
cfg "github.com/tendermint/tendermint/config" cfg "github.com/tendermint/tendermint/config"
cstypes "github.com/tendermint/tendermint/consensus/types"
"github.com/tendermint/tendermint/proxy" "github.com/tendermint/tendermint/proxy"
sm "github.com/tendermint/tendermint/state" sm "github.com/tendermint/tendermint/state"
"github.com/tendermint/tendermint/types" "github.com/tendermint/tendermint/types"
@ -39,124 +40,6 @@ var (
ErrVoteHeightMismatch = errors.New("Error vote height mismatch") ErrVoteHeightMismatch = errors.New("Error vote height mismatch")
) )
//-----------------------------------------------------------------------------
// RoundStepType enum type
// RoundStepType enumerates the state of the consensus state machine
type RoundStepType uint8 // These must be numeric, ordered.
const (
RoundStepNewHeight = RoundStepType(0x01) // Wait til CommitTime + timeoutCommit
RoundStepNewRound = RoundStepType(0x02) // Setup new round and go to RoundStepPropose
RoundStepPropose = RoundStepType(0x03) // Did propose, gossip proposal
RoundStepPrevote = RoundStepType(0x04) // Did prevote, gossip prevotes
RoundStepPrevoteWait = RoundStepType(0x05) // Did receive any +2/3 prevotes, start timeout
RoundStepPrecommit = RoundStepType(0x06) // Did precommit, gossip precommits
RoundStepPrecommitWait = RoundStepType(0x07) // Did receive any +2/3 precommits, start timeout
RoundStepCommit = RoundStepType(0x08) // Entered commit state machine
// NOTE: RoundStepNewHeight acts as RoundStepCommitWait.
)
// String returns a string
func (rs RoundStepType) String() string {
switch rs {
case RoundStepNewHeight:
return "RoundStepNewHeight"
case RoundStepNewRound:
return "RoundStepNewRound"
case RoundStepPropose:
return "RoundStepPropose"
case RoundStepPrevote:
return "RoundStepPrevote"
case RoundStepPrevoteWait:
return "RoundStepPrevoteWait"
case RoundStepPrecommit:
return "RoundStepPrecommit"
case RoundStepPrecommitWait:
return "RoundStepPrecommitWait"
case RoundStepCommit:
return "RoundStepCommit"
default:
return "RoundStepUnknown" // Cannot panic.
}
}
//-----------------------------------------------------------------------------
// RoundState defines the internal consensus state.
// It is Immutable when returned from ConsensusState.GetRoundState()
// TODO: Actually, only the top pointer is copied,
// so access to field pointers is still racey
type RoundState struct {
Height int // Height we are working on
Round int
Step RoundStepType
StartTime time.Time
CommitTime time.Time // Subjective time when +2/3 precommits for Block at Round were found
Validators *types.ValidatorSet
Proposal *types.Proposal
ProposalBlock *types.Block
ProposalBlockParts *types.PartSet
LockedRound int
LockedBlock *types.Block
LockedBlockParts *types.PartSet
Votes *HeightVoteSet
CommitRound int //
LastCommit *types.VoteSet // Last precommits at Height-1
LastValidators *types.ValidatorSet
}
// RoundStateEvent returns the H/R/S of the RoundState as an event.
func (rs *RoundState) RoundStateEvent() types.EventDataRoundState {
edrs := types.EventDataRoundState{
Height: rs.Height,
Round: rs.Round,
Step: rs.Step.String(),
RoundState: rs,
}
return edrs
}
// String returns a string
func (rs *RoundState) String() string {
return rs.StringIndented("")
}
// StringIndented returns a string
func (rs *RoundState) StringIndented(indent string) string {
return fmt.Sprintf(`RoundState{
%s H:%v R:%v S:%v
%s StartTime: %v
%s CommitTime: %v
%s Validators: %v
%s Proposal: %v
%s ProposalBlock: %v %v
%s LockedRound: %v
%s LockedBlock: %v %v
%s Votes: %v
%s LastCommit: %v
%s LastValidators: %v
%s}`,
indent, rs.Height, rs.Round, rs.Step,
indent, rs.StartTime,
indent, rs.CommitTime,
indent, rs.Validators.StringIndented(indent+" "),
indent, rs.Proposal,
indent, rs.ProposalBlockParts.StringShort(), rs.ProposalBlock.StringShort(),
indent, rs.LockedRound,
indent, rs.LockedBlockParts.StringShort(), rs.LockedBlock.StringShort(),
indent, rs.Votes.StringIndented(indent+" "),
indent, rs.LastCommit.StringShort(),
indent, rs.LastValidators.StringIndented(indent+" "),
indent)
}
// StringShort returns a string
func (rs *RoundState) StringShort() string {
return fmt.Sprintf(`RoundState{H:%v R:%v S:%v ST:%v}`,
rs.Height, rs.Round, rs.Step, rs.StartTime)
}
//----------------------------------------------------------------------------- //-----------------------------------------------------------------------------
var ( var (
@ -171,10 +54,10 @@ type msgInfo struct {
// internally generated messages which may update the state // internally generated messages which may update the state
type timeoutInfo struct { type timeoutInfo struct {
Duration time.Duration `json:"duration"`
Height int `json:"height"`
Round int `json:"round"`
Step RoundStepType `json:"step"`
Duration time.Duration `json:"duration"`
Height int `json:"height"`
Round int `json:"round"`
Step cstypes.RoundStepType `json:"step"`
} }
func (ti *timeoutInfo) String() string { func (ti *timeoutInfo) String() string {
@ -199,7 +82,7 @@ type ConsensusState struct {
// internal state // internal state
mtx sync.Mutex mtx sync.Mutex
RoundState
cstypes.RoundState
state *sm.State // State until height-1. state *sm.State // State until height-1.
// state changes may be triggered by msgs from peers, // state changes may be triggered by msgs from peers,
@ -282,13 +165,13 @@ func (cs *ConsensusState) GetState() *sm.State {
} }
// GetRoundState returns a copy of the internal consensus state. // GetRoundState returns a copy of the internal consensus state.
func (cs *ConsensusState) GetRoundState() *RoundState {
func (cs *ConsensusState) GetRoundState() *cstypes.RoundState {
cs.mtx.Lock() cs.mtx.Lock()
defer cs.mtx.Unlock() defer cs.mtx.Unlock()
return cs.getRoundState() return cs.getRoundState()
} }
func (cs *ConsensusState) getRoundState() *RoundState {
func (cs *ConsensusState) getRoundState() *cstypes.RoundState {
rs := cs.RoundState // copy rs := cs.RoundState // copy
return &rs return &rs
} }
@ -469,20 +352,20 @@ func (cs *ConsensusState) updateHeight(height int) {
cs.Height = height cs.Height = height
} }
func (cs *ConsensusState) updateRoundStep(round int, step RoundStepType) {
func (cs *ConsensusState) updateRoundStep(round int, step cstypes.RoundStepType) {
cs.Round = round cs.Round = round
cs.Step = step cs.Step = step
} }
// enterNewRound(height, 0) at cs.StartTime. // enterNewRound(height, 0) at cs.StartTime.
func (cs *ConsensusState) scheduleRound0(rs *RoundState) {
func (cs *ConsensusState) scheduleRound0(rs *cstypes.RoundState) {
//cs.Logger.Info("scheduleRound0", "now", time.Now(), "startTime", cs.StartTime) //cs.Logger.Info("scheduleRound0", "now", time.Now(), "startTime", cs.StartTime)
sleepDuration := rs.StartTime.Sub(time.Now()) sleepDuration := rs.StartTime.Sub(time.Now())
cs.scheduleTimeout(sleepDuration, rs.Height, 0, RoundStepNewHeight)
cs.scheduleTimeout(sleepDuration, rs.Height, 0, cstypes.RoundStepNewHeight)
} }
// Attempt to schedule a timeout (by sending timeoutInfo on the tickChan) // Attempt to schedule a timeout (by sending timeoutInfo on the tickChan)
func (cs *ConsensusState) scheduleTimeout(duration time.Duration, height, round int, step RoundStepType) {
func (cs *ConsensusState) scheduleTimeout(duration time.Duration, height, round int, step cstypes.RoundStepType) {
cs.timeoutTicker.ScheduleTimeout(timeoutInfo{duration, height, round, step}) cs.timeoutTicker.ScheduleTimeout(timeoutInfo{duration, height, round, step})
} }
@ -524,7 +407,7 @@ func (cs *ConsensusState) reconstructLastCommit(state *sm.State) {
} }
// Updates ConsensusState and increments height to match that of state. // Updates ConsensusState and increments height to match that of state.
// The round becomes 0 and cs.Step becomes RoundStepNewHeight.
// The round becomes 0 and cs.Step becomes cstypes.RoundStepNewHeight.
func (cs *ConsensusState) updateToState(state *sm.State) { func (cs *ConsensusState) updateToState(state *sm.State) {
if cs.CommitRound > -1 && 0 < cs.Height && cs.Height != state.LastBlockHeight { if cs.CommitRound > -1 && 0 < cs.Height && cs.Height != state.LastBlockHeight {
cmn.PanicSanity(cmn.Fmt("updateToState() expected state height of %v but found %v", cmn.PanicSanity(cmn.Fmt("updateToState() expected state height of %v but found %v",
@ -560,7 +443,7 @@ func (cs *ConsensusState) updateToState(state *sm.State) {
// RoundState fields // RoundState fields
cs.updateHeight(height) cs.updateHeight(height)
cs.updateRoundStep(0, RoundStepNewHeight)
cs.updateRoundStep(0, cstypes.RoundStepNewHeight)
if cs.CommitTime.IsZero() { if cs.CommitTime.IsZero() {
// "Now" makes it easier to sync up dev nodes. // "Now" makes it easier to sync up dev nodes.
// We add timeoutCommit to allow transactions // We add timeoutCommit to allow transactions
@ -578,7 +461,7 @@ func (cs *ConsensusState) updateToState(state *sm.State) {
cs.LockedRound = 0 cs.LockedRound = 0
cs.LockedBlock = nil cs.LockedBlock = nil
cs.LockedBlockParts = nil cs.LockedBlockParts = nil
cs.Votes = NewHeightVoteSet(state.ChainID, height, validators)
cs.Votes = cstypes.NewHeightVoteSet(state.ChainID, height, validators)
cs.CommitRound = -1 cs.CommitRound = -1
cs.LastCommit = lastPrecommits cs.LastCommit = lastPrecommits
cs.LastValidators = state.LastValidators cs.LastValidators = state.LastValidators
@ -699,7 +582,7 @@ func (cs *ConsensusState) handleMsg(mi msgInfo) {
} }
} }
func (cs *ConsensusState) handleTimeout(ti timeoutInfo, rs RoundState) {
func (cs *ConsensusState) handleTimeout(ti timeoutInfo, rs cstypes.RoundState) {
cs.Logger.Debug("Received tock", "timeout", ti.Duration, "height", ti.Height, "round", ti.Round, "step", ti.Step) cs.Logger.Debug("Received tock", "timeout", ti.Duration, "height", ti.Height, "round", ti.Round, "step", ti.Step)
// timeouts must be for current height, round, step // timeouts must be for current height, round, step
@ -713,19 +596,19 @@ func (cs *ConsensusState) handleTimeout(ti timeoutInfo, rs RoundState) {
defer cs.mtx.Unlock() defer cs.mtx.Unlock()
switch ti.Step { switch ti.Step {
case RoundStepNewHeight:
case cstypes.RoundStepNewHeight:
// NewRound event fired from enterNewRound. // NewRound event fired from enterNewRound.
// XXX: should we fire timeout here (for timeout commit)? // XXX: should we fire timeout here (for timeout commit)?
cs.enterNewRound(ti.Height, 0) cs.enterNewRound(ti.Height, 0)
case RoundStepNewRound:
case cstypes.RoundStepNewRound:
cs.enterPropose(ti.Height, 0) cs.enterPropose(ti.Height, 0)
case RoundStepPropose:
case cstypes.RoundStepPropose:
types.FireEventTimeoutPropose(cs.evsw, cs.RoundStateEvent()) types.FireEventTimeoutPropose(cs.evsw, cs.RoundStateEvent())
cs.enterPrevote(ti.Height, ti.Round) cs.enterPrevote(ti.Height, ti.Round)
case RoundStepPrevoteWait:
case cstypes.RoundStepPrevoteWait:
types.FireEventTimeoutWait(cs.evsw, cs.RoundStateEvent()) types.FireEventTimeoutWait(cs.evsw, cs.RoundStateEvent())
cs.enterPrecommit(ti.Height, ti.Round) cs.enterPrecommit(ti.Height, ti.Round)
case RoundStepPrecommitWait:
case cstypes.RoundStepPrecommitWait:
types.FireEventTimeoutWait(cs.evsw, cs.RoundStateEvent()) types.FireEventTimeoutWait(cs.evsw, cs.RoundStateEvent())
cs.enterNewRound(ti.Height, ti.Round+1) cs.enterNewRound(ti.Height, ti.Round+1)
default: default:
@ -752,7 +635,7 @@ func (cs *ConsensusState) handleTxsAvailable(height int) {
// Enter: +2/3 prevotes any or +2/3 precommits for block or any from (height, round) // Enter: +2/3 prevotes any or +2/3 precommits for block or any from (height, round)
// NOTE: cs.StartTime was already set for height. // NOTE: cs.StartTime was already set for height.
func (cs *ConsensusState) enterNewRound(height int, round int) { func (cs *ConsensusState) enterNewRound(height int, round int) {
if cs.Height != height || round < cs.Round || (cs.Round == round && cs.Step != RoundStepNewHeight) {
if cs.Height != height || round < cs.Round || (cs.Round == round && cs.Step != cstypes.RoundStepNewHeight) {
cs.Logger.Debug(cmn.Fmt("enterNewRound(%v/%v): Invalid args. Current step: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step)) cs.Logger.Debug(cmn.Fmt("enterNewRound(%v/%v): Invalid args. Current step: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step))
return return
} }
@ -773,7 +656,7 @@ func (cs *ConsensusState) enterNewRound(height int, round int) {
// Setup new round // Setup new round
// we don't fire newStep for this step, // we don't fire newStep for this step,
// but we fire an event, so update the round step first // but we fire an event, so update the round step first
cs.updateRoundStep(round, RoundStepNewRound)
cs.updateRoundStep(round, cstypes.RoundStepNewRound)
cs.Validators = validators cs.Validators = validators
if round == 0 { if round == 0 {
// We've already reset these upon new height, // We've already reset these upon new height,
@ -794,7 +677,7 @@ func (cs *ConsensusState) enterNewRound(height int, round int) {
waitForTxs := cs.config.WaitForTxs() && round == 0 && !cs.needProofBlock(height) waitForTxs := cs.config.WaitForTxs() && round == 0 && !cs.needProofBlock(height)
if waitForTxs { if waitForTxs {
if cs.config.CreateEmptyBlocksInterval > 0 { if cs.config.CreateEmptyBlocksInterval > 0 {
cs.scheduleTimeout(cs.config.EmptyBlocksInterval(), height, round, RoundStepNewRound)
cs.scheduleTimeout(cs.config.EmptyBlocksInterval(), height, round, cstypes.RoundStepNewRound)
} }
go cs.proposalHeartbeat(height, round) go cs.proposalHeartbeat(height, round)
} else { } else {
@ -827,7 +710,7 @@ func (cs *ConsensusState) proposalHeartbeat(height, round int) {
for { for {
rs := cs.GetRoundState() rs := cs.GetRoundState()
// if we've already moved on, no need to send more heartbeats // if we've already moved on, no need to send more heartbeats
if rs.Step > RoundStepNewRound || rs.Round > round || rs.Height > height {
if rs.Step > cstypes.RoundStepNewRound || rs.Round > round || rs.Height > height {
return return
} }
heartbeat := &types.Heartbeat{ heartbeat := &types.Heartbeat{
@ -849,7 +732,7 @@ func (cs *ConsensusState) proposalHeartbeat(height, round int) {
// Enter (CreateEmptyBlocks, CreateEmptyBlocksInterval > 0 ): after enterNewRound(height,round), after timeout of CreateEmptyBlocksInterval // Enter (CreateEmptyBlocks, CreateEmptyBlocksInterval > 0 ): after enterNewRound(height,round), after timeout of CreateEmptyBlocksInterval
// Enter (!CreateEmptyBlocks) : after enterNewRound(height,round), once txs are in the mempool // Enter (!CreateEmptyBlocks) : after enterNewRound(height,round), once txs are in the mempool
func (cs *ConsensusState) enterPropose(height int, round int) { func (cs *ConsensusState) enterPropose(height int, round int) {
if cs.Height != height || round < cs.Round || (cs.Round == round && RoundStepPropose <= cs.Step) {
if cs.Height != height || round < cs.Round || (cs.Round == round && cstypes.RoundStepPropose <= cs.Step) {
cs.Logger.Debug(cmn.Fmt("enterPropose(%v/%v): Invalid args. Current step: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step)) cs.Logger.Debug(cmn.Fmt("enterPropose(%v/%v): Invalid args. Current step: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step))
return return
} }
@ -857,7 +740,7 @@ func (cs *ConsensusState) enterPropose(height int, round int) {
defer func() { defer func() {
// Done enterPropose: // Done enterPropose:
cs.updateRoundStep(round, RoundStepPropose)
cs.updateRoundStep(round, cstypes.RoundStepPropose)
cs.newStep() cs.newStep()
// If we have the whole proposal + POL, then goto Prevote now. // If we have the whole proposal + POL, then goto Prevote now.
@ -869,7 +752,7 @@ func (cs *ConsensusState) enterPropose(height int, round int) {
}() }()
// If we don't get the proposal and all block parts quick enough, enterPrevote // If we don't get the proposal and all block parts quick enough, enterPrevote
cs.scheduleTimeout(cs.config.Propose(round), height, round, RoundStepPropose)
cs.scheduleTimeout(cs.config.Propose(round), height, round, cstypes.RoundStepPropose)
// Nothing more to do if we're not a validator // Nothing more to do if we're not a validator
if cs.privValidator == nil { if cs.privValidator == nil {
@ -986,14 +869,14 @@ func (cs *ConsensusState) createProposalBlock() (block *types.Block, blockParts
// Prevote for LockedBlock if we're locked, or ProposalBlock if valid. // Prevote for LockedBlock if we're locked, or ProposalBlock if valid.
// Otherwise vote nil. // Otherwise vote nil.
func (cs *ConsensusState) enterPrevote(height int, round int) { func (cs *ConsensusState) enterPrevote(height int, round int) {
if cs.Height != height || round < cs.Round || (cs.Round == round && RoundStepPrevote <= cs.Step) {
if cs.Height != height || round < cs.Round || (cs.Round == round && cstypes.RoundStepPrevote <= cs.Step) {
cs.Logger.Debug(cmn.Fmt("enterPrevote(%v/%v): Invalid args. Current step: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step)) cs.Logger.Debug(cmn.Fmt("enterPrevote(%v/%v): Invalid args. Current step: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step))
return return
} }
defer func() { defer func() {
// Done enterPrevote: // Done enterPrevote:
cs.updateRoundStep(round, RoundStepPrevote)
cs.updateRoundStep(round, cstypes.RoundStepPrevote)
cs.newStep() cs.newStep()
}() }()
@ -1048,7 +931,7 @@ func (cs *ConsensusState) defaultDoPrevote(height int, round int) {
// Enter: any +2/3 prevotes at next round. // Enter: any +2/3 prevotes at next round.
func (cs *ConsensusState) enterPrevoteWait(height int, round int) { func (cs *ConsensusState) enterPrevoteWait(height int, round int) {
if cs.Height != height || round < cs.Round || (cs.Round == round && RoundStepPrevoteWait <= cs.Step) {
if cs.Height != height || round < cs.Round || (cs.Round == round && cstypes.RoundStepPrevoteWait <= cs.Step) {
cs.Logger.Debug(cmn.Fmt("enterPrevoteWait(%v/%v): Invalid args. Current step: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step)) cs.Logger.Debug(cmn.Fmt("enterPrevoteWait(%v/%v): Invalid args. Current step: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step))
return return
} }
@ -1059,12 +942,12 @@ func (cs *ConsensusState) enterPrevoteWait(height int, round int) {
defer func() { defer func() {
// Done enterPrevoteWait: // Done enterPrevoteWait:
cs.updateRoundStep(round, RoundStepPrevoteWait)
cs.updateRoundStep(round, cstypes.RoundStepPrevoteWait)
cs.newStep() cs.newStep()
}() }()
// Wait for some more prevotes; enterPrecommit // Wait for some more prevotes; enterPrecommit
cs.scheduleTimeout(cs.config.Prevote(round), height, round, RoundStepPrevoteWait)
cs.scheduleTimeout(cs.config.Prevote(round), height, round, cstypes.RoundStepPrevoteWait)
} }
// Enter: `timeoutPrevote` after any +2/3 prevotes. // Enter: `timeoutPrevote` after any +2/3 prevotes.
@ -1074,7 +957,7 @@ func (cs *ConsensusState) enterPrevoteWait(height int, round int) {
// else, unlock an existing lock and precommit nil if +2/3 of prevotes were nil, // else, unlock an existing lock and precommit nil if +2/3 of prevotes were nil,
// else, precommit nil otherwise. // else, precommit nil otherwise.
func (cs *ConsensusState) enterPrecommit(height int, round int) { func (cs *ConsensusState) enterPrecommit(height int, round int) {
if cs.Height != height || round < cs.Round || (cs.Round == round && RoundStepPrecommit <= cs.Step) {
if cs.Height != height || round < cs.Round || (cs.Round == round && cstypes.RoundStepPrecommit <= cs.Step) {
cs.Logger.Debug(cmn.Fmt("enterPrecommit(%v/%v): Invalid args. Current step: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step)) cs.Logger.Debug(cmn.Fmt("enterPrecommit(%v/%v): Invalid args. Current step: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step))
return return
} }
@ -1083,7 +966,7 @@ func (cs *ConsensusState) enterPrecommit(height int, round int) {
defer func() { defer func() {
// Done enterPrecommit: // Done enterPrecommit:
cs.updateRoundStep(round, RoundStepPrecommit)
cs.updateRoundStep(round, cstypes.RoundStepPrecommit)
cs.newStep() cs.newStep()
}() }()
@ -1167,7 +1050,7 @@ func (cs *ConsensusState) enterPrecommit(height int, round int) {
// Enter: any +2/3 precommits for next round. // Enter: any +2/3 precommits for next round.
func (cs *ConsensusState) enterPrecommitWait(height int, round int) { func (cs *ConsensusState) enterPrecommitWait(height int, round int) {
if cs.Height != height || round < cs.Round || (cs.Round == round && RoundStepPrecommitWait <= cs.Step) {
if cs.Height != height || round < cs.Round || (cs.Round == round && cstypes.RoundStepPrecommitWait <= cs.Step) {
cs.Logger.Debug(cmn.Fmt("enterPrecommitWait(%v/%v): Invalid args. Current step: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step)) cs.Logger.Debug(cmn.Fmt("enterPrecommitWait(%v/%v): Invalid args. Current step: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step))
return return
} }
@ -1178,18 +1061,18 @@ func (cs *ConsensusState) enterPrecommitWait(height int, round int) {
defer func() { defer func() {
// Done enterPrecommitWait: // Done enterPrecommitWait:
cs.updateRoundStep(round, RoundStepPrecommitWait)
cs.updateRoundStep(round, cstypes.RoundStepPrecommitWait)
cs.newStep() cs.newStep()
}() }()
// Wait for some more precommits; enterNewRound // Wait for some more precommits; enterNewRound
cs.scheduleTimeout(cs.config.Precommit(round), height, round, RoundStepPrecommitWait)
cs.scheduleTimeout(cs.config.Precommit(round), height, round, cstypes.RoundStepPrecommitWait)
} }
// Enter: +2/3 precommits for block // Enter: +2/3 precommits for block
func (cs *ConsensusState) enterCommit(height int, commitRound int) { func (cs *ConsensusState) enterCommit(height int, commitRound int) {
if cs.Height != height || RoundStepCommit <= cs.Step {
if cs.Height != height || cstypes.RoundStepCommit <= cs.Step {
cs.Logger.Debug(cmn.Fmt("enterCommit(%v/%v): Invalid args. Current step: %v/%v/%v", height, commitRound, cs.Height, cs.Round, cs.Step)) cs.Logger.Debug(cmn.Fmt("enterCommit(%v/%v): Invalid args. Current step: %v/%v/%v", height, commitRound, cs.Height, cs.Round, cs.Step))
return return
} }
@ -1198,7 +1081,7 @@ func (cs *ConsensusState) enterCommit(height int, commitRound int) {
defer func() { defer func() {
// Done enterCommit: // Done enterCommit:
// keep cs.Round the same, commitRound points to the right Precommits set. // keep cs.Round the same, commitRound points to the right Precommits set.
cs.updateRoundStep(cs.Round, RoundStepCommit)
cs.updateRoundStep(cs.Round, cstypes.RoundStepCommit)
cs.CommitRound = commitRound cs.CommitRound = commitRound
cs.CommitTime = time.Now() cs.CommitTime = time.Now()
cs.newStep() cs.newStep()
@ -1255,9 +1138,9 @@ func (cs *ConsensusState) tryFinalizeCommit(height int) {
cs.finalizeCommit(height) cs.finalizeCommit(height)
} }
// Increment height and goto RoundStepNewHeight
// Increment height and goto cstypes.RoundStepNewHeight
func (cs *ConsensusState) finalizeCommit(height int) { func (cs *ConsensusState) finalizeCommit(height int) {
if cs.Height != height || cs.Step != RoundStepCommit {
if cs.Height != height || cs.Step != cstypes.RoundStepCommit {
cs.Logger.Debug(cmn.Fmt("finalizeCommit(%v): Invalid args. Current step: %v/%v/%v", height, cs.Height, cs.Round, cs.Step)) cs.Logger.Debug(cmn.Fmt("finalizeCommit(%v): Invalid args. Current step: %v/%v/%v", height, cs.Height, cs.Round, cs.Step))
return return
} }
@ -1351,7 +1234,7 @@ func (cs *ConsensusState) finalizeCommit(height int) {
// By here, // By here,
// * cs.Height has been increment to height+1 // * cs.Height has been increment to height+1
// * cs.Step is now RoundStepNewHeight
// * cs.Step is now cstypes.RoundStepNewHeight
// * cs.StartTime is set to when we will start round0. // * cs.StartTime is set to when we will start round0.
} }
@ -1369,8 +1252,8 @@ func (cs *ConsensusState) defaultSetProposal(proposal *types.Proposal) error {
return nil return nil
} }
// We don't care about the proposal if we're already in RoundStepCommit.
if RoundStepCommit <= cs.Step {
// We don't care about the proposal if we're already in cstypes.RoundStepCommit.
if cstypes.RoundStepCommit <= cs.Step {
return nil return nil
} }
@ -1415,10 +1298,10 @@ func (cs *ConsensusState) addProposalBlockPart(height int, part *types.Part, ver
cs.state.Params().BlockSizeParams.MaxBytes, &n, &err).(*types.Block) cs.state.Params().BlockSizeParams.MaxBytes, &n, &err).(*types.Block)
// NOTE: it's possible to receive complete proposal blocks for future rounds without having the proposal // NOTE: it's possible to receive complete proposal blocks for future rounds without having the proposal
cs.Logger.Info("Received complete proposal block", "height", cs.ProposalBlock.Height, "hash", cs.ProposalBlock.Hash()) cs.Logger.Info("Received complete proposal block", "height", cs.ProposalBlock.Height, "hash", cs.ProposalBlock.Hash())
if cs.Step == RoundStepPropose && cs.isProposalComplete() {
if cs.Step == cstypes.RoundStepPropose && cs.isProposalComplete() {
// Move onto the next step // Move onto the next step
cs.enterPrevote(height, cs.Round) cs.enterPrevote(height, cs.Round)
} else if cs.Step == RoundStepCommit {
} else if cs.Step == cstypes.RoundStepCommit {
// If we're waiting on the proposal block... // If we're waiting on the proposal block...
cs.tryFinalizeCommit(height) cs.tryFinalizeCommit(height)
} }
@ -1463,7 +1346,7 @@ func (cs *ConsensusState) addVote(vote *types.Vote, peerKey string) (added bool,
// A precommit for the previous height? // A precommit for the previous height?
// These come in while we wait timeoutCommit // These come in while we wait timeoutCommit
if vote.Height+1 == cs.Height { if vote.Height+1 == cs.Height {
if !(cs.Step == RoundStepNewHeight && vote.Type == types.VoteTypePrecommit) {
if !(cs.Step == cstypes.RoundStepNewHeight && vote.Type == types.VoteTypePrecommit) {
// TODO: give the reason .. // TODO: give the reason ..
// fmt.Errorf("tryAddVote: Wrong height, not a LastCommit straggler commit.") // fmt.Errorf("tryAddVote: Wrong height, not a LastCommit straggler commit.")
return added, ErrVoteHeightMismatch return added, ErrVoteHeightMismatch
@ -1476,7 +1359,7 @@ func (cs *ConsensusState) addVote(vote *types.Vote, peerKey string) (added bool,
// if we can skip timeoutCommit and have all the votes now, // if we can skip timeoutCommit and have all the votes now,
if cs.config.SkipTimeoutCommit && cs.LastCommit.HasAll() { if cs.config.SkipTimeoutCommit && cs.LastCommit.HasAll() {
// go straight to new round (skip timeout commit) // go straight to new round (skip timeout commit)
// cs.scheduleTimeout(time.Duration(0), cs.Height, 0, RoundStepNewHeight)
// cs.scheduleTimeout(time.Duration(0), cs.Height, 0, cstypes.RoundStepNewHeight)
cs.enterNewRound(cs.Height, 0) cs.enterNewRound(cs.Height, 0)
} }
} }
@ -1540,7 +1423,7 @@ func (cs *ConsensusState) addVote(vote *types.Vote, peerKey string) (added bool,
if cs.config.SkipTimeoutCommit && precommits.HasAll() { if cs.config.SkipTimeoutCommit && precommits.HasAll() {
// if we have all the votes now, // if we have all the votes now,
// go straight to new round (skip timeout commit) // go straight to new round (skip timeout commit)
// cs.scheduleTimeout(time.Duration(0), cs.Height, 0, RoundStepNewHeight)
// cs.scheduleTimeout(time.Duration(0), cs.Height, 0, cstypes.RoundStepNewHeight)
cs.enterNewRound(cs.Height, 0) cs.enterNewRound(cs.Height, 0)
} }
@ -1601,7 +1484,7 @@ func (cs *ConsensusState) signAddVote(type_ byte, hash []byte, header types.Part
//--------------------------------------------------------- //---------------------------------------------------------
func CompareHRS(h1, r1 int, s1 RoundStepType, h2, r2 int, s2 RoundStepType) int {
func CompareHRS(h1, r1 int, s1 cstypes.RoundStepType, h2, r2 int, s2 cstypes.RoundStepType) int {
if h1 < h2 { if h1 < h2 {
return -1 return -1
} else if h1 > h2 { } else if h1 > h2 {


+ 16
- 15
consensus/state_test.go View File

@ -6,6 +6,7 @@ import (
"testing" "testing"
"time" "time"
cstypes "github.com/tendermint/tendermint/consensus/types"
"github.com/tendermint/tendermint/types" "github.com/tendermint/tendermint/types"
cmn "github.com/tendermint/tmlibs/common" cmn "github.com/tendermint/tmlibs/common"
) )
@ -247,7 +248,7 @@ func TestFullRound1(t *testing.T) {
// grab proposal // grab proposal
re := <-propCh re := <-propCh
propBlockHash := re.(types.TMEventData).Unwrap().(types.EventDataRoundState).RoundState.(*RoundState).ProposalBlock.Hash()
propBlockHash := re.(types.TMEventData).Unwrap().(types.EventDataRoundState).RoundState.(*cstypes.RoundState).ProposalBlock.Hash()
<-voteCh // wait for prevote <-voteCh // wait for prevote
// NOTE: voteChan cap of 0 ensures we can complete this // NOTE: voteChan cap of 0 ensures we can complete this
@ -344,7 +345,7 @@ func TestLockNoPOL(t *testing.T) {
cs1.startRoutines(0) cs1.startRoutines(0)
re := <-proposalCh re := <-proposalCh
rs := re.(types.TMEventData).Unwrap().(types.EventDataRoundState).RoundState.(*RoundState)
rs := re.(types.TMEventData).Unwrap().(types.EventDataRoundState).RoundState.(*cstypes.RoundState)
theBlockHash := rs.ProposalBlock.Hash() theBlockHash := rs.ProposalBlock.Hash()
<-voteCh // prevote <-voteCh // prevote
@ -384,7 +385,7 @@ func TestLockNoPOL(t *testing.T) {
// now we're on a new round and not the proposer, so wait for timeout // now we're on a new round and not the proposer, so wait for timeout
re = <-timeoutProposeCh re = <-timeoutProposeCh
rs = re.(types.TMEventData).Unwrap().(types.EventDataRoundState).RoundState.(*RoundState)
rs = re.(types.TMEventData).Unwrap().(types.EventDataRoundState).RoundState.(*cstypes.RoundState)
if rs.ProposalBlock != nil { if rs.ProposalBlock != nil {
panic("Expected proposal block to be nil") panic("Expected proposal block to be nil")
@ -428,7 +429,7 @@ func TestLockNoPOL(t *testing.T) {
incrementRound(vs2) incrementRound(vs2)
re = <-proposalCh re = <-proposalCh
rs = re.(types.TMEventData).Unwrap().(types.EventDataRoundState).RoundState.(*RoundState)
rs = re.(types.TMEventData).Unwrap().(types.EventDataRoundState).RoundState.(*cstypes.RoundState)
// now we're on a new round and are the proposer // now we're on a new round and are the proposer
if !bytes.Equal(rs.ProposalBlock.Hash(), rs.LockedBlock.Hash()) { if !bytes.Equal(rs.ProposalBlock.Hash(), rs.LockedBlock.Hash()) {
@ -515,7 +516,7 @@ func TestLockPOLRelock(t *testing.T) {
<-newRoundCh <-newRoundCh
re := <-proposalCh re := <-proposalCh
rs := re.(types.TMEventData).Unwrap().(types.EventDataRoundState).RoundState.(*RoundState)
rs := re.(types.TMEventData).Unwrap().(types.EventDataRoundState).RoundState.(*cstypes.RoundState)
theBlockHash := rs.ProposalBlock.Hash() theBlockHash := rs.ProposalBlock.Hash()
<-voteCh // prevote <-voteCh // prevote
@ -591,7 +592,7 @@ func TestLockPOLRelock(t *testing.T) {
be := <-newBlockCh be := <-newBlockCh
b := be.(types.TMEventData).Unwrap().(types.EventDataNewBlockHeader) b := be.(types.TMEventData).Unwrap().(types.EventDataNewBlockHeader)
re = <-newRoundCh re = <-newRoundCh
rs = re.(types.TMEventData).Unwrap().(types.EventDataRoundState).RoundState.(*RoundState)
rs = re.(types.TMEventData).Unwrap().(types.EventDataRoundState).RoundState.(*cstypes.RoundState)
if rs.Height != 2 { if rs.Height != 2 {
panic("Expected height to increment") panic("Expected height to increment")
} }
@ -627,7 +628,7 @@ func TestLockPOLUnlock(t *testing.T) {
startTestRound(cs1, cs1.Height, 0) startTestRound(cs1, cs1.Height, 0)
<-newRoundCh <-newRoundCh
re := <-proposalCh re := <-proposalCh
rs := re.(types.TMEventData).Unwrap().(types.EventDataRoundState).RoundState.(*RoundState)
rs := re.(types.TMEventData).Unwrap().(types.EventDataRoundState).RoundState.(*cstypes.RoundState)
theBlockHash := rs.ProposalBlock.Hash() theBlockHash := rs.ProposalBlock.Hash()
<-voteCh // prevote <-voteCh // prevote
@ -653,7 +654,7 @@ func TestLockPOLUnlock(t *testing.T) {
// timeout to new round // timeout to new round
re = <-timeoutWaitCh re = <-timeoutWaitCh
rs = re.(types.TMEventData).Unwrap().(types.EventDataRoundState).RoundState.(*RoundState)
rs = re.(types.TMEventData).Unwrap().(types.EventDataRoundState).RoundState.(*cstypes.RoundState)
lockedBlockHash := rs.LockedBlock.Hash() lockedBlockHash := rs.LockedBlock.Hash()
//XXX: this isnt guaranteed to get there before the timeoutPropose ... //XXX: this isnt guaranteed to get there before the timeoutPropose ...
@ -713,7 +714,7 @@ func TestLockPOLSafety1(t *testing.T) {
startTestRound(cs1, cs1.Height, 0) startTestRound(cs1, cs1.Height, 0)
<-newRoundCh <-newRoundCh
re := <-proposalCh re := <-proposalCh
rs := re.(types.TMEventData).Unwrap().(types.EventDataRoundState).RoundState.(*RoundState)
rs := re.(types.TMEventData).Unwrap().(types.EventDataRoundState).RoundState.(*cstypes.RoundState)
propBlock := rs.ProposalBlock propBlock := rs.ProposalBlock
<-voteCh // prevote <-voteCh // prevote
@ -761,7 +762,7 @@ func TestLockPOLSafety1(t *testing.T) {
re = <-proposalCh re = <-proposalCh
} }
rs = re.(types.TMEventData).Unwrap().(types.EventDataRoundState).RoundState.(*RoundState)
rs = re.(types.TMEventData).Unwrap().(types.EventDataRoundState).RoundState.(*cstypes.RoundState)
if rs.LockedBlock != nil { if rs.LockedBlock != nil {
panic("we should not be locked!") panic("we should not be locked!")
@ -848,7 +849,7 @@ func TestLockPOLSafety2(t *testing.T) {
incrementRound(vs2, vs3, vs4) incrementRound(vs2, vs3, vs4)
cs1.updateRoundStep(0, RoundStepPrecommitWait)
cs1.updateRoundStep(0, cstypes.RoundStepPrecommitWait)
t.Log("### ONTO Round 1") t.Log("### ONTO Round 1")
// jump in at round 1 // jump in at round 1
@ -929,7 +930,7 @@ func TestSlashingPrevotes(t *testing.T) {
re := <-proposalCh re := <-proposalCh
<-voteCh // prevote <-voteCh // prevote
rs := re.(types.EventDataRoundState).RoundState.(*RoundState)
rs := re.(types.EventDataRoundState).RoundState.(*cstypes.RoundState)
// we should now be stuck in limbo forever, waiting for more prevotes // we should now be stuck in limbo forever, waiting for more prevotes
// add one for a different block should cause us to go into prevote wait // add one for a different block should cause us to go into prevote wait
@ -1009,7 +1010,7 @@ func TestHalt1(t *testing.T) {
startTestRound(cs1, cs1.Height, 0) startTestRound(cs1, cs1.Height, 0)
<-newRoundCh <-newRoundCh
re := <-proposalCh re := <-proposalCh
rs := re.(types.TMEventData).Unwrap().(types.EventDataRoundState).RoundState.(*RoundState)
rs := re.(types.TMEventData).Unwrap().(types.EventDataRoundState).RoundState.(*cstypes.RoundState)
propBlock := rs.ProposalBlock propBlock := rs.ProposalBlock
propBlockParts := propBlock.MakePartSet(partSize) propBlockParts := propBlock.MakePartSet(partSize)
@ -1032,7 +1033,7 @@ func TestHalt1(t *testing.T) {
// timeout to new round // timeout to new round
<-timeoutWaitCh <-timeoutWaitCh
re = <-newRoundCh re = <-newRoundCh
rs = re.(types.TMEventData).Unwrap().(types.EventDataRoundState).RoundState.(*RoundState)
rs = re.(types.TMEventData).Unwrap().(types.EventDataRoundState).RoundState.(*cstypes.RoundState)
t.Log("### ONTO ROUND 1") t.Log("### ONTO ROUND 1")
/*Round2 /*Round2
@ -1050,7 +1051,7 @@ func TestHalt1(t *testing.T) {
// receiving that precommit should take us straight to commit // receiving that precommit should take us straight to commit
<-newBlockCh <-newBlockCh
re = <-newRoundCh re = <-newRoundCh
rs = re.(types.TMEventData).Unwrap().(types.EventDataRoundState).RoundState.(*RoundState)
rs = re.(types.TMEventData).Unwrap().(types.EventDataRoundState).RoundState.(*cstypes.RoundState)
if rs.Height != 2 { if rs.Height != 2 {
panic("expected height to increment") panic("expected height to increment")


consensus/height_vote_set.go → consensus/types/height_vote_set.go View File


consensus/height_vote_set_test.go → consensus/types/height_vote_set_test.go View File


+ 57
- 0
consensus/types/reactor.go View File

@ -0,0 +1,57 @@
package types
import (
"fmt"
"time"
"github.com/tendermint/tendermint/types"
cmn "github.com/tendermint/tmlibs/common"
)
//-----------------------------------------------------------------------------
// PeerRoundState contains the known state of a peer.
// NOTE: Read-only when returned by PeerState.GetRoundState().
type PeerRoundState struct {
Height int // Height peer is at
Round int // Round peer is at, -1 if unknown.
Step RoundStepType // Step peer is at
StartTime time.Time // Estimated start of round 0 at this height
Proposal bool // True if peer has proposal for this round
ProposalBlockPartsHeader types.PartSetHeader //
ProposalBlockParts *cmn.BitArray //
ProposalPOLRound int // Proposal's POL round. -1 if none.
ProposalPOL *cmn.BitArray // nil until ProposalPOLMessage received.
Prevotes *cmn.BitArray // All votes peer has for this round
Precommits *cmn.BitArray // All precommits peer has for this round
LastCommitRound int // Round of commit for last height. -1 if none.
LastCommit *cmn.BitArray // All commit precommits of commit for last height.
CatchupCommitRound int // Round that we have commit for. Not necessarily unique. -1 if none.
CatchupCommit *cmn.BitArray // All commit precommits peer has for this height & CatchupCommitRound
}
// String returns a string representation of the PeerRoundState
func (prs PeerRoundState) String() string {
return prs.StringIndented("")
}
// StringIndented returns a string representation of the PeerRoundState
func (prs PeerRoundState) StringIndented(indent string) string {
return fmt.Sprintf(`PeerRoundState{
%s %v/%v/%v @%v
%s Proposal %v -> %v
%s POL %v (round %v)
%s Prevotes %v
%s Precommits %v
%s LastCommit %v (round %v)
%s Catchup %v (round %v)
%s}`,
indent, prs.Height, prs.Round, prs.Step, prs.StartTime,
indent, prs.ProposalBlockPartsHeader, prs.ProposalBlockParts,
indent, prs.ProposalPOL, prs.ProposalPOLRound,
indent, prs.Prevotes,
indent, prs.Precommits,
indent, prs.LastCommit, prs.LastCommitRound,
indent, prs.CatchupCommit, prs.CatchupCommitRound,
indent)
}

+ 126
- 0
consensus/types/state.go View File

@ -0,0 +1,126 @@
package types
import (
"fmt"
"time"
"github.com/tendermint/tendermint/types"
)
//-----------------------------------------------------------------------------
// RoundStepType enum type
// RoundStepType enumerates the state of the consensus state machine
type RoundStepType uint8 // These must be numeric, ordered.
const (
RoundStepNewHeight = RoundStepType(0x01) // Wait til CommitTime + timeoutCommit
RoundStepNewRound = RoundStepType(0x02) // Setup new round and go to RoundStepPropose
RoundStepPropose = RoundStepType(0x03) // Did propose, gossip proposal
RoundStepPrevote = RoundStepType(0x04) // Did prevote, gossip prevotes
RoundStepPrevoteWait = RoundStepType(0x05) // Did receive any +2/3 prevotes, start timeout
RoundStepPrecommit = RoundStepType(0x06) // Did precommit, gossip precommits
RoundStepPrecommitWait = RoundStepType(0x07) // Did receive any +2/3 precommits, start timeout
RoundStepCommit = RoundStepType(0x08) // Entered commit state machine
// NOTE: RoundStepNewHeight acts as RoundStepCommitWait.
)
// String returns a string
func (rs RoundStepType) String() string {
switch rs {
case RoundStepNewHeight:
return "RoundStepNewHeight"
case RoundStepNewRound:
return "RoundStepNewRound"
case RoundStepPropose:
return "RoundStepPropose"
case RoundStepPrevote:
return "RoundStepPrevote"
case RoundStepPrevoteWait:
return "RoundStepPrevoteWait"
case RoundStepPrecommit:
return "RoundStepPrecommit"
case RoundStepPrecommitWait:
return "RoundStepPrecommitWait"
case RoundStepCommit:
return "RoundStepCommit"
default:
return "RoundStepUnknown" // Cannot panic.
}
}
//-----------------------------------------------------------------------------
// RoundState defines the internal consensus state.
// It is Immutable when returned from ConsensusState.GetRoundState()
// TODO: Actually, only the top pointer is copied,
// so access to field pointers is still racey
type RoundState struct {
Height int // Height we are working on
Round int
Step RoundStepType
StartTime time.Time
CommitTime time.Time // Subjective time when +2/3 precommits for Block at Round were found
Validators *types.ValidatorSet
Proposal *types.Proposal
ProposalBlock *types.Block
ProposalBlockParts *types.PartSet
LockedRound int
LockedBlock *types.Block
LockedBlockParts *types.PartSet
Votes *HeightVoteSet
CommitRound int //
LastCommit *types.VoteSet // Last precommits at Height-1
LastValidators *types.ValidatorSet
}
// RoundStateEvent returns the H/R/S of the RoundState as an event.
func (rs *RoundState) RoundStateEvent() types.EventDataRoundState {
edrs := types.EventDataRoundState{
Height: rs.Height,
Round: rs.Round,
Step: rs.Step.String(),
RoundState: rs,
}
return edrs
}
// String returns a string
func (rs *RoundState) String() string {
return rs.StringIndented("")
}
// StringIndented returns a string
func (rs *RoundState) StringIndented(indent string) string {
return fmt.Sprintf(`RoundState{
%s H:%v R:%v S:%v
%s StartTime: %v
%s CommitTime: %v
%s Validators: %v
%s Proposal: %v
%s ProposalBlock: %v %v
%s LockedRound: %v
%s LockedBlock: %v %v
%s Votes: %v
%s LastCommit: %v
%s LastValidators: %v
%s}`,
indent, rs.Height, rs.Round, rs.Step,
indent, rs.StartTime,
indent, rs.CommitTime,
indent, rs.Validators.StringIndented(indent+" "),
indent, rs.Proposal,
indent, rs.ProposalBlockParts.StringShort(), rs.ProposalBlock.StringShort(),
indent, rs.LockedRound,
indent, rs.LockedBlockParts.StringShort(), rs.LockedBlock.StringShort(),
indent, rs.Votes.StringIndented(indent+" "),
indent, rs.LastCommit.StringShort(),
indent, rs.LastValidators.StringIndented(indent+" "),
indent)
}
// StringShort returns a string
func (rs *RoundState) StringShort() string {
return fmt.Sprintf(`RoundState{H:%v R:%v S:%v ST:%v}`,
rs.Height, rs.Round, rs.Step, rs.StartTime)
}

+ 5
- 18
rpc/core/consensus.go View File

@ -1,13 +1,10 @@
package core package core
import ( import (
"encoding/json"
cm "github.com/tendermint/tendermint/consensus" cm "github.com/tendermint/tendermint/consensus"
cstypes "github.com/tendermint/tendermint/consensus/types"
ctypes "github.com/tendermint/tendermint/rpc/core/types" ctypes "github.com/tendermint/tendermint/rpc/core/types"
"github.com/tendermint/tendermint/types" "github.com/tendermint/tendermint/types"
"github.com/pkg/errors"
) )
// Get the validator set at the given block height. // Get the validator set at the given block height.
@ -85,22 +82,12 @@ func Validators(heightPtr *int) (*ctypes.ResultValidators, error) {
// } // }
// ``` // ```
func DumpConsensusState() (*ctypes.ResultDumpConsensusState, error) { func DumpConsensusState() (*ctypes.ResultDumpConsensusState, error) {
roundStateBytes, err := json.Marshal(consensusState.GetRoundState())
if err != nil {
return nil, errors.Wrap(err, "failed to marshal round state")
}
peerRoundStates := make(map[string]json.RawMessage)
for i, peer := range p2pSwitch.Peers().List() {
peerRoundStates := make(map[string]*cstypes.PeerRoundState)
for _, peer := range p2pSwitch.Peers().List() {
// TODO: clean this up? // TODO: clean this up?
peerState := peer.Get(types.PeerStateKey).(*cm.PeerState) peerState := peer.Get(types.PeerStateKey).(*cm.PeerState)
peerRoundState := peerState.GetRoundState() peerRoundState := peerState.GetRoundState()
peerRoundStateBytes, err := json.Marshal(peerRoundState)
if err != nil {
return nil, errors.Wrapf(err, "failed to marshal peer#%d round state", i)
}
peerRoundStates[peer.Key()] = json.RawMessage(peerRoundStateBytes)
peerRoundStates[peer.Key()] = peerRoundState
} }
return &ctypes.ResultDumpConsensusState{json.RawMessage(roundStateBytes), peerRoundStates}, nil
return &ctypes.ResultDumpConsensusState{consensusState.GetRoundState(), peerRoundStates}, nil
} }

+ 2
- 1
rpc/core/pipe.go View File

@ -3,6 +3,7 @@ package core
import ( import (
crypto "github.com/tendermint/go-crypto" crypto "github.com/tendermint/go-crypto"
"github.com/tendermint/tendermint/consensus" "github.com/tendermint/tendermint/consensus"
cstypes "github.com/tendermint/tendermint/consensus/types"
p2p "github.com/tendermint/tendermint/p2p" p2p "github.com/tendermint/tendermint/p2p"
"github.com/tendermint/tendermint/proxy" "github.com/tendermint/tendermint/proxy"
sm "github.com/tendermint/tendermint/state" sm "github.com/tendermint/tendermint/state"
@ -17,7 +18,7 @@ import (
type Consensus interface { type Consensus interface {
GetState() *sm.State GetState() *sm.State
GetValidators() (int, []*types.Validator) GetValidators() (int, []*types.Validator)
GetRoundState() *consensus.RoundState
GetRoundState() *cstypes.RoundState
} }
type P2P interface { type P2P interface {


+ 3
- 3
rpc/core/types/responses.go View File

@ -1,12 +1,12 @@
package core_types package core_types
import ( import (
"encoding/json"
"strings" "strings"
abci "github.com/tendermint/abci/types" abci "github.com/tendermint/abci/types"
crypto "github.com/tendermint/go-crypto" crypto "github.com/tendermint/go-crypto"
"github.com/tendermint/go-wire/data" "github.com/tendermint/go-wire/data"
cstypes "github.com/tendermint/tendermint/consensus/types"
"github.com/tendermint/tendermint/p2p" "github.com/tendermint/tendermint/p2p"
"github.com/tendermint/tendermint/types" "github.com/tendermint/tendermint/types"
) )
@ -76,8 +76,8 @@ type ResultValidators struct {
} }
type ResultDumpConsensusState struct { type ResultDumpConsensusState struct {
RoundState json.RawMessage `json:"round_state"`
PeerRoundStates map[string]json.RawMessage `json:"peer_round_states"`
RoundState *cstypes.RoundState `json:"round_state"`
PeerRoundStates map[string]*cstypes.PeerRoundState `json:"peer_round_states"`
} }
type ResultBroadcastTx struct { type ResultBroadcastTx struct {


Loading…
Cancel
Save