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.

450 lines
13 KiB

8 years ago
8 years ago
8 years ago
8 years ago
8 years ago
8 years ago
8 years ago
9 years ago
8 years ago
8 years ago
8 years ago
8 years ago
  1. package consensus
  2. import (
  3. "bytes"
  4. "fmt"
  5. "io/ioutil"
  6. "os"
  7. "path"
  8. "sort"
  9. "sync"
  10. "testing"
  11. "time"
  12. abcicli "github.com/tendermint/abci/client"
  13. abci "github.com/tendermint/abci/types"
  14. . "github.com/tendermint/go-common"
  15. cfg "github.com/tendermint/go-config"
  16. dbm "github.com/tendermint/go-db"
  17. "github.com/tendermint/go-p2p"
  18. bc "github.com/tendermint/tendermint/blockchain"
  19. "github.com/tendermint/tendermint/config/tendermint_test"
  20. mempl "github.com/tendermint/tendermint/mempool"
  21. sm "github.com/tendermint/tendermint/state"
  22. "github.com/tendermint/tendermint/types"
  23. "github.com/tendermint/abci/example/counter"
  24. "github.com/tendermint/abci/example/dummy"
  25. )
  26. var config cfg.Config // NOTE: must be reset for each _test.go file
  27. var ensureTimeout = time.Duration(2)
  28. func ensureDir(dir string, mode os.FileMode) {
  29. if err := EnsureDir(dir, mode); err != nil {
  30. panic(err)
  31. }
  32. }
  33. //-------------------------------------------------------------------------------
  34. // validator stub (a dummy consensus peer we control)
  35. type validatorStub struct {
  36. Index int // Validator index. NOTE: we don't assume validator set changes.
  37. Height int
  38. Round int
  39. *types.PrivValidator
  40. }
  41. var testMinPower = 10
  42. func NewValidatorStub(privValidator *types.PrivValidator, valIndex int) *validatorStub {
  43. return &validatorStub{
  44. Index: valIndex,
  45. PrivValidator: privValidator,
  46. }
  47. }
  48. func (vs *validatorStub) signVote(voteType byte, hash []byte, header types.PartSetHeader) (*types.Vote, error) {
  49. vote := &types.Vote{
  50. ValidatorIndex: vs.Index,
  51. ValidatorAddress: vs.PrivValidator.Address,
  52. Height: vs.Height,
  53. Round: vs.Round,
  54. Type: voteType,
  55. BlockID: types.BlockID{hash, header},
  56. }
  57. err := vs.PrivValidator.SignVote(config.GetString("chain_id"), vote)
  58. return vote, err
  59. }
  60. // Sign vote for type/hash/header
  61. func signVote(vs *validatorStub, voteType byte, hash []byte, header types.PartSetHeader) *types.Vote {
  62. v, err := vs.signVote(voteType, hash, header)
  63. if err != nil {
  64. panic(fmt.Errorf("failed to sign vote: %v", err))
  65. }
  66. return v
  67. }
  68. func signVotes(voteType byte, hash []byte, header types.PartSetHeader, vss ...*validatorStub) []*types.Vote {
  69. votes := make([]*types.Vote, len(vss))
  70. for i, vs := range vss {
  71. votes[i] = signVote(vs, voteType, hash, header)
  72. }
  73. return votes
  74. }
  75. func incrementHeight(vss ...*validatorStub) {
  76. for _, vs := range vss {
  77. vs.Height += 1
  78. }
  79. }
  80. func incrementRound(vss ...*validatorStub) {
  81. for _, vs := range vss {
  82. vs.Round += 1
  83. }
  84. }
  85. //-------------------------------------------------------------------------------
  86. // Functions for transitioning the consensus state
  87. func startTestRound(cs *ConsensusState, height, round int) {
  88. cs.enterNewRound(height, round)
  89. cs.startRoutines(0)
  90. }
  91. // Create proposal block from cs1 but sign it with vs
  92. func decideProposal(cs1 *ConsensusState, vs *validatorStub, height, round int) (proposal *types.Proposal, block *types.Block) {
  93. block, blockParts := cs1.createProposalBlock()
  94. if block == nil { // on error
  95. panic("error creating proposal block")
  96. }
  97. // Make proposal
  98. polRound, polBlockID := cs1.Votes.POLInfo()
  99. proposal = types.NewProposal(height, round, blockParts.Header(), polRound, polBlockID)
  100. if err := vs.SignProposal(config.GetString("chain_id"), proposal); err != nil {
  101. panic(err)
  102. }
  103. return
  104. }
  105. func addVotes(to *ConsensusState, votes ...*types.Vote) {
  106. for _, vote := range votes {
  107. to.peerMsgQueue <- msgInfo{Msg: &VoteMessage{vote}}
  108. }
  109. }
  110. func signAddVotes(to *ConsensusState, voteType byte, hash []byte, header types.PartSetHeader, vss ...*validatorStub) {
  111. votes := signVotes(voteType, hash, header, vss...)
  112. addVotes(to, votes...)
  113. }
  114. func validatePrevote(t *testing.T, cs *ConsensusState, round int, privVal *validatorStub, blockHash []byte) {
  115. prevotes := cs.Votes.Prevotes(round)
  116. var vote *types.Vote
  117. if vote = prevotes.GetByAddress(privVal.Address); vote == nil {
  118. panic("Failed to find prevote from validator")
  119. }
  120. if blockHash == nil {
  121. if vote.BlockID.Hash != nil {
  122. panic(fmt.Sprintf("Expected prevote to be for nil, got %X", vote.BlockID.Hash))
  123. }
  124. } else {
  125. if !bytes.Equal(vote.BlockID.Hash, blockHash) {
  126. panic(fmt.Sprintf("Expected prevote to be for %X, got %X", blockHash, vote.BlockID.Hash))
  127. }
  128. }
  129. }
  130. func validateLastPrecommit(t *testing.T, cs *ConsensusState, privVal *validatorStub, blockHash []byte) {
  131. votes := cs.LastCommit
  132. var vote *types.Vote
  133. if vote = votes.GetByAddress(privVal.Address); vote == nil {
  134. panic("Failed to find precommit from validator")
  135. }
  136. if !bytes.Equal(vote.BlockID.Hash, blockHash) {
  137. panic(fmt.Sprintf("Expected precommit to be for %X, got %X", blockHash, vote.BlockID.Hash))
  138. }
  139. }
  140. func validatePrecommit(t *testing.T, cs *ConsensusState, thisRound, lockRound int, privVal *validatorStub, votedBlockHash, lockedBlockHash []byte) {
  141. precommits := cs.Votes.Precommits(thisRound)
  142. var vote *types.Vote
  143. if vote = precommits.GetByAddress(privVal.Address); vote == nil {
  144. panic("Failed to find precommit from validator")
  145. }
  146. if votedBlockHash == nil {
  147. if vote.BlockID.Hash != nil {
  148. panic("Expected precommit to be for nil")
  149. }
  150. } else {
  151. if !bytes.Equal(vote.BlockID.Hash, votedBlockHash) {
  152. panic("Expected precommit to be for proposal block")
  153. }
  154. }
  155. if lockedBlockHash == nil {
  156. if cs.LockedRound != lockRound || cs.LockedBlock != nil {
  157. panic(fmt.Sprintf("Expected to be locked on nil at round %d. Got locked at round %d with block %v", lockRound, cs.LockedRound, cs.LockedBlock))
  158. }
  159. } else {
  160. if cs.LockedRound != lockRound || !bytes.Equal(cs.LockedBlock.Hash(), lockedBlockHash) {
  161. panic(fmt.Sprintf("Expected block to be locked on round %d, got %d. Got locked block %X, expected %X", lockRound, cs.LockedRound, cs.LockedBlock.Hash(), lockedBlockHash))
  162. }
  163. }
  164. }
  165. func validatePrevoteAndPrecommit(t *testing.T, cs *ConsensusState, thisRound, lockRound int, privVal *validatorStub, votedBlockHash, lockedBlockHash []byte) {
  166. // verify the prevote
  167. validatePrevote(t, cs, thisRound, privVal, votedBlockHash)
  168. // verify precommit
  169. cs.mtx.Lock()
  170. validatePrecommit(t, cs, thisRound, lockRound, privVal, votedBlockHash, lockedBlockHash)
  171. cs.mtx.Unlock()
  172. }
  173. // genesis
  174. func subscribeToVoter(cs *ConsensusState, addr []byte) chan interface{} {
  175. voteCh0 := subscribeToEvent(cs.evsw, "tester", types.EventStringVote(), 1)
  176. voteCh := make(chan interface{})
  177. go func() {
  178. for {
  179. v := <-voteCh0
  180. vote := v.(types.EventDataVote)
  181. // we only fire for our own votes
  182. if bytes.Equal(addr, vote.Vote.ValidatorAddress) {
  183. voteCh <- v
  184. }
  185. }
  186. }()
  187. return voteCh
  188. }
  189. func readVotes(ch chan interface{}, reads int) chan struct{} {
  190. wg := make(chan struct{})
  191. go func() {
  192. for i := 0; i < reads; i++ {
  193. <-ch // read the precommit event
  194. }
  195. close(wg)
  196. }()
  197. return wg
  198. }
  199. //-------------------------------------------------------------------------------
  200. // consensus states
  201. func newConsensusState(state *sm.State, pv *types.PrivValidator, app abci.Application) *ConsensusState {
  202. return newConsensusStateWithConfig(config, state, pv, app)
  203. }
  204. func newConsensusStateWithConfig(thisConfig cfg.Config, state *sm.State, pv *types.PrivValidator, app abci.Application) *ConsensusState {
  205. // Get BlockStore
  206. blockDB := dbm.NewMemDB()
  207. blockStore := bc.NewBlockStore(blockDB)
  208. // one for mempool, one for consensus
  209. mtx := new(sync.Mutex)
  210. proxyAppConnMem := abcicli.NewLocalClient(mtx, app)
  211. proxyAppConnCon := abcicli.NewLocalClient(mtx, app)
  212. // Make Mempool
  213. mempool := mempl.NewMempool(thisConfig, proxyAppConnMem)
  214. // Make ConsensusReactor
  215. cs := NewConsensusState(thisConfig, state, proxyAppConnCon, blockStore, mempool)
  216. cs.SetPrivValidator(pv)
  217. evsw := types.NewEventSwitch()
  218. cs.SetEventSwitch(evsw)
  219. evsw.Start()
  220. return cs
  221. }
  222. func loadPrivValidator(conf cfg.Config) *types.PrivValidator {
  223. privValidatorFile := conf.GetString("priv_validator_file")
  224. ensureDir(path.Dir(privValidatorFile), 0700)
  225. privValidator := types.LoadOrGenPrivValidator(privValidatorFile)
  226. privValidator.Reset()
  227. return privValidator
  228. }
  229. func fixedConsensusState() *ConsensusState {
  230. stateDB := dbm.NewMemDB()
  231. state := sm.MakeGenesisStateFromFile(stateDB, config.GetString("genesis_file"))
  232. privValidator := loadPrivValidator(config)
  233. cs := newConsensusState(state, privValidator, counter.NewCounterApplication(true))
  234. return cs
  235. }
  236. func fixedConsensusStateDummy() *ConsensusState {
  237. stateDB := dbm.NewMemDB()
  238. state := sm.MakeGenesisStateFromFile(stateDB, config.GetString("genesis_file"))
  239. privValidator := loadPrivValidator(config)
  240. cs := newConsensusState(state, privValidator, dummy.NewDummyApplication())
  241. return cs
  242. }
  243. func randConsensusState(nValidators int) (*ConsensusState, []*validatorStub) {
  244. // Get State
  245. state, privVals := randGenesisState(nValidators, false, 10)
  246. vss := make([]*validatorStub, nValidators)
  247. cs := newConsensusState(state, privVals[0], counter.NewCounterApplication(true))
  248. for i := 0; i < nValidators; i++ {
  249. vss[i] = NewValidatorStub(privVals[i], i)
  250. }
  251. // since cs1 starts at 1
  252. incrementHeight(vss[1:]...)
  253. return cs, vss
  254. }
  255. //-------------------------------------------------------------------------------
  256. func ensureNoNewStep(stepCh chan interface{}) {
  257. timeout := time.NewTicker(ensureTimeout * time.Second)
  258. select {
  259. case <-timeout.C:
  260. break
  261. case <-stepCh:
  262. panic("We should be stuck waiting for more votes, not moving to the next step")
  263. }
  264. }
  265. //-------------------------------------------------------------------------------
  266. // consensus nets
  267. func randConsensusNet(nValidators int, testName string, tickerFunc func() TimeoutTicker, appFunc func() abci.Application) []*ConsensusState {
  268. genDoc, privVals := randGenesisDoc(nValidators, false, 10)
  269. css := make([]*ConsensusState, nValidators)
  270. for i := 0; i < nValidators; i++ {
  271. db := dbm.NewMemDB() // each state needs its own db
  272. state := sm.MakeGenesisState(db, genDoc)
  273. state.Save()
  274. thisConfig := tendermint_test.ResetConfig(Fmt("%s_%d", testName, i))
  275. ensureDir(path.Dir(thisConfig.GetString("cs_wal_file")), 0700) // dir for wal
  276. css[i] = newConsensusStateWithConfig(thisConfig, state, privVals[i], appFunc())
  277. css[i].SetTimeoutTicker(tickerFunc())
  278. }
  279. return css
  280. }
  281. // nPeers = nValidators + nNotValidator
  282. func randConsensusNetWithPeers(nValidators, nPeers int, testName string, tickerFunc func() TimeoutTicker, appFunc func() abci.Application) []*ConsensusState {
  283. genDoc, privVals := randGenesisDoc(nValidators, false, int64(testMinPower))
  284. css := make([]*ConsensusState, nPeers)
  285. for i := 0; i < nPeers; i++ {
  286. db := dbm.NewMemDB() // each state needs its own db
  287. state := sm.MakeGenesisState(db, genDoc)
  288. state.Save()
  289. thisConfig := tendermint_test.ResetConfig(Fmt("%s_%d", testName, i))
  290. ensureDir(path.Dir(thisConfig.GetString("cs_wal_file")), 0700) // dir for wal
  291. var privVal *types.PrivValidator
  292. if i < nValidators {
  293. privVal = privVals[i]
  294. } else {
  295. privVal = types.GenPrivValidator()
  296. _, tempFilePath := Tempfile("priv_validator_")
  297. privVal.SetFile(tempFilePath)
  298. }
  299. css[i] = newConsensusStateWithConfig(thisConfig, state, privVal, appFunc())
  300. css[i].SetTimeoutTicker(tickerFunc())
  301. }
  302. return css
  303. }
  304. func getSwitchIndex(switches []*p2p.Switch, peer *p2p.Peer) int {
  305. for i, s := range switches {
  306. if bytes.Equal(peer.NodeInfo.PubKey.Address(), s.NodeInfo().PubKey.Address()) {
  307. return i
  308. }
  309. }
  310. panic("didnt find peer in switches")
  311. return -1
  312. }
  313. //-------------------------------------------------------------------------------
  314. // genesis
  315. func randGenesisDoc(numValidators int, randPower bool, minPower int64) (*types.GenesisDoc, []*types.PrivValidator) {
  316. validators := make([]types.GenesisValidator, numValidators)
  317. privValidators := make([]*types.PrivValidator, numValidators)
  318. for i := 0; i < numValidators; i++ {
  319. val, privVal := types.RandValidator(randPower, minPower)
  320. validators[i] = types.GenesisValidator{
  321. PubKey: val.PubKey,
  322. Amount: val.VotingPower,
  323. }
  324. privValidators[i] = privVal
  325. }
  326. sort.Sort(types.PrivValidatorsByAddress(privValidators))
  327. return &types.GenesisDoc{
  328. GenesisTime: time.Now(),
  329. ChainID: config.GetString("chain_id"),
  330. Validators: validators,
  331. }, privValidators
  332. }
  333. func randGenesisState(numValidators int, randPower bool, minPower int64) (*sm.State, []*types.PrivValidator) {
  334. genDoc, privValidators := randGenesisDoc(numValidators, randPower, minPower)
  335. db := dbm.NewMemDB()
  336. s0 := sm.MakeGenesisState(db, genDoc)
  337. s0.Save()
  338. return s0, privValidators
  339. }
  340. //------------------------------------
  341. // mock ticker
  342. func newMockTickerFunc(onlyOnce bool) func() TimeoutTicker {
  343. return func() TimeoutTicker {
  344. return &mockTicker{
  345. c: make(chan timeoutInfo, 10),
  346. onlyOnce: onlyOnce,
  347. }
  348. }
  349. }
  350. // mock ticker only fires on RoundStepNewHeight
  351. // and only once if onlyOnce=true
  352. type mockTicker struct {
  353. c chan timeoutInfo
  354. mtx sync.Mutex
  355. onlyOnce bool
  356. fired bool
  357. }
  358. func (m *mockTicker) Start() (bool, error) {
  359. return true, nil
  360. }
  361. func (m *mockTicker) Stop() bool {
  362. return true
  363. }
  364. func (m *mockTicker) ScheduleTimeout(ti timeoutInfo) {
  365. m.mtx.Lock()
  366. defer m.mtx.Unlock()
  367. if m.onlyOnce && m.fired {
  368. return
  369. }
  370. if ti.Step == RoundStepNewHeight {
  371. m.c <- ti
  372. m.fired = true
  373. }
  374. }
  375. func (m *mockTicker) Chan() <-chan timeoutInfo {
  376. return m.c
  377. }
  378. //------------------------------------
  379. func newCounter() abci.Application {
  380. return counter.NewCounterApplication(true)
  381. }
  382. func newPersistentDummy() abci.Application {
  383. dir, _ := ioutil.TempDir("/tmp", "persistent-dummy")
  384. return dummy.NewPersistentDummyApplication(dir)
  385. }