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.

685 lines
20 KiB

  1. package consensus
  2. import (
  3. "bytes"
  4. "context"
  5. "fmt"
  6. "io"
  7. "io/ioutil"
  8. "os"
  9. "path"
  10. "runtime"
  11. "testing"
  12. "time"
  13. "github.com/stretchr/testify/assert"
  14. "github.com/stretchr/testify/require"
  15. "github.com/tendermint/tendermint/abci/example/kvstore"
  16. abci "github.com/tendermint/tendermint/abci/types"
  17. crypto "github.com/tendermint/tendermint/crypto"
  18. auto "github.com/tendermint/tendermint/libs/autofile"
  19. dbm "github.com/tendermint/tendermint/libs/db"
  20. "github.com/tendermint/tendermint/version"
  21. cfg "github.com/tendermint/tendermint/config"
  22. "github.com/tendermint/tendermint/libs/log"
  23. "github.com/tendermint/tendermint/privval"
  24. "github.com/tendermint/tendermint/proxy"
  25. sm "github.com/tendermint/tendermint/state"
  26. "github.com/tendermint/tendermint/types"
  27. )
  28. var consensusReplayConfig *cfg.Config
  29. func init() {
  30. consensusReplayConfig = ResetConfig("consensus_replay_test")
  31. }
  32. // These tests ensure we can always recover from failure at any part of the consensus process.
  33. // There are two general failure scenarios: failure during consensus, and failure while applying the block.
  34. // Only the latter interacts with the app and store,
  35. // but the former has to deal with restrictions on re-use of priv_validator keys.
  36. // The `WAL Tests` are for failures during the consensus;
  37. // the `Handshake Tests` are for failures in applying the block.
  38. // With the help of the WAL, we can recover from it all!
  39. //------------------------------------------------------------------------------------------
  40. // WAL Tests
  41. // TODO: It would be better to verify explicitly which states we can recover from without the wal
  42. // and which ones we need the wal for - then we'd also be able to only flush the
  43. // wal writer when we need to, instead of with every message.
  44. func startNewConsensusStateAndWaitForBlock(t *testing.T, lastBlockHeight int64, blockDB dbm.DB, stateDB dbm.DB) {
  45. logger := log.TestingLogger()
  46. state, _ := sm.LoadStateFromDBOrGenesisFile(stateDB, consensusReplayConfig.GenesisFile())
  47. privValidator := loadPrivValidator(consensusReplayConfig)
  48. cs := newConsensusStateWithConfigAndBlockStore(consensusReplayConfig, state, privValidator, kvstore.NewKVStoreApplication(), blockDB)
  49. cs.SetLogger(logger)
  50. bytes, _ := ioutil.ReadFile(cs.config.WalFile())
  51. // fmt.Printf("====== WAL: \n\r%s\n", bytes)
  52. t.Logf("====== WAL: \n\r%X\n", bytes)
  53. err := cs.Start()
  54. require.NoError(t, err)
  55. defer cs.Stop()
  56. // This is just a signal that we haven't halted; its not something contained
  57. // in the WAL itself. Assuming the consensus state is running, replay of any
  58. // WAL, including the empty one, should eventually be followed by a new
  59. // block, or else something is wrong.
  60. newBlockCh := make(chan interface{}, 1)
  61. err = cs.eventBus.Subscribe(context.Background(), testSubscriber, types.EventQueryNewBlock, newBlockCh)
  62. require.NoError(t, err)
  63. select {
  64. case <-newBlockCh:
  65. case <-time.After(60 * time.Second):
  66. t.Fatalf("Timed out waiting for new block (see trace above)")
  67. }
  68. }
  69. func sendTxs(cs *ConsensusState, ctx context.Context) {
  70. for i := 0; i < 256; i++ {
  71. select {
  72. case <-ctx.Done():
  73. return
  74. default:
  75. tx := []byte{byte(i)}
  76. cs.mempool.CheckTx(tx, nil)
  77. i++
  78. }
  79. }
  80. }
  81. // TestWALCrash uses crashing WAL to test we can recover from any WAL failure.
  82. func TestWALCrash(t *testing.T) {
  83. testCases := []struct {
  84. name string
  85. initFn func(dbm.DB, *ConsensusState, context.Context)
  86. heightToStop int64
  87. }{
  88. {"empty block",
  89. func(stateDB dbm.DB, cs *ConsensusState, ctx context.Context) {},
  90. 1},
  91. {"many non-empty blocks",
  92. func(stateDB dbm.DB, cs *ConsensusState, ctx context.Context) {
  93. go sendTxs(cs, ctx)
  94. },
  95. 3},
  96. }
  97. for _, tc := range testCases {
  98. t.Run(tc.name, func(t *testing.T) {
  99. crashWALandCheckLiveness(t, tc.initFn, tc.heightToStop)
  100. })
  101. }
  102. }
  103. func crashWALandCheckLiveness(t *testing.T, initFn func(dbm.DB, *ConsensusState, context.Context), heightToStop int64) {
  104. walPaniced := make(chan error)
  105. crashingWal := &crashingWAL{panicCh: walPaniced, heightToStop: heightToStop}
  106. i := 1
  107. LOOP:
  108. for {
  109. // fmt.Printf("====== LOOP %d\n", i)
  110. t.Logf("====== LOOP %d\n", i)
  111. // create consensus state from a clean slate
  112. logger := log.NewNopLogger()
  113. stateDB := dbm.NewMemDB()
  114. state, _ := sm.MakeGenesisStateFromFile(consensusReplayConfig.GenesisFile())
  115. privValidator := loadPrivValidator(consensusReplayConfig)
  116. blockDB := dbm.NewMemDB()
  117. cs := newConsensusStateWithConfigAndBlockStore(consensusReplayConfig, state, privValidator, kvstore.NewKVStoreApplication(), blockDB)
  118. cs.SetLogger(logger)
  119. // start sending transactions
  120. ctx, cancel := context.WithCancel(context.Background())
  121. initFn(stateDB, cs, ctx)
  122. // clean up WAL file from the previous iteration
  123. walFile := cs.config.WalFile()
  124. os.Remove(walFile)
  125. // set crashing WAL
  126. csWal, err := cs.OpenWAL(walFile)
  127. require.NoError(t, err)
  128. crashingWal.next = csWal
  129. // reset the message counter
  130. crashingWal.msgIndex = 1
  131. cs.wal = crashingWal
  132. // start consensus state
  133. err = cs.Start()
  134. require.NoError(t, err)
  135. i++
  136. select {
  137. case err := <-walPaniced:
  138. t.Logf("WAL paniced: %v", err)
  139. // make sure we can make blocks after a crash
  140. startNewConsensusStateAndWaitForBlock(t, cs.Height, blockDB, stateDB)
  141. // stop consensus state and transactions sender (initFn)
  142. cs.Stop()
  143. cancel()
  144. // if we reached the required height, exit
  145. if _, ok := err.(ReachedHeightToStopError); ok {
  146. break LOOP
  147. }
  148. case <-time.After(10 * time.Second):
  149. t.Fatal("WAL did not panic for 10 seconds (check the log)")
  150. }
  151. }
  152. }
  153. // crashingWAL is a WAL which crashes or rather simulates a crash during Save
  154. // (before and after). It remembers a message for which we last panicked
  155. // (lastPanicedForMsgIndex), so we don't panic for it in subsequent iterations.
  156. type crashingWAL struct {
  157. next WAL
  158. panicCh chan error
  159. heightToStop int64
  160. msgIndex int // current message index
  161. lastPanicedForMsgIndex int // last message for which we panicked
  162. }
  163. // WALWriteError indicates a WAL crash.
  164. type WALWriteError struct {
  165. msg string
  166. }
  167. func (e WALWriteError) Error() string {
  168. return e.msg
  169. }
  170. // ReachedHeightToStopError indicates we've reached the required consensus
  171. // height and may exit.
  172. type ReachedHeightToStopError struct {
  173. height int64
  174. }
  175. func (e ReachedHeightToStopError) Error() string {
  176. return fmt.Sprintf("reached height to stop %d", e.height)
  177. }
  178. // Write simulate WAL's crashing by sending an error to the panicCh and then
  179. // exiting the cs.receiveRoutine.
  180. func (w *crashingWAL) Write(m WALMessage) {
  181. if endMsg, ok := m.(EndHeightMessage); ok {
  182. if endMsg.Height == w.heightToStop {
  183. w.panicCh <- ReachedHeightToStopError{endMsg.Height}
  184. runtime.Goexit()
  185. } else {
  186. w.next.Write(m)
  187. }
  188. return
  189. }
  190. if w.msgIndex > w.lastPanicedForMsgIndex {
  191. w.lastPanicedForMsgIndex = w.msgIndex
  192. _, file, line, _ := runtime.Caller(1)
  193. w.panicCh <- WALWriteError{fmt.Sprintf("failed to write %T to WAL (fileline: %s:%d)", m, file, line)}
  194. runtime.Goexit()
  195. } else {
  196. w.msgIndex++
  197. w.next.Write(m)
  198. }
  199. }
  200. func (w *crashingWAL) WriteSync(m WALMessage) {
  201. w.Write(m)
  202. }
  203. func (w *crashingWAL) Group() *auto.Group { return w.next.Group() }
  204. func (w *crashingWAL) SearchForEndHeight(height int64, options *WALSearchOptions) (gr *auto.GroupReader, found bool, err error) {
  205. return w.next.SearchForEndHeight(height, options)
  206. }
  207. func (w *crashingWAL) Start() error { return w.next.Start() }
  208. func (w *crashingWAL) Stop() error { return w.next.Stop() }
  209. func (w *crashingWAL) Wait() { w.next.Wait() }
  210. //------------------------------------------------------------------------------------------
  211. // Handshake Tests
  212. const (
  213. NUM_BLOCKS = 6
  214. )
  215. var (
  216. mempool = sm.MockMempool{}
  217. evpool = sm.MockEvidencePool{}
  218. )
  219. //---------------------------------------
  220. // Test handshake/replay
  221. // 0 - all synced up
  222. // 1 - saved block but app and state are behind
  223. // 2 - save block and committed but state is behind
  224. var modes = []uint{0, 1, 2}
  225. // Sync from scratch
  226. func TestHandshakeReplayAll(t *testing.T) {
  227. for _, m := range modes {
  228. testHandshakeReplay(t, 0, m)
  229. }
  230. }
  231. // Sync many, not from scratch
  232. func TestHandshakeReplaySome(t *testing.T) {
  233. for _, m := range modes {
  234. testHandshakeReplay(t, 1, m)
  235. }
  236. }
  237. // Sync from lagging by one
  238. func TestHandshakeReplayOne(t *testing.T) {
  239. for _, m := range modes {
  240. testHandshakeReplay(t, NUM_BLOCKS-1, m)
  241. }
  242. }
  243. // Sync from caught up
  244. func TestHandshakeReplayNone(t *testing.T) {
  245. for _, m := range modes {
  246. testHandshakeReplay(t, NUM_BLOCKS, m)
  247. }
  248. }
  249. func tempWALWithData(data []byte) string {
  250. walFile, err := ioutil.TempFile("", "wal")
  251. if err != nil {
  252. panic(fmt.Errorf("failed to create temp WAL file: %v", err))
  253. }
  254. _, err = walFile.Write(data)
  255. if err != nil {
  256. panic(fmt.Errorf("failed to write to temp WAL file: %v", err))
  257. }
  258. if err := walFile.Close(); err != nil {
  259. panic(fmt.Errorf("failed to close temp WAL file: %v", err))
  260. }
  261. return walFile.Name()
  262. }
  263. // Make some blocks. Start a fresh app and apply nBlocks blocks. Then restart the app and sync it up with the remaining blocks
  264. func testHandshakeReplay(t *testing.T, nBlocks int, mode uint) {
  265. config := ResetConfig("proxy_test_")
  266. walBody, err := WALWithNBlocks(NUM_BLOCKS)
  267. if err != nil {
  268. t.Fatal(err)
  269. }
  270. walFile := tempWALWithData(walBody)
  271. config.Consensus.SetWalFile(walFile)
  272. privVal := privval.LoadFilePV(config.PrivValidatorFile())
  273. wal, err := NewWAL(walFile)
  274. if err != nil {
  275. t.Fatal(err)
  276. }
  277. wal.SetLogger(log.TestingLogger())
  278. if err := wal.Start(); err != nil {
  279. t.Fatal(err)
  280. }
  281. defer wal.Stop()
  282. chain, commits, err := makeBlockchainFromWAL(wal)
  283. if err != nil {
  284. t.Fatalf(err.Error())
  285. }
  286. stateDB, state, store := stateAndStore(config, privVal.GetPubKey(), kvstore.ProtocolVersion)
  287. store.chain = chain
  288. store.commits = commits
  289. // run the chain through state.ApplyBlock to build up the tendermint state
  290. state = buildTMStateFromChain(config, stateDB, state, chain, mode)
  291. latestAppHash := state.AppHash
  292. // make a new client creator
  293. kvstoreApp := kvstore.NewPersistentKVStoreApplication(path.Join(config.DBDir(), "2"))
  294. clientCreator2 := proxy.NewLocalClientCreator(kvstoreApp)
  295. if nBlocks > 0 {
  296. // run nBlocks against a new client to build up the app state.
  297. // use a throwaway tendermint state
  298. proxyApp := proxy.NewAppConns(clientCreator2)
  299. stateDB, state, _ := stateAndStore(config, privVal.GetPubKey(), kvstore.ProtocolVersion)
  300. buildAppStateFromChain(proxyApp, stateDB, state, chain, nBlocks, mode)
  301. }
  302. // now start the app using the handshake - it should sync
  303. genDoc, _ := sm.MakeGenesisDocFromFile(config.GenesisFile())
  304. handshaker := NewHandshaker(stateDB, state, store, genDoc)
  305. proxyApp := proxy.NewAppConns(clientCreator2)
  306. if err := proxyApp.Start(); err != nil {
  307. t.Fatalf("Error starting proxy app connections: %v", err)
  308. }
  309. defer proxyApp.Stop()
  310. if err := handshaker.Handshake(proxyApp); err != nil {
  311. t.Fatalf("Error on abci handshake: %v", err)
  312. }
  313. // get the latest app hash from the app
  314. res, err := proxyApp.Query().InfoSync(abci.RequestInfo{Version: ""})
  315. if err != nil {
  316. t.Fatal(err)
  317. }
  318. // the app hash should be synced up
  319. if !bytes.Equal(latestAppHash, res.LastBlockAppHash) {
  320. t.Fatalf("Expected app hashes to match after handshake/replay. got %X, expected %X", res.LastBlockAppHash, latestAppHash)
  321. }
  322. expectedBlocksToSync := NUM_BLOCKS - nBlocks
  323. if nBlocks == NUM_BLOCKS && mode > 0 {
  324. expectedBlocksToSync++
  325. } else if nBlocks > 0 && mode == 1 {
  326. expectedBlocksToSync++
  327. }
  328. if handshaker.NBlocks() != expectedBlocksToSync {
  329. t.Fatalf("Expected handshake to sync %d blocks, got %d", expectedBlocksToSync, handshaker.NBlocks())
  330. }
  331. }
  332. func applyBlock(stateDB dbm.DB, st sm.State, blk *types.Block, proxyApp proxy.AppConns) sm.State {
  333. testPartSize := types.BlockPartSizeBytes
  334. blockExec := sm.NewBlockExecutor(stateDB, log.TestingLogger(), proxyApp.Consensus(), mempool, evpool)
  335. blkID := types.BlockID{blk.Hash(), blk.MakePartSet(testPartSize).Header()}
  336. newState, err := blockExec.ApplyBlock(st, blkID, blk)
  337. if err != nil {
  338. panic(err)
  339. }
  340. return newState
  341. }
  342. func buildAppStateFromChain(proxyApp proxy.AppConns, stateDB dbm.DB,
  343. state sm.State, chain []*types.Block, nBlocks int, mode uint) {
  344. // start a new app without handshake, play nBlocks blocks
  345. if err := proxyApp.Start(); err != nil {
  346. panic(err)
  347. }
  348. defer proxyApp.Stop()
  349. validators := types.TM2PB.ValidatorUpdates(state.Validators)
  350. if _, err := proxyApp.Consensus().InitChainSync(abci.RequestInitChain{
  351. Validators: validators,
  352. }); err != nil {
  353. panic(err)
  354. }
  355. switch mode {
  356. case 0:
  357. for i := 0; i < nBlocks; i++ {
  358. block := chain[i]
  359. state = applyBlock(stateDB, state, block, proxyApp)
  360. }
  361. case 1, 2:
  362. for i := 0; i < nBlocks-1; i++ {
  363. block := chain[i]
  364. state = applyBlock(stateDB, state, block, proxyApp)
  365. }
  366. if mode == 2 {
  367. // update the kvstore height and apphash
  368. // as if we ran commit but not
  369. state = applyBlock(stateDB, state, chain[nBlocks-1], proxyApp)
  370. }
  371. }
  372. }
  373. func buildTMStateFromChain(config *cfg.Config, stateDB dbm.DB, state sm.State, chain []*types.Block, mode uint) sm.State {
  374. // run the whole chain against this client to build up the tendermint state
  375. clientCreator := proxy.NewLocalClientCreator(kvstore.NewPersistentKVStoreApplication(path.Join(config.DBDir(), "1")))
  376. proxyApp := proxy.NewAppConns(clientCreator)
  377. if err := proxyApp.Start(); err != nil {
  378. panic(err)
  379. }
  380. defer proxyApp.Stop()
  381. validators := types.TM2PB.ValidatorUpdates(state.Validators)
  382. if _, err := proxyApp.Consensus().InitChainSync(abci.RequestInitChain{
  383. Validators: validators,
  384. }); err != nil {
  385. panic(err)
  386. }
  387. switch mode {
  388. case 0:
  389. // sync right up
  390. for _, block := range chain {
  391. state = applyBlock(stateDB, state, block, proxyApp)
  392. }
  393. case 1, 2:
  394. // sync up to the penultimate as if we stored the block.
  395. // whether we commit or not depends on the appHash
  396. for _, block := range chain[:len(chain)-1] {
  397. state = applyBlock(stateDB, state, block, proxyApp)
  398. }
  399. // apply the final block to a state copy so we can
  400. // get the right next appHash but keep the state back
  401. applyBlock(stateDB, state, chain[len(chain)-1], proxyApp)
  402. }
  403. return state
  404. }
  405. //--------------------------
  406. // utils for making blocks
  407. func makeBlockchainFromWAL(wal WAL) ([]*types.Block, []*types.Commit, error) {
  408. // Search for height marker
  409. gr, found, err := wal.SearchForEndHeight(0, &WALSearchOptions{})
  410. if err != nil {
  411. return nil, nil, err
  412. }
  413. if !found {
  414. return nil, nil, fmt.Errorf("WAL does not contain height %d.", 1)
  415. }
  416. defer gr.Close() // nolint: errcheck
  417. // log.Notice("Build a blockchain by reading from the WAL")
  418. var blocks []*types.Block
  419. var commits []*types.Commit
  420. var thisBlockParts *types.PartSet
  421. var thisBlockCommit *types.Commit
  422. var height int64
  423. dec := NewWALDecoder(gr)
  424. for {
  425. msg, err := dec.Decode()
  426. if err == io.EOF {
  427. break
  428. } else if err != nil {
  429. return nil, nil, err
  430. }
  431. piece := readPieceFromWAL(msg)
  432. if piece == nil {
  433. continue
  434. }
  435. switch p := piece.(type) {
  436. case EndHeightMessage:
  437. // if its not the first one, we have a full block
  438. if thisBlockParts != nil {
  439. var block = new(types.Block)
  440. _, err = cdc.UnmarshalBinaryLengthPrefixedReader(thisBlockParts.GetReader(), block, 0)
  441. if err != nil {
  442. panic(err)
  443. }
  444. if block.Height != height+1 {
  445. panic(fmt.Sprintf("read bad block from wal. got height %d, expected %d", block.Height, height+1))
  446. }
  447. commitHeight := thisBlockCommit.Precommits[0].Height
  448. if commitHeight != height+1 {
  449. panic(fmt.Sprintf("commit doesnt match. got height %d, expected %d", commitHeight, height+1))
  450. }
  451. blocks = append(blocks, block)
  452. commits = append(commits, thisBlockCommit)
  453. height++
  454. }
  455. case *types.PartSetHeader:
  456. thisBlockParts = types.NewPartSetFromHeader(*p)
  457. case *types.Part:
  458. _, err := thisBlockParts.AddPart(p)
  459. if err != nil {
  460. return nil, nil, err
  461. }
  462. case *types.Vote:
  463. if p.Type == types.PrecommitType {
  464. thisBlockCommit = &types.Commit{
  465. BlockID: p.BlockID,
  466. Precommits: []*types.Vote{p},
  467. }
  468. }
  469. }
  470. }
  471. // grab the last block too
  472. var block = new(types.Block)
  473. _, err = cdc.UnmarshalBinaryLengthPrefixedReader(thisBlockParts.GetReader(), block, 0)
  474. if err != nil {
  475. panic(err)
  476. }
  477. if block.Height != height+1 {
  478. panic(fmt.Sprintf("read bad block from wal. got height %d, expected %d", block.Height, height+1))
  479. }
  480. commitHeight := thisBlockCommit.Precommits[0].Height
  481. if commitHeight != height+1 {
  482. panic(fmt.Sprintf("commit doesnt match. got height %d, expected %d", commitHeight, height+1))
  483. }
  484. blocks = append(blocks, block)
  485. commits = append(commits, thisBlockCommit)
  486. return blocks, commits, nil
  487. }
  488. func readPieceFromWAL(msg *TimedWALMessage) interface{} {
  489. // for logging
  490. switch m := msg.Msg.(type) {
  491. case msgInfo:
  492. switch msg := m.Msg.(type) {
  493. case *ProposalMessage:
  494. return &msg.Proposal.BlockPartsHeader
  495. case *BlockPartMessage:
  496. return msg.Part
  497. case *VoteMessage:
  498. return msg.Vote
  499. }
  500. case EndHeightMessage:
  501. return m
  502. }
  503. return nil
  504. }
  505. // fresh state and mock store
  506. func stateAndStore(config *cfg.Config, pubKey crypto.PubKey, appVersion version.Protocol) (dbm.DB, sm.State, *mockBlockStore) {
  507. stateDB := dbm.NewMemDB()
  508. state, _ := sm.MakeGenesisStateFromFile(config.GenesisFile())
  509. state.Version.Consensus.App = appVersion
  510. store := NewMockBlockStore(config, state.ConsensusParams)
  511. return stateDB, state, store
  512. }
  513. //----------------------------------
  514. // mock block store
  515. type mockBlockStore struct {
  516. config *cfg.Config
  517. params types.ConsensusParams
  518. chain []*types.Block
  519. commits []*types.Commit
  520. }
  521. // TODO: NewBlockStore(db.NewMemDB) ...
  522. func NewMockBlockStore(config *cfg.Config, params types.ConsensusParams) *mockBlockStore {
  523. return &mockBlockStore{config, params, nil, nil}
  524. }
  525. func (bs *mockBlockStore) Height() int64 { return int64(len(bs.chain)) }
  526. func (bs *mockBlockStore) LoadBlock(height int64) *types.Block { return bs.chain[height-1] }
  527. func (bs *mockBlockStore) LoadBlockMeta(height int64) *types.BlockMeta {
  528. block := bs.chain[height-1]
  529. return &types.BlockMeta{
  530. BlockID: types.BlockID{block.Hash(), block.MakePartSet(types.BlockPartSizeBytes).Header()},
  531. Header: block.Header,
  532. }
  533. }
  534. func (bs *mockBlockStore) LoadBlockPart(height int64, index int) *types.Part { return nil }
  535. func (bs *mockBlockStore) SaveBlock(block *types.Block, blockParts *types.PartSet, seenCommit *types.Commit) {
  536. }
  537. func (bs *mockBlockStore) LoadBlockCommit(height int64) *types.Commit {
  538. return bs.commits[height-1]
  539. }
  540. func (bs *mockBlockStore) LoadSeenCommit(height int64) *types.Commit {
  541. return bs.commits[height-1]
  542. }
  543. //----------------------------------------
  544. func TestInitChainUpdateValidators(t *testing.T) {
  545. val, _ := types.RandValidator(true, 10)
  546. vals := types.NewValidatorSet([]*types.Validator{val})
  547. app := &initChainApp{vals: types.TM2PB.ValidatorUpdates(vals)}
  548. clientCreator := proxy.NewLocalClientCreator(app)
  549. config := ResetConfig("proxy_test_")
  550. privVal := privval.LoadFilePV(config.PrivValidatorFile())
  551. stateDB, state, store := stateAndStore(config, privVal.GetPubKey(), 0x0)
  552. oldValAddr := state.Validators.Validators[0].Address
  553. // now start the app using the handshake - it should sync
  554. genDoc, _ := sm.MakeGenesisDocFromFile(config.GenesisFile())
  555. handshaker := NewHandshaker(stateDB, state, store, genDoc)
  556. proxyApp := proxy.NewAppConns(clientCreator)
  557. if err := proxyApp.Start(); err != nil {
  558. t.Fatalf("Error starting proxy app connections: %v", err)
  559. }
  560. defer proxyApp.Stop()
  561. if err := handshaker.Handshake(proxyApp); err != nil {
  562. t.Fatalf("Error on abci handshake: %v", err)
  563. }
  564. // reload the state, check the validator set was updated
  565. state = sm.LoadState(stateDB)
  566. newValAddr := state.Validators.Validators[0].Address
  567. expectValAddr := val.Address
  568. assert.NotEqual(t, oldValAddr, newValAddr)
  569. assert.Equal(t, newValAddr, expectValAddr)
  570. }
  571. func newInitChainApp(vals []abci.ValidatorUpdate) *initChainApp {
  572. return &initChainApp{
  573. vals: vals,
  574. }
  575. }
  576. // returns the vals on InitChain
  577. type initChainApp struct {
  578. abci.BaseApplication
  579. vals []abci.ValidatorUpdate
  580. }
  581. func (ica *initChainApp) InitChain(req abci.RequestInitChain) abci.ResponseInitChain {
  582. return abci.ResponseInitChain{
  583. Validators: ica.vals,
  584. }
  585. }