@ -1,30 +1,22 @@
package consensus
import (
"bytes"
"encoding/binary"
"encoding/hex"
"sync"
"time"
"github.com/harmony-one/harmony/crypto/bls"
"github.com/ethereum/go-ethereum/rlp"
"github.com/harmony-one/harmony/core/types"
"github.com/ethereum/go-ethereum/common"
bls_core "github.com/harmony-one/bls/ffi/go/bls"
msg_pb "github.com/harmony-one/harmony/api/proto/message"
"github.com/harmony-one/harmony/consensus/quorum"
"github.com/harmony-one/harmony/consensus/signature"
bls_cosi "github.com/harmony-one/harmony/crypto/bls"
nodeconfig "github.com/harmony-one/harmony/internal/configs/node"
"github.com/harmony-one/harmony/internal/utils"
"github.com/harmony-one/harmony/p2p"
)
// MaxViewIDDiff limits the received view ID to only 100 further from the current view ID
const MaxViewIDDiff = 100
// MaxViewIDDiff limits the received view ID to only 249 further from the current view ID
const MaxViewIDDiff = 249
// State contains current mode and current viewID
type State struct {
@ -98,30 +90,15 @@ func (pm *State) GetViewChangeDuraion() time.Duration {
return time . Duration ( diff * diff * int64 ( viewChangeDuration ) )
}
// switchPhase will switch FBFTPhase to nextPhase if the desirePhase equals the nextPhase
func ( consensus * Consensus ) switchPhase ( desired FBFTPhase , override bool ) {
if override {
consensus . phase = desired
return
}
var nextPhase FBFTPhase
switch consensus . phase {
case FBFTAnnounce :
nextPhase = FBFTPrepare
case FBFTPrepare :
nextPhase = FBFTCommit
case FBFTCommit :
nextPhase = FBFTAnnounce
}
if nextPhase == desired {
consensus . phase = nextPhase
}
}
// GetNextLeaderKey uniquely determine who is the leader for given viewID
func ( consensus * Consensus ) GetNextLeaderKey ( ) * bls . PublicKeyWrapper {
wasFound , next := consensus . Decider . NextAfter ( consensus . LeaderPubKey )
func ( consensus * Consensus ) GetNextLeaderKey ( viewID uint64 ) * bls . PublicKeyWrapper {
gap := 1
consensus . getLogger ( ) . Info ( ) .
Str ( "leaderPubKey" , consensus . LeaderPubKey . Bytes . Hex ( ) ) .
Uint64 ( "newViewID" , viewID ) .
Uint64 ( "myCurBlockViewID" , consensus . GetCurBlockViewID ( ) ) .
Msg ( "[GetNextLeaderKey] got leaderPubKey from coinbase" )
wasFound , next := consensus . Decider . NthNext ( consensus . LeaderPubKey , gap )
if ! wasFound {
consensus . getLogger ( ) . Warn ( ) .
Str ( "key" , consensus . LeaderPubKey . Bytes . Hex ( ) ) .
@ -130,22 +107,6 @@ func (consensus *Consensus) GetNextLeaderKey() *bls.PublicKeyWrapper {
return next
}
// ResetViewChangeState reset the state for viewchange
func ( consensus * Consensus ) ResetViewChangeState ( ) {
consensus . getLogger ( ) . Debug ( ) .
Str ( "Phase" , consensus . phase . String ( ) ) .
Msg ( "[ResetViewChangeState] Resetting view change state" )
consensus . current . SetMode ( Normal )
consensus . m1Payload = [ ] byte { }
consensus . bhpSigs = map [ uint64 ] map [ string ] * bls_core . Sign { }
consensus . nilSigs = map [ uint64 ] map [ string ] * bls_core . Sign { }
consensus . viewIDSigs = map [ uint64 ] map [ string ] * bls_core . Sign { }
consensus . bhpBitmap = map [ uint64 ] * bls_cosi . Mask { }
consensus . nilBitmap = map [ uint64 ] * bls_cosi . Mask { }
consensus . viewIDBitmap = map [ uint64 ] * bls_cosi . Mask { }
consensus . Decider . ResetViewChangeVotes ( )
}
func createTimeout ( ) map [ TimeoutType ] * utils . Timeout {
timeouts := make ( map [ TimeoutType ] * utils . Timeout )
timeouts [ timeoutConsensus ] = utils . NewTimeout ( phaseDuration )
@ -154,7 +115,8 @@ func createTimeout() map[TimeoutType]*utils.Timeout {
return timeouts
}
// startViewChange send a new view change
// startViewChange send a new view change
// the viewID is the current viewID
func ( consensus * Consensus ) startViewChange ( viewID uint64 ) {
if consensus . disableViewChange {
return
@ -163,15 +125,33 @@ func (consensus *Consensus) startViewChange(viewID uint64) {
consensus . consensusTimeout [ timeoutBootstrap ] . Stop ( )
consensus . current . SetMode ( ViewChanging )
consensus . SetViewChangingID ( viewID )
consensus . LeaderPubKey = consensus . GetNextLeaderKey ( )
consensus . LeaderPubKey = consensus . GetNextLeaderKey ( viewID )
duration := consensus . current . GetViewChangeDuraion ( )
consensus . getLogger ( ) . Warn ( ) .
Uint64 ( "viewID" , viewID ) .
Uint64 ( "viewChangingID" , consensus . GetViewChangingID ( ) ) .
Dur ( "timeoutDuration" , duration ) .
Str ( "NextLeader" , consensus . LeaderPubKey . Bytes . Hex ( ) ) .
Msg ( "[startViewChange]" )
consensus . consensusTimeout [ timeoutViewChange ] . SetDuration ( duration )
defer consensus . consensusTimeout [ timeoutViewChange ] . Start ( )
// update the dictionary key if the viewID is first time received
consensus . vc . AddViewIDKeyIfNotExist ( viewID , consensus . Decider . Participants ( ) )
// init my own payload
if err := consensus . vc . InitPayload (
consensus . FBFTLog ,
viewID ,
consensus . blockNum ,
consensus . priKey ) ; err != nil {
consensus . getLogger ( ) . Error ( ) . Err ( err ) . Msg ( "Init Payload Error" )
}
// for view change, send separate view change per public key
// do not do multi-sign of view change message
for _ , key := range consensus . priKey {
if ! consensus . IsValidatorInCommittee ( key . Pub . Bytes ) {
continue
@ -183,21 +163,26 @@ func (consensus *Consensus) startViewChange(viewID uint64) {
p2p . ConstructMessage ( msgToSend ) ,
)
}
consensus . consensusTimeout [ timeoutViewChange ] . SetDuration ( duration )
consensus . consensusTimeout [ timeoutViewChange ] . Start ( )
}
// onViewChange is called when the view change message is received.
func ( consensus * Consensus ) onViewChange ( msg * msg_pb . Message ) {
consensus . getLogger ( ) . Info ( ) . Msg ( "[onViewChange] Received ViewChange Message" )
recvMsg , err := ParseViewChangeMessage ( msg )
if err != nil {
consensus . getLogger ( ) . Warn ( ) . Msg ( "[onViewChange] Unable To Parse Viewchange Message" )
consensus . getLogger ( ) . Warn ( ) . Err ( err ) . Msg ( "[onViewChange] Unable To Parse Viewchange Message" )
return
}
// if not leader, noop
newLeaderKey := recvMsg . LeaderPubkey
newLeaderPriKey , err := consensus . GetLeaderPrivateKey ( newLeaderKey . Object )
if err != nil {
consensus . getLogger ( ) . Info ( ) .
Err ( err ) .
Interface ( "SenderPubkeys" , recvMsg . SenderPubkeys ) .
Str ( "NextLeader" , recvMsg . LeaderPubkey . Bytes . Hex ( ) ) .
Str ( "myBLSPubKey" , consensus . priKey . GetPublicKeys ( ) . SerializeToHexStr ( ) ) .
Msg ( "[onViewChange] I am not the Leader" )
return
}
@ -205,7 +190,8 @@ func (consensus *Consensus) onViewChange(msg *msg_pb.Message) {
consensus . getLogger ( ) . Info ( ) .
Int64 ( "have" , consensus . Decider . SignersCount ( quorum . ViewChange ) ) .
Int64 ( "need" , consensus . Decider . TwoThirdsSignersCount ( ) ) .
Interface ( "validatorPubKeys" , recvMsg . SenderPubkeys ) .
Interface ( "SenderPubkeys" , recvMsg . SenderPubkeys ) .
Str ( "newLeaderKey" , newLeaderKey . Bytes . Hex ( ) ) .
Msg ( "[onViewChange] Received Enough View Change Messages" )
return
}
@ -217,212 +203,36 @@ func (consensus *Consensus) onViewChange(msg *msg_pb.Message) {
// already checked the length of SenderPubkeys in onViewChangeSanityCheck
senderKey := recvMsg . SenderPubkeys [ 0 ]
consensus . vcLock . Lock ( )
defer consensus . vcLock . Unlock ( )
// update the dictionary key if the viewID is first time received
consensus . addViewIDKeyIfNotExist ( recvMsg . ViewID )
// TODO: remove NIL type message
// add self m1 or m2 type message signature and bitmap
_ , ok1 := consensus . nilSigs [ recvMsg . ViewID ] [ newLeaderKey . Bytes . Hex ( ) ]
_ , ok2 := consensus . bhpSigs [ recvMsg . ViewID ] [ newLeaderKey . Bytes . Hex ( ) ]
if ! ( ok1 || ok2 ) {
// add own signature for newview message
preparedMsgs := consensus . FBFTLog . GetMessagesByTypeSeq (
msg_pb . MessageType_PREPARED , recvMsg . BlockNum ,
)
preparedMsg := consensus . FBFTLog . FindMessageByMaxViewID ( preparedMsgs )
hasBlock := false
if preparedMsg != nil {
if preparedBlock := consensus . FBFTLog . GetBlockByHash (
preparedMsg . BlockHash ,
) ; preparedBlock != nil {
if consensus . BlockVerifier ( preparedBlock ) ; err != nil {
consensus . getLogger ( ) . Error ( ) . Err ( err ) . Msg ( "[onViewChange] My own prepared block verification failed" )
} else {
hasBlock = true
}
}
}
if hasBlock {
consensus . getLogger ( ) . Info ( ) . Msg ( "[onViewChange] add my M1 type messaage" )
msgToSign := append ( preparedMsg . BlockHash [ : ] , preparedMsg . Payload ... )
for i , key := range consensus . priKey {
if err := consensus . bhpBitmap [ recvMsg . ViewID ] . SetKey ( key . Pub . Bytes , true ) ; err != nil {
consensus . getLogger ( ) . Warn ( ) . Msgf ( "[onViewChange] bhpBitmap setkey failed for key at index %d" , i )
continue
}
consensus . bhpSigs [ recvMsg . ViewID ] [ key . Pub . Bytes . Hex ( ) ] = key . Pri . SignHash ( msgToSign )
}
// if m1Payload is empty, we just add one
if len ( consensus . m1Payload ) == 0 {
consensus . m1Payload = append ( preparedMsg . BlockHash [ : ] , preparedMsg . Payload ... )
}
} else {
consensus . getLogger ( ) . Info ( ) . Msg ( "[onViewChange] add my M2(NIL) type messaage" )
for i , key := range consensus . priKey {
if err := consensus . nilBitmap [ recvMsg . ViewID ] . SetKey ( key . Pub . Bytes , true ) ; err != nil {
consensus . getLogger ( ) . Warn ( ) . Msgf ( "[onViewChange] nilBitmap setkey failed for key at index %d" , i )
continue
}
consensus . nilSigs [ recvMsg . ViewID ] [ key . Pub . Bytes . Hex ( ) ] = key . Pri . SignHash ( NIL )
}
}
}
// add self m3 type message signature and bitmap
_ , ok3 := consensus . viewIDSigs [ recvMsg . ViewID ] [ newLeaderKey . Bytes . Hex ( ) ]
if ! ok3 {
viewIDBytes := make ( [ ] byte , 8 )
binary . LittleEndian . PutUint64 ( viewIDBytes , recvMsg . ViewID )
for i , key := range consensus . priKey {
if err := consensus . viewIDBitmap [ recvMsg . ViewID ] . SetKey ( key . Pub . Bytes , true ) ; err != nil {
consensus . getLogger ( ) . Warn ( ) . Msgf ( "[onViewChange] viewIDBitmap setkey failed for key at index %d" , i )
continue
}
consensus . viewIDSigs [ recvMsg . ViewID ] [ key . Pub . Bytes . Hex ( ) ] = key . Pri . SignHash ( viewIDBytes )
}
}
preparedBlock := & types . Block { }
hasBlock := false
if len ( recvMsg . Payload ) != 0 && len ( recvMsg . Block ) != 0 {
if err := rlp . DecodeBytes ( recvMsg . Block , preparedBlock ) ; err != nil {
consensus . getLogger ( ) . Warn ( ) .
Err ( err ) .
Uint64 ( "MsgBlockNum" , recvMsg . BlockNum ) .
Msg ( "[onViewChange] Unparseable prepared block data" )
return
}
hasBlock = true
}
// m2 type message
if ! hasBlock {
_ , ok := consensus . nilSigs [ recvMsg . ViewID ] [ senderKey . Bytes . Hex ( ) ]
if ok {
consensus . getLogger ( ) . Debug ( ) .
Str ( "validatorPubKey" , senderKey . Bytes . Hex ( ) ) .
Msg ( "[onViewChange] Already Received M2 message from validator" )
return
}
if ! recvMsg . ViewchangeSig . VerifyHash ( senderKey . Object , NIL ) {
consensus . getLogger ( ) . Warn ( ) . Msg ( "[onViewChange] Failed To Verify Signature For M2 Type Viewchange Message" )
return
}
consensus . getLogger ( ) . Info ( ) .
Str ( "validatorPubKey" , senderKey . Bytes . Hex ( ) ) .
Msg ( "[onViewChange] Add M2 (NIL) type message" )
consensus . nilSigs [ recvMsg . ViewID ] [ senderKey . Bytes . Hex ( ) ] = recvMsg . ViewchangeSig
consensus . nilBitmap [ recvMsg . ViewID ] . SetKey ( senderKey . Bytes , true ) // Set the bitmap indicating that this validator signed.
} else { // m1 type message
if consensus . BlockVerifier ( preparedBlock ) ; err != nil {
consensus . getLogger ( ) . Error ( ) . Err ( err ) . Msg ( "[onViewChange] Prepared block verification failed" )
return
}
_ , ok := consensus . bhpSigs [ recvMsg . ViewID ] [ senderKey . Bytes . Hex ( ) ]
if ok {
consensus . getLogger ( ) . Debug ( ) .
Str ( "validatorPubKey" , senderKey . Bytes . Hex ( ) ) .
Msg ( "[onViewChange] Already Received M1 Message From the Validator" )
return
}
if ! recvMsg . ViewchangeSig . VerifyHash ( senderKey . Object , recvMsg . Payload ) {
consensus . getLogger ( ) . Warn ( ) . Msg ( "[onViewChange] Failed to Verify Signature for M1 Type Viewchange Message" )
return
}
// first time receive m1 type message, need verify validity of prepared message
if len ( consensus . m1Payload ) == 0 || ! bytes . Equal ( consensus . m1Payload , recvMsg . Payload ) {
if len ( recvMsg . Payload ) <= 32 {
consensus . getLogger ( ) . Warn ( ) .
Int ( "len" , len ( recvMsg . Payload ) ) .
Msg ( "[onViewChange] M1 RecvMsg Payload Not Enough Length" )
return
}
blockHash := recvMsg . Payload [ : 32 ]
aggSig , mask , err := consensus . ReadSignatureBitmapPayload ( recvMsg . Payload , 32 )
if err != nil {
consensus . getLogger ( ) . Error ( ) . Err ( err ) . Msg ( "[onViewChange] M1 RecvMsg Payload Read Error" )
return
}
if ! consensus . Decider . IsQuorumAchievedByMask ( mask ) {
consensus . getLogger ( ) . Warn ( ) .
Msgf ( "[onViewChange] Quorum Not achieved" )
return
}
// Verify the multi-sig for prepare phase
if ! aggSig . VerifyHash ( mask . AggregatePublic , blockHash [ : ] ) {
consensus . getLogger ( ) . Warn ( ) .
Hex ( "blockHash" , blockHash ) .
Msg ( "[onViewChange] failed to verify multi signature for m1 prepared payload" )
return
}
// if m1Payload is empty, we just add one
if len ( consensus . m1Payload ) == 0 {
consensus . m1Payload = append ( recvMsg . Payload [ : 0 : 0 ] , recvMsg . Payload ... )
// create prepared message for new leader
preparedMsg := FBFTMessage {
MessageType : msg_pb . MessageType_PREPARED ,
ViewID : recvMsg . ViewID ,
BlockNum : recvMsg . BlockNum ,
}
preparedMsg . BlockHash = common . Hash { }
copy ( preparedMsg . BlockHash [ : ] , recvMsg . Payload [ : 32 ] )
preparedMsg . Payload = make ( [ ] byte , len ( recvMsg . Payload ) - 32 )
copy ( preparedMsg . Payload [ : ] , recvMsg . Payload [ 32 : ] )
preparedMsg . SenderPubkeys = [ ] * bls . PublicKeyWrapper { newLeaderKey }
consensus . getLogger ( ) . Info ( ) . Msg ( "[onViewChange] New Leader Prepared Message Added" )
consensus . FBFTLog . AddMessage ( & preparedMsg )
consensus . FBFTLog . AddBlock ( preparedBlock )
}
}
consensus . getLogger ( ) . Info ( ) .
Str ( "validatorPubKey" , senderKey . Bytes . Hex ( ) ) .
Msg ( "[onViewChange] Add M1 (prepared) type message" )
consensus . bhpSigs [ recvMsg . ViewID ] [ senderKey . Bytes . Hex ( ) ] = recvMsg . ViewchangeSig
consensus . bhpBitmap [ recvMsg . ViewID ] . SetKey ( senderKey . Bytes , true ) // Set the bitmap indicating that this validator signed.
}
// check and add viewID (m3 type) message signature
if _ , ok := consensus . viewIDSigs [ recvMsg . ViewID ] [ senderKey . Bytes . Hex ( ) ] ; ok {
consensus . getLogger ( ) . Debug ( ) .
Str ( "validatorPubKey" , senderKey . Bytes . Hex ( ) ) .
Msg ( "[onViewChange] Already Received M3(ViewID) message from the validator" )
members := consensus . Decider . Participants ( )
consensus . vc . AddViewIDKeyIfNotExist ( recvMsg . ViewID , members )
// do it once only per viewID/Leader
if err := consensus . vc . InitPayload ( consensus . FBFTLog ,
recvMsg . ViewID ,
recvMsg . BlockNum ,
consensus . priKey ) ; err != nil {
consensus . getLogger ( ) . Error ( ) . Err ( err ) . Msg ( "Init Payload Error" )
return
}
viewIDHash := make ( [ ] byte , 8 )
binary . LittleEndian . PutUint64 ( viewIDHash , recvMsg . ViewID )
if ! recvMsg . ViewidSig . VerifyHash ( senderKey . Object , viewIDHash ) {
consensus . getLogger ( ) . Warn ( ) .
Uint64 ( "MsgViewID" , recvMsg . ViewID ) .
Msg ( "[onViewChange] Failed to Verify M3 Message Signature" )
err = consensus . vc . ProcessViewChangeMsg ( consensus . FBFTLog , consensus . Decider , recvMsg )
if err != nil {
consensus . getLogger ( ) . Error ( ) . Err ( err ) .
Uint64 ( "viewID" , recvMsg . ViewID ) .
Uint64 ( "blockNum" , recvMsg . BlockNum ) .
Str ( "msgSender" , senderKey . Bytes . Hex ( ) ) .
Msg ( "Verify View Change Message Error" )
return
}
consensus . getLogger ( ) . Info ( ) .
Str ( "validatorPubKey" , senderKey . Bytes . Hex ( ) ) .
Msg ( "[onViewChange] Add M3 (ViewID) type message" )
consensus . viewIDSigs [ recvMsg . ViewID ] [ senderKey . Bytes . Hex ( ) ] = recvMsg . ViewidSig
// Set the bitmap indicating that this validator signed.
consensus . viewIDBitmap [ recvMsg . ViewID ] . SetKey ( senderKey . Bytes , true )
consensus . getLogger ( ) . Info ( ) .
Int ( "have" , len ( consensus . viewIDSigs [ recvMsg . ViewID ] ) ) .
Int64 ( "total" , consensus . Decider . ParticipantsCount ( ) ) .
Msg ( "[onViewChange]" )
// received enough view change messages, change state to normal consensus
if consensus . Decider . IsQuorumAchievedByMask ( consensus . viewIDBitmap [ recvMsg . ViewID ] ) {
if consensus . Decider . IsQuorumAchievedByMask ( consensus . vc . GetViewIDBitmap ( recvMsg . ViewID ) ) {
consensus . getLogger ( ) . Info ( ) . Msg ( "[onViewChange] View Change Message Quorum Reached" )
consensus . current . SetMode ( Normal )
consensus . LeaderPubKey = newLeaderKey
consensus . ResetState ( )
if len ( consensus . m1Payload ) == 0 {
if consensus . vc . IsM1PayloadEmpty ( ) {
// TODO(Chao): explain why ReadySignal is sent only in this case but not the other case.
// Make sure the newly proposed block have the correct view ID
consensus . SetCurBlockViewID ( recvMsg . ViewID )
@ -430,13 +240,10 @@ func (consensus *Consensus) onViewChange(msg *msg_pb.Message) {
consensus . ReadySignal <- struct { } { }
} ( )
} else {
consensus . getLogger ( ) . Info ( ) .
Str ( "From" , consensus . phase . String ( ) ) .
Str ( "To" , FBFTCommit . String ( ) ) .
Msg ( "[OnViewChange] Switching phase" )
consensus . switchPhase ( FBFTCommit , true )
copy ( consensus . blockHash [ : ] , consensus . m1Payload [ : 32 ] )
aggSig , mask , err := consensus . ReadSignatureBitmapPayload ( consensus . m1Payload , 32 )
consensus . switchPhase ( "onViewChange" , FBFTCommit )
payload := consensus . vc . GetM1Payload ( )
copy ( consensus . blockHash [ : ] , payload [ : 32 ] )
aggSig , mask , err := consensus . ReadSignatureBitmapPayload ( payload , 32 )
if err != nil {
consensus . getLogger ( ) . Error ( ) . Err ( err ) .
@ -456,7 +263,7 @@ func (consensus *Consensus) onViewChange(msg *msg_pb.Message) {
block . Epoch ( ) , block . Hash ( ) , block . NumberU64 ( ) , block . Header ( ) . ViewID ( ) . Uint64 ( ) )
for i , key := range consensus . priKey {
if err := consensus . commitBitmap . SetKey ( key . Pub . Bytes , true ) ; err != nil {
consensus . getLogger ( ) . Warn ( ) .
consensus . getLogger ( ) . Warn ( ) . Err ( err ) .
Msgf ( "[OnViewChange] New Leader commit bitmap set failed for key at index %d" , i )
continue
}
@ -469,22 +276,18 @@ func (consensus *Consensus) onViewChange(msg *msg_pb.Message) {
block . NumberU64 ( ) ,
block . Header ( ) . ViewID ( ) . Uint64 ( ) ,
) ; err != nil {
consensus . getLogger ( ) . Warn ( ) . Msg ( "submit vote on viewchange commit failed" )
consensus . getLogger ( ) . Warn ( ) . Err ( err ) . Msg ( "submit vote on viewchange commit failed" )
return
}
}
}
consensus . SetViewChanging ID ( recvMsg . ViewID )
consensus . SetViewIDs ( recvMsg . ViewID )
msgToSend := consensus . constructNewViewMessage (
recvMsg . ViewID , newLeaderPriKey ,
)
consensus . getLogger ( ) . Warn ( ) .
Int ( "payloadSize" , len ( consensus . m1Payload ) ) .
Hex ( "M1Payload" , consensus . m1Payload ) .
Msg ( "[onViewChange] Sent NewView Message" )
if err := consensus . msgSender . SendWithRetry (
consensus . blockNum ,
msg_pb . MessageType_NEWVIEW ,
@ -495,25 +298,42 @@ func (consensus *Consensus) onViewChange(msg *msg_pb.Message) {
consensus . getLogger ( ) . Err ( err ) .
Msg ( "could not send out the NEWVIEW message" )
}
consensus . getLogger ( ) . Info ( ) .
Str ( "myKey" , newLeaderKey . Bytes . Hex ( ) ) .
Hex ( "M1Payload" , consensus . vc . GetM1Payload ( ) ) .
Msg ( "[onViewChange] Sent NewView Messge" )
consensus . SetCurBlockViewID ( recvMsg . ViewID )
consensus . ResetViewChangeState ( )
consensus . consensusTimeout [ timeoutViewChange ] . Stop ( )
consensus . consensusTimeout [ timeoutConsensus ] . Start ( )
consensus . getLogger ( ) . Info ( ) . Str ( "myKey" , newLeaderKey . Bytes . Hex ( ) ) . Msg ( "[onViewChange] I am the New Leader" )
consensus . getLogger ( ) . Info ( ) .
Str ( "myKey" , newLeaderKey . Bytes . Hex ( ) ) .
Msg ( "[onViewChange] I am the New Leader" )
}
}
// TODO: move to consensus_leader.go later
// onNewView is called when validators received newView message from the new leader
// the validator needs to check the m3bitmap to see if the quorum is reached
// If the new view message contains payload (block), and at least one m1 message was
// collected by the new leader (m3count > m2count), the validator will create a new
// prepared message from the payload and commit it to the block
// Or the validator will enter announce phase to wait for the new block proposed
// from the new leader
func ( consensus * Consensus ) onNewView ( msg * msg_pb . Message ) {
consensus . getLogger ( ) . Info ( ) . Msg ( "[onNewView] Received NewView Message" )
recvMsg , err := consensus . ParseNewViewMessage ( msg )
members := consensus . Decider . Participants ( )
recvMsg , err := ParseNewViewMessage ( msg , members )
if err != nil {
consensus . getLogger ( ) . Warn ( ) . Err ( err ) . Msg ( "[onNewView] Unable to Parse NewView Message" )
return
}
if ! consensus . onNewViewSanityCheck ( recvMsg ) {
// change view and leaderKey to keep in sync with network
if consensus . blockNum != recvMsg . BlockNum {
consensus . getLogger ( ) . Warn ( ) .
Uint64 ( "MsgBlockNum" , recvMsg . BlockNum ) .
Uint64 ( "myBlockNum" , consensus . blockNum ) .
Msg ( "[onNewView] Invalid block number" )
return
}
@ -523,82 +343,26 @@ func (consensus *Consensus) onNewView(msg *msg_pb.Message) {
}
senderKey := recvMsg . SenderPubkeys [ 0 ]
consensus . vcLock . Lock ( )
defer consensus . vcLock . Unlock ( )
if recvMsg . M3AggSig == nil || recvMsg . M3Bitmap == nil {
consensus . getLogger ( ) . Error ( ) . Msg ( "[onNewView] M3AggSig or M3Bitmap is nil" )
if ! consensus . onNewViewSanityCheck ( recvMsg ) {
return
}
m3Sig := recvMsg . M3AggSig
m3Mask := recvMsg . M3Bitmap
viewIDBytes := make ( [ ] byte , 8 )
binary . LittleEndian . PutUint64 ( viewIDBytes , recvMsg . ViewID )
if ! consensus . Decider . IsQuorumAchievedByMask ( m3Mask ) {
consensus . getLogger ( ) . Warn ( ) .
Msgf ( "[onNewView] Quorum Not achieved" )
preparedBlock , err := consensus . vc . VerifyNewViewMsg ( recvMsg )
if err != nil {
consensus . getLogger ( ) . Warn ( ) . Err ( err ) . Msg ( "[onNewView] Verify New View Msg Failed" )
return
}
if ! m3Sig . VerifyHash ( m3Mask . AggregatePublic , viewIDBytes ) {
m3Mask := recvMsg . M3Bitmap
if ! consensus . Decider . IsQuorumAchievedByMask ( m3Mask ) {
consensus . getLogger ( ) . Warn ( ) .
Str ( "m3Sig" , m3Sig . SerializeToHexStr ( ) ) .
Hex ( "m3Mask" , m3Mask . Bitmap ) .
Uint64 ( "MsgViewID" , recvMsg . ViewID ) .
Msg ( "[onNewView] Unable to Verify Aggregated Signature of M3 (ViewID) payload" )
Msgf ( "[onNewView] Quorum Not achieved" )
return
}
m2Mask := recvMsg . M2Bitmap
if recvMsg . M2AggSig != nil {
consensus . getLogger ( ) . Info ( ) . Msg ( "[onNewView] M2AggSig (NIL) is Not Empty" )
m2Sig := recvMsg . M2AggSig
if ! m2Sig . VerifyHash ( m2Mask . AggregatePublic , NIL ) {
consensus . getLogger ( ) . Warn ( ) .
Msg ( "[onNewView] Unable to Verify Aggregated Signature of M2 (NIL) payload" )
return
}
}
// check when M3 sigs > M2 sigs, then M1 (recvMsg.Payload) should not be empty
preparedBlock := & types . Block { }
hasBlock := false
if len ( recvMsg . Payload ) != 0 && len ( recvMsg . Block ) != 0 {
if err := rlp . DecodeBytes ( recvMsg . Block , preparedBlock ) ; err != nil {
consensus . getLogger ( ) . Warn ( ) .
Err ( err ) .
Uint64 ( "MsgBlockNum" , recvMsg . BlockNum ) .
Msg ( "[onNewView] Unparseable prepared block data" )
return
}
blockHash := recvMsg . Payload [ : 32 ]
preparedBlockHash := preparedBlock . Hash ( )
if ! bytes . Equal ( preparedBlockHash [ : ] , blockHash ) {
consensus . getLogger ( ) . Warn ( ) .
Err ( err ) .
Str ( "blockHash" , preparedBlock . Hash ( ) . Hex ( ) ) .
Str ( "payloadBlockHash" , hex . EncodeToString ( blockHash ) ) .
Msg ( "[onNewView] Prepared block hash doesn't match msg block hash." )
return
}
hasBlock = true
if consensus . BlockVerifier ( preparedBlock ) ; err != nil {
consensus . getLogger ( ) . Error ( ) . Err ( err ) . Msg ( "[onNewView] Prepared block verification failed" )
return
}
}
if m2Mask == nil || m2Mask . Bitmap == nil ||
( m2Mask != nil && m2Mask . Bitmap != nil &&
utils . CountOneBits ( m3Mask . Bitmap ) > utils . CountOneBits ( m2Mask . Bitmap ) ) {
if len ( recvMsg . Payload ) <= 32 {
consensus . getLogger ( ) . Info ( ) .
Msg ( "[onNewView] M1 (prepared) Type Payload Not Have Enough Length" )
return
}
// m1 is not empty, check it's valid
blockHash := recvMsg . Payload [ : 32 ]
aggSig , mask , err := consensus . ReadSignatureBitmapPayload ( recvMsg . Payload , 32 )
@ -625,30 +389,27 @@ func (consensus *Consensus) onNewView(msg *msg_pb.Message) {
copy ( preparedMsg . BlockHash [ : ] , blockHash [ : ] )
preparedMsg . Payload = make ( [ ] byte , len ( recvMsg . Payload ) - 32 )
copy ( preparedMsg . Payload [ : ] , recvMsg . Payload [ 32 : ] )
preparedMsg . SenderPubkeys = [ ] * bls . PublicKeyWrapper { senderKey }
consensus . FBFTLog . AddMessage ( & preparedMsg )
if hasBlock {
if preparedBlock != nil {
consensus . FBFTLog . AddBlock ( preparedBlock )
}
}
if ! consensus . IsViewChangingMode ( ) {
consensus . getLogger ( ) . Info ( ) . Msg ( "Not in ViewChanging Mode." )
return
}
// newView message verified success, override my state
consensus . SetViewIDs ( recvMsg . ViewID )
consensus . LeaderPubKey = senderKey
consensus . ResetViewChangeState ( )
// change view and leaderKey to keep in sync with network
if consensus . blockNum != recvMsg . BlockNum {
consensus . getLogger ( ) . Info ( ) .
Str ( "newLeaderKey" , consensus . LeaderPubKey . Bytes . Hex ( ) ) .
Uint64 ( "MsgBlockNum" , recvMsg . BlockNum ) .
Msg ( "[onNewView] New Leader Changed" )
return
}
// NewView message is verified, change state to normal consensus
if hasBlock {
if preparedBlock != nil {
// Construct and send the commit message
commitPayload := signature . ConstructCommitPayload ( consensus . ChainReader ,
preparedBlock . Epoch ( ) , preparedBlock . Hash ( ) , preparedBlock . NumberU64 ( ) , preparedBlock . Header ( ) . ViewID ( ) . Uint64 ( ) )
@ -666,7 +427,7 @@ func (consensus *Consensus) onNewView(msg *msg_pb.Message) {
)
if err != nil {
consensus . getLogger ( ) . Err ( err ) . Msg ( "could not create commit message" )
return
continue
}
consensus . getLogger ( ) . Info ( ) . Msg ( "onNewView === commit" )
@ -675,11 +436,7 @@ func (consensus *Consensus) onNewView(msg *msg_pb.Message) {
p2p . ConstructMessage ( p2pMsg . Bytes ) ,
)
}
consensus . getLogger ( ) . Info ( ) .
Str ( "From" , consensus . phase . String ( ) ) .
Str ( "To" , FBFTCommit . String ( ) ) .
Msg ( "[OnViewChange] Switching phase" )
consensus . switchPhase ( FBFTCommit , true )
consensus . switchPhase ( "onNewView" , FBFTCommit )
} else {
consensus . ResetState ( )
consensus . getLogger ( ) . Info ( ) . Msg ( "onNewView === announce" )
@ -687,8 +444,16 @@ func (consensus *Consensus) onNewView(msg *msg_pb.Message) {
consensus . getLogger ( ) . Info ( ) .
Str ( "newLeaderKey" , consensus . LeaderPubKey . Bytes . Hex ( ) ) .
Msg ( "new leader changed" )
consensus . getLogger ( ) . Info ( ) .
Msg ( "validator start consensus timer and stop view change timer" )
consensus . consensusTimeout [ timeoutConsensus ] . Start ( )
consensus . consensusTimeout [ timeoutViewChange ] . Stop ( )
}
// ResetViewChangeState resets the view change structure
func ( consensus * Consensus ) ResetViewChangeState ( ) {
consensus . getLogger ( ) . Info ( ) .
Str ( "Phase" , consensus . phase . String ( ) ) .
Msg ( "[ResetViewChangeState] Resetting view change state" )
consensus . current . SetMode ( Normal )
consensus . vc . Reset ( )
consensus . Decider . ResetViewChangeVotes ( )
}