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.

588 lines
18 KiB

7 years ago
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. "errors"
  5. "fmt"
  6. "sort"
  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. clist "github.com/tendermint/tendermint/libs/clist"
  15. "github.com/tendermint/tendermint/libs/log"
  16. tmproto "github.com/tendermint/tendermint/proto/tendermint/types"
  17. sm "github.com/tendermint/tendermint/state"
  18. "github.com/tendermint/tendermint/types"
  19. )
  20. const (
  21. // prefixes are unique across all tm db's
  22. prefixCommitted = int64(8)
  23. prefixPending = int64(9)
  24. )
  25. // Pool maintains a pool of valid evidence to be broadcasted and committed
  26. type Pool struct {
  27. logger log.Logger
  28. evidenceStore dbm.DB
  29. evidenceList *clist.CList // concurrent linked-list of evidence
  30. evidenceSize uint32 // amount of pending evidence
  31. // needed to load validators to verify evidence
  32. stateDB sm.Store
  33. // needed to load headers and commits to verify evidence
  34. blockStore BlockStore
  35. mtx sync.Mutex
  36. // latest state
  37. state sm.State
  38. // evidence from consensus if buffered to this slice, awaiting until the next height
  39. // before being flushed to the pool. This prevents broadcasting and proposing of
  40. // evidence before the height with which the evidence happened is finished.
  41. consensusBuffer []types.Evidence
  42. pruningHeight int64
  43. pruningTime time.Time
  44. }
  45. // NewPool creates an evidence pool. If using an existing evidence store,
  46. // it will add all pending evidence to the concurrent list.
  47. func NewPool(logger log.Logger, evidenceDB dbm.DB, stateDB sm.Store, blockStore BlockStore) (*Pool, error) {
  48. state, err := stateDB.Load()
  49. if err != nil {
  50. return nil, fmt.Errorf("failed to load state: %w", err)
  51. }
  52. pool := &Pool{
  53. stateDB: stateDB,
  54. blockStore: blockStore,
  55. state: state,
  56. logger: logger,
  57. evidenceStore: evidenceDB,
  58. evidenceList: clist.New(),
  59. consensusBuffer: make([]types.Evidence, 0),
  60. }
  61. // If pending evidence already in db, in event of prior failure, then check
  62. // for expiration, update the size and load it back to the evidenceList.
  63. pool.pruningHeight, pool.pruningTime = pool.removeExpiredPendingEvidence()
  64. evList, _, err := pool.listEvidence(prefixPending, -1)
  65. if err != nil {
  66. return nil, err
  67. }
  68. atomic.StoreUint32(&pool.evidenceSize, uint32(len(evList)))
  69. for _, ev := range evList {
  70. pool.evidenceList.PushBack(ev)
  71. }
  72. return pool, nil
  73. }
  74. // PendingEvidence is used primarily as part of block proposal and returns up to
  75. // maxNum of uncommitted evidence.
  76. func (evpool *Pool) PendingEvidence(maxBytes int64) ([]types.Evidence, int64) {
  77. if evpool.Size() == 0 {
  78. return []types.Evidence{}, 0
  79. }
  80. evidence, size, err := evpool.listEvidence(prefixPending, maxBytes)
  81. if err != nil {
  82. evpool.logger.Error("failed to retrieve pending evidence", "err", err)
  83. }
  84. return evidence, size
  85. }
  86. // Update pulls the latest state to be used for expiration and evidence params
  87. // and then prunes all expired evidence.
  88. func (evpool *Pool) Update(state sm.State, ev types.EvidenceList) {
  89. // sanity check
  90. if state.LastBlockHeight <= evpool.state.LastBlockHeight {
  91. panic(fmt.Sprintf(
  92. "failed EvidencePool.Update new state height is less than or equal to previous state height: %d <= %d",
  93. state.LastBlockHeight,
  94. evpool.state.LastBlockHeight,
  95. ))
  96. }
  97. evpool.logger.Info(
  98. "updating evidence pool",
  99. "last_block_height", state.LastBlockHeight,
  100. "last_block_time", state.LastBlockTime,
  101. )
  102. evpool.mtx.Lock()
  103. // flush awaiting evidence from consensus into pool
  104. evpool.flushConsensusBuffer()
  105. // update state
  106. evpool.state = state
  107. evpool.mtx.Unlock()
  108. // move committed evidence out from the pending pool and into the committed pool
  109. evpool.markEvidenceAsCommitted(ev)
  110. // Prune pending evidence when it has expired. This also updates when the next
  111. // evidence will expire.
  112. if evpool.Size() > 0 && state.LastBlockHeight > evpool.pruningHeight &&
  113. state.LastBlockTime.After(evpool.pruningTime) {
  114. evpool.pruningHeight, evpool.pruningTime = evpool.removeExpiredPendingEvidence()
  115. }
  116. }
  117. // AddEvidence checks the evidence is valid and adds it to the pool.
  118. func (evpool *Pool) AddEvidence(ev types.Evidence) error {
  119. evpool.logger.Debug("attempting to add evidence", "evidence", ev)
  120. // We have already verified this piece of evidence - no need to do it again
  121. if evpool.isPending(ev) {
  122. evpool.logger.Info("evidence already pending; ignoring", "evidence", ev)
  123. return nil
  124. }
  125. // check that the evidence isn't already committed
  126. if evpool.isCommitted(ev) {
  127. // This can happen if the peer that sent us the evidence is behind so we
  128. // shouldn't punish the peer.
  129. evpool.logger.Debug("evidence was already committed; ignoring", "evidence", ev)
  130. return nil
  131. }
  132. // 1) Verify against state.
  133. if err := evpool.verify(ev); err != nil {
  134. return err
  135. }
  136. // 2) Save to store.
  137. if err := evpool.addPendingEvidence(ev); err != nil {
  138. return fmt.Errorf("failed to add evidence to pending list: %w", err)
  139. }
  140. // 3) Add evidence to clist.
  141. evpool.evidenceList.PushBack(ev)
  142. evpool.logger.Info("verified new evidence of byzantine behavior", "evidence", ev)
  143. return nil
  144. }
  145. // AddEvidenceFromConsensus should be exposed only to the consensus reactor so
  146. // it can add evidence to the pool directly without the need for verification.
  147. func (evpool *Pool) AddEvidenceFromConsensus(ev types.Evidence) error {
  148. // we already have this evidence, log this but don't return an error.
  149. if evpool.isPending(ev) {
  150. evpool.logger.Info("evidence already pending; ignoring", "evidence", ev)
  151. return nil
  152. }
  153. // add evidence to a buffer which will pass the evidence to the pool at the following height.
  154. // This avoids the issue of some nodes verifying and proposing evidence at a height where the
  155. // block hasn't been committed on cause others to potentially fail.
  156. evpool.mtx.Lock()
  157. defer evpool.mtx.Unlock()
  158. evpool.consensusBuffer = append(evpool.consensusBuffer, ev)
  159. evpool.logger.Info("received new evidence of byzantine behavior from consensus", "evidence", ev)
  160. return nil
  161. }
  162. // CheckEvidence takes an array of evidence from a block and verifies all the evidence there.
  163. // If it has already verified the evidence then it jumps to the next one. It ensures that no
  164. // evidence has already been committed or is being proposed twice. It also adds any
  165. // evidence that it doesn't currently have so that it can quickly form ABCI Evidence later.
  166. func (evpool *Pool) CheckEvidence(evList types.EvidenceList) error {
  167. hashes := make([][]byte, len(evList))
  168. for idx, ev := range evList {
  169. ok := evpool.fastCheck(ev)
  170. if !ok {
  171. // check that the evidence isn't already committed
  172. if evpool.isCommitted(ev) {
  173. return &types.ErrInvalidEvidence{Evidence: ev, Reason: errors.New("evidence was already committed")}
  174. }
  175. err := evpool.verify(ev)
  176. if err != nil {
  177. return &types.ErrInvalidEvidence{Evidence: ev, Reason: err}
  178. }
  179. if err := evpool.addPendingEvidence(ev); err != nil {
  180. // Something went wrong with adding the evidence but we already know it is valid
  181. // hence we log an error and continue
  182. evpool.logger.Error("failed to add evidence to pending list", "err", err, "evidence", ev)
  183. }
  184. evpool.logger.Info("verified new evidence of byzantine behavior", "evidence", ev)
  185. }
  186. // check for duplicate evidence. We cache hashes so we don't have to work them out again.
  187. hashes[idx] = ev.Hash()
  188. for i := idx - 1; i >= 0; i-- {
  189. if bytes.Equal(hashes[i], hashes[idx]) {
  190. return &types.ErrInvalidEvidence{Evidence: ev, Reason: errors.New("duplicate evidence")}
  191. }
  192. }
  193. }
  194. return nil
  195. }
  196. // EvidenceFront goes to the first evidence in the clist
  197. func (evpool *Pool) EvidenceFront() *clist.CElement {
  198. return evpool.evidenceList.Front()
  199. }
  200. // EvidenceWaitChan is a channel that closes once the first evidence in the list
  201. // is there. i.e Front is not nil.
  202. func (evpool *Pool) EvidenceWaitChan() <-chan struct{} {
  203. return evpool.evidenceList.WaitChan()
  204. }
  205. // Size returns the number of evidence in the pool.
  206. func (evpool *Pool) Size() uint32 {
  207. return atomic.LoadUint32(&evpool.evidenceSize)
  208. }
  209. // State returns the current state of the evpool.
  210. func (evpool *Pool) State() sm.State {
  211. evpool.mtx.Lock()
  212. defer evpool.mtx.Unlock()
  213. return evpool.state
  214. }
  215. // fastCheck leverages the fact that the evidence pool may have already verified
  216. // the evidence to see if it can quickly conclude that the evidence is already
  217. // valid.
  218. func (evpool *Pool) fastCheck(ev types.Evidence) bool {
  219. if lcae, ok := ev.(*types.LightClientAttackEvidence); ok {
  220. key := keyPending(ev)
  221. evBytes, err := evpool.evidenceStore.Get(key)
  222. if evBytes == nil { // the evidence is not in the nodes pending list
  223. return false
  224. }
  225. if err != nil {
  226. evpool.logger.Error("failed to load light client attack evidence", "err", err, "key(height/hash)", key)
  227. return false
  228. }
  229. var trustedPb tmproto.LightClientAttackEvidence
  230. if err = trustedPb.Unmarshal(evBytes); err != nil {
  231. evpool.logger.Error(
  232. "failed to convert light client attack evidence from bytes",
  233. "key(height/hash)", key,
  234. "err", err,
  235. )
  236. return false
  237. }
  238. trustedEv, err := types.LightClientAttackEvidenceFromProto(&trustedPb)
  239. if err != nil {
  240. evpool.logger.Error(
  241. "failed to convert light client attack evidence from protobuf",
  242. "key(height/hash)", key,
  243. "err", err,
  244. )
  245. return false
  246. }
  247. // Ensure that all the byzantine validators that the evidence pool has match
  248. // the byzantine validators in this evidence.
  249. if trustedEv.ByzantineValidators == nil && lcae.ByzantineValidators != nil {
  250. return false
  251. }
  252. if len(trustedEv.ByzantineValidators) != len(lcae.ByzantineValidators) {
  253. return false
  254. }
  255. byzValsCopy := make([]*types.Validator, len(lcae.ByzantineValidators))
  256. for i, v := range lcae.ByzantineValidators {
  257. byzValsCopy[i] = v.Copy()
  258. }
  259. // ensure that both validator arrays are in the same order
  260. sort.Sort(types.ValidatorsByVotingPower(byzValsCopy))
  261. for idx, val := range trustedEv.ByzantineValidators {
  262. if !bytes.Equal(byzValsCopy[idx].Address, val.Address) {
  263. return false
  264. }
  265. if byzValsCopy[idx].VotingPower != val.VotingPower {
  266. return false
  267. }
  268. }
  269. return true
  270. }
  271. // For all other evidence the evidence pool just checks if it is already in
  272. // the pending db.
  273. return evpool.isPending(ev)
  274. }
  275. // IsExpired checks whether evidence or a polc is expired by checking whether a height and time is older
  276. // than set by the evidence consensus parameters
  277. func (evpool *Pool) isExpired(height int64, time time.Time) bool {
  278. var (
  279. params = evpool.State().ConsensusParams.Evidence
  280. ageDuration = evpool.State().LastBlockTime.Sub(time)
  281. ageNumBlocks = evpool.State().LastBlockHeight - height
  282. )
  283. return ageNumBlocks > params.MaxAgeNumBlocks &&
  284. ageDuration > params.MaxAgeDuration
  285. }
  286. // IsCommitted returns true if we have already seen this exact evidence and it is already marked as committed.
  287. func (evpool *Pool) isCommitted(evidence types.Evidence) bool {
  288. key := keyCommitted(evidence)
  289. ok, err := evpool.evidenceStore.Has(key)
  290. if err != nil {
  291. evpool.logger.Error("failed to find committed evidence", "err", err)
  292. }
  293. return ok
  294. }
  295. // IsPending checks whether the evidence is already pending. DB errors are passed to the logger.
  296. func (evpool *Pool) isPending(evidence types.Evidence) bool {
  297. key := keyPending(evidence)
  298. ok, err := evpool.evidenceStore.Has(key)
  299. if err != nil {
  300. evpool.logger.Error("failed to find pending evidence", "err", err)
  301. }
  302. return ok
  303. }
  304. func (evpool *Pool) addPendingEvidence(ev types.Evidence) error {
  305. evpb, err := types.EvidenceToProto(ev)
  306. if err != nil {
  307. return fmt.Errorf("failed to convert to proto: %w", err)
  308. }
  309. evBytes, err := evpb.Marshal()
  310. if err != nil {
  311. return fmt.Errorf("failed to marshal evidence: %w", err)
  312. }
  313. key := keyPending(ev)
  314. err = evpool.evidenceStore.Set(key, evBytes)
  315. if err != nil {
  316. return fmt.Errorf("failed to persist evidence: %w", err)
  317. }
  318. atomic.AddUint32(&evpool.evidenceSize, 1)
  319. return nil
  320. }
  321. func (evpool *Pool) removePendingEvidence(evidence types.Evidence) {
  322. key := keyPending(evidence)
  323. if err := evpool.evidenceStore.Delete(key); err != nil {
  324. evpool.logger.Error("failed to delete pending evidence", "err", err)
  325. } else {
  326. atomic.AddUint32(&evpool.evidenceSize, ^uint32(0))
  327. evpool.logger.Info("deleted pending evidence", "evidence", evidence)
  328. }
  329. }
  330. // markEvidenceAsCommitted processes all the evidence in the block, marking it as
  331. // committed and removing it from the pending database.
  332. func (evpool *Pool) markEvidenceAsCommitted(evidence types.EvidenceList) {
  333. blockEvidenceMap := make(map[string]struct{}, len(evidence))
  334. for _, ev := range evidence {
  335. if evpool.isPending(ev) {
  336. evpool.removePendingEvidence(ev)
  337. blockEvidenceMap[evMapKey(ev)] = struct{}{}
  338. }
  339. // Add evidence to the committed list. As the evidence is stored in the block store
  340. // we only need to record the height that it was saved at.
  341. key := keyCommitted(ev)
  342. h := gogotypes.Int64Value{Value: ev.Height()}
  343. evBytes, err := proto.Marshal(&h)
  344. if err != nil {
  345. evpool.logger.Error("failed to marshal committed evidence", "key(height/hash)", key, "err", err)
  346. continue
  347. }
  348. if err := evpool.evidenceStore.Set(key, evBytes); err != nil {
  349. evpool.logger.Error("failed to save committed evidence", "key(height/hash)", key, "err", err)
  350. }
  351. evpool.logger.Info("marked evidence as committed", "evidence", ev)
  352. }
  353. // remove committed evidence from the clist
  354. if len(blockEvidenceMap) != 0 {
  355. evpool.removeEvidenceFromList(blockEvidenceMap)
  356. }
  357. }
  358. // listEvidence retrieves lists evidence from oldest to newest within maxBytes.
  359. // If maxBytes is -1, there's no cap on the size of returned evidence.
  360. func (evpool *Pool) listEvidence(prefixKey int64, maxBytes int64) ([]types.Evidence, int64, error) {
  361. var (
  362. evSize int64
  363. totalSize int64
  364. evidence []types.Evidence
  365. evList tmproto.EvidenceList // used for calculating the bytes size
  366. )
  367. iter, err := dbm.IteratePrefix(evpool.evidenceStore, prefixToBytes(prefixKey))
  368. if err != nil {
  369. return nil, totalSize, fmt.Errorf("database error: %v", err)
  370. }
  371. defer iter.Close()
  372. for ; iter.Valid(); iter.Next() {
  373. var evpb tmproto.Evidence
  374. if err := evpb.Unmarshal(iter.Value()); err != nil {
  375. return evidence, totalSize, err
  376. }
  377. evList.Evidence = append(evList.Evidence, evpb)
  378. evSize = int64(evList.Size())
  379. if maxBytes != -1 && evSize > maxBytes {
  380. if err := iter.Error(); err != nil {
  381. return evidence, totalSize, err
  382. }
  383. return evidence, totalSize, nil
  384. }
  385. ev, err := types.EvidenceFromProto(&evpb)
  386. if err != nil {
  387. return nil, totalSize, err
  388. }
  389. totalSize = evSize
  390. evidence = append(evidence, ev)
  391. }
  392. if err := iter.Error(); err != nil {
  393. return evidence, totalSize, err
  394. }
  395. return evidence, totalSize, nil
  396. }
  397. func (evpool *Pool) removeExpiredPendingEvidence() (int64, time.Time) {
  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
  402. }
  403. defer iter.Close()
  404. blockEvidenceMap := make(map[string]struct{})
  405. for ; iter.Valid(); iter.Next() {
  406. ev, err := bytesToEv(iter.Value())
  407. if err != nil {
  408. evpool.logger.Error("failed to transition evidence from protobuf", "err", err)
  409. continue
  410. }
  411. if !evpool.isExpired(ev.Height(), ev.Time()) {
  412. if len(blockEvidenceMap) != 0 {
  413. evpool.removeEvidenceFromList(blockEvidenceMap)
  414. }
  415. // Return the height and time with which this evidence will have expired
  416. // so we know when to prune next.
  417. return ev.Height() + evpool.State().ConsensusParams.Evidence.MaxAgeNumBlocks + 1,
  418. ev.Time().Add(evpool.State().ConsensusParams.Evidence.MaxAgeDuration).Add(time.Second)
  419. }
  420. evpool.removePendingEvidence(ev)
  421. blockEvidenceMap[evMapKey(ev)] = struct{}{}
  422. }
  423. // we either have no pending evidence or all evidence has expired
  424. if len(blockEvidenceMap) != 0 {
  425. evpool.removeEvidenceFromList(blockEvidenceMap)
  426. }
  427. return evpool.State().LastBlockHeight, evpool.State().LastBlockTime
  428. }
  429. func (evpool *Pool) removeEvidenceFromList(
  430. blockEvidenceMap map[string]struct{}) {
  431. for e := evpool.evidenceList.Front(); e != nil; e = e.Next() {
  432. // Remove from clist
  433. ev := e.Value.(types.Evidence)
  434. if _, ok := blockEvidenceMap[evMapKey(ev)]; ok {
  435. evpool.evidenceList.Remove(e)
  436. e.DetachPrev()
  437. }
  438. }
  439. }
  440. // flushConsensusBuffer moves the evidence produced from consensus into the evidence pool
  441. // and list so that it can be broadcasted and proposed
  442. func (evpool *Pool) flushConsensusBuffer() {
  443. for _, ev := range evpool.consensusBuffer {
  444. if err := evpool.addPendingEvidence(ev); err != nil {
  445. evpool.logger.Error("failed to flush evidence from consensus buffer to pending list: %w", err)
  446. continue
  447. }
  448. evpool.evidenceList.PushBack(ev)
  449. }
  450. // reset consensus buffer
  451. evpool.consensusBuffer = make([]types.Evidence, 0)
  452. }
  453. func bytesToEv(evBytes []byte) (types.Evidence, error) {
  454. var evpb tmproto.Evidence
  455. err := evpb.Unmarshal(evBytes)
  456. if err != nil {
  457. return &types.DuplicateVoteEvidence{}, err
  458. }
  459. return types.EvidenceFromProto(&evpb)
  460. }
  461. func evMapKey(ev types.Evidence) string {
  462. return string(ev.Hash())
  463. }
  464. func prefixToBytes(prefix int64) []byte {
  465. key, err := orderedcode.Append(nil, prefix)
  466. if err != nil {
  467. panic(err)
  468. }
  469. return key
  470. }
  471. func keyCommitted(evidence types.Evidence) []byte {
  472. var height int64 = evidence.Height()
  473. key, err := orderedcode.Append(nil, prefixCommitted, height, string(evidence.Hash()))
  474. if err != nil {
  475. panic(err)
  476. }
  477. return key
  478. }
  479. func keyPending(evidence types.Evidence) []byte {
  480. var height int64 = evidence.Height()
  481. key, err := orderedcode.Append(nil, prefixPending, height, string(evidence.Hash()))
  482. if err != nil {
  483. panic(err)
  484. }
  485. return key
  486. }