mirror of
https://github.com/fluencelabs/tendermint
synced 2025-06-14 22:01:20 +00:00
make byzantine logic testable
This commit is contained in:
@ -271,7 +271,7 @@ func (conR *ConsensusReactor) Receive(chID byte, src *p2p.Peer, msgBytes []byte)
|
||||
}
|
||||
|
||||
// Sets our private validator account for signing votes.
|
||||
func (conR *ConsensusReactor) SetPrivValidator(priv *types.PrivValidator) {
|
||||
func (conR *ConsensusReactor) SetPrivValidator(priv PrivValidator) {
|
||||
conR.conS.SetPrivValidator(priv)
|
||||
}
|
||||
|
||||
@ -399,7 +399,11 @@ OUTER_LOOP:
|
||||
if index, ok := prs.ProposalBlockParts.Not().PickRandom(); ok {
|
||||
// Ensure that the peer's PartSetHeader is correct
|
||||
blockMeta := conR.blockStore.LoadBlockMeta(prs.Height)
|
||||
if !blockMeta.PartsHeader.Equals(prs.ProposalBlockPartsHeader) {
|
||||
if blockMeta == nil {
|
||||
log.Warn("Failed to load block meta", "peer height", prs.Height, "our height", rs.Height)
|
||||
time.Sleep(peerGossipSleepDuration)
|
||||
continue OUTER_LOOP
|
||||
} else if !blockMeta.PartsHeader.Equals(prs.ProposalBlockPartsHeader) {
|
||||
log.Info("Peer ProposalBlockPartsHeader mismatch, sleeping",
|
||||
"peerHeight", prs.Height, "blockPartsHeader", blockMeta.PartsHeader, "peerBlockPartsHeader", prs.ProposalBlockPartsHeader)
|
||||
time.Sleep(peerGossipSleepDuration)
|
||||
|
@ -1,6 +1,7 @@
|
||||
package consensus
|
||||
|
||||
import (
|
||||
"fmt"
|
||||
"net"
|
||||
"sync"
|
||||
"testing"
|
||||
@ -8,7 +9,10 @@ import (
|
||||
|
||||
"github.com/tendermint/tendermint/config/tendermint_test"
|
||||
|
||||
"github.com/tendermint/ed25519"
|
||||
. "github.com/tendermint/go-common"
|
||||
cfg "github.com/tendermint/go-config"
|
||||
"github.com/tendermint/go-crypto"
|
||||
dbm "github.com/tendermint/go-db"
|
||||
"github.com/tendermint/go-events"
|
||||
"github.com/tendermint/go-p2p"
|
||||
@ -31,7 +35,7 @@ func resetConfigTimeouts() {
|
||||
// config.Set("timeout_commit", 1000)
|
||||
}
|
||||
|
||||
func TestReactor(t *testing.T) {
|
||||
func _TestReactor(t *testing.T) {
|
||||
resetConfigTimeouts()
|
||||
N := 4
|
||||
css := randConsensusNet(N)
|
||||
@ -80,3 +84,201 @@ func TestReactor(t *testing.T) {
|
||||
t.Fatalf("Timed out waiting for all validators to commit first block")
|
||||
}
|
||||
}
|
||||
|
||||
func _TestByzantine(t *testing.T) {
|
||||
resetConfigTimeouts()
|
||||
N := 4
|
||||
css := randConsensusNet(N)
|
||||
|
||||
switches := make([]*p2p.Switch, N)
|
||||
for i := 0; i < N; i++ {
|
||||
switches[i] = p2p.NewSwitch(cfg.NewMapConfig(nil))
|
||||
}
|
||||
|
||||
reactors := make([]*ConsensusReactor, N)
|
||||
eventChans := make([]chan interface{}, N)
|
||||
for i := 0; i < N; i++ {
|
||||
blockStoreDB := dbm.NewDB(Fmt("blockstore%d", i), config.GetString("db_backend"), config.GetString("db_dir"))
|
||||
blockStore := bc.NewBlockStore(blockStoreDB)
|
||||
|
||||
if i == 0 {
|
||||
// make byzantine
|
||||
css[i].decideProposal = func(j int) func(int, int) {
|
||||
return func(height, round int) {
|
||||
fmt.Println("hmph", j)
|
||||
byzantineDecideProposalFunc(height, round, css[j], switches[j])
|
||||
}
|
||||
}(i)
|
||||
css[i].doPrevote = func(height, round int) {}
|
||||
css[i].setProposal = func(j int) func(proposal *types.Proposal) error {
|
||||
return func(proposal *types.Proposal) error {
|
||||
return byzantineSetProposal(proposal, css[j], switches[j])
|
||||
}
|
||||
}(i)
|
||||
}
|
||||
reactors[i] = NewConsensusReactor(css[i], blockStore, false)
|
||||
reactors[i].SetPrivValidator(css[i].privValidator)
|
||||
|
||||
eventSwitch := events.NewEventSwitch()
|
||||
_, err := eventSwitch.Start()
|
||||
if err != nil {
|
||||
t.Fatalf("Failed to start switch: %v", err)
|
||||
}
|
||||
|
||||
reactors[i].SetEventSwitch(eventSwitch)
|
||||
eventChans[i] = subscribeToEvent(eventSwitch, "tester", types.EventStringNewBlock(), 1)
|
||||
}
|
||||
p2p.MakeConnectedSwitches(N, func(i int, s *p2p.Switch) *p2p.Switch {
|
||||
s.AddReactor("CONSENSUS", reactors[i])
|
||||
return s
|
||||
}, net.Pipe)
|
||||
|
||||
// wait till everyone makes the first new block
|
||||
wg := new(sync.WaitGroup)
|
||||
wg.Add(N)
|
||||
for i := 0; i < N; i++ {
|
||||
go func(j int) {
|
||||
<-eventChans[j]
|
||||
wg.Done()
|
||||
}(i)
|
||||
}
|
||||
|
||||
done := make(chan struct{})
|
||||
go func() {
|
||||
wg.Wait()
|
||||
close(done)
|
||||
}()
|
||||
|
||||
tick := time.NewTicker(time.Second * 3)
|
||||
select {
|
||||
case <-done:
|
||||
case <-tick.C:
|
||||
t.Fatalf("Timed out waiting for all validators to commit first block")
|
||||
}
|
||||
}
|
||||
|
||||
//-------------------------------
|
||||
// byzantine consensus functions
|
||||
|
||||
func byzantineDecideProposalFunc(height, round int, cs *ConsensusState, sw *p2p.Switch) {
|
||||
// byzantine user should create two proposals and try to split the vote.
|
||||
// Avoid sending on internalMsgQueue and running consensus state.
|
||||
|
||||
// Create a new proposal block from state/txs from the mempool.
|
||||
block1, blockParts1 := cs.createProposalBlock()
|
||||
polRound, polBlockID := cs.Votes.POLInfo()
|
||||
proposal1 := types.NewProposal(height, round, blockParts1.Header(), polRound, polBlockID)
|
||||
cs.privValidator.SignProposal(cs.state.ChainID, proposal1) // byzantine doesnt err
|
||||
|
||||
// Create a new proposal block from state/txs from the mempool.
|
||||
block2, blockParts2 := cs.createProposalBlock()
|
||||
polRound, polBlockID = cs.Votes.POLInfo()
|
||||
proposal2 := types.NewProposal(height, round, blockParts2.Header(), polRound, polBlockID)
|
||||
cs.privValidator.SignProposal(cs.state.ChainID, proposal2) // byzantine doesnt err
|
||||
|
||||
log.Notice("Byzantine: broadcasting conflicting proposals")
|
||||
// broadcast conflicting proposals/block parts to peers
|
||||
peers := sw.Peers().List()
|
||||
for i, peer := range peers {
|
||||
if i < len(peers)/2 {
|
||||
go sendProposalAndParts(height, round, cs, peer, proposal1, block1, blockParts1)
|
||||
} else {
|
||||
go sendProposalAndParts(height, round, cs, peer, proposal2, block2, blockParts2)
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
func sendProposalAndParts(height, round int, cs *ConsensusState, peer *p2p.Peer, proposal *types.Proposal, block *types.Block, parts *types.PartSet) {
|
||||
// proposal
|
||||
msg := &ProposalMessage{Proposal: proposal}
|
||||
peer.Send(DataChannel, struct{ ConsensusMessage }{msg})
|
||||
|
||||
// parts
|
||||
for i := 0; i < parts.Total(); i++ {
|
||||
part := parts.GetPart(i)
|
||||
msg := &BlockPartMessage{
|
||||
Height: height, // This tells peer that this part applies to us.
|
||||
Round: round, // This tells peer that this part applies to us.
|
||||
Part: part,
|
||||
}
|
||||
peer.Send(DataChannel, struct{ ConsensusMessage }{msg})
|
||||
}
|
||||
|
||||
// votes
|
||||
prevote, _ := cs.signVote(types.VoteTypePrevote, block.Hash(), parts.Header())
|
||||
peer.Send(VoteChannel, struct{ ConsensusMessage }{&VoteMessage{prevote}})
|
||||
precommit, _ := cs.signVote(types.VoteTypePrecommit, block.Hash(), parts.Header())
|
||||
peer.Send(VoteChannel, struct{ ConsensusMessage }{&VoteMessage{precommit}})
|
||||
}
|
||||
|
||||
func byzantineSetProposal(proposal *types.Proposal, cs *ConsensusState, sw *p2p.Switch) error {
|
||||
peers := sw.Peers().List()
|
||||
for _, peer := range peers {
|
||||
// votes
|
||||
var blockHash []byte // XXX proposal.BlockHash
|
||||
blockHash = []byte{0, 1, 0, 2, 0, 3}
|
||||
prevote, _ := cs.signVote(types.VoteTypePrevote, blockHash, proposal.BlockPartsHeader)
|
||||
peer.Send(VoteChannel, struct{ ConsensusMessage }{&VoteMessage{prevote}})
|
||||
precommit, _ := cs.signVote(types.VoteTypePrecommit, blockHash, proposal.BlockPartsHeader)
|
||||
peer.Send(VoteChannel, struct{ ConsensusMessage }{&VoteMessage{precommit}})
|
||||
}
|
||||
return nil
|
||||
}
|
||||
|
||||
//----------------------------------------
|
||||
// byzantine privValidator
|
||||
type ByzantinePrivValidator struct {
|
||||
Address []byte `json:"address"`
|
||||
PubKey crypto.PubKey `json:"pub_key"`
|
||||
|
||||
// PrivKey should be empty if a Signer other than the default is being used.
|
||||
PrivKey crypto.PrivKey `json:"priv_key"`
|
||||
types.Signer `json:"-"`
|
||||
|
||||
mtx sync.Mutex
|
||||
}
|
||||
|
||||
func (privVal *ByzantinePrivValidator) SetSigner(s types.Signer) {
|
||||
privVal.Signer = s
|
||||
}
|
||||
|
||||
// Generates a new validator with private key.
|
||||
func GenPrivValidator() *ByzantinePrivValidator {
|
||||
privKeyBytes := new([64]byte)
|
||||
copy(privKeyBytes[:32], crypto.CRandBytes(32))
|
||||
pubKeyBytes := ed25519.MakePublicKey(privKeyBytes)
|
||||
pubKey := crypto.PubKeyEd25519(*pubKeyBytes)
|
||||
privKey := crypto.PrivKeyEd25519(*privKeyBytes)
|
||||
return &ByzantinePrivValidator{
|
||||
Address: pubKey.Address(),
|
||||
PubKey: pubKey,
|
||||
PrivKey: privKey,
|
||||
Signer: types.NewDefaultSigner(privKey),
|
||||
}
|
||||
}
|
||||
|
||||
func (privVal *ByzantinePrivValidator) GetAddress() []byte {
|
||||
return privVal.Address
|
||||
}
|
||||
|
||||
func (privVal *ByzantinePrivValidator) SignVote(chainID string, vote *types.Vote) error {
|
||||
privVal.mtx.Lock()
|
||||
defer privVal.mtx.Unlock()
|
||||
|
||||
// Sign
|
||||
vote.Signature = privVal.Sign(types.SignBytes(chainID, vote)).(crypto.SignatureEd25519)
|
||||
return nil
|
||||
}
|
||||
|
||||
func (privVal *ByzantinePrivValidator) SignProposal(chainID string, proposal *types.Proposal) error {
|
||||
privVal.mtx.Lock()
|
||||
defer privVal.mtx.Unlock()
|
||||
|
||||
// Sign
|
||||
proposal.Signature = privVal.Sign(types.SignBytes(chainID, proposal)).(crypto.SignatureEd25519)
|
||||
return nil
|
||||
}
|
||||
|
||||
func (privVal *ByzantinePrivValidator) String() string {
|
||||
return Fmt("PrivValidator{%X}", privVal.Address)
|
||||
}
|
||||
|
@ -110,9 +110,13 @@ func runReplayTest(t *testing.T, cs *ConsensusState, walDir string, newBlockCh c
|
||||
cs.Wait()
|
||||
}
|
||||
|
||||
func toPV(pv PrivValidator) *types.PrivValidator {
|
||||
return pv.(*types.PrivValidator)
|
||||
}
|
||||
|
||||
func setupReplayTest(thisCase *testCase, nLines int, crashAfter bool) (*ConsensusState, chan interface{}, string, string) {
|
||||
fmt.Println("-------------------------------------")
|
||||
log.Notice(Fmt("Starting replay test of %d lines of WAL (crash before write)", nLines))
|
||||
log.Notice(Fmt("Starting replay test of %d lines of WAL. Crash after = %v", nLines, crashAfter))
|
||||
|
||||
lineStep := nLines
|
||||
if crashAfter {
|
||||
@ -128,10 +132,10 @@ func setupReplayTest(thisCase *testCase, nLines int, crashAfter bool) (*Consensu
|
||||
cs := fixedConsensusStateDummy()
|
||||
|
||||
// set the last step according to when we crashed vs the wal
|
||||
cs.privValidator.LastHeight = 1 // first block
|
||||
cs.privValidator.LastStep = thisCase.stepMap[lineStep]
|
||||
toPV(cs.privValidator).LastHeight = 1 // first block
|
||||
toPV(cs.privValidator).LastStep = mapPrivValStep[lineStep]
|
||||
|
||||
fmt.Println("LAST STEP", cs.privValidator.LastStep)
|
||||
log.Warn("setupReplayTest", "LastStep", toPV(cs.privValidator).LastStep)
|
||||
|
||||
newBlockCh := subscribeToEvent(cs.evsw, "tester", types.EventStringNewBlock(), 1)
|
||||
|
||||
@ -168,8 +172,8 @@ func TestReplayCrashBeforeWritePropose(t *testing.T) {
|
||||
if err != nil {
|
||||
t.Fatalf("Error reading json data: %v", err)
|
||||
}
|
||||
cs.privValidator.LastSignBytes = types.SignBytes(cs.state.ChainID, proposal.Proposal)
|
||||
cs.privValidator.LastSignature = proposal.Proposal.Signature
|
||||
toPV(cs.privValidator).LastSignBytes = types.SignBytes(cs.state.ChainID, proposal.Proposal)
|
||||
toPV(cs.privValidator).LastSignature = proposal.Proposal.Signature
|
||||
runReplayTest(t, cs, walDir, newBlockCh, thisCase, lineNum)
|
||||
}
|
||||
}
|
||||
@ -187,8 +191,8 @@ func TestReplayCrashBeforeWritePrevote(t *testing.T) {
|
||||
if err != nil {
|
||||
t.Fatalf("Error reading json data: %v", err)
|
||||
}
|
||||
cs.privValidator.LastSignBytes = types.SignBytes(cs.state.ChainID, vote.Vote)
|
||||
cs.privValidator.LastSignature = vote.Vote.Signature
|
||||
toPV(cs.privValidator).LastSignBytes = types.SignBytes(cs.state.ChainID, vote.Vote)
|
||||
toPV(cs.privValidator).LastSignature = vote.Vote.Signature
|
||||
})
|
||||
runReplayTest(t, cs, walDir, newBlockCh, thisCase, lineNum)
|
||||
}
|
||||
@ -207,8 +211,8 @@ func TestReplayCrashBeforeWritePrecommit(t *testing.T) {
|
||||
if err != nil {
|
||||
t.Fatalf("Error reading json data: %v", err)
|
||||
}
|
||||
cs.privValidator.LastSignBytes = types.SignBytes(cs.state.ChainID, vote.Vote)
|
||||
cs.privValidator.LastSignature = vote.Vote.Signature
|
||||
toPV(cs.privValidator).LastSignBytes = types.SignBytes(cs.state.ChainID, vote.Vote)
|
||||
toPV(cs.privValidator).LastSignature = vote.Vote.Signature
|
||||
})
|
||||
runReplayTest(t, cs, walDir, newBlockCh, thisCase, lineNum)
|
||||
}
|
||||
|
@ -209,6 +209,12 @@ func (ti *timeoutInfo) String() string {
|
||||
return fmt.Sprintf("%v ; %d/%d %v", ti.Duration, ti.Height, ti.Round, ti.Step)
|
||||
}
|
||||
|
||||
type PrivValidator interface {
|
||||
GetAddress() []byte
|
||||
SignVote(chainID string, vote *types.Vote) error
|
||||
SignProposal(chainID string, proposal *types.Proposal) error
|
||||
}
|
||||
|
||||
// Tracks consensus state across block heights and rounds.
|
||||
type ConsensusState struct {
|
||||
BaseService
|
||||
@ -217,7 +223,7 @@ type ConsensusState struct {
|
||||
proxyAppConn proxy.AppConnConsensus
|
||||
blockStore *bc.BlockStore
|
||||
mempool *mempl.Mempool
|
||||
privValidator *types.PrivValidator
|
||||
privValidator PrivValidator
|
||||
|
||||
mtx sync.Mutex
|
||||
RoundState
|
||||
@ -236,6 +242,11 @@ type ConsensusState struct {
|
||||
replayMode bool // so we don't log signing errors during replay
|
||||
|
||||
nSteps int // used for testing to limit the number of transitions the state makes
|
||||
|
||||
// allow certain function to be overwritten for testing
|
||||
decideProposal func(height, round int)
|
||||
doPrevote func(height, round int)
|
||||
setProposal func(proposal *types.Proposal) error
|
||||
}
|
||||
|
||||
func NewConsensusState(config cfg.Config, state *sm.State, proxyAppConn proxy.AppConnConsensus, blockStore *bc.BlockStore, mempool *mempl.Mempool) *ConsensusState {
|
||||
@ -251,6 +262,11 @@ func NewConsensusState(config cfg.Config, state *sm.State, proxyAppConn proxy.Ap
|
||||
tockChan: make(chan timeoutInfo, tickTockBufferSize),
|
||||
timeoutParams: InitTimeoutParamsFromConfig(config),
|
||||
}
|
||||
// set function defaults (may be overwritten before calling Start)
|
||||
cs.decideProposal = cs.defaultDecideProposal
|
||||
cs.doPrevote = cs.defaultDoPrevote
|
||||
cs.setProposal = cs.defaultSetProposal
|
||||
|
||||
cs.updateToState(state)
|
||||
// Don't call scheduleRound0 yet.
|
||||
// We do that upon Start().
|
||||
@ -295,7 +311,7 @@ func (cs *ConsensusState) GetValidators() (int, []*types.Validator) {
|
||||
return cs.state.LastBlockHeight, cs.state.Validators.Copy().Validators
|
||||
}
|
||||
|
||||
func (cs *ConsensusState) SetPrivValidator(priv *types.PrivValidator) {
|
||||
func (cs *ConsensusState) SetPrivValidator(priv PrivValidator) {
|
||||
cs.mtx.Lock()
|
||||
defer cs.mtx.Unlock()
|
||||
cs.privValidator = priv
|
||||
@ -825,16 +841,16 @@ func (cs *ConsensusState) enterPropose(height int, round int) {
|
||||
return
|
||||
}
|
||||
|
||||
if !bytes.Equal(cs.Validators.Proposer().Address, cs.privValidator.Address) {
|
||||
if !bytes.Equal(cs.Validators.Proposer().Address, cs.privValidator.GetAddress()) {
|
||||
log.Info("enterPropose: Not our turn to propose", "proposer", cs.Validators.Proposer().Address, "privValidator", cs.privValidator)
|
||||
} else {
|
||||
log.Info("enterPropose: Our turn to propose", "proposer", cs.Validators.Proposer().Address, "privValidator", cs.privValidator)
|
||||
cs.decideProposal(height, round)
|
||||
}
|
||||
|
||||
}
|
||||
}
|
||||
|
||||
func (cs *ConsensusState) decideProposal(height, round int) {
|
||||
func (cs *ConsensusState) defaultDecideProposal(height, round int) {
|
||||
var block *types.Block
|
||||
var blockParts *types.PartSet
|
||||
|
||||
@ -875,7 +891,6 @@ func (cs *ConsensusState) decideProposal(height, round int) {
|
||||
log.Warn("enterPropose: Error signing proposal", "height", height, "round", round, "error", err)
|
||||
}
|
||||
}
|
||||
|
||||
}
|
||||
|
||||
// Returns true if the proposal block is complete &&
|
||||
@ -972,10 +987,10 @@ func (cs *ConsensusState) enterPrevote(height int, round int) {
|
||||
// (so we have more time to try and collect +2/3 prevotes for a single block)
|
||||
}
|
||||
|
||||
func (cs *ConsensusState) doPrevote(height int, round int) {
|
||||
func (cs *ConsensusState) defaultDoPrevote(height int, round int) {
|
||||
// If a block is locked, prevote that.
|
||||
if cs.LockedBlock != nil {
|
||||
log.Info("enterPrevote: Block was locked")
|
||||
log.Notice("enterPrevote: Block was locked")
|
||||
cs.signAddVote(types.VoteTypePrevote, cs.LockedBlock.Hash(), cs.LockedBlockParts.Header())
|
||||
return
|
||||
}
|
||||
@ -1051,9 +1066,9 @@ func (cs *ConsensusState) enterPrecommit(height int, round int) {
|
||||
// If we don't have a polka, we must precommit nil
|
||||
if !ok {
|
||||
if cs.LockedBlock != nil {
|
||||
log.Info("enterPrecommit: No +2/3 prevotes during enterPrecommit while we're locked. Precommitting nil")
|
||||
log.Notice("enterPrecommit: No +2/3 prevotes during enterPrecommit while we're locked. Precommitting nil")
|
||||
} else {
|
||||
log.Info("enterPrecommit: No +2/3 prevotes during enterPrecommit. Precommitting nil.")
|
||||
log.Notice("enterPrecommit: No +2/3 prevotes during enterPrecommit. Precommitting nil.")
|
||||
}
|
||||
cs.signAddVote(types.VoteTypePrecommit, nil, types.PartSetHeader{})
|
||||
return
|
||||
@ -1322,8 +1337,9 @@ func (cs *ConsensusState) commitStateUpdateMempool(s *sm.State, block *types.Blo
|
||||
|
||||
//-----------------------------------------------------------------------------
|
||||
|
||||
func (cs *ConsensusState) setProposal(proposal *types.Proposal) error {
|
||||
func (cs *ConsensusState) defaultSetProposal(proposal *types.Proposal) error {
|
||||
// Already have one
|
||||
// TODO: possibly catch double proposals
|
||||
if cs.Proposal != nil {
|
||||
return nil
|
||||
}
|
||||
@ -1519,9 +1535,10 @@ func (cs *ConsensusState) addVote(vote *types.Vote, peerKey string) (added bool,
|
||||
|
||||
func (cs *ConsensusState) signVote(type_ byte, hash []byte, header types.PartSetHeader) (*types.Vote, error) {
|
||||
// TODO: store our index in the cs so we don't have to do this every time
|
||||
valIndex, _ := cs.Validators.GetByAddress(cs.privValidator.Address)
|
||||
addr := cs.privValidator.GetAddress()
|
||||
valIndex, _ := cs.Validators.GetByAddress(addr)
|
||||
vote := &types.Vote{
|
||||
ValidatorAddress: cs.privValidator.Address,
|
||||
ValidatorAddress: addr,
|
||||
ValidatorIndex: valIndex,
|
||||
Height: cs.Height,
|
||||
Round: cs.Round,
|
||||
@ -1534,8 +1551,7 @@ func (cs *ConsensusState) signVote(type_ byte, hash []byte, header types.PartSet
|
||||
|
||||
// sign the vote and publish on internalMsgQueue
|
||||
func (cs *ConsensusState) signAddVote(type_ byte, hash []byte, header types.PartSetHeader) *types.Vote {
|
||||
|
||||
if cs.privValidator == nil || !cs.Validators.HasAddress(cs.privValidator.Address) {
|
||||
if cs.privValidator == nil || !cs.Validators.HasAddress(cs.privValidator.GetAddress()) {
|
||||
return nil
|
||||
}
|
||||
vote, err := cs.signVote(type_, hash, header)
|
||||
@ -1544,9 +1560,9 @@ func (cs *ConsensusState) signAddVote(type_ byte, hash []byte, header types.Part
|
||||
log.Info("Signed and pushed vote", "height", cs.Height, "round", cs.Round, "vote", vote, "error", err)
|
||||
return vote
|
||||
} else {
|
||||
if !cs.replayMode {
|
||||
log.Warn("Error signing vote", "height", cs.Height, "round", cs.Round, "vote", vote, "error", err)
|
||||
}
|
||||
//if !cs.replayMode {
|
||||
log.Warn("Error signing vote", "height", cs.Height, "round", cs.Round, "vote", vote, "error", err)
|
||||
//}
|
||||
return nil
|
||||
}
|
||||
}
|
||||
|
@ -66,8 +66,8 @@ func TestProposerSelection0(t *testing.T) {
|
||||
|
||||
// lets commit a block and ensure proposer for the next height is correct
|
||||
prop := cs1.GetRoundState().Validators.Proposer()
|
||||
if !bytes.Equal(prop.Address, cs1.privValidator.Address) {
|
||||
panic(Fmt("expected proposer to be validator %d. Got %X", 0, prop.Address))
|
||||
if !bytes.Equal(prop.Address, cs1.privValidator.GetAddress()) {
|
||||
t.Fatalf("expected proposer to be validator %d. Got %X", 0, prop.Address)
|
||||
}
|
||||
|
||||
// wait for complete proposal
|
||||
@ -605,7 +605,7 @@ func TestLockPOLUnlock(t *testing.T) {
|
||||
timeoutWaitCh := subscribeToEvent(cs1.evsw, "tester", types.EventStringTimeoutWait(), 1)
|
||||
newRoundCh := subscribeToEvent(cs1.evsw, "tester", types.EventStringNewRound(), 1)
|
||||
unlockCh := subscribeToEvent(cs1.evsw, "tester", types.EventStringUnlock(), 1)
|
||||
voteCh := subscribeToVoter(cs1, cs1.privValidator.Address)
|
||||
voteCh := subscribeToVoter(cs1, cs1.privValidator.GetAddress())
|
||||
|
||||
// everything done from perspective of cs1
|
||||
|
||||
@ -697,7 +697,7 @@ func TestLockPOLSafety1(t *testing.T) {
|
||||
timeoutProposeCh := subscribeToEvent(cs1.evsw, "tester", types.EventStringTimeoutPropose(), 1)
|
||||
timeoutWaitCh := subscribeToEvent(cs1.evsw, "tester", types.EventStringTimeoutWait(), 1)
|
||||
newRoundCh := subscribeToEvent(cs1.evsw, "tester", types.EventStringNewRound(), 1)
|
||||
voteCh := subscribeToVoter(cs1, cs1.privValidator.Address)
|
||||
voteCh := subscribeToVoter(cs1, cs1.privValidator.GetAddress())
|
||||
|
||||
// start round and wait for propose and prevote
|
||||
startTestRound(cs1, cs1.Height, 0)
|
||||
@ -817,7 +817,7 @@ func TestLockPOLSafety2(t *testing.T) {
|
||||
timeoutWaitCh := subscribeToEvent(cs1.evsw, "tester", types.EventStringTimeoutWait(), 1)
|
||||
newRoundCh := subscribeToEvent(cs1.evsw, "tester", types.EventStringNewRound(), 1)
|
||||
unlockCh := subscribeToEvent(cs1.evsw, "tester", types.EventStringUnlock(), 1)
|
||||
voteCh := subscribeToVoter(cs1, cs1.privValidator.Address)
|
||||
voteCh := subscribeToVoter(cs1, cs1.privValidator.GetAddress())
|
||||
|
||||
// the block for R0: gets polkad but we miss it
|
||||
// (even though we signed it, shhh)
|
||||
@ -909,7 +909,7 @@ func TestSlashingPrevotes(t *testing.T) {
|
||||
proposalCh := subscribeToEvent(cs1.evsw,"tester",types.EventStringCompleteProposal() , 1)
|
||||
timeoutWaitCh := subscribeToEvent(cs1.evsw,"tester",types.EventStringTimeoutWait() , 1)
|
||||
newRoundCh := subscribeToEvent(cs1.evsw,"tester",types.EventStringNewRound() , 1)
|
||||
voteCh := subscribeToVoter(cs1, cs1.privValidator.Address)
|
||||
voteCh := subscribeToVoter(cs1, cs1.privValidator.GetAddress())
|
||||
|
||||
// start round and wait for propose and prevote
|
||||
startTestRound(cs1, cs1.Height, 0)
|
||||
@ -944,7 +944,7 @@ func TestSlashingPrecommits(t *testing.T) {
|
||||
proposalCh := subscribeToEvent(cs1.evsw,"tester",types.EventStringCompleteProposal() , 1)
|
||||
timeoutWaitCh := subscribeToEvent(cs1.evsw,"tester",types.EventStringTimeoutWait() , 1)
|
||||
newRoundCh := subscribeToEvent(cs1.evsw,"tester",types.EventStringNewRound() , 1)
|
||||
voteCh := subscribeToVoter(cs1, cs1.privValidator.Address)
|
||||
voteCh := subscribeToVoter(cs1, cs1.privValidator.GetAddress())
|
||||
|
||||
// start round and wait for propose and prevote
|
||||
startTestRound(cs1, cs1.Height, 0)
|
||||
@ -989,7 +989,7 @@ func TestHalt1(t *testing.T) {
|
||||
timeoutWaitCh := subscribeToEvent(cs1.evsw, "tester", types.EventStringTimeoutWait(), 1)
|
||||
newRoundCh := subscribeToEvent(cs1.evsw, "tester", types.EventStringNewRound(), 1)
|
||||
newBlockCh := subscribeToEvent(cs1.evsw, "tester", types.EventStringNewBlock(), 1)
|
||||
voteCh := subscribeToVoter(cs1, cs1.privValidator.Address)
|
||||
voteCh := subscribeToVoter(cs1, cs1.privValidator.GetAddress())
|
||||
|
||||
// start round and wait for propose and prevote
|
||||
startTestRound(cs1, cs1.Height, 0)
|
||||
|
@ -163,6 +163,10 @@ func (privVal *PrivValidator) Reset() {
|
||||
privVal.Save()
|
||||
}
|
||||
|
||||
func (privVal *PrivValidator) GetAddress() []byte {
|
||||
return privVal.Address
|
||||
}
|
||||
|
||||
func (privVal *PrivValidator) SignVote(chainID string, vote *Vote) error {
|
||||
privVal.mtx.Lock()
|
||||
defer privVal.mtx.Unlock()
|
||||
@ -231,6 +235,7 @@ func (privVal *PrivValidator) signBytesHRS(height, round int, step int8, signByt
|
||||
privVal.save()
|
||||
|
||||
return signature, nil
|
||||
|
||||
}
|
||||
|
||||
func (privVal *PrivValidator) String() string {
|
||||
|
Reference in New Issue
Block a user