Browse Source

addVote takes index

pull/142/head
Ethan Buchman 9 years ago
parent
commit
2b8157ce2a
6 changed files with 149 additions and 100 deletions
  1. +2
    -2
      consensus/height_vote_set.go
  2. +12
    -9
      consensus/reactor.go
  3. +54
    -82
      consensus/state.go
  4. +75
    -1
      consensus/test.go
  5. +0
    -1
      rpc/core_client/ws_client.go
  6. +6
    -5
      types/vote_set.go

+ 2
- 2
consensus/height_vote_set.go View File

@ -88,7 +88,7 @@ func (hvs *HeightVoteSet) addRound(round int) {
// Duplicate votes return added=false, err=nil. // Duplicate votes return added=false, err=nil.
// By convention, peerKey is "" if origin is self. // By convention, peerKey is "" if origin is self.
func (hvs *HeightVoteSet) AddByAddress(address []byte, vote *types.Vote, peerKey string) (added bool, index int, err error) {
func (hvs *HeightVoteSet) AddByIndex(valIndex int, vote *types.Vote, peerKey string) (added bool, address []byte, err error) {
hvs.mtx.Lock() hvs.mtx.Lock()
defer hvs.mtx.Unlock() defer hvs.mtx.Unlock()
voteSet := hvs.getVoteSet(vote.Round, vote.Type) voteSet := hvs.getVoteSet(vote.Round, vote.Type)
@ -104,7 +104,7 @@ func (hvs *HeightVoteSet) AddByAddress(address []byte, vote *types.Vote, peerKey
} }
return return
} }
added, index, err = voteSet.AddByAddress(address, vote)
added, address, err = voteSet.AddByIndex(valIndex, vote)
return return
} }


+ 12
- 9
consensus/reactor.go View File

@ -137,8 +137,7 @@ func (conR *ConsensusReactor) Receive(chID byte, peer *p2p.Peer, msgBytes []byte
return return
} }
// Get round state
rs := conR.conS.GetRoundState()
// Get peer states
ps := peer.Data.Get(PeerStateKey).(*PeerState) ps := peer.Data.Get(PeerStateKey).(*PeerState)
_, msg, err := DecodeMessage(msgBytes) _, msg, err := DecodeMessage(msgBytes)
if err != nil { if err != nil {
@ -146,13 +145,13 @@ func (conR *ConsensusReactor) Receive(chID byte, peer *p2p.Peer, msgBytes []byte
// TODO punish peer? // TODO punish peer?
return return
} }
log.Debug("Receive", "channel", chID, "peer", peer, "msg", msg, "rsHeight", rs.Height)
log.Debug("Receive", "channel", chId, "peer", peer, "msg", msg)
switch chID { switch chID {
case StateChannel: case StateChannel:
switch msg := msg.(type) { switch msg := msg.(type) {
case *NewRoundStepMessage: case *NewRoundStepMessage:
ps.ApplyNewRoundStepMessage(msg, rs)
ps.ApplyNewRoundStepMessage(msg)
case *CommitStepMessage: case *CommitStepMessage:
ps.ApplyCommitStepMessage(msg) ps.ApplyCommitStepMessage(msg)
case *HasVoteMessage: case *HasVoteMessage:
@ -189,15 +188,19 @@ func (conR *ConsensusReactor) Receive(chID byte, peer *p2p.Peer, msgBytes []byte
vote, valIndex := msg.Vote, msg.ValidatorIndex vote, valIndex := msg.Vote, msg.ValidatorIndex
// attempt to add the vote and dupeout the validator if its a duplicate signature // attempt to add the vote and dupeout the validator if its a duplicate signature
added, err := conR.conS.TryAddVote(rs, vote, valIndex, peer.Key)
added, err := conR.conS.TryAddVote(valIndex, vote, peer.Key)
if err == ErrAddingVote { if err == ErrAddingVote {
// TODO: punish peer // TODO: punish peer
} else if err != nil { } else if err != nil {
return return
} }
ps.EnsureVoteBitArrays(rs.Height, rs.Validators.Size())
ps.EnsureVoteBitArrays(rs.Height-1, rs.LastCommit.Size())
cs := conR.conS
cs.mtx.Lock()
height, valSize, lastCommitSize := cs.Height, cs.Validators.Size(), cs.LastCommit.Size()
cs.mtx.Unlock()
ps.EnsureVoteBitArrays(height, valSize)
ps.EnsureVoteBitArrays(height-1, lastCommitSize)
ps.SetHasVote(vote, valIndex) ps.SetHasVote(vote, valIndex)
if added { if added {
@ -208,7 +211,7 @@ func (conR *ConsensusReactor) Receive(chID byte, peer *p2p.Peer, msgBytes []byte
} }
default: default:
// TODO: should these be punishable?
// don't punish (leave room for soft upgrades)
log.Warn(Fmt("Unknown message type %v", reflect.TypeOf(msg))) log.Warn(Fmt("Unknown message type %v", reflect.TypeOf(msg)))
} }
default: default:
@ -793,7 +796,7 @@ func (ps *PeerState) setHasVote(height int, round int, type_ byte, index int) {
} }
} }
func (ps *PeerState) ApplyNewRoundStepMessage(msg *NewRoundStepMessage, rs *RoundState) {
func (ps *PeerState) ApplyNewRoundStepMessage(msg *NewRoundStepMessage) {
ps.mtx.Lock() ps.mtx.Lock()
defer ps.mtx.Unlock() defer ps.mtx.Unlock()


+ 54
- 82
consensus/state.go View File

@ -180,6 +180,7 @@ var (
ErrInvalidProposalSignature = errors.New("Error invalid proposal signature") ErrInvalidProposalSignature = errors.New("Error invalid proposal signature")
ErrInvalidProposalPOLRound = errors.New("Error invalid proposal POL round") ErrInvalidProposalPOLRound = errors.New("Error invalid proposal POL round")
ErrAddingVote = errors.New("Error adding vote") ErrAddingVote = errors.New("Error adding vote")
ErrVoteHeightMismatch = errors.New("Error vote height mismatch")
) )
//----------------------------------------------------------------------------- //-----------------------------------------------------------------------------
@ -300,7 +301,7 @@ type ConsensusState struct {
evsw events.Fireable evsw events.Fireable
evc *events.EventCache // set in stageBlock and passed into state evc *events.EventCache // set in stageBlock and passed into state
timeoutChan chan TimeoutEvent // RoundState instead?
timeoutChan chan TimeoutEvent // so we can track timeouts
timeoutQuitChan chan struct{} timeoutQuitChan chan struct{}
} }
@ -499,7 +500,7 @@ func (cs *ConsensusState) EnterNewRound(height int, round int) {
cs.mtx.Lock() cs.mtx.Lock()
defer cs.mtx.Unlock() defer cs.mtx.Unlock()
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 != RoundStepNewHeight) {
log.Info(Fmt("EnterNewRound(%v/%v): Invalid args. Current step: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step))
log.Debug(Fmt("EnterNewRound(%v/%v): Invalid args. Current step: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step))
return return
} }
if now := time.Now(); cs.StartTime.After(now) { if now := time.Now(); cs.StartTime.After(now) {
@ -538,39 +539,29 @@ func (cs *ConsensusState) EnterPropose(height int, round int) {
cs.mtx.Lock() cs.mtx.Lock()
defer cs.mtx.Unlock() defer cs.mtx.Unlock()
if cs.Height != height || round < cs.Round || (cs.Round == round && RoundStepPropose <= cs.Step) { if cs.Height != height || round < cs.Round || (cs.Round == round && RoundStepPropose <= cs.Step) {
log.Info(Fmt("EnterPropose(%v/%v): Invalid args. Current step: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step))
log.Debug(Fmt("EnterPropose(%v/%v): Invalid args. Current step: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step))
return return
} }
log.Info(Fmt("EnterPropose(%v/%v). Current: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step)) log.Info(Fmt("EnterPropose(%v/%v). Current: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step))
var enterPrevote = make(chan struct{}, 1)
defer func() { defer func() {
// Done EnterPropose: // Done EnterPropose:
cs.Round = round cs.Round = round
cs.Step = RoundStepPropose cs.Step = RoundStepPropose
enterPrevote <- struct{}{}
cs.newStepCh <- cs.getRoundState()
// If we have the whole proposal + POL, then goto Prevote now.
// else, we'll EnterPrevote when the rest of the proposal is received (in AddProposalBlockPart),
// or else after timeoutPropose
if cs.isProposalComplete() {
go cs.EnterPrevote(height, cs.Round)
}
}() }()
// EnterPrevote after timeoutPropose or if the proposal is complete
// This step times out after `timeoutPropose`
go func() { go func() {
ticker := time.NewTicker(timeoutPropose)
LOOP:
for {
select {
case <-ticker.C:
enterPrevote <- struct{}{}
cs.timeoutChan <- TimeoutEvent{RoundStepPropose, height, round}
break LOOP
case <-enterPrevote:
// If we already have the proposal + POL, then goto Prevote
if cs.isProposalComplete() {
break LOOP
}
}
}
cs.newStepCh <- cs.getRoundState()
go cs.EnterPrevote(height, round)
time.Sleep(timeoutPropose)
cs.EnterPrevote(height, round)
}() }()
// Nothing more to do if we're not a validator // Nothing more to do if we're not a validator
@ -582,7 +573,7 @@ func (cs *ConsensusState) EnterPropose(height int, round int) {
log.Info("EnterPropose: Not our turn to propose", "proposer", cs.Validators.Proposer().Address, "privValidator", cs.privValidator) log.Info("EnterPropose: Not our turn to propose", "proposer", cs.Validators.Proposer().Address, "privValidator", cs.privValidator)
} else { } else {
log.Info("EnterPropose: Our turn to propose", "proposer", cs.Validators.Proposer().Address, "privValidator", cs.privValidator) log.Info("EnterPropose: Our turn to propose", "proposer", cs.Validators.Proposer().Address, "privValidator", cs.privValidator)
cs.decideProposal(height, round) // if this takes longer than timeoutPropose we'll catch it in a later EnterPropose
cs.decideProposal(height, round)
} }
} }
@ -633,8 +624,7 @@ func (cs *ConsensusState) isProposalComplete() bool {
} }
// Create the next block to propose and return it. // Create the next block to propose and return it.
// NOTE: make it side-effect free for clarity.
// XXX: where are the side-effects?
// NOTE: keep it side-effect free for clarity.
func (cs *ConsensusState) createProposalBlock() (block *types.Block, blockParts *types.PartSet) { func (cs *ConsensusState) createProposalBlock() (block *types.Block, blockParts *types.PartSet) {
var validation *types.Validation var validation *types.Validation
if cs.Height == 1 { if cs.Height == 1 {
@ -688,7 +678,7 @@ func (cs *ConsensusState) EnterPrevote(height int, round int) {
cs.mtx.Lock() cs.mtx.Lock()
defer cs.mtx.Unlock() defer cs.mtx.Unlock()
if cs.Height != height || round < cs.Round || (cs.Round == round && RoundStepPrevote <= cs.Step) { if cs.Height != height || round < cs.Round || (cs.Round == round && RoundStepPrevote <= cs.Step) {
log.Info(Fmt("EnterPrevote(%v/%v): Invalid args. Current step: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step))
log.Debug(Fmt("EnterPrevote(%v/%v): Invalid args. Current step: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step))
return return
} }
log.Info(Fmt("EnterPrevote(%v/%v). Current: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step)) log.Info(Fmt("EnterPrevote(%v/%v). Current: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step))
@ -742,7 +732,7 @@ func (cs *ConsensusState) EnterPrevoteWait(height int, round int) {
cs.mtx.Lock() cs.mtx.Lock()
defer cs.mtx.Unlock() defer cs.mtx.Unlock()
if cs.Height != height || round < cs.Round || (cs.Round == round && RoundStepPrevoteWait <= cs.Step) { if cs.Height != height || round < cs.Round || (cs.Round == round && RoundStepPrevoteWait <= cs.Step) {
log.Info(Fmt("EnterPrevoteWait(%v/%v): Invalid args. Current step: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step))
log.Debug(Fmt("EnterPrevoteWait(%v/%v): Invalid args. Current step: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step))
return return
} }
if !cs.Votes.Prevotes(round).HasTwoThirdsAny() { if !cs.Votes.Prevotes(round).HasTwoThirdsAny() {
@ -766,15 +756,14 @@ func (cs *ConsensusState) EnterPrevoteWait(height int, round int) {
// Enter: +2/3 precomits for block or nil. // Enter: +2/3 precomits for block or nil.
// Enter: `timeoutPrevote` after any +2/3 prevotes. // Enter: `timeoutPrevote` after any +2/3 prevotes.
// Enter: any +2/3 precommits for next round. // Enter: any +2/3 precommits for next round.
// Lock & precommit the ProposalBlock if we have enough prevotes for it,
// Lock & precommit the ProposalBlock if we have enough prevotes for it (a POL in this round)
// 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.
// NOTE: we don't precommit our locked block (unless theres another POL for it) because it complicates unlocking and accountability
func (cs *ConsensusState) EnterPrecommit(height int, round int) { func (cs *ConsensusState) EnterPrecommit(height int, round int) {
cs.mtx.Lock() cs.mtx.Lock()
defer cs.mtx.Unlock() defer cs.mtx.Unlock()
if cs.Height != height || round < cs.Round || (cs.Round == round && RoundStepPrecommit <= cs.Step) { if cs.Height != height || round < cs.Round || (cs.Round == round && RoundStepPrecommit <= cs.Step) {
log.Info(Fmt("EnterPrecommit(%v/%v): Invalid args. Current step: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step))
log.Debug(Fmt("EnterPrecommit(%v/%v): Invalid args. Current step: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step))
return return
} }
log.Info(Fmt("EnterPrecommit(%v/%v). Current: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step)) log.Info(Fmt("EnterPrecommit(%v/%v). Current: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step))
@ -792,9 +781,7 @@ func (cs *ConsensusState) EnterPrecommit(height int, round int) {
hash, partsHeader, ok := cs.Votes.Prevotes(round).TwoThirdsMajority() hash, partsHeader, ok := cs.Votes.Prevotes(round).TwoThirdsMajority()
// If we don't have two thirds of prevotes, just precommit nil
// NOTE: alternatively, if we have seen a POL since our last precommit,
// we could precommit that
// If we don't have two thirds of prevotes, we must precommit nil
if !ok { if !ok {
if cs.LockedBlock != nil { if cs.LockedBlock != nil {
log.Info("EnterPrecommit: No +2/3 prevotes during EnterPrecommit while we're locked. Precommitting nil") log.Info("EnterPrecommit: No +2/3 prevotes during EnterPrecommit while we're locked. Precommitting nil")
@ -814,7 +801,7 @@ func (cs *ConsensusState) EnterPrecommit(height int, round int) {
log.Info("EnterPrecommit: +2/3 prevoted for nil.") log.Info("EnterPrecommit: +2/3 prevoted for nil.")
} else { } else {
log.Info("EnterPrecommit: +2/3 prevoted for nil. Unlocking") log.Info("EnterPrecommit: +2/3 prevoted for nil. Unlocking")
cs.LockedRound = 0 //XXX: should be this round
cs.LockedRound = 0
cs.LockedBlock = nil cs.LockedBlock = nil
cs.LockedBlockParts = nil cs.LockedBlockParts = nil
} }
@ -834,7 +821,7 @@ func (cs *ConsensusState) EnterPrecommit(height int, round int) {
// If +2/3 prevoted for proposal block, stage and precommit it // If +2/3 prevoted for proposal block, stage and precommit it
if cs.ProposalBlock.HashesTo(hash) { if cs.ProposalBlock.HashesTo(hash) {
log.Info("EnterPrecommit: +2/3 prevoted proposal block.", "hash", fmt.Sprintf("%X", hash))
log.Info("EnterPrecommit: +2/3 prevoted proposal block.", "hash", hash)
// Validate the block. // Validate the block.
if err := cs.stageBlock(cs.ProposalBlock, cs.ProposalBlockParts); err != nil { if err := cs.stageBlock(cs.ProposalBlock, cs.ProposalBlockParts); err != nil {
PanicConsensus(Fmt("EnterPrecommit: +2/3 prevoted for an invalid block: %v", err)) PanicConsensus(Fmt("EnterPrecommit: +2/3 prevoted for an invalid block: %v", err))
@ -872,7 +859,7 @@ func (cs *ConsensusState) EnterPrecommitWait(height int, round int) {
cs.mtx.Lock() cs.mtx.Lock()
defer cs.mtx.Unlock() defer cs.mtx.Unlock()
if cs.Height != height || round < cs.Round || (cs.Round == round && RoundStepPrecommitWait <= cs.Step) { if cs.Height != height || round < cs.Round || (cs.Round == round && RoundStepPrecommitWait <= cs.Step) {
log.Info(Fmt("EnterPrecommitWait(%v/%v): Invalid args. Current step: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step))
log.Debug(Fmt("EnterPrecommitWait(%v/%v): Invalid args. Current step: %v/%v/%v", height, round, cs.Height, cs.Round, cs.Step))
return return
} }
if !cs.Votes.Precommits(round).HasTwoThirdsAny() { if !cs.Votes.Precommits(round).HasTwoThirdsAny() {
@ -902,7 +889,7 @@ func (cs *ConsensusState) EnterCommit(height int, commitRound int) {
cs.mtx.Lock() cs.mtx.Lock()
defer cs.mtx.Unlock() defer cs.mtx.Unlock()
if cs.Height != height || RoundStepCommit <= cs.Step { if cs.Height != height || RoundStepCommit <= cs.Step {
log.Info(Fmt("EnterCommit(%v/%v): Invalid args. Current step: %v/%v/%v", height, commitRound, cs.Height, cs.Round, cs.Step))
log.Debug(Fmt("EnterCommit(%v/%v): Invalid args. Current step: %v/%v/%v", height, commitRound, cs.Height, cs.Round, cs.Step))
return return
} }
log.Info(Fmt("EnterCommit(%v/%v). Current: %v/%v/%v", height, commitRound, cs.Height, cs.Round, cs.Step)) log.Info(Fmt("EnterCommit(%v/%v). Current: %v/%v/%v", height, commitRound, cs.Height, cs.Round, cs.Step))
@ -924,20 +911,10 @@ func (cs *ConsensusState) EnterCommit(height int, commitRound int) {
// The Locked* fields no longer matter. // The Locked* fields no longer matter.
// Move them over to ProposalBlock if they match the commit hash, // Move them over to ProposalBlock if they match the commit hash,
// otherwise they can now be cleared.
// XXX: can't we just wait to clear them in updateToState ?
// XXX: it's causing a race condition in tests where they get cleared
// before we can check the lock!
// otherwise they'll be cleared in updateToState.
if cs.LockedBlock.HashesTo(hash) { if cs.LockedBlock.HashesTo(hash) {
cs.ProposalBlock = cs.LockedBlock cs.ProposalBlock = cs.LockedBlock
cs.ProposalBlockParts = cs.LockedBlockParts cs.ProposalBlockParts = cs.LockedBlockParts
/*cs.LockedRound = 0
cs.LockedBlock = nil
cs.LockedBlockParts = nil*/
} else {
/*cs.LockedRound = 0
cs.LockedBlock = nil
cs.LockedBlockParts = nil*/
} }
// If we don't have the block being committed, set up to get it. // If we don't have the block being committed, set up to get it.
@ -977,7 +954,7 @@ func (cs *ConsensusState) FinalizeCommit(height, round int) {
defer cs.mtx.Unlock() defer cs.mtx.Unlock()
if cs.Height != height || cs.Step != RoundStepCommit { if cs.Height != height || cs.Step != RoundStepCommit {
log.Info(Fmt("FinalizeCommit(%v): Invalid args. Current step: %v/%v/%v", height, cs.Height, cs.Round, cs.Step))
log.Debug(Fmt("FinalizeCommit(%v): Invalid args. Current step: %v/%v/%v", height, cs.Height, cs.Round, cs.Step))
return return
} }
@ -1078,7 +1055,6 @@ func (cs *ConsensusState) AddProposalBlockPart(height int, part *types.Part) (ad
log.Info("Received complete proposal", "hash", cs.ProposalBlock.Hash()) log.Info("Received complete proposal", "hash", cs.ProposalBlock.Hash())
if cs.Step == RoundStepPropose && cs.isProposalComplete() { if cs.Step == RoundStepPropose && cs.isProposalComplete() {
// Move onto the next step // Move onto the next step
// XXX: isn't this unecessary since propose will either do this or timeout into it
go cs.EnterPrevote(height, cs.Round) go cs.EnterPrevote(height, cs.Round)
} else if cs.Step == RoundStepCommit { } else if cs.Step == RoundStepCommit {
// If we're waiting on the proposal block... // If we're waiting on the proposal block...
@ -1089,35 +1065,23 @@ func (cs *ConsensusState) AddProposalBlockPart(height int, part *types.Part) (ad
return added, nil return added, nil
} }
func (cs *ConsensusState) AddVote(address []byte, vote *types.Vote, peerKey string) (added bool, index int, err error) {
func (cs *ConsensusState) AddVote(valIndex int, vote *types.Vote, peerKey string) (added bool, address []byte, err error) {
cs.mtx.Lock() cs.mtx.Lock()
defer cs.mtx.Unlock() defer cs.mtx.Unlock()
return cs.addVote(address, vote, peerKey)
return cs.addVote(valIndex, vote, peerKey)
} }
// Attempt to add the vote. if its a duplicate signature, dupeout the validator // Attempt to add the vote. if its a duplicate signature, dupeout the validator
func (cs *ConsensusState) TryAddVote(rs *RoundState, vote *types.Vote, valIndex int, peerKey string) (bool, error) {
var validators *types.ValidatorSet
if rs.Height == vote.Height {
validators = rs.Validators
} else if rs.Height == vote.Height+1 {
if !(rs.Step == RoundStepNewHeight && vote.Type == types.VoteTypePrecommit) {
return false, fmt.Errorf("TryAddVote: Wrong height, not a LastCommit straggler commit.")
}
validators = rs.LastValidators
} else {
return false, fmt.Errorf("TryAddVote: Wrong height. Not necessarily a bad peer.")
}
// We have vote/validators. Height may not be rs.Height
address, _ := validators.GetByIndex(valIndex)
added, index, err := cs.AddVote(address, vote, peerKey)
_ = index // should be same as valIndex
func (cs *ConsensusState) TryAddVote(valIndex int, vote *types.Vote, peerKey string) (bool, error) {
added, address, err := cs.AddVote(valIndex, vote, peerKey)
if err != nil { if err != nil {
// If conflicting sig, broadcast evidence tx for slashing. Else punish peer.
if errDupe, ok := err.(*types.ErrVoteConflictingSignature); ok {
// If the vote height is off, we'll just ignore it,
// But if it's a conflicting sig, broadcast evidence tx for slashing
// and otherwise punish peer.
if err == ErrVoteHeightMismatch {
return added, err
} else if errDupe, ok := err.(*types.ErrVoteConflictingSignature); ok {
log.Warn("Found conflicting vote. Publish evidence") log.Warn("Found conflicting vote. Publish evidence")
evidenceTx := &types.DupeoutTx{ evidenceTx := &types.DupeoutTx{
Address: address, Address: address,
@ -1137,12 +1101,17 @@ func (cs *ConsensusState) TryAddVote(rs *RoundState, vote *types.Vote, valIndex
//----------------------------------------------------------------------------- //-----------------------------------------------------------------------------
func (cs *ConsensusState) addVote(address []byte, vote *types.Vote, peerKey string) (added bool, index int, err error) {
func (cs *ConsensusState) addVote(valIndex int, vote *types.Vote, peerKey string) (added bool, address []byte, err error) {
log.Debug("addVote", "voteHeight", vote.Height, "voteType", vote.Type, "csHeight", cs.Height) log.Debug("addVote", "voteHeight", vote.Height, "voteType", vote.Type, "csHeight", cs.Height)
// A precommit for the previous height? // A precommit for the previous height?
if vote.Height+1 == cs.Height && vote.Type == types.VoteTypePrecommit {
added, index, err = cs.LastCommit.AddByAddress(address, vote)
if vote.Height+1 == cs.Height {
if !(cs.Step == RoundStepNewHeight && vote.Type == types.VoteTypePrecommit) {
// TODO: give the reason ..
// fmt.Errorf("TryAddVote: Wrong height, not a LastCommit straggler commit.")
return added, nil, ErrVoteHeightMismatch
}
added, address, err = cs.LastCommit.AddByIndex(valIndex, vote)
if added { if added {
log.Info(Fmt("Added to lastPrecommits: %v", cs.LastCommit.StringShort())) log.Info(Fmt("Added to lastPrecommits: %v", cs.LastCommit.StringShort()))
} }
@ -1152,7 +1121,7 @@ func (cs *ConsensusState) addVote(address []byte, vote *types.Vote, peerKey stri
// A prevote/precommit for this height? // A prevote/precommit for this height?
if vote.Height == cs.Height { if vote.Height == cs.Height {
height := cs.Height height := cs.Height
added, index, err = cs.Votes.AddByAddress(address, vote, peerKey)
added, address, err = cs.Votes.AddByIndex(valIndex, vote, peerKey)
if added { if added {
switch vote.Type { switch vote.Type {
case types.VoteTypePrevote: case types.VoteTypePrevote:
@ -1167,7 +1136,7 @@ func (cs *ConsensusState) addVote(address []byte, vote *types.Vote, peerKey stri
hash, _, ok := prevotes.TwoThirdsMajority() hash, _, ok := prevotes.TwoThirdsMajority()
if ok && !cs.LockedBlock.HashesTo(hash) { if ok && !cs.LockedBlock.HashesTo(hash) {
log.Notice("Unlocking because of POL.", "lockedRound", cs.LockedRound, "POLRound", vote.Round) log.Notice("Unlocking because of POL.", "lockedRound", cs.LockedRound, "POLRound", vote.Round)
cs.LockedRound = 0 // XXX: shouldn't we set this to the current round?
cs.LockedRound = 0
cs.LockedBlock = nil cs.LockedBlock = nil
cs.LockedBlockParts = nil cs.LockedBlockParts = nil
} }
@ -1214,8 +1183,10 @@ func (cs *ConsensusState) addVote(address []byte, vote *types.Vote, peerKey stri
PanicSanity(Fmt("Unexpected vote type %X", vote.Type)) // Should not happen. PanicSanity(Fmt("Unexpected vote type %X", vote.Type)) // Should not happen.
} }
} }
// Either duplicate, or error upon cs.Votes.AddByAddress()
// Either duplicate, or error upon cs.Votes.AddByIndex()
return return
} else {
err = ErrVoteHeightMismatch
} }
// Height mismatch, bad peer? // Height mismatch, bad peer?
@ -1270,7 +1241,9 @@ func (cs *ConsensusState) signAddVote(type_ byte, hash []byte, header types.Part
} }
vote, err := cs.signVote(type_, hash, header) vote, err := cs.signVote(type_, hash, header)
if err == nil { if err == nil {
_, _, err := cs.addVote(cs.privValidator.Address, vote, "")
// NOTE: store our index in the cs so we don't have to do this every time
valIndex, _ := cs.Validators.GetByAddress(cs.privValidator.Address)
_, _, err := cs.addVote(valIndex, vote, "")
log.Notice("Signed and added vote", "height", cs.Height, "round", cs.Round, "vote", vote, "error", err) log.Notice("Signed and added vote", "height", cs.Height, "round", cs.Round, "vote", vote, "error", err)
return vote return vote
} else { } else {
@ -1333,7 +1306,6 @@ func (cs *ConsensusState) logTimeouts(timeoutChan chan TimeoutEvent, quitChan <-
select { select {
case timeout := <-timeoutChan: case timeout := <-timeoutChan:
log.Info("Timeout in consensus state", "height", timeout.Height, "round", timeout.Round, "step", timeout.Type.String()) log.Info("Timeout in consensus state", "height", timeout.Height, "round", timeout.Round, "step", timeout.Type.String())
timeoutChan <- timeout
case <-quitChan: case <-quitChan:
return return


+ 75
- 1
consensus/test.go View File

@ -17,9 +17,47 @@ import (
//------------------------------------------------------------------------------- //-------------------------------------------------------------------------------
// utils // utils
func changeProposer(t *testing.T, perspectiveOf, newProposer *ConsensusState) *types.Block {
_, v1 := perspectiveOf.Validators.GetByAddress(perspectiveOf.privValidator.Address)
v1.Accum, v1.VotingPower = 0, 0
if updated := perspectiveOf.Validators.Update(v1); !updated {
t.Fatal("failed to update validator")
}
_, v2 := perspectiveOf.Validators.GetByAddress(newProposer.privValidator.Address)
v2.Accum, v2.VotingPower = 100, 100
if updated := perspectiveOf.Validators.Update(v2); !updated {
t.Fatal("failed to update validator")
}
// make the proposal
propBlock, _ := newProposer.createProposalBlock()
if propBlock == nil {
t.Fatal("Failed to create proposal block with cs2")
}
return propBlock
}
func fixVotingPower(t *testing.T, cs1 *ConsensusState, addr2 []byte) {
_, v1 := cs1.Validators.GetByAddress(cs1.privValidator.Address)
_, v2 := cs1.Validators.GetByAddress(addr2)
v1.Accum, v1.VotingPower = v2.Accum, v2.VotingPower
if updated := cs1.Validators.Update(v1); !updated {
t.Fatal("failed to update validator")
}
}
func addVoteToFromMany(to *ConsensusState, votes []*types.Vote, froms ...*ConsensusState) {
if len(votes) != len(froms) {
panic("len(votes) and len(froms) must match")
}
for i, from := range froms {
addVoteToFrom(to, from, votes[i])
}
}
func addVoteToFrom(to, from *ConsensusState, vote *types.Vote) { func addVoteToFrom(to, from *ConsensusState, vote *types.Vote) {
valIndex, _ := to.Validators.GetByAddress(from.privValidator.Address) valIndex, _ := to.Validators.GetByAddress(from.privValidator.Address)
added, err := to.TryAddVote(to.GetRoundState(), vote, valIndex, "")
added, err := to.TryAddVote(valIndex, vote, "")
if _, ok := err.(*types.ErrVoteConflictingSignature); ok { if _, ok := err.(*types.ErrVoteConflictingSignature); ok {
// let it fly // let it fly
} else if !added { } else if !added {
@ -37,7 +75,22 @@ func signVote(from *ConsensusState, voteType byte, hash []byte, header types.Par
return vote return vote
} }
func signVoteMany(voteType byte, hash []byte, header types.PartSetHeader, css ...*ConsensusState) []*types.Vote {
votes := make([]*types.Vote, len(css))
for i, cs := range css {
votes[i] = signVote(cs, voteType, hash, header)
}
return votes
}
// add vote to one cs from another // add vote to one cs from another
func signAddVoteToFromMany(voteType byte, to *ConsensusState, hash []byte, header types.PartSetHeader, froms ...*ConsensusState) {
for _, from := range froms {
vote := signVote(from, voteType, hash, header)
addVoteToFrom(to, from, vote)
}
}
func signAddVoteToFrom(voteType byte, to, from *ConsensusState, hash []byte, header types.PartSetHeader) *types.Vote { func signAddVoteToFrom(voteType byte, to, from *ConsensusState, hash []byte, header types.PartSetHeader) *types.Vote {
vote := signVote(from, voteType, hash, header) vote := signVote(from, voteType, hash, header)
addVoteToFrom(to, from, vote) addVoteToFrom(to, from, vote)
@ -81,6 +134,12 @@ func validatePrevote(t *testing.T, cs *ConsensusState, round int, privVal *types
} }
} }
func incrementRound(css ...*ConsensusState) {
for _, cs := range css {
cs.Round += 1
}
}
func validatePrecommit(t *testing.T, cs *ConsensusState, thisRound, lockRound int, privVal *types.PrivValidator, votedBlockHash, lockedBlockHash []byte) { func validatePrecommit(t *testing.T, cs *ConsensusState, thisRound, lockRound int, privVal *types.PrivValidator, votedBlockHash, lockedBlockHash []byte) {
precommits := cs.Votes.Precommits(thisRound) precommits := cs.Votes.Precommits(thisRound)
var vote *types.Vote var vote *types.Vote
@ -110,6 +169,20 @@ func validatePrecommit(t *testing.T, cs *ConsensusState, thisRound, lockRound in
} }
func validatePrevoteAndPrecommit(t *testing.T, cs *ConsensusState, thisRound, lockRound int, privVal *types.PrivValidator, votedBlockHash, lockedBlockHash []byte, f func()) {
// verify the prevote
validatePrevote(t, cs, thisRound, privVal, votedBlockHash)
if f != nil {
f()
}
// wait to finish precommit
<-cs.NewStepCh()
// verify precommit
cs.mtx.Lock()
validatePrecommit(t, cs, thisRound, lockRound, privVal, votedBlockHash, lockedBlockHash)
cs.mtx.Unlock()
}
func simpleConsensusState(nValidators int) ([]*ConsensusState, []*types.PrivValidator) { func simpleConsensusState(nValidators int) ([]*ConsensusState, []*types.PrivValidator) {
// Get State // Get State
state, privAccs, privVals := sm.RandGenesisState(10, true, 1000, nValidators, false, 10) state, privAccs, privVals := sm.RandGenesisState(10, true, 1000, nValidators, false, 10)
@ -131,6 +204,7 @@ func simpleConsensusState(nValidators int) ([]*ConsensusState, []*types.PrivVali
// Make ConsensusReactor // Make ConsensusReactor
cs := NewConsensusState(state, blockStore, mempoolReactor) cs := NewConsensusState(state, blockStore, mempoolReactor)
cs.SetPrivValidator(privVals[i])
// read off the NewHeightStep // read off the NewHeightStep
<-cs.NewStepCh() <-cs.NewStepCh()


+ 0
- 1
rpc/core_client/ws_client.go View File

@ -7,7 +7,6 @@ import (
"github.com/tendermint/tendermint/Godeps/_workspace/src/github.com/gorilla/websocket" "github.com/tendermint/tendermint/Godeps/_workspace/src/github.com/gorilla/websocket"
. "github.com/tendermint/tendermint/common" . "github.com/tendermint/tendermint/common"
_ "github.com/tendermint/tendermint/config/tendermint_test"
ctypes "github.com/tendermint/tendermint/rpc/core/types" ctypes "github.com/tendermint/tendermint/rpc/core/types"
"github.com/tendermint/tendermint/rpc/types" "github.com/tendermint/tendermint/rpc/types"
"github.com/tendermint/tendermint/wire" "github.com/tendermint/tendermint/wire"


+ 6
- 5
types/vote_set.go View File

@ -85,7 +85,7 @@ func (voteSet *VoteSet) Size() int {
// Otherwise returns err=ErrVote[UnexpectedStep|InvalidAccount|InvalidSignature|InvalidBlockHash|ConflictingSignature] // Otherwise returns err=ErrVote[UnexpectedStep|InvalidAccount|InvalidSignature|InvalidBlockHash|ConflictingSignature]
// Duplicate votes return added=false, err=nil. // Duplicate votes return added=false, err=nil.
// NOTE: vote should not be mutated after adding. // NOTE: vote should not be mutated after adding.
func (voteSet *VoteSet) AddByIndex(valIndex int, vote *Vote) (added bool, index int, err error) {
func (voteSet *VoteSet) AddByIndex(valIndex int, vote *Vote) (added bool, address []byte, err error) {
voteSet.mtx.Lock() voteSet.mtx.Lock()
defer voteSet.mtx.Unlock() defer voteSet.mtx.Unlock()
@ -109,14 +109,15 @@ func (voteSet *VoteSet) AddByAddress(address []byte, vote *Vote) (added bool, in
return voteSet.addVote(val, valIndex, vote) return voteSet.addVote(val, valIndex, vote)
} }
func (voteSet *VoteSet) addByIndex(valIndex int, vote *Vote) (bool, int, error) {
func (voteSet *VoteSet) addByIndex(valIndex int, vote *Vote) (added bool, address []byte, err error) {
// Ensure that signer is a validator. // Ensure that signer is a validator.
_, val := voteSet.valSet.GetByIndex(valIndex)
address, val := voteSet.valSet.GetByIndex(valIndex)
if val == nil { if val == nil {
return false, 0, ErrVoteInvalidAccount
return false, nil, ErrVoteInvalidAccount
} }
return voteSet.addVote(val, valIndex, vote)
added, _, err = voteSet.addVote(val, valIndex, vote)
return
} }
func (voteSet *VoteSet) addVote(val *Validator, valIndex int, vote *Vote) (bool, int, error) { func (voteSet *VoteSet) addVote(val *Validator, valIndex int, vote *Vote) (bool, int, error) {


Loading…
Cancel
Save