You can not select more than 25 topics Topics must start with a letter or number, can include dashes ('-') and can be up to 35 characters long.

1459 lines
44 KiB

  1. package consensus
  2. import (
  3. "fmt"
  4. "reflect"
  5. "sync"
  6. "time"
  7. "github.com/pkg/errors"
  8. amino "github.com/tendermint/go-amino"
  9. cmn "github.com/tendermint/tmlibs/common"
  10. "github.com/tendermint/tmlibs/log"
  11. cstypes "github.com/tendermint/tendermint/consensus/types"
  12. tmevents "github.com/tendermint/tendermint/libs/events"
  13. "github.com/tendermint/tendermint/p2p"
  14. sm "github.com/tendermint/tendermint/state"
  15. "github.com/tendermint/tendermint/types"
  16. )
  17. const (
  18. StateChannel = byte(0x20)
  19. DataChannel = byte(0x21)
  20. VoteChannel = byte(0x22)
  21. VoteSetBitsChannel = byte(0x23)
  22. maxMsgSize = 1048576 // 1MB; NOTE/TODO: keep in sync with types.PartSet sizes.
  23. blocksToContributeToBecomeGoodPeer = 10000
  24. )
  25. //-----------------------------------------------------------------------------
  26. // ConsensusReactor defines a reactor for the consensus service.
  27. type ConsensusReactor struct {
  28. p2p.BaseReactor // BaseService + p2p.Switch
  29. conS *ConsensusState
  30. mtx sync.RWMutex
  31. fastSync bool
  32. eventBus *types.EventBus
  33. }
  34. // NewConsensusReactor returns a new ConsensusReactor with the given
  35. // consensusState.
  36. func NewConsensusReactor(consensusState *ConsensusState, fastSync bool) *ConsensusReactor {
  37. conR := &ConsensusReactor{
  38. conS: consensusState,
  39. fastSync: fastSync,
  40. }
  41. conR.BaseReactor = *p2p.NewBaseReactor("ConsensusReactor", conR)
  42. return conR
  43. }
  44. // OnStart implements BaseService by subscribing to events, which later will be
  45. // broadcasted to other peers and starting state if we're not in fast sync.
  46. func (conR *ConsensusReactor) OnStart() error {
  47. conR.Logger.Info("ConsensusReactor ", "fastSync", conR.FastSync())
  48. if err := conR.BaseReactor.OnStart(); err != nil {
  49. return err
  50. }
  51. conR.subscribeToBroadcastEvents()
  52. if !conR.FastSync() {
  53. err := conR.conS.Start()
  54. if err != nil {
  55. return err
  56. }
  57. }
  58. return nil
  59. }
  60. // OnStop implements BaseService by unsubscribing from events and stopping
  61. // state.
  62. func (conR *ConsensusReactor) OnStop() {
  63. conR.BaseReactor.OnStop()
  64. conR.unsubscribeFromBroadcastEvents()
  65. conR.conS.Stop()
  66. }
  67. // SwitchToConsensus switches from fast_sync mode to consensus mode.
  68. // It resets the state, turns off fast_sync, and starts the consensus state-machine
  69. func (conR *ConsensusReactor) SwitchToConsensus(state sm.State, blocksSynced int) {
  70. conR.Logger.Info("SwitchToConsensus")
  71. conR.conS.reconstructLastCommit(state)
  72. // NOTE: The line below causes broadcastNewRoundStepRoutine() to
  73. // broadcast a NewRoundStepMessage.
  74. conR.conS.updateToState(state)
  75. conR.mtx.Lock()
  76. conR.fastSync = false
  77. conR.mtx.Unlock()
  78. if blocksSynced > 0 {
  79. // dont bother with the WAL if we fast synced
  80. conR.conS.doWALCatchup = false
  81. }
  82. err := conR.conS.Start()
  83. if err != nil {
  84. conR.Logger.Error("Error starting conS", "err", err)
  85. return
  86. }
  87. }
  88. // GetChannels implements Reactor
  89. func (conR *ConsensusReactor) GetChannels() []*p2p.ChannelDescriptor {
  90. // TODO optimize
  91. return []*p2p.ChannelDescriptor{
  92. {
  93. ID: StateChannel,
  94. Priority: 5,
  95. SendQueueCapacity: 100,
  96. RecvMessageCapacity: maxMsgSize,
  97. },
  98. {
  99. ID: DataChannel, // maybe split between gossiping current block and catchup stuff
  100. Priority: 10, // once we gossip the whole block there's nothing left to send until next height or round
  101. SendQueueCapacity: 100,
  102. RecvBufferCapacity: 50 * 4096,
  103. RecvMessageCapacity: maxMsgSize,
  104. },
  105. {
  106. ID: VoteChannel,
  107. Priority: 5,
  108. SendQueueCapacity: 100,
  109. RecvBufferCapacity: 100 * 100,
  110. RecvMessageCapacity: maxMsgSize,
  111. },
  112. {
  113. ID: VoteSetBitsChannel,
  114. Priority: 1,
  115. SendQueueCapacity: 2,
  116. RecvBufferCapacity: 1024,
  117. RecvMessageCapacity: maxMsgSize,
  118. },
  119. }
  120. }
  121. // AddPeer implements Reactor
  122. func (conR *ConsensusReactor) AddPeer(peer p2p.Peer) {
  123. if !conR.IsRunning() {
  124. return
  125. }
  126. // Create peerState for peer
  127. peerState := NewPeerState(peer).SetLogger(conR.Logger)
  128. peer.Set(types.PeerStateKey, peerState)
  129. // Begin routines for this peer.
  130. go conR.gossipDataRoutine(peer, peerState)
  131. go conR.gossipVotesRoutine(peer, peerState)
  132. go conR.queryMaj23Routine(peer, peerState)
  133. // Send our state to peer.
  134. // If we're fast_syncing, broadcast a RoundStepMessage later upon SwitchToConsensus().
  135. if !conR.FastSync() {
  136. conR.sendNewRoundStepMessages(peer)
  137. }
  138. }
  139. // RemovePeer implements Reactor
  140. func (conR *ConsensusReactor) RemovePeer(peer p2p.Peer, reason interface{}) {
  141. if !conR.IsRunning() {
  142. return
  143. }
  144. // TODO
  145. //peer.Get(PeerStateKey).(*PeerState).Disconnect()
  146. }
  147. // Receive implements Reactor
  148. // NOTE: We process these messages even when we're fast_syncing.
  149. // Messages affect either a peer state or the consensus state.
  150. // Peer state updates can happen in parallel, but processing of
  151. // proposals, block parts, and votes are ordered by the receiveRoutine
  152. // NOTE: blocks on consensus state for proposals, block parts, and votes
  153. func (conR *ConsensusReactor) Receive(chID byte, src p2p.Peer, msgBytes []byte) {
  154. if !conR.IsRunning() {
  155. conR.Logger.Debug("Receive", "src", src, "chId", chID, "bytes", msgBytes)
  156. return
  157. }
  158. msg, err := DecodeMessage(msgBytes)
  159. if err != nil {
  160. conR.Logger.Error("Error decoding message", "src", src, "chId", chID, "msg", msg, "err", err, "bytes", msgBytes)
  161. conR.Switch.StopPeerForError(src, err)
  162. return
  163. }
  164. conR.Logger.Debug("Receive", "src", src, "chId", chID, "msg", msg)
  165. // Get peer states
  166. ps := src.Get(types.PeerStateKey).(*PeerState)
  167. switch chID {
  168. case StateChannel:
  169. switch msg := msg.(type) {
  170. case *NewRoundStepMessage:
  171. ps.ApplyNewRoundStepMessage(msg)
  172. case *CommitStepMessage:
  173. ps.ApplyCommitStepMessage(msg)
  174. case *HasVoteMessage:
  175. ps.ApplyHasVoteMessage(msg)
  176. case *VoteSetMaj23Message:
  177. cs := conR.conS
  178. cs.mtx.Lock()
  179. height, votes := cs.Height, cs.Votes
  180. cs.mtx.Unlock()
  181. if height != msg.Height {
  182. return
  183. }
  184. // Peer claims to have a maj23 for some BlockID at H,R,S,
  185. err := votes.SetPeerMaj23(msg.Round, msg.Type, ps.peer.ID(), msg.BlockID)
  186. if err != nil {
  187. conR.Switch.StopPeerForError(src, err)
  188. return
  189. }
  190. // Respond with a VoteSetBitsMessage showing which votes we have.
  191. // (and consequently shows which we don't have)
  192. var ourVotes *cmn.BitArray
  193. switch msg.Type {
  194. case types.VoteTypePrevote:
  195. ourVotes = votes.Prevotes(msg.Round).BitArrayByBlockID(msg.BlockID)
  196. case types.VoteTypePrecommit:
  197. ourVotes = votes.Precommits(msg.Round).BitArrayByBlockID(msg.BlockID)
  198. default:
  199. conR.Logger.Error("Bad VoteSetBitsMessage field Type")
  200. return
  201. }
  202. src.TrySend(VoteSetBitsChannel, cdc.MustMarshalBinaryBare(&VoteSetBitsMessage{
  203. Height: msg.Height,
  204. Round: msg.Round,
  205. Type: msg.Type,
  206. BlockID: msg.BlockID,
  207. Votes: ourVotes,
  208. }))
  209. case *ProposalHeartbeatMessage:
  210. hb := msg.Heartbeat
  211. conR.Logger.Debug("Received proposal heartbeat message",
  212. "height", hb.Height, "round", hb.Round, "sequence", hb.Sequence,
  213. "valIdx", hb.ValidatorIndex, "valAddr", hb.ValidatorAddress)
  214. default:
  215. conR.Logger.Error(cmn.Fmt("Unknown message type %v", reflect.TypeOf(msg)))
  216. }
  217. case DataChannel:
  218. if conR.FastSync() {
  219. conR.Logger.Info("Ignoring message received during fastSync", "msg", msg)
  220. return
  221. }
  222. switch msg := msg.(type) {
  223. case *ProposalMessage:
  224. ps.SetHasProposal(msg.Proposal)
  225. conR.conS.peerMsgQueue <- msgInfo{msg, src.ID()}
  226. case *ProposalPOLMessage:
  227. ps.ApplyProposalPOLMessage(msg)
  228. case *BlockPartMessage:
  229. ps.SetHasProposalBlockPart(msg.Height, msg.Round, msg.Part.Index)
  230. if numBlocks := ps.RecordBlockPart(msg); numBlocks%blocksToContributeToBecomeGoodPeer == 0 {
  231. conR.Switch.MarkPeerAsGood(src)
  232. }
  233. conR.conS.peerMsgQueue <- msgInfo{msg, src.ID()}
  234. default:
  235. conR.Logger.Error(cmn.Fmt("Unknown message type %v", reflect.TypeOf(msg)))
  236. }
  237. case VoteChannel:
  238. if conR.FastSync() {
  239. conR.Logger.Info("Ignoring message received during fastSync", "msg", msg)
  240. return
  241. }
  242. switch msg := msg.(type) {
  243. case *VoteMessage:
  244. cs := conR.conS
  245. cs.mtx.Lock()
  246. height, valSize, lastCommitSize := cs.Height, cs.Validators.Size(), cs.LastCommit.Size()
  247. cs.mtx.Unlock()
  248. ps.EnsureVoteBitArrays(height, valSize)
  249. ps.EnsureVoteBitArrays(height-1, lastCommitSize)
  250. ps.SetHasVote(msg.Vote)
  251. if blocks := ps.RecordVote(msg.Vote); blocks%blocksToContributeToBecomeGoodPeer == 0 {
  252. conR.Switch.MarkPeerAsGood(src)
  253. }
  254. cs.peerMsgQueue <- msgInfo{msg, src.ID()}
  255. default:
  256. // don't punish (leave room for soft upgrades)
  257. conR.Logger.Error(cmn.Fmt("Unknown message type %v", reflect.TypeOf(msg)))
  258. }
  259. case VoteSetBitsChannel:
  260. if conR.FastSync() {
  261. conR.Logger.Info("Ignoring message received during fastSync", "msg", msg)
  262. return
  263. }
  264. switch msg := msg.(type) {
  265. case *VoteSetBitsMessage:
  266. cs := conR.conS
  267. cs.mtx.Lock()
  268. height, votes := cs.Height, cs.Votes
  269. cs.mtx.Unlock()
  270. if height == msg.Height {
  271. var ourVotes *cmn.BitArray
  272. switch msg.Type {
  273. case types.VoteTypePrevote:
  274. ourVotes = votes.Prevotes(msg.Round).BitArrayByBlockID(msg.BlockID)
  275. case types.VoteTypePrecommit:
  276. ourVotes = votes.Precommits(msg.Round).BitArrayByBlockID(msg.BlockID)
  277. default:
  278. conR.Logger.Error("Bad VoteSetBitsMessage field Type")
  279. return
  280. }
  281. ps.ApplyVoteSetBitsMessage(msg, ourVotes)
  282. } else {
  283. ps.ApplyVoteSetBitsMessage(msg, nil)
  284. }
  285. default:
  286. // don't punish (leave room for soft upgrades)
  287. conR.Logger.Error(cmn.Fmt("Unknown message type %v", reflect.TypeOf(msg)))
  288. }
  289. default:
  290. conR.Logger.Error(cmn.Fmt("Unknown chId %X", chID))
  291. }
  292. if err != nil {
  293. conR.Logger.Error("Error in Receive()", "err", err)
  294. }
  295. }
  296. // SetEventBus sets event bus.
  297. func (conR *ConsensusReactor) SetEventBus(b *types.EventBus) {
  298. conR.eventBus = b
  299. conR.conS.SetEventBus(b)
  300. }
  301. // FastSync returns whether the consensus reactor is in fast-sync mode.
  302. func (conR *ConsensusReactor) FastSync() bool {
  303. conR.mtx.RLock()
  304. defer conR.mtx.RUnlock()
  305. return conR.fastSync
  306. }
  307. //--------------------------------------
  308. // subscribeToBroadcastEvents subscribes for new round steps, votes and
  309. // proposal heartbeats using internal pubsub defined on state to broadcast
  310. // them to peers upon receiving.
  311. func (conR *ConsensusReactor) subscribeToBroadcastEvents() {
  312. const subscriber = "consensus-reactor"
  313. conR.conS.evsw.AddListenerForEvent(subscriber, types.EventNewRoundStep,
  314. func(data tmevents.EventData) {
  315. conR.broadcastNewRoundStepMessages(data.(*cstypes.RoundState))
  316. })
  317. conR.conS.evsw.AddListenerForEvent(subscriber, types.EventVote,
  318. func(data tmevents.EventData) {
  319. conR.broadcastHasVoteMessage(data.(*types.Vote))
  320. })
  321. conR.conS.evsw.AddListenerForEvent(subscriber, types.EventProposalHeartbeat,
  322. func(data tmevents.EventData) {
  323. conR.broadcastProposalHeartbeatMessage(data.(*types.Heartbeat))
  324. })
  325. }
  326. func (conR *ConsensusReactor) unsubscribeFromBroadcastEvents() {
  327. const subscriber = "consensus-reactor"
  328. conR.conS.evsw.RemoveListener(subscriber)
  329. }
  330. func (conR *ConsensusReactor) broadcastProposalHeartbeatMessage(hb *types.Heartbeat) {
  331. conR.Logger.Debug("Broadcasting proposal heartbeat message",
  332. "height", hb.Height, "round", hb.Round, "sequence", hb.Sequence)
  333. msg := &ProposalHeartbeatMessage{hb}
  334. conR.Switch.Broadcast(StateChannel, cdc.MustMarshalBinaryBare(msg))
  335. }
  336. func (conR *ConsensusReactor) broadcastNewRoundStepMessages(rs *cstypes.RoundState) {
  337. nrsMsg, csMsg := makeRoundStepMessages(rs)
  338. if nrsMsg != nil {
  339. conR.Switch.Broadcast(StateChannel, cdc.MustMarshalBinaryBare(nrsMsg))
  340. }
  341. if csMsg != nil {
  342. conR.Switch.Broadcast(StateChannel, cdc.MustMarshalBinaryBare(csMsg))
  343. }
  344. }
  345. // Broadcasts HasVoteMessage to peers that care.
  346. func (conR *ConsensusReactor) broadcastHasVoteMessage(vote *types.Vote) {
  347. msg := &HasVoteMessage{
  348. Height: vote.Height,
  349. Round: vote.Round,
  350. Type: vote.Type,
  351. Index: vote.ValidatorIndex,
  352. }
  353. conR.Switch.Broadcast(StateChannel, cdc.MustMarshalBinaryBare(msg))
  354. /*
  355. // TODO: Make this broadcast more selective.
  356. for _, peer := range conR.Switch.Peers().List() {
  357. ps := peer.Get(PeerStateKey).(*PeerState)
  358. prs := ps.GetRoundState()
  359. if prs.Height == vote.Height {
  360. // TODO: Also filter on round?
  361. peer.TrySend(StateChannel, struct{ ConsensusMessage }{msg})
  362. } else {
  363. // Height doesn't match
  364. // TODO: check a field, maybe CatchupCommitRound?
  365. // TODO: But that requires changing the struct field comment.
  366. }
  367. }
  368. */
  369. }
  370. func makeRoundStepMessages(rs *cstypes.RoundState) (nrsMsg *NewRoundStepMessage, csMsg *CommitStepMessage) {
  371. nrsMsg = &NewRoundStepMessage{
  372. Height: rs.Height,
  373. Round: rs.Round,
  374. Step: rs.Step,
  375. SecondsSinceStartTime: int(time.Since(rs.StartTime).Seconds()),
  376. LastCommitRound: rs.LastCommit.Round(),
  377. }
  378. if rs.Step == cstypes.RoundStepCommit {
  379. csMsg = &CommitStepMessage{
  380. Height: rs.Height,
  381. BlockPartsHeader: rs.ProposalBlockParts.Header(),
  382. BlockParts: rs.ProposalBlockParts.BitArray(),
  383. }
  384. }
  385. return
  386. }
  387. func (conR *ConsensusReactor) sendNewRoundStepMessages(peer p2p.Peer) {
  388. rs := conR.conS.GetRoundState()
  389. nrsMsg, csMsg := makeRoundStepMessages(rs)
  390. if nrsMsg != nil {
  391. peer.Send(StateChannel, cdc.MustMarshalBinaryBare(nrsMsg))
  392. }
  393. if csMsg != nil {
  394. peer.Send(StateChannel, cdc.MustMarshalBinaryBare(csMsg))
  395. }
  396. }
  397. func (conR *ConsensusReactor) gossipDataRoutine(peer p2p.Peer, ps *PeerState) {
  398. logger := conR.Logger.With("peer", peer)
  399. OUTER_LOOP:
  400. for {
  401. // Manage disconnects from self or peer.
  402. if !peer.IsRunning() || !conR.IsRunning() {
  403. logger.Info("Stopping gossipDataRoutine for peer")
  404. return
  405. }
  406. rs := conR.conS.GetRoundState()
  407. prs := ps.GetRoundState()
  408. // Send proposal Block parts?
  409. if rs.ProposalBlockParts.HasHeader(prs.ProposalBlockPartsHeader) {
  410. if index, ok := rs.ProposalBlockParts.BitArray().Sub(prs.ProposalBlockParts.Copy()).PickRandom(); ok {
  411. part := rs.ProposalBlockParts.GetPart(index)
  412. msg := &BlockPartMessage{
  413. Height: rs.Height, // This tells peer that this part applies to us.
  414. Round: rs.Round, // This tells peer that this part applies to us.
  415. Part: part,
  416. }
  417. logger.Debug("Sending block part", "height", prs.Height, "round", prs.Round)
  418. if peer.Send(DataChannel, cdc.MustMarshalBinaryBare(msg)) {
  419. ps.SetHasProposalBlockPart(prs.Height, prs.Round, index)
  420. }
  421. continue OUTER_LOOP
  422. }
  423. }
  424. // If the peer is on a previous height, help catch up.
  425. if (0 < prs.Height) && (prs.Height < rs.Height) {
  426. heightLogger := logger.With("height", prs.Height)
  427. // if we never received the commit message from the peer, the block parts wont be initialized
  428. if prs.ProposalBlockParts == nil {
  429. blockMeta := conR.conS.blockStore.LoadBlockMeta(prs.Height)
  430. if blockMeta == nil {
  431. cmn.PanicCrisis(cmn.Fmt("Failed to load block %d when blockStore is at %d",
  432. prs.Height, conR.conS.blockStore.Height()))
  433. }
  434. ps.InitProposalBlockParts(blockMeta.BlockID.PartsHeader)
  435. // continue the loop since prs is a copy and not effected by this initialization
  436. continue OUTER_LOOP
  437. }
  438. conR.gossipDataForCatchup(heightLogger, rs, prs, ps, peer)
  439. continue OUTER_LOOP
  440. }
  441. // If height and round don't match, sleep.
  442. if (rs.Height != prs.Height) || (rs.Round != prs.Round) {
  443. //logger.Info("Peer Height|Round mismatch, sleeping", "peerHeight", prs.Height, "peerRound", prs.Round, "peer", peer)
  444. time.Sleep(conR.conS.config.PeerGossipSleep())
  445. continue OUTER_LOOP
  446. }
  447. // By here, height and round match.
  448. // Proposal block parts were already matched and sent if any were wanted.
  449. // (These can match on hash so the round doesn't matter)
  450. // Now consider sending other things, like the Proposal itself.
  451. // Send Proposal && ProposalPOL BitArray?
  452. if rs.Proposal != nil && !prs.Proposal {
  453. // Proposal: share the proposal metadata with peer.
  454. {
  455. msg := &ProposalMessage{Proposal: rs.Proposal}
  456. logger.Debug("Sending proposal", "height", prs.Height, "round", prs.Round)
  457. if peer.Send(DataChannel, cdc.MustMarshalBinaryBare(msg)) {
  458. ps.SetHasProposal(rs.Proposal)
  459. }
  460. }
  461. // ProposalPOL: lets peer know which POL votes we have so far.
  462. // Peer must receive ProposalMessage first.
  463. // rs.Proposal was validated, so rs.Proposal.POLRound <= rs.Round,
  464. // so we definitely have rs.Votes.Prevotes(rs.Proposal.POLRound).
  465. if 0 <= rs.Proposal.POLRound {
  466. msg := &ProposalPOLMessage{
  467. Height: rs.Height,
  468. ProposalPOLRound: rs.Proposal.POLRound,
  469. ProposalPOL: rs.Votes.Prevotes(rs.Proposal.POLRound).BitArray(),
  470. }
  471. logger.Debug("Sending POL", "height", prs.Height, "round", prs.Round)
  472. peer.Send(DataChannel, cdc.MustMarshalBinaryBare(msg))
  473. }
  474. continue OUTER_LOOP
  475. }
  476. // Nothing to do. Sleep.
  477. time.Sleep(conR.conS.config.PeerGossipSleep())
  478. continue OUTER_LOOP
  479. }
  480. }
  481. func (conR *ConsensusReactor) gossipDataForCatchup(logger log.Logger, rs *cstypes.RoundState,
  482. prs *cstypes.PeerRoundState, ps *PeerState, peer p2p.Peer) {
  483. if index, ok := prs.ProposalBlockParts.Not().PickRandom(); ok {
  484. // Ensure that the peer's PartSetHeader is correct
  485. blockMeta := conR.conS.blockStore.LoadBlockMeta(prs.Height)
  486. if blockMeta == nil {
  487. logger.Error("Failed to load block meta",
  488. "ourHeight", rs.Height, "blockstoreHeight", conR.conS.blockStore.Height())
  489. time.Sleep(conR.conS.config.PeerGossipSleep())
  490. return
  491. } else if !blockMeta.BlockID.PartsHeader.Equals(prs.ProposalBlockPartsHeader) {
  492. logger.Info("Peer ProposalBlockPartsHeader mismatch, sleeping",
  493. "blockPartsHeader", blockMeta.BlockID.PartsHeader, "peerBlockPartsHeader", prs.ProposalBlockPartsHeader)
  494. time.Sleep(conR.conS.config.PeerGossipSleep())
  495. return
  496. }
  497. // Load the part
  498. part := conR.conS.blockStore.LoadBlockPart(prs.Height, index)
  499. if part == nil {
  500. logger.Error("Could not load part", "index", index,
  501. "blockPartsHeader", blockMeta.BlockID.PartsHeader, "peerBlockPartsHeader", prs.ProposalBlockPartsHeader)
  502. time.Sleep(conR.conS.config.PeerGossipSleep())
  503. return
  504. }
  505. // Send the part
  506. msg := &BlockPartMessage{
  507. Height: prs.Height, // Not our height, so it doesn't matter.
  508. Round: prs.Round, // Not our height, so it doesn't matter.
  509. Part: part,
  510. }
  511. logger.Debug("Sending block part for catchup", "round", prs.Round, "index", index)
  512. if peer.Send(DataChannel, cdc.MustMarshalBinaryBare(msg)) {
  513. ps.SetHasProposalBlockPart(prs.Height, prs.Round, index)
  514. } else {
  515. logger.Debug("Sending block part for catchup failed")
  516. }
  517. return
  518. }
  519. //logger.Info("No parts to send in catch-up, sleeping")
  520. time.Sleep(conR.conS.config.PeerGossipSleep())
  521. }
  522. func (conR *ConsensusReactor) gossipVotesRoutine(peer p2p.Peer, ps *PeerState) {
  523. logger := conR.Logger.With("peer", peer)
  524. // Simple hack to throttle logs upon sleep.
  525. var sleeping = 0
  526. OUTER_LOOP:
  527. for {
  528. // Manage disconnects from self or peer.
  529. if !peer.IsRunning() || !conR.IsRunning() {
  530. logger.Info("Stopping gossipVotesRoutine for peer")
  531. return
  532. }
  533. rs := conR.conS.GetRoundState()
  534. prs := ps.GetRoundState()
  535. switch sleeping {
  536. case 1: // First sleep
  537. sleeping = 2
  538. case 2: // No more sleep
  539. sleeping = 0
  540. }
  541. //logger.Debug("gossipVotesRoutine", "rsHeight", rs.Height, "rsRound", rs.Round,
  542. // "prsHeight", prs.Height, "prsRound", prs.Round, "prsStep", prs.Step)
  543. // If height matches, then send LastCommit, Prevotes, Precommits.
  544. if rs.Height == prs.Height {
  545. heightLogger := logger.With("height", prs.Height)
  546. if conR.gossipVotesForHeight(heightLogger, rs, prs, ps) {
  547. continue OUTER_LOOP
  548. }
  549. }
  550. // Special catchup logic.
  551. // If peer is lagging by height 1, send LastCommit.
  552. if prs.Height != 0 && rs.Height == prs.Height+1 {
  553. if ps.PickSendVote(rs.LastCommit) {
  554. logger.Debug("Picked rs.LastCommit to send", "height", prs.Height)
  555. continue OUTER_LOOP
  556. }
  557. }
  558. // Catchup logic
  559. // If peer is lagging by more than 1, send Commit.
  560. if prs.Height != 0 && rs.Height >= prs.Height+2 {
  561. // Load the block commit for prs.Height,
  562. // which contains precommit signatures for prs.Height.
  563. commit := conR.conS.blockStore.LoadBlockCommit(prs.Height)
  564. if ps.PickSendVote(commit) {
  565. logger.Debug("Picked Catchup commit to send", "height", prs.Height)
  566. continue OUTER_LOOP
  567. }
  568. }
  569. if sleeping == 0 {
  570. // We sent nothing. Sleep...
  571. sleeping = 1
  572. logger.Debug("No votes to send, sleeping", "rs.Height", rs.Height, "prs.Height", prs.Height,
  573. "localPV", rs.Votes.Prevotes(rs.Round).BitArray(), "peerPV", prs.Prevotes,
  574. "localPC", rs.Votes.Precommits(rs.Round).BitArray(), "peerPC", prs.Precommits)
  575. } else if sleeping == 2 {
  576. // Continued sleep...
  577. sleeping = 1
  578. }
  579. time.Sleep(conR.conS.config.PeerGossipSleep())
  580. continue OUTER_LOOP
  581. }
  582. }
  583. func (conR *ConsensusReactor) gossipVotesForHeight(logger log.Logger, rs *cstypes.RoundState, prs *cstypes.PeerRoundState, ps *PeerState) bool {
  584. // If there are lastCommits to send...
  585. if prs.Step == cstypes.RoundStepNewHeight {
  586. if ps.PickSendVote(rs.LastCommit) {
  587. logger.Debug("Picked rs.LastCommit to send")
  588. return true
  589. }
  590. }
  591. // If there are POL prevotes to send...
  592. if prs.Step <= cstypes.RoundStepPropose && prs.Round != -1 && prs.Round <= rs.Round && prs.ProposalPOLRound != -1 {
  593. if polPrevotes := rs.Votes.Prevotes(prs.ProposalPOLRound); polPrevotes != nil {
  594. if ps.PickSendVote(polPrevotes) {
  595. logger.Debug("Picked rs.Prevotes(prs.ProposalPOLRound) to send",
  596. "round", prs.ProposalPOLRound)
  597. return true
  598. }
  599. }
  600. }
  601. // If there are prevotes to send...
  602. if prs.Step <= cstypes.RoundStepPrevoteWait && prs.Round != -1 && prs.Round <= rs.Round {
  603. if ps.PickSendVote(rs.Votes.Prevotes(prs.Round)) {
  604. logger.Debug("Picked rs.Prevotes(prs.Round) to send", "round", prs.Round)
  605. return true
  606. }
  607. }
  608. // If there are precommits to send...
  609. if prs.Step <= cstypes.RoundStepPrecommitWait && prs.Round != -1 && prs.Round <= rs.Round {
  610. if ps.PickSendVote(rs.Votes.Precommits(prs.Round)) {
  611. logger.Debug("Picked rs.Precommits(prs.Round) to send", "round", prs.Round)
  612. return true
  613. }
  614. }
  615. // If there are prevotes to send...Needed because of validBlock mechanism
  616. if prs.Round != -1 && prs.Round <= rs.Round {
  617. if ps.PickSendVote(rs.Votes.Prevotes(prs.Round)) {
  618. logger.Debug("Picked rs.Prevotes(prs.Round) to send", "round", prs.Round)
  619. return true
  620. }
  621. }
  622. // If there are POLPrevotes to send...
  623. if prs.ProposalPOLRound != -1 {
  624. if polPrevotes := rs.Votes.Prevotes(prs.ProposalPOLRound); polPrevotes != nil {
  625. if ps.PickSendVote(polPrevotes) {
  626. logger.Debug("Picked rs.Prevotes(prs.ProposalPOLRound) to send",
  627. "round", prs.ProposalPOLRound)
  628. return true
  629. }
  630. }
  631. }
  632. return false
  633. }
  634. // NOTE: `queryMaj23Routine` has a simple crude design since it only comes
  635. // into play for liveness when there's a signature DDoS attack happening.
  636. func (conR *ConsensusReactor) queryMaj23Routine(peer p2p.Peer, ps *PeerState) {
  637. logger := conR.Logger.With("peer", peer)
  638. OUTER_LOOP:
  639. for {
  640. // Manage disconnects from self or peer.
  641. if !peer.IsRunning() || !conR.IsRunning() {
  642. logger.Info("Stopping queryMaj23Routine for peer")
  643. return
  644. }
  645. // Maybe send Height/Round/Prevotes
  646. {
  647. rs := conR.conS.GetRoundState()
  648. prs := ps.GetRoundState()
  649. if rs.Height == prs.Height {
  650. if maj23, ok := rs.Votes.Prevotes(prs.Round).TwoThirdsMajority(); ok {
  651. peer.TrySend(StateChannel, cdc.MustMarshalBinaryBare(&VoteSetMaj23Message{
  652. Height: prs.Height,
  653. Round: prs.Round,
  654. Type: types.VoteTypePrevote,
  655. BlockID: maj23,
  656. }))
  657. time.Sleep(conR.conS.config.PeerQueryMaj23Sleep())
  658. }
  659. }
  660. }
  661. // Maybe send Height/Round/Precommits
  662. {
  663. rs := conR.conS.GetRoundState()
  664. prs := ps.GetRoundState()
  665. if rs.Height == prs.Height {
  666. if maj23, ok := rs.Votes.Precommits(prs.Round).TwoThirdsMajority(); ok {
  667. peer.TrySend(StateChannel, cdc.MustMarshalBinaryBare(&VoteSetMaj23Message{
  668. Height: prs.Height,
  669. Round: prs.Round,
  670. Type: types.VoteTypePrecommit,
  671. BlockID: maj23,
  672. }))
  673. time.Sleep(conR.conS.config.PeerQueryMaj23Sleep())
  674. }
  675. }
  676. }
  677. // Maybe send Height/Round/ProposalPOL
  678. {
  679. rs := conR.conS.GetRoundState()
  680. prs := ps.GetRoundState()
  681. if rs.Height == prs.Height && prs.ProposalPOLRound >= 0 {
  682. if maj23, ok := rs.Votes.Prevotes(prs.ProposalPOLRound).TwoThirdsMajority(); ok {
  683. peer.TrySend(StateChannel, cdc.MustMarshalBinaryBare(&VoteSetMaj23Message{
  684. Height: prs.Height,
  685. Round: prs.ProposalPOLRound,
  686. Type: types.VoteTypePrevote,
  687. BlockID: maj23,
  688. }))
  689. time.Sleep(conR.conS.config.PeerQueryMaj23Sleep())
  690. }
  691. }
  692. }
  693. // Little point sending LastCommitRound/LastCommit,
  694. // These are fleeting and non-blocking.
  695. // Maybe send Height/CatchupCommitRound/CatchupCommit.
  696. {
  697. prs := ps.GetRoundState()
  698. if prs.CatchupCommitRound != -1 && 0 < prs.Height && prs.Height <= conR.conS.blockStore.Height() {
  699. commit := conR.conS.LoadCommit(prs.Height)
  700. peer.TrySend(StateChannel, cdc.MustMarshalBinaryBare(&VoteSetMaj23Message{
  701. Height: prs.Height,
  702. Round: commit.Round(),
  703. Type: types.VoteTypePrecommit,
  704. BlockID: commit.BlockID,
  705. }))
  706. time.Sleep(conR.conS.config.PeerQueryMaj23Sleep())
  707. }
  708. }
  709. time.Sleep(conR.conS.config.PeerQueryMaj23Sleep())
  710. continue OUTER_LOOP
  711. }
  712. }
  713. // String returns a string representation of the ConsensusReactor.
  714. // NOTE: For now, it is just a hard-coded string to avoid accessing unprotected shared variables.
  715. // TODO: improve!
  716. func (conR *ConsensusReactor) String() string {
  717. // better not to access shared variables
  718. return "ConsensusReactor" // conR.StringIndented("")
  719. }
  720. // StringIndented returns an indented string representation of the ConsensusReactor
  721. func (conR *ConsensusReactor) StringIndented(indent string) string {
  722. s := "ConsensusReactor{\n"
  723. s += indent + " " + conR.conS.StringIndented(indent+" ") + "\n"
  724. for _, peer := range conR.Switch.Peers().List() {
  725. ps := peer.Get(types.PeerStateKey).(*PeerState)
  726. s += indent + " " + ps.StringIndented(indent+" ") + "\n"
  727. }
  728. s += indent + "}"
  729. return s
  730. }
  731. //-----------------------------------------------------------------------------
  732. var (
  733. ErrPeerStateHeightRegression = errors.New("Error peer state height regression")
  734. ErrPeerStateInvalidStartTime = errors.New("Error peer state invalid startTime")
  735. )
  736. // PeerState contains the known state of a peer, including its connection and
  737. // threadsafe access to its PeerRoundState.
  738. // NOTE: THIS GETS DUMPED WITH rpc/core/consensus.go.
  739. // Be mindful of what you Expose.
  740. type PeerState struct {
  741. peer p2p.Peer
  742. logger log.Logger
  743. mtx sync.Mutex `json:"-"` // NOTE: Modify below using setters, never directly.
  744. PRS cstypes.PeerRoundState `json:"round_state"` // Exposed.
  745. Stats *peerStateStats `json:"stats"` // Exposed.
  746. }
  747. // peerStateStats holds internal statistics for a peer.
  748. type peerStateStats struct {
  749. LastVoteHeight int64 `json:"last_vote_height"`
  750. Votes int `json:"votes"`
  751. LastBlockPartHeight int64 `json:"last_block_part_height"`
  752. BlockParts int `json:"block_parts"`
  753. }
  754. func (pss peerStateStats) String() string {
  755. return fmt.Sprintf("peerStateStats{lvh: %d, votes: %d, lbph: %d, blockParts: %d}",
  756. pss.LastVoteHeight, pss.Votes, pss.LastBlockPartHeight, pss.BlockParts)
  757. }
  758. // NewPeerState returns a new PeerState for the given Peer
  759. func NewPeerState(peer p2p.Peer) *PeerState {
  760. return &PeerState{
  761. peer: peer,
  762. logger: log.NewNopLogger(),
  763. PRS: cstypes.PeerRoundState{
  764. Round: -1,
  765. ProposalPOLRound: -1,
  766. LastCommitRound: -1,
  767. CatchupCommitRound: -1,
  768. },
  769. Stats: &peerStateStats{},
  770. }
  771. }
  772. // SetLogger allows to set a logger on the peer state. Returns the peer state
  773. // itself.
  774. func (ps *PeerState) SetLogger(logger log.Logger) *PeerState {
  775. ps.logger = logger
  776. return ps
  777. }
  778. // GetRoundState returns an shallow copy of the PeerRoundState.
  779. // There's no point in mutating it since it won't change PeerState.
  780. func (ps *PeerState) GetRoundState() *cstypes.PeerRoundState {
  781. ps.mtx.Lock()
  782. defer ps.mtx.Unlock()
  783. prs := ps.PRS // copy
  784. return &prs
  785. }
  786. // ToJSON returns a json of PeerState, marshalled using go-amino.
  787. func (ps *PeerState) ToJSON() ([]byte, error) {
  788. ps.mtx.Lock()
  789. defer ps.mtx.Unlock()
  790. return cdc.MarshalJSON(ps)
  791. }
  792. // GetHeight returns an atomic snapshot of the PeerRoundState's height
  793. // used by the mempool to ensure peers are caught up before broadcasting new txs
  794. func (ps *PeerState) GetHeight() int64 {
  795. ps.mtx.Lock()
  796. defer ps.mtx.Unlock()
  797. return ps.PRS.Height
  798. }
  799. // SetHasProposal sets the given proposal as known for the peer.
  800. func (ps *PeerState) SetHasProposal(proposal *types.Proposal) {
  801. ps.mtx.Lock()
  802. defer ps.mtx.Unlock()
  803. if ps.PRS.Height != proposal.Height || ps.PRS.Round != proposal.Round {
  804. return
  805. }
  806. if ps.PRS.Proposal {
  807. return
  808. }
  809. ps.PRS.Proposal = true
  810. ps.PRS.ProposalBlockPartsHeader = proposal.BlockPartsHeader
  811. ps.PRS.ProposalBlockParts = cmn.NewBitArray(proposal.BlockPartsHeader.Total)
  812. ps.PRS.ProposalPOLRound = proposal.POLRound
  813. ps.PRS.ProposalPOL = nil // Nil until ProposalPOLMessage received.
  814. }
  815. // InitProposalBlockParts initializes the peer's proposal block parts header and bit array.
  816. func (ps *PeerState) InitProposalBlockParts(partsHeader types.PartSetHeader) {
  817. ps.mtx.Lock()
  818. defer ps.mtx.Unlock()
  819. if ps.PRS.ProposalBlockParts != nil {
  820. return
  821. }
  822. ps.PRS.ProposalBlockPartsHeader = partsHeader
  823. ps.PRS.ProposalBlockParts = cmn.NewBitArray(partsHeader.Total)
  824. }
  825. // SetHasProposalBlockPart sets the given block part index as known for the peer.
  826. func (ps *PeerState) SetHasProposalBlockPart(height int64, round int, index int) {
  827. ps.mtx.Lock()
  828. defer ps.mtx.Unlock()
  829. if ps.PRS.Height != height || ps.PRS.Round != round {
  830. return
  831. }
  832. ps.PRS.ProposalBlockParts.SetIndex(index, true)
  833. }
  834. // PickSendVote picks a vote and sends it to the peer.
  835. // Returns true if vote was sent.
  836. func (ps *PeerState) PickSendVote(votes types.VoteSetReader) bool {
  837. if vote, ok := ps.PickVoteToSend(votes); ok {
  838. msg := &VoteMessage{vote}
  839. ps.logger.Debug("Sending vote message", "ps", ps, "vote", vote)
  840. return ps.peer.Send(VoteChannel, cdc.MustMarshalBinaryBare(msg))
  841. }
  842. return false
  843. }
  844. // PickVoteToSend picks a vote to send to the peer.
  845. // Returns true if a vote was picked.
  846. // NOTE: `votes` must be the correct Size() for the Height().
  847. func (ps *PeerState) PickVoteToSend(votes types.VoteSetReader) (vote *types.Vote, ok bool) {
  848. ps.mtx.Lock()
  849. defer ps.mtx.Unlock()
  850. if votes.Size() == 0 {
  851. return nil, false
  852. }
  853. height, round, type_, size := votes.Height(), votes.Round(), votes.Type(), votes.Size()
  854. // Lazily set data using 'votes'.
  855. if votes.IsCommit() {
  856. ps.ensureCatchupCommitRound(height, round, size)
  857. }
  858. ps.ensureVoteBitArrays(height, size)
  859. psVotes := ps.getVoteBitArray(height, round, type_)
  860. if psVotes == nil {
  861. return nil, false // Not something worth sending
  862. }
  863. if index, ok := votes.BitArray().Sub(psVotes).PickRandom(); ok {
  864. ps.setHasVote(height, round, type_, index)
  865. return votes.GetByIndex(index), true
  866. }
  867. return nil, false
  868. }
  869. func (ps *PeerState) getVoteBitArray(height int64, round int, type_ byte) *cmn.BitArray {
  870. if !types.IsVoteTypeValid(type_) {
  871. return nil
  872. }
  873. if ps.PRS.Height == height {
  874. if ps.PRS.Round == round {
  875. switch type_ {
  876. case types.VoteTypePrevote:
  877. return ps.PRS.Prevotes
  878. case types.VoteTypePrecommit:
  879. return ps.PRS.Precommits
  880. }
  881. }
  882. if ps.PRS.CatchupCommitRound == round {
  883. switch type_ {
  884. case types.VoteTypePrevote:
  885. return nil
  886. case types.VoteTypePrecommit:
  887. return ps.PRS.CatchupCommit
  888. }
  889. }
  890. if ps.PRS.ProposalPOLRound == round {
  891. switch type_ {
  892. case types.VoteTypePrevote:
  893. return ps.PRS.ProposalPOL
  894. case types.VoteTypePrecommit:
  895. return nil
  896. }
  897. }
  898. return nil
  899. }
  900. if ps.PRS.Height == height+1 {
  901. if ps.PRS.LastCommitRound == round {
  902. switch type_ {
  903. case types.VoteTypePrevote:
  904. return nil
  905. case types.VoteTypePrecommit:
  906. return ps.PRS.LastCommit
  907. }
  908. }
  909. return nil
  910. }
  911. return nil
  912. }
  913. // 'round': A round for which we have a +2/3 commit.
  914. func (ps *PeerState) ensureCatchupCommitRound(height int64, round int, numValidators int) {
  915. if ps.PRS.Height != height {
  916. return
  917. }
  918. /*
  919. NOTE: This is wrong, 'round' could change.
  920. e.g. if orig round is not the same as block LastCommit round.
  921. if ps.CatchupCommitRound != -1 && ps.CatchupCommitRound != round {
  922. cmn.PanicSanity(cmn.Fmt("Conflicting CatchupCommitRound. Height: %v, Orig: %v, New: %v", height, ps.CatchupCommitRound, round))
  923. }
  924. */
  925. if ps.PRS.CatchupCommitRound == round {
  926. return // Nothing to do!
  927. }
  928. ps.PRS.CatchupCommitRound = round
  929. if round == ps.PRS.Round {
  930. ps.PRS.CatchupCommit = ps.PRS.Precommits
  931. } else {
  932. ps.PRS.CatchupCommit = cmn.NewBitArray(numValidators)
  933. }
  934. }
  935. // EnsureVoteBitArrays ensures the bit-arrays have been allocated for tracking
  936. // what votes this peer has received.
  937. // NOTE: It's important to make sure that numValidators actually matches
  938. // what the node sees as the number of validators for height.
  939. func (ps *PeerState) EnsureVoteBitArrays(height int64, numValidators int) {
  940. ps.mtx.Lock()
  941. defer ps.mtx.Unlock()
  942. ps.ensureVoteBitArrays(height, numValidators)
  943. }
  944. func (ps *PeerState) ensureVoteBitArrays(height int64, numValidators int) {
  945. if ps.PRS.Height == height {
  946. if ps.PRS.Prevotes == nil {
  947. ps.PRS.Prevotes = cmn.NewBitArray(numValidators)
  948. }
  949. if ps.PRS.Precommits == nil {
  950. ps.PRS.Precommits = cmn.NewBitArray(numValidators)
  951. }
  952. if ps.PRS.CatchupCommit == nil {
  953. ps.PRS.CatchupCommit = cmn.NewBitArray(numValidators)
  954. }
  955. if ps.PRS.ProposalPOL == nil {
  956. ps.PRS.ProposalPOL = cmn.NewBitArray(numValidators)
  957. }
  958. } else if ps.PRS.Height == height+1 {
  959. if ps.PRS.LastCommit == nil {
  960. ps.PRS.LastCommit = cmn.NewBitArray(numValidators)
  961. }
  962. }
  963. }
  964. // RecordVote updates internal statistics for this peer by recording the vote.
  965. // It returns the total number of votes (1 per block). This essentially means
  966. // the number of blocks for which peer has been sending us votes.
  967. func (ps *PeerState) RecordVote(vote *types.Vote) int {
  968. ps.mtx.Lock()
  969. defer ps.mtx.Unlock()
  970. if ps.Stats.LastVoteHeight >= vote.Height {
  971. return ps.Stats.Votes
  972. }
  973. ps.Stats.LastVoteHeight = vote.Height
  974. ps.Stats.Votes++
  975. return ps.Stats.Votes
  976. }
  977. // VotesSent returns the number of blocks for which peer has been sending us
  978. // votes.
  979. func (ps *PeerState) VotesSent() int {
  980. ps.mtx.Lock()
  981. defer ps.mtx.Unlock()
  982. return ps.Stats.Votes
  983. }
  984. // RecordBlockPart updates internal statistics for this peer by recording the
  985. // block part. It returns the total number of block parts (1 per block). This
  986. // essentially means the number of blocks for which peer has been sending us
  987. // block parts.
  988. func (ps *PeerState) RecordBlockPart(bp *BlockPartMessage) int {
  989. ps.mtx.Lock()
  990. defer ps.mtx.Unlock()
  991. if ps.Stats.LastBlockPartHeight >= bp.Height {
  992. return ps.Stats.BlockParts
  993. }
  994. ps.Stats.LastBlockPartHeight = bp.Height
  995. ps.Stats.BlockParts++
  996. return ps.Stats.BlockParts
  997. }
  998. // BlockPartsSent returns the number of blocks for which peer has been sending
  999. // us block parts.
  1000. func (ps *PeerState) BlockPartsSent() int {
  1001. ps.mtx.Lock()
  1002. defer ps.mtx.Unlock()
  1003. return ps.Stats.BlockParts
  1004. }
  1005. // SetHasVote sets the given vote as known by the peer
  1006. func (ps *PeerState) SetHasVote(vote *types.Vote) {
  1007. ps.mtx.Lock()
  1008. defer ps.mtx.Unlock()
  1009. ps.setHasVote(vote.Height, vote.Round, vote.Type, vote.ValidatorIndex)
  1010. }
  1011. func (ps *PeerState) setHasVote(height int64, round int, type_ byte, index int) {
  1012. logger := ps.logger.With("peerH/R", cmn.Fmt("%d/%d", ps.PRS.Height, ps.PRS.Round), "H/R", cmn.Fmt("%d/%d", height, round))
  1013. logger.Debug("setHasVote", "type", type_, "index", index)
  1014. // NOTE: some may be nil BitArrays -> no side effects.
  1015. psVotes := ps.getVoteBitArray(height, round, type_)
  1016. if psVotes != nil {
  1017. psVotes.SetIndex(index, true)
  1018. }
  1019. }
  1020. // ApplyNewRoundStepMessage updates the peer state for the new round.
  1021. func (ps *PeerState) ApplyNewRoundStepMessage(msg *NewRoundStepMessage) {
  1022. ps.mtx.Lock()
  1023. defer ps.mtx.Unlock()
  1024. // Ignore duplicates or decreases
  1025. if CompareHRS(msg.Height, msg.Round, msg.Step, ps.PRS.Height, ps.PRS.Round, ps.PRS.Step) <= 0 {
  1026. return
  1027. }
  1028. // Just remember these values.
  1029. psHeight := ps.PRS.Height
  1030. psRound := ps.PRS.Round
  1031. //psStep := ps.PRS.Step
  1032. psCatchupCommitRound := ps.PRS.CatchupCommitRound
  1033. psCatchupCommit := ps.PRS.CatchupCommit
  1034. startTime := time.Now().Add(-1 * time.Duration(msg.SecondsSinceStartTime) * time.Second)
  1035. ps.PRS.Height = msg.Height
  1036. ps.PRS.Round = msg.Round
  1037. ps.PRS.Step = msg.Step
  1038. ps.PRS.StartTime = startTime
  1039. if psHeight != msg.Height || psRound != msg.Round {
  1040. ps.PRS.Proposal = false
  1041. ps.PRS.ProposalBlockPartsHeader = types.PartSetHeader{}
  1042. ps.PRS.ProposalBlockParts = nil
  1043. ps.PRS.ProposalPOLRound = -1
  1044. ps.PRS.ProposalPOL = nil
  1045. // We'll update the BitArray capacity later.
  1046. ps.PRS.Prevotes = nil
  1047. ps.PRS.Precommits = nil
  1048. }
  1049. if psHeight == msg.Height && psRound != msg.Round && msg.Round == psCatchupCommitRound {
  1050. // Peer caught up to CatchupCommitRound.
  1051. // Preserve psCatchupCommit!
  1052. // NOTE: We prefer to use prs.Precommits if
  1053. // pr.Round matches pr.CatchupCommitRound.
  1054. ps.PRS.Precommits = psCatchupCommit
  1055. }
  1056. if psHeight != msg.Height {
  1057. // Shift Precommits to LastCommit.
  1058. if psHeight+1 == msg.Height && psRound == msg.LastCommitRound {
  1059. ps.PRS.LastCommitRound = msg.LastCommitRound
  1060. ps.PRS.LastCommit = ps.PRS.Precommits
  1061. } else {
  1062. ps.PRS.LastCommitRound = msg.LastCommitRound
  1063. ps.PRS.LastCommit = nil
  1064. }
  1065. // We'll update the BitArray capacity later.
  1066. ps.PRS.CatchupCommitRound = -1
  1067. ps.PRS.CatchupCommit = nil
  1068. }
  1069. }
  1070. // ApplyCommitStepMessage updates the peer state for the new commit.
  1071. func (ps *PeerState) ApplyCommitStepMessage(msg *CommitStepMessage) {
  1072. ps.mtx.Lock()
  1073. defer ps.mtx.Unlock()
  1074. if ps.PRS.Height != msg.Height {
  1075. return
  1076. }
  1077. ps.PRS.ProposalBlockPartsHeader = msg.BlockPartsHeader
  1078. ps.PRS.ProposalBlockParts = msg.BlockParts
  1079. }
  1080. // ApplyProposalPOLMessage updates the peer state for the new proposal POL.
  1081. func (ps *PeerState) ApplyProposalPOLMessage(msg *ProposalPOLMessage) {
  1082. ps.mtx.Lock()
  1083. defer ps.mtx.Unlock()
  1084. if ps.PRS.Height != msg.Height {
  1085. return
  1086. }
  1087. if ps.PRS.ProposalPOLRound != msg.ProposalPOLRound {
  1088. return
  1089. }
  1090. // TODO: Merge onto existing ps.PRS.ProposalPOL?
  1091. // We might have sent some prevotes in the meantime.
  1092. ps.PRS.ProposalPOL = msg.ProposalPOL
  1093. }
  1094. // ApplyHasVoteMessage updates the peer state for the new vote.
  1095. func (ps *PeerState) ApplyHasVoteMessage(msg *HasVoteMessage) {
  1096. ps.mtx.Lock()
  1097. defer ps.mtx.Unlock()
  1098. if ps.PRS.Height != msg.Height {
  1099. return
  1100. }
  1101. ps.setHasVote(msg.Height, msg.Round, msg.Type, msg.Index)
  1102. }
  1103. // ApplyVoteSetBitsMessage updates the peer state for the bit-array of votes
  1104. // it claims to have for the corresponding BlockID.
  1105. // `ourVotes` is a BitArray of votes we have for msg.BlockID
  1106. // NOTE: if ourVotes is nil (e.g. msg.Height < rs.Height),
  1107. // we conservatively overwrite ps's votes w/ msg.Votes.
  1108. func (ps *PeerState) ApplyVoteSetBitsMessage(msg *VoteSetBitsMessage, ourVotes *cmn.BitArray) {
  1109. ps.mtx.Lock()
  1110. defer ps.mtx.Unlock()
  1111. votes := ps.getVoteBitArray(msg.Height, msg.Round, msg.Type)
  1112. if votes != nil {
  1113. if ourVotes == nil {
  1114. votes.Update(msg.Votes)
  1115. } else {
  1116. otherVotes := votes.Sub(ourVotes)
  1117. hasVotes := otherVotes.Or(msg.Votes)
  1118. votes.Update(hasVotes)
  1119. }
  1120. }
  1121. }
  1122. // String returns a string representation of the PeerState
  1123. func (ps *PeerState) String() string {
  1124. return ps.StringIndented("")
  1125. }
  1126. // StringIndented returns a string representation of the PeerState
  1127. func (ps *PeerState) StringIndented(indent string) string {
  1128. ps.mtx.Lock()
  1129. defer ps.mtx.Unlock()
  1130. return fmt.Sprintf(`PeerState{
  1131. %s Key %v
  1132. %s RoundState %v
  1133. %s Stats %v
  1134. %s}`,
  1135. indent, ps.peer.ID(),
  1136. indent, ps.PRS.StringIndented(indent+" "),
  1137. indent, ps.Stats,
  1138. indent)
  1139. }
  1140. //-----------------------------------------------------------------------------
  1141. // Messages
  1142. // ConsensusMessage is a message that can be sent and received on the ConsensusReactor
  1143. type ConsensusMessage interface{}
  1144. func RegisterConsensusMessages(cdc *amino.Codec) {
  1145. cdc.RegisterInterface((*ConsensusMessage)(nil), nil)
  1146. cdc.RegisterConcrete(&NewRoundStepMessage{}, "tendermint/NewRoundStepMessage", nil)
  1147. cdc.RegisterConcrete(&CommitStepMessage{}, "tendermint/CommitStep", nil)
  1148. cdc.RegisterConcrete(&ProposalMessage{}, "tendermint/Proposal", nil)
  1149. cdc.RegisterConcrete(&ProposalPOLMessage{}, "tendermint/ProposalPOL", nil)
  1150. cdc.RegisterConcrete(&BlockPartMessage{}, "tendermint/BlockPart", nil)
  1151. cdc.RegisterConcrete(&VoteMessage{}, "tendermint/Vote", nil)
  1152. cdc.RegisterConcrete(&HasVoteMessage{}, "tendermint/HasVote", nil)
  1153. cdc.RegisterConcrete(&VoteSetMaj23Message{}, "tendermint/VoteSetMaj23", nil)
  1154. cdc.RegisterConcrete(&VoteSetBitsMessage{}, "tendermint/VoteSetBits", nil)
  1155. cdc.RegisterConcrete(&ProposalHeartbeatMessage{}, "tendermint/ProposalHeartbeat", nil)
  1156. }
  1157. // DecodeMessage decodes the given bytes into a ConsensusMessage.
  1158. func DecodeMessage(bz []byte) (msg ConsensusMessage, err error) {
  1159. if len(bz) > maxMsgSize {
  1160. return msg, fmt.Errorf("Msg exceeds max size (%d > %d)",
  1161. len(bz), maxMsgSize)
  1162. }
  1163. err = cdc.UnmarshalBinaryBare(bz, &msg)
  1164. return
  1165. }
  1166. //-------------------------------------
  1167. // NewRoundStepMessage is sent for every step taken in the ConsensusState.
  1168. // For every height/round/step transition
  1169. type NewRoundStepMessage struct {
  1170. Height int64
  1171. Round int
  1172. Step cstypes.RoundStepType
  1173. SecondsSinceStartTime int
  1174. LastCommitRound int
  1175. }
  1176. // String returns a string representation.
  1177. func (m *NewRoundStepMessage) String() string {
  1178. return fmt.Sprintf("[NewRoundStep H:%v R:%v S:%v LCR:%v]",
  1179. m.Height, m.Round, m.Step, m.LastCommitRound)
  1180. }
  1181. //-------------------------------------
  1182. // CommitStepMessage is sent when a block is committed.
  1183. type CommitStepMessage struct {
  1184. Height int64
  1185. BlockPartsHeader types.PartSetHeader
  1186. BlockParts *cmn.BitArray
  1187. }
  1188. // String returns a string representation.
  1189. func (m *CommitStepMessage) String() string {
  1190. return fmt.Sprintf("[CommitStep H:%v BP:%v BA:%v]", m.Height, m.BlockPartsHeader, m.BlockParts)
  1191. }
  1192. //-------------------------------------
  1193. // ProposalMessage is sent when a new block is proposed.
  1194. type ProposalMessage struct {
  1195. Proposal *types.Proposal
  1196. }
  1197. // String returns a string representation.
  1198. func (m *ProposalMessage) String() string {
  1199. return fmt.Sprintf("[Proposal %v]", m.Proposal)
  1200. }
  1201. //-------------------------------------
  1202. // ProposalPOLMessage is sent when a previous proposal is re-proposed.
  1203. type ProposalPOLMessage struct {
  1204. Height int64
  1205. ProposalPOLRound int
  1206. ProposalPOL *cmn.BitArray
  1207. }
  1208. // String returns a string representation.
  1209. func (m *ProposalPOLMessage) String() string {
  1210. return fmt.Sprintf("[ProposalPOL H:%v POLR:%v POL:%v]", m.Height, m.ProposalPOLRound, m.ProposalPOL)
  1211. }
  1212. //-------------------------------------
  1213. // BlockPartMessage is sent when gossipping a piece of the proposed block.
  1214. type BlockPartMessage struct {
  1215. Height int64
  1216. Round int
  1217. Part *types.Part
  1218. }
  1219. // String returns a string representation.
  1220. func (m *BlockPartMessage) String() string {
  1221. return fmt.Sprintf("[BlockPart H:%v R:%v P:%v]", m.Height, m.Round, m.Part)
  1222. }
  1223. //-------------------------------------
  1224. // VoteMessage is sent when voting for a proposal (or lack thereof).
  1225. type VoteMessage struct {
  1226. Vote *types.Vote
  1227. }
  1228. // String returns a string representation.
  1229. func (m *VoteMessage) String() string {
  1230. return fmt.Sprintf("[Vote %v]", m.Vote)
  1231. }
  1232. //-------------------------------------
  1233. // HasVoteMessage is sent to indicate that a particular vote has been received.
  1234. type HasVoteMessage struct {
  1235. Height int64
  1236. Round int
  1237. Type byte
  1238. Index int
  1239. }
  1240. // String returns a string representation.
  1241. func (m *HasVoteMessage) String() string {
  1242. return fmt.Sprintf("[HasVote VI:%v V:{%v/%02d/%v}]", m.Index, m.Height, m.Round, m.Type)
  1243. }
  1244. //-------------------------------------
  1245. // VoteSetMaj23Message is sent to indicate that a given BlockID has seen +2/3 votes.
  1246. type VoteSetMaj23Message struct {
  1247. Height int64
  1248. Round int
  1249. Type byte
  1250. BlockID types.BlockID
  1251. }
  1252. // String returns a string representation.
  1253. func (m *VoteSetMaj23Message) String() string {
  1254. return fmt.Sprintf("[VSM23 %v/%02d/%v %v]", m.Height, m.Round, m.Type, m.BlockID)
  1255. }
  1256. //-------------------------------------
  1257. // VoteSetBitsMessage is sent to communicate the bit-array of votes seen for the BlockID.
  1258. type VoteSetBitsMessage struct {
  1259. Height int64
  1260. Round int
  1261. Type byte
  1262. BlockID types.BlockID
  1263. Votes *cmn.BitArray
  1264. }
  1265. // String returns a string representation.
  1266. func (m *VoteSetBitsMessage) String() string {
  1267. return fmt.Sprintf("[VSB %v/%02d/%v %v %v]", m.Height, m.Round, m.Type, m.BlockID, m.Votes)
  1268. }
  1269. //-------------------------------------
  1270. // ProposalHeartbeatMessage is sent to signal that a node is alive and waiting for transactions for a proposal.
  1271. type ProposalHeartbeatMessage struct {
  1272. Heartbeat *types.Heartbeat
  1273. }
  1274. // String returns a string representation.
  1275. func (m *ProposalHeartbeatMessage) String() string {
  1276. return fmt.Sprintf("[HEARTBEAT %v]", m.Heartbeat)
  1277. }