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.

665 lines
20 KiB

7 years ago
7 years ago
7 years ago
lint: Enable Golint (#4212) * Fix many golint errors * Fix golint errors in the 'lite' package * Don't export Pool.store * Fix typo * Revert unwanted changes * Fix errors in counter package * Fix linter errors in kvstore package * Fix linter error in example package * Fix error in tests package * Fix linter errors in v2 package * Fix linter errors in consensus package * Fix linter errors in evidence package * Fix linter error in fail package * Fix linter errors in query package * Fix linter errors in core package * Fix linter errors in node package * Fix linter errors in mempool package * Fix linter error in conn package * Fix linter errors in pex package * Rename PEXReactor export to Reactor * Fix linter errors in trust package * Fix linter errors in upnp package * Fix linter errors in p2p package * Fix linter errors in proxy package * Fix linter errors in mock_test package * Fix linter error in client_test package * Fix linter errors in coretypes package * Fix linter errors in coregrpc package * Fix linter errors in rpcserver package * Fix linter errors in rpctypes package * Fix linter errors in rpctest package * Fix linter error in json2wal script * Fix linter error in wal2json script * Fix linter errors in kv package * Fix linter error in state package * Fix linter error in grpc_client * Fix linter errors in types package * Fix linter error in version package * Fix remaining errors * Address review comments * Fix broken tests * Reconcile package coregrpc * Fix golangci bot error * Fix new golint errors * Fix broken reference * Enable golint linter * minor changes to bring golint into line * fix failing test * fix pex reactor naming * address PR comments
5 years ago
7 years ago
7 years ago
7 years ago
7 years ago
7 years ago
7 years ago
7 years ago
  1. package evidence
  2. import (
  3. "bytes"
  4. "context"
  5. "errors"
  6. "fmt"
  7. "sync"
  8. "sync/atomic"
  9. "time"
  10. "github.com/gogo/protobuf/proto"
  11. gogotypes "github.com/gogo/protobuf/types"
  12. "github.com/google/orderedcode"
  13. dbm "github.com/tendermint/tm-db"
  14. "github.com/tendermint/tendermint/internal/eventbus"
  15. clist "github.com/tendermint/tendermint/internal/libs/clist"
  16. sm "github.com/tendermint/tendermint/internal/state"
  17. "github.com/tendermint/tendermint/libs/log"
  18. tmproto "github.com/tendermint/tendermint/proto/tendermint/types"
  19. "github.com/tendermint/tendermint/types"
  20. )
  21. const (
  22. // prefixes are unique across all tm db's
  23. prefixCommitted = int64(9)
  24. prefixPending = int64(10)
  25. )
  26. // Pool maintains a pool of valid evidence to be broadcasted and committed
  27. type Pool struct {
  28. logger log.Logger
  29. evidenceStore dbm.DB
  30. evidenceList *clist.CList // concurrent linked-list of evidence
  31. evidenceSize uint32 // amount of pending evidence
  32. // needed to load validators to verify evidence
  33. stateDB sm.Store
  34. // needed to load headers and commits to verify evidence
  35. blockStore BlockStore
  36. mtx sync.Mutex
  37. // latest state
  38. state sm.State
  39. // evidence from consensus is buffered to this slice, awaiting until the next height
  40. // before being flushed to the pool. This prevents broadcasting and proposing of
  41. // evidence before the height with which the evidence happened is finished.
  42. consensusBuffer []duplicateVoteSet
  43. pruningHeight int64
  44. pruningTime time.Time
  45. // Eventbus to emit events when evidence is validated
  46. // Not part of the constructor, use SetEventBus to set it
  47. // The eventBus must be started in order for event publishing not to block
  48. eventBus *eventbus.EventBus
  49. Metrics *Metrics
  50. }
  51. func (evpool *Pool) SetEventBus(e *eventbus.EventBus) {
  52. evpool.eventBus = e
  53. }
  54. // NewPool creates an evidence pool. If using an existing evidence store,
  55. // it will add all pending evidence to the concurrent list.
  56. func NewPool(logger log.Logger, evidenceDB dbm.DB, stateDB sm.Store, blockStore BlockStore, metrics *Metrics) (*Pool, error) {
  57. state, err := stateDB.Load()
  58. if err != nil {
  59. return nil, fmt.Errorf("failed to load state: %w", err)
  60. }
  61. pool := &Pool{
  62. stateDB: stateDB,
  63. blockStore: blockStore,
  64. state: state,
  65. logger: logger,
  66. evidenceStore: evidenceDB,
  67. evidenceList: clist.New(),
  68. consensusBuffer: make([]duplicateVoteSet, 0),
  69. Metrics: metrics,
  70. }
  71. // If pending evidence already in db, in event of prior failure, then check
  72. // for expiration, update the size and load it back to the evidenceList.
  73. pool.pruningHeight, pool.pruningTime = pool.removeExpiredPendingEvidence()
  74. evList, _, err := pool.listEvidence(prefixPending, -1)
  75. if err != nil {
  76. return nil, err
  77. }
  78. atomic.StoreUint32(&pool.evidenceSize, uint32(len(evList)))
  79. pool.Metrics.NumEvidence.Set(float64(pool.evidenceSize))
  80. for _, ev := range evList {
  81. pool.evidenceList.PushBack(ev)
  82. }
  83. pool.eventBus = nil
  84. return pool, nil
  85. }
  86. // PendingEvidence is used primarily as part of block proposal and returns up to
  87. // maxNum of uncommitted evidence.
  88. func (evpool *Pool) PendingEvidence(maxBytes int64) ([]types.Evidence, int64) {
  89. if evpool.Size() == 0 {
  90. return []types.Evidence{}, 0
  91. }
  92. evidence, size, err := evpool.listEvidence(prefixPending, maxBytes)
  93. if err != nil {
  94. evpool.logger.Error("failed to retrieve pending evidence", "err", err)
  95. }
  96. return evidence, size
  97. }
  98. // Update takes both the new state and the evidence committed at that height and performs
  99. // the following operations:
  100. // 1. Take any conflicting votes from consensus and use the state's LastBlockTime to form
  101. // DuplicateVoteEvidence and add it to the pool.
  102. // 2. Update the pool's state which contains evidence params relating to expiry.
  103. // 3. Moves pending evidence that has now been committed into the committed pool.
  104. // 4. Removes any expired evidence based on both height and time.
  105. func (evpool *Pool) Update(ctx context.Context, state sm.State, ev types.EvidenceList) {
  106. // sanity check
  107. if state.LastBlockHeight <= evpool.state.LastBlockHeight {
  108. panic(fmt.Sprintf(
  109. "failed EvidencePool.Update new state height is less than or equal to previous state height: %d <= %d",
  110. state.LastBlockHeight,
  111. evpool.state.LastBlockHeight,
  112. ))
  113. }
  114. evpool.logger.Debug(
  115. "updating evidence pool",
  116. "last_block_height", state.LastBlockHeight,
  117. "last_block_time", state.LastBlockTime,
  118. )
  119. // flush conflicting vote pairs from the buffer, producing DuplicateVoteEvidence and
  120. // adding it to the pool
  121. evpool.processConsensusBuffer(ctx, state)
  122. // update state
  123. evpool.updateState(state)
  124. // move committed evidence out from the pending pool and into the committed pool
  125. evpool.markEvidenceAsCommitted(ev, state.LastBlockHeight)
  126. // Prune pending evidence when it has expired. This also updates when the next
  127. // evidence will expire.
  128. if evpool.Size() > 0 && state.LastBlockHeight > evpool.pruningHeight &&
  129. state.LastBlockTime.After(evpool.pruningTime) {
  130. evpool.pruningHeight, evpool.pruningTime = evpool.removeExpiredPendingEvidence()
  131. }
  132. }
  133. // AddEvidence checks the evidence is valid and adds it to the pool.
  134. func (evpool *Pool) AddEvidence(ctx context.Context, ev types.Evidence) error {
  135. evpool.logger.Debug("attempting to add evidence", "evidence", ev)
  136. // We have already verified this piece of evidence - no need to do it again
  137. if evpool.isPending(ev) {
  138. evpool.logger.Debug("evidence already pending; ignoring", "evidence", ev)
  139. return nil
  140. }
  141. // check that the evidence isn't already committed
  142. if evpool.isCommitted(ev) {
  143. // This can happen if the peer that sent us the evidence is behind so we
  144. // shouldn't punish the peer.
  145. evpool.logger.Debug("evidence was already committed; ignoring", "evidence", ev)
  146. return nil
  147. }
  148. // 1) Verify against state.
  149. if err := evpool.verify(ctx, ev); err != nil {
  150. return err
  151. }
  152. // 2) Save to store.
  153. if err := evpool.addPendingEvidence(ctx, ev); err != nil {
  154. return fmt.Errorf("failed to add evidence to pending list: %w", err)
  155. }
  156. // 3) Add evidence to clist.
  157. evpool.evidenceList.PushBack(ev)
  158. evpool.logger.Info("verified new evidence of byzantine behavior", "evidence", ev)
  159. return nil
  160. }
  161. // ReportConflictingVotes takes two conflicting votes and forms duplicate vote evidence,
  162. // adding it eventually to the evidence pool.
  163. //
  164. // Duplicate vote attacks happen before the block is committed and the timestamp is
  165. // finalized, thus the evidence pool holds these votes in a buffer, forming the
  166. // evidence from them once consensus at that height has been reached and `Update()` with
  167. // the new state called.
  168. //
  169. // Votes are not verified.
  170. func (evpool *Pool) ReportConflictingVotes(voteA, voteB *types.Vote) {
  171. evpool.mtx.Lock()
  172. defer evpool.mtx.Unlock()
  173. evpool.consensusBuffer = append(evpool.consensusBuffer, duplicateVoteSet{
  174. VoteA: voteA,
  175. VoteB: voteB,
  176. })
  177. }
  178. // CheckEvidence takes an array of evidence from a block and verifies all the evidence there.
  179. // If it has already verified the evidence then it jumps to the next one. It ensures that no
  180. // evidence has already been committed or is being proposed twice. It also adds any
  181. // evidence that it doesn't currently have so that it can quickly form ABCI Evidence later.
  182. func (evpool *Pool) CheckEvidence(ctx context.Context, evList types.EvidenceList) error {
  183. hashes := make([][]byte, len(evList))
  184. for idx, ev := range evList {
  185. _, isLightEv := ev.(*types.LightClientAttackEvidence)
  186. // We must verify light client attack evidence regardless because there could be a
  187. // different conflicting block with the same hash.
  188. if isLightEv || !evpool.isPending(ev) {
  189. // check that the evidence isn't already committed
  190. if evpool.isCommitted(ev) {
  191. return &types.ErrInvalidEvidence{Evidence: ev, Reason: errors.New("evidence was already committed")}
  192. }
  193. err := evpool.verify(ctx, ev)
  194. if err != nil {
  195. return err
  196. }
  197. if err := evpool.addPendingEvidence(ctx, ev); err != nil {
  198. // Something went wrong with adding the evidence but we already know it is valid
  199. // hence we log an error and continue
  200. evpool.logger.Error("failed to add evidence to pending list", "err", err, "evidence", ev)
  201. }
  202. evpool.logger.Info("check evidence: verified evidence of byzantine behavior", "evidence", ev)
  203. }
  204. // check for duplicate evidence. We cache hashes so we don't have to work them out again.
  205. hashes[idx] = ev.Hash()
  206. for i := idx - 1; i >= 0; i-- {
  207. if bytes.Equal(hashes[i], hashes[idx]) {
  208. return &types.ErrInvalidEvidence{Evidence: ev, Reason: errors.New("duplicate evidence")}
  209. }
  210. }
  211. }
  212. return nil
  213. }
  214. // EvidenceFront goes to the first evidence in the clist
  215. func (evpool *Pool) EvidenceFront() *clist.CElement {
  216. return evpool.evidenceList.Front()
  217. }
  218. // EvidenceWaitChan is a channel that closes once the first evidence in the list
  219. // is there. i.e Front is not nil.
  220. func (evpool *Pool) EvidenceWaitChan() <-chan struct{} {
  221. return evpool.evidenceList.WaitChan()
  222. }
  223. // Size returns the number of evidence in the pool.
  224. func (evpool *Pool) Size() uint32 {
  225. return atomic.LoadUint32(&evpool.evidenceSize)
  226. }
  227. // State returns the current state of the evpool.
  228. func (evpool *Pool) State() sm.State {
  229. evpool.mtx.Lock()
  230. defer evpool.mtx.Unlock()
  231. return evpool.state
  232. }
  233. func (evpool *Pool) Close() error {
  234. return evpool.evidenceStore.Close()
  235. }
  236. // IsExpired checks whether evidence or a polc is expired by checking whether a height and time is older
  237. // than set by the evidence consensus parameters
  238. func (evpool *Pool) isExpired(height int64, time time.Time) bool {
  239. var (
  240. params = evpool.State().ConsensusParams.Evidence
  241. ageDuration = evpool.State().LastBlockTime.Sub(time)
  242. ageNumBlocks = evpool.State().LastBlockHeight - height
  243. )
  244. return ageNumBlocks > params.MaxAgeNumBlocks &&
  245. ageDuration > params.MaxAgeDuration
  246. }
  247. // IsCommitted returns true if we have already seen this exact evidence and it is already marked as committed.
  248. func (evpool *Pool) isCommitted(evidence types.Evidence) bool {
  249. key := keyCommitted(evidence)
  250. ok, err := evpool.evidenceStore.Has(key)
  251. if err != nil {
  252. evpool.logger.Error("failed to find committed evidence", "err", err)
  253. }
  254. return ok
  255. }
  256. // IsPending checks whether the evidence is already pending. DB errors are passed to the logger.
  257. func (evpool *Pool) isPending(evidence types.Evidence) bool {
  258. key := keyPending(evidence)
  259. ok, err := evpool.evidenceStore.Has(key)
  260. if err != nil {
  261. evpool.logger.Error("failed to find pending evidence", "err", err)
  262. }
  263. return ok
  264. }
  265. func (evpool *Pool) addPendingEvidence(ctx context.Context, ev types.Evidence) error {
  266. evpb, err := types.EvidenceToProto(ev)
  267. if err != nil {
  268. return fmt.Errorf("failed to convert to proto: %w", err)
  269. }
  270. evBytes, err := evpb.Marshal()
  271. if err != nil {
  272. return fmt.Errorf("failed to marshal evidence: %w", err)
  273. }
  274. key := keyPending(ev)
  275. err = evpool.evidenceStore.Set(key, evBytes)
  276. if err != nil {
  277. return fmt.Errorf("failed to persist evidence: %w", err)
  278. }
  279. atomic.AddUint32(&evpool.evidenceSize, 1)
  280. evpool.Metrics.NumEvidence.Set(float64(evpool.evidenceSize))
  281. // This should normally never be true
  282. if evpool.eventBus == nil {
  283. evpool.logger.Debug("event bus is not configured")
  284. return nil
  285. }
  286. return evpool.eventBus.PublishEventEvidenceValidated(ctx, types.EventDataEvidenceValidated{
  287. Evidence: ev,
  288. Height: ev.Height(),
  289. })
  290. }
  291. // markEvidenceAsCommitted processes all the evidence in the block, marking it as
  292. // committed and removing it from the pending database.
  293. func (evpool *Pool) markEvidenceAsCommitted(evidence types.EvidenceList, height int64) {
  294. blockEvidenceMap := make(map[string]struct{}, len(evidence))
  295. batch := evpool.evidenceStore.NewBatch()
  296. defer batch.Close()
  297. for _, ev := range evidence {
  298. if evpool.isPending(ev) {
  299. if err := batch.Delete(keyPending(ev)); err != nil {
  300. evpool.logger.Error("failed to batch delete pending evidence", "err", err)
  301. }
  302. blockEvidenceMap[evMapKey(ev)] = struct{}{}
  303. }
  304. // Add evidence to the committed list. As the evidence is stored in the block store
  305. // we only need to record the height that it was saved at.
  306. key := keyCommitted(ev)
  307. h := gogotypes.Int64Value{Value: height}
  308. evBytes, err := proto.Marshal(&h)
  309. if err != nil {
  310. evpool.logger.Error("failed to marshal committed evidence", "key(height/hash)", key, "err", err)
  311. continue
  312. }
  313. if err := evpool.evidenceStore.Set(key, evBytes); err != nil {
  314. evpool.logger.Error("failed to save committed evidence", "key(height/hash)", key, "err", err)
  315. }
  316. evpool.logger.Debug("marked evidence as committed", "evidence", ev)
  317. }
  318. // check if we need to remove any pending evidence
  319. if len(blockEvidenceMap) == 0 {
  320. return
  321. }
  322. // remove committed evidence from pending bucket
  323. if err := batch.WriteSync(); err != nil {
  324. evpool.logger.Error("failed to batch delete pending evidence", "err", err)
  325. return
  326. }
  327. // remove committed evidence from the clist
  328. evpool.removeEvidenceFromList(blockEvidenceMap)
  329. // update the evidence size
  330. atomic.AddUint32(&evpool.evidenceSize, ^uint32(len(blockEvidenceMap)-1))
  331. evpool.Metrics.NumEvidence.Set(float64(evpool.evidenceSize))
  332. }
  333. // listEvidence retrieves lists evidence from oldest to newest within maxBytes.
  334. // If maxBytes is -1, there's no cap on the size of returned evidence.
  335. func (evpool *Pool) listEvidence(prefixKey int64, maxBytes int64) ([]types.Evidence, int64, error) {
  336. var (
  337. evSize int64
  338. totalSize int64
  339. evidence []types.Evidence
  340. evList tmproto.EvidenceList // used for calculating the bytes size
  341. )
  342. iter, err := dbm.IteratePrefix(evpool.evidenceStore, prefixToBytes(prefixKey))
  343. if err != nil {
  344. return nil, totalSize, fmt.Errorf("database error: %w", err)
  345. }
  346. defer iter.Close()
  347. for ; iter.Valid(); iter.Next() {
  348. var evpb tmproto.Evidence
  349. if err := evpb.Unmarshal(iter.Value()); err != nil {
  350. return evidence, totalSize, err
  351. }
  352. evList.Evidence = append(evList.Evidence, evpb)
  353. evSize = int64(evList.Size())
  354. if maxBytes != -1 && evSize > maxBytes {
  355. if err := iter.Error(); err != nil {
  356. return evidence, totalSize, err
  357. }
  358. return evidence, totalSize, nil
  359. }
  360. ev, err := types.EvidenceFromProto(&evpb)
  361. if err != nil {
  362. return nil, totalSize, err
  363. }
  364. totalSize = evSize
  365. evidence = append(evidence, ev)
  366. }
  367. if err := iter.Error(); err != nil {
  368. return evidence, totalSize, err
  369. }
  370. return evidence, totalSize, nil
  371. }
  372. func (evpool *Pool) removeExpiredPendingEvidence() (int64, time.Time) {
  373. batch := evpool.evidenceStore.NewBatch()
  374. defer batch.Close()
  375. height, time, blockEvidenceMap := evpool.batchExpiredPendingEvidence(batch)
  376. // if we haven't removed any evidence then return early
  377. if len(blockEvidenceMap) == 0 {
  378. return height, time
  379. }
  380. evpool.logger.Debug("removing expired evidence",
  381. "height", evpool.State().LastBlockHeight,
  382. "time", evpool.State().LastBlockTime,
  383. "expired evidence", len(blockEvidenceMap),
  384. )
  385. // remove expired evidence from pending bucket
  386. if err := batch.WriteSync(); err != nil {
  387. evpool.logger.Error("failed to batch delete pending evidence", "err", err)
  388. return evpool.State().LastBlockHeight, evpool.State().LastBlockTime
  389. }
  390. // remove evidence from the clist
  391. evpool.removeEvidenceFromList(blockEvidenceMap)
  392. // update the evidence size
  393. atomic.AddUint32(&evpool.evidenceSize, ^uint32(len(blockEvidenceMap)-1))
  394. return height, time
  395. }
  396. func (evpool *Pool) batchExpiredPendingEvidence(batch dbm.Batch) (int64, time.Time, map[string]struct{}) {
  397. blockEvidenceMap := make(map[string]struct{})
  398. iter, err := dbm.IteratePrefix(evpool.evidenceStore, prefixToBytes(prefixPending))
  399. if err != nil {
  400. evpool.logger.Error("failed to iterate over pending evidence", "err", err)
  401. return evpool.State().LastBlockHeight, evpool.State().LastBlockTime, blockEvidenceMap
  402. }
  403. defer iter.Close()
  404. for ; iter.Valid(); iter.Next() {
  405. ev, err := bytesToEv(iter.Value())
  406. if err != nil {
  407. evpool.logger.Error("failed to transition evidence from protobuf", "err", err, "ev", ev)
  408. continue
  409. }
  410. // if true, we have looped through all expired evidence
  411. if !evpool.isExpired(ev.Height(), ev.Time()) {
  412. // Return the height and time with which this evidence will have expired
  413. // so we know when to prune next.
  414. return ev.Height() + evpool.State().ConsensusParams.Evidence.MaxAgeNumBlocks + 1,
  415. ev.Time().Add(evpool.State().ConsensusParams.Evidence.MaxAgeDuration).Add(time.Second),
  416. blockEvidenceMap
  417. }
  418. // else add to the batch
  419. if err := batch.Delete(iter.Key()); err != nil {
  420. evpool.logger.Error("failed to batch delete evidence", "err", err, "ev", ev)
  421. continue
  422. }
  423. // and add to the map to remove the evidence from the clist
  424. blockEvidenceMap[evMapKey(ev)] = struct{}{}
  425. }
  426. return evpool.State().LastBlockHeight, evpool.State().LastBlockTime, blockEvidenceMap
  427. }
  428. func (evpool *Pool) removeEvidenceFromList(
  429. blockEvidenceMap map[string]struct{}) {
  430. for e := evpool.evidenceList.Front(); e != nil; e = e.Next() {
  431. // Remove from clist
  432. ev := e.Value.(types.Evidence)
  433. if _, ok := blockEvidenceMap[evMapKey(ev)]; ok {
  434. evpool.evidenceList.Remove(e)
  435. e.DetachPrev()
  436. }
  437. }
  438. }
  439. func (evpool *Pool) updateState(state sm.State) {
  440. evpool.mtx.Lock()
  441. defer evpool.mtx.Unlock()
  442. evpool.state = state
  443. }
  444. // processConsensusBuffer converts all the duplicate votes witnessed from consensus
  445. // into DuplicateVoteEvidence. It sets the evidence timestamp to the block height
  446. // from the most recently committed block.
  447. // Evidence is then added to the pool so as to be ready to be broadcasted and proposed.
  448. func (evpool *Pool) processConsensusBuffer(ctx context.Context, state sm.State) {
  449. evpool.mtx.Lock()
  450. defer evpool.mtx.Unlock()
  451. for _, voteSet := range evpool.consensusBuffer {
  452. // Check the height of the conflicting votes and fetch the corresponding time and validator set
  453. // to produce the valid evidence
  454. var (
  455. dve *types.DuplicateVoteEvidence
  456. err error
  457. )
  458. switch {
  459. case voteSet.VoteA.Height == state.LastBlockHeight:
  460. dve, err = types.NewDuplicateVoteEvidence(
  461. voteSet.VoteA,
  462. voteSet.VoteB,
  463. state.LastBlockTime,
  464. state.LastValidators,
  465. )
  466. case voteSet.VoteA.Height < state.LastBlockHeight:
  467. valSet, dbErr := evpool.stateDB.LoadValidators(voteSet.VoteA.Height)
  468. if dbErr != nil {
  469. evpool.logger.Error("failed to load validator set for conflicting votes",
  470. "height", voteSet.VoteA.Height, "err", err)
  471. continue
  472. }
  473. blockMeta := evpool.blockStore.LoadBlockMeta(voteSet.VoteA.Height)
  474. if blockMeta == nil {
  475. evpool.logger.Error("failed to load block time for conflicting votes", "height", voteSet.VoteA.Height)
  476. continue
  477. }
  478. dve, err = types.NewDuplicateVoteEvidence(
  479. voteSet.VoteA,
  480. voteSet.VoteB,
  481. blockMeta.Header.Time,
  482. valSet,
  483. )
  484. default:
  485. // evidence pool shouldn't expect to get votes from consensus of a height that is above the current
  486. // state. If this error is seen then perhaps consider keeping the votes in the buffer and retry
  487. // in following heights
  488. evpool.logger.Error("inbound duplicate votes from consensus are of a greater height than current state",
  489. "duplicate vote height", voteSet.VoteA.Height,
  490. "state.LastBlockHeight", state.LastBlockHeight)
  491. continue
  492. }
  493. if err != nil {
  494. evpool.logger.Error("error in generating evidence from votes", "err", err)
  495. continue
  496. }
  497. // check if we already have this evidence
  498. if evpool.isPending(dve) {
  499. evpool.logger.Debug("evidence already pending; ignoring", "evidence", dve)
  500. continue
  501. }
  502. // check that the evidence is not already committed on chain
  503. if evpool.isCommitted(dve) {
  504. evpool.logger.Debug("evidence already committed; ignoring", "evidence", dve)
  505. continue
  506. }
  507. if err := evpool.addPendingEvidence(ctx, dve); err != nil {
  508. evpool.logger.Error("failed to flush evidence from consensus buffer to pending list: %w", err)
  509. continue
  510. }
  511. evpool.evidenceList.PushBack(dve)
  512. evpool.logger.Info("verified new evidence of byzantine behavior", "evidence", dve)
  513. }
  514. // reset consensus buffer
  515. evpool.consensusBuffer = make([]duplicateVoteSet, 0)
  516. }
  517. type duplicateVoteSet struct {
  518. VoteA *types.Vote
  519. VoteB *types.Vote
  520. }
  521. func bytesToEv(evBytes []byte) (types.Evidence, error) {
  522. var evpb tmproto.Evidence
  523. err := evpb.Unmarshal(evBytes)
  524. if err != nil {
  525. return &types.DuplicateVoteEvidence{}, err
  526. }
  527. return types.EvidenceFromProto(&evpb)
  528. }
  529. func evMapKey(ev types.Evidence) string {
  530. return string(ev.Hash())
  531. }
  532. func prefixToBytes(prefix int64) []byte {
  533. key, err := orderedcode.Append(nil, prefix)
  534. if err != nil {
  535. panic(err)
  536. }
  537. return key
  538. }
  539. func keyCommitted(evidence types.Evidence) []byte {
  540. height := evidence.Height()
  541. key, err := orderedcode.Append(nil, prefixCommitted, height, string(evidence.Hash()))
  542. if err != nil {
  543. panic(err)
  544. }
  545. return key
  546. }
  547. func keyPending(evidence types.Evidence) []byte {
  548. height := evidence.Height()
  549. key, err := orderedcode.Append(nil, prefixPending, height, string(evidence.Hash()))
  550. if err != nil {
  551. panic(err)
  552. }
  553. return key
  554. }