diff --git a/blockchain/pool.go b/blockchain/pool.go index f3148e6c5..164d3b3b5 100644 --- a/blockchain/pool.go +++ b/blockchain/pool.go @@ -5,15 +5,15 @@ import ( "sync" "time" - "github.com/tendermint/tendermint/p2p" - "github.com/tendermint/tendermint/types" cmn "github.com/tendermint/tmlibs/common" flow "github.com/tendermint/tmlibs/flowrate" "github.com/tendermint/tmlibs/log" + + "github.com/tendermint/tendermint/p2p" + "github.com/tendermint/tendermint/types" ) /* - eg, L = latency = 0.1s P = num peers = 10 FN = num full nodes @@ -23,7 +23,6 @@ eg, L = latency = 0.1s B/S = CB/P/BS = 12.8 blocks/s 12.8 * 0.1 = 1.28 blocks on conn - */ const ( @@ -503,7 +502,7 @@ func (bpr *bpRequester) requestRoutine() { OUTER_LOOP: for { // Pick a peer to send request to. - var peer *bpPeer = nil + var peer *bpPeer PICK_PEER_LOOP: for { if !bpr.IsRunning() || !bpr.pool.IsRunning() { diff --git a/blockchain/pool_test.go b/blockchain/pool_test.go index 0cdbc3a9e..ce16899a7 100644 --- a/blockchain/pool_test.go +++ b/blockchain/pool_test.go @@ -5,10 +5,11 @@ import ( "testing" "time" - "github.com/tendermint/tendermint/p2p" - "github.com/tendermint/tendermint/types" cmn "github.com/tendermint/tmlibs/common" "github.com/tendermint/tmlibs/log" + + "github.com/tendermint/tendermint/p2p" + "github.com/tendermint/tendermint/types" ) func init() { diff --git a/blockchain/reactor.go b/blockchain/reactor.go index 701e04f68..ee84a794f 100644 --- a/blockchain/reactor.go +++ b/blockchain/reactor.go @@ -8,11 +8,13 @@ import ( "time" wire "github.com/tendermint/go-wire" + + cmn "github.com/tendermint/tmlibs/common" + "github.com/tendermint/tmlibs/log" + "github.com/tendermint/tendermint/p2p" sm "github.com/tendermint/tendermint/state" "github.com/tendermint/tendermint/types" - cmn "github.com/tendermint/tmlibs/common" - "github.com/tendermint/tmlibs/log" ) const ( @@ -47,18 +49,22 @@ type BlockchainReactor struct { // immutable initialState sm.State - blockExec *sm.BlockExecutor - store *BlockStore - pool *BlockPool - fastSync bool - requestsCh chan BlockRequest - timeoutsCh chan p2p.ID + blockExec *sm.BlockExecutor + store *BlockStore + pool *BlockPool + fastSync bool + + requestsCh <-chan BlockRequest + timeoutsCh <-chan p2p.ID } // NewBlockchainReactor returns new reactor instance. -func NewBlockchainReactor(state sm.State, blockExec *sm.BlockExecutor, store *BlockStore, fastSync bool) *BlockchainReactor { +func NewBlockchainReactor(state sm.State, blockExec *sm.BlockExecutor, store *BlockStore, + fastSync bool) *BlockchainReactor { + if state.LastBlockHeight != store.Height() { - cmn.PanicSanity(cmn.Fmt("state (%v) and store (%v) height mismatch", state.LastBlockHeight, store.Height())) + cmn.PanicSanity(cmn.Fmt("state (%v) and store (%v) height mismatch", state.LastBlockHeight, + store.Height())) } requestsCh := make(chan BlockRequest, defaultChannelCapacity) @@ -122,7 +128,8 @@ func (bcR *BlockchainReactor) GetChannels() []*p2p.ChannelDescriptor { // AddPeer implements Reactor by sending our state to peer. func (bcR *BlockchainReactor) AddPeer(peer p2p.Peer) { - if !peer.Send(BlockchainChannel, struct{ BlockchainMessage }{&bcStatusResponseMessage{bcR.store.Height()}}) { + if !peer.Send(BlockchainChannel, + struct{ BlockchainMessage }{&bcStatusResponseMessage{bcR.store.Height()}}) { // doing nothing, will try later in `poolRoutine` } // peer is added to the pool once we receive the first @@ -138,7 +145,9 @@ func (bcR *BlockchainReactor) RemovePeer(peer p2p.Peer, reason interface{}) { // if we have it. Otherwise, we'll respond saying we don't have it. // According to the Tendermint spec, if all nodes are honest, // no node should be requesting for a block that's non-existent. -func (bcR *BlockchainReactor) respondToPeer(msg *bcBlockRequestMessage, src p2p.Peer) (queued bool) { +func (bcR *BlockchainReactor) respondToPeer(msg *bcBlockRequestMessage, + src p2p.Peer) (queued bool) { + block := bcR.store.LoadBlock(msg.Height) if block != nil { msg := &bcBlockResponseMessage{Block: block} @@ -173,7 +182,8 @@ func (bcR *BlockchainReactor) Receive(chID byte, src p2p.Peer, msgBytes []byte) bcR.pool.AddBlock(src.ID(), msg.Block, len(msgBytes)) case *bcStatusRequestMessage: // Send peer our state. - queued := src.TrySend(BlockchainChannel, struct{ BlockchainMessage }{&bcStatusResponseMessage{bcR.store.Height()}}) + queued := src.TrySend(BlockchainChannel, + struct{ BlockchainMessage }{&bcStatusResponseMessage{bcR.store.Height()}}) if !queued { // sorry } @@ -291,9 +301,10 @@ FOR_LOOP: state, err = bcR.blockExec.ApplyBlock(state, firstID, first) if err != nil { // TODO This is bad, are we zombie? - cmn.PanicQ(cmn.Fmt("Failed to process committed block (%d:%X): %v", first.Height, first.Hash(), err)) + cmn.PanicQ(cmn.Fmt("Failed to process committed block (%d:%X): %v", + first.Height, first.Hash(), err)) } - blocksSynced += 1 + blocksSynced++ // update the consensus params bcR.updateConsensusParams(state.ConsensusParams) @@ -315,7 +326,8 @@ FOR_LOOP: // BroadcastStatusRequest broadcasts `BlockStore` height. func (bcR *BlockchainReactor) BroadcastStatusRequest() error { - bcR.Switch.Broadcast(BlockchainChannel, struct{ BlockchainMessage }{&bcStatusRequestMessage{bcR.store.Height()}}) + bcR.Switch.Broadcast(BlockchainChannel, + struct{ BlockchainMessage }{&bcStatusRequestMessage{bcR.store.Height()}}) return nil } diff --git a/blockchain/reactor_test.go b/blockchain/reactor_test.go index 06f6c36c5..6f5b14ff3 100644 --- a/blockchain/reactor_test.go +++ b/blockchain/reactor_test.go @@ -4,6 +4,7 @@ import ( "testing" wire "github.com/tendermint/go-wire" + cmn "github.com/tendermint/tmlibs/common" dbm "github.com/tendermint/tmlibs/db" "github.com/tendermint/tmlibs/log" @@ -28,7 +29,8 @@ func newBlockchainReactor(logger log.Logger, maxBlockHeight int64) *BlockchainRe // Make the blockchainReactor itself fastSync := true var nilApp proxy.AppConnConsensus - blockExec := sm.NewBlockExecutor(dbm.NewMemDB(), log.TestingLogger(), nilApp, types.MockMempool{}, types.MockEvidencePool{}) + blockExec := sm.NewBlockExecutor(dbm.NewMemDB(), log.TestingLogger(), nilApp, + types.MockMempool{}, types.MockEvidencePool{}) bcReactor := NewBlockchainReactor(state.Copy(), blockExec, blockStore, fastSync) bcReactor.SetLogger(logger.With("module", "blockchain")) @@ -130,7 +132,8 @@ func newbcrTestPeer(id p2p.ID) *bcrTestPeer { func (tp *bcrTestPeer) lastValue() interface{} { return <-tp.ch } func (tp *bcrTestPeer) TrySend(chID byte, value interface{}) bool { - if _, ok := value.(struct{ BlockchainMessage }).BlockchainMessage.(*bcStatusResponseMessage); ok { + if _, ok := value.(struct{ BlockchainMessage }). + BlockchainMessage.(*bcStatusResponseMessage); ok { // Discard status response messages since they skew our results // We only want to deal with: // + bcBlockResponseMessage diff --git a/blockchain/store.go b/blockchain/store.go index 1033999fe..91d2b220f 100644 --- a/blockchain/store.go +++ b/blockchain/store.go @@ -8,9 +8,11 @@ import ( "sync" wire "github.com/tendermint/go-wire" - "github.com/tendermint/tendermint/types" + cmn "github.com/tendermint/tmlibs/common" dbm "github.com/tendermint/tmlibs/db" + + "github.com/tendermint/tendermint/types" ) /* diff --git a/blockchain/store_test.go b/blockchain/store_test.go index 1fd88dac3..933329c4b 100644 --- a/blockchain/store_test.go +++ b/blockchain/store_test.go @@ -13,9 +13,11 @@ import ( "github.com/stretchr/testify/require" wire "github.com/tendermint/go-wire" - "github.com/tendermint/tendermint/types" + "github.com/tendermint/tmlibs/db" "github.com/tendermint/tmlibs/log" + + "github.com/tendermint/tendermint/types" ) func TestLoadBlockStoreStateJSON(t *testing.T) { @@ -104,7 +106,8 @@ var ( partSet = block.MakePartSet(2) part1 = partSet.GetPart(0) part2 = partSet.GetPart(1) - seenCommit1 = &types.Commit{Precommits: []*types.Vote{{Height: 10, Timestamp: time.Now().UTC()}}} + seenCommit1 = &types.Commit{Precommits: []*types.Vote{{Height: 10, + Timestamp: time.Now().UTC()}}} ) // TODO: This test should be simplified ... @@ -124,7 +127,8 @@ func TestBlockStoreSaveLoadBlock(t *testing.T) { // save a block block := makeBlock(bs.Height()+1, state) validPartSet := block.MakePartSet(2) - seenCommit := &types.Commit{Precommits: []*types.Vote{{Height: 10, Timestamp: time.Now().UTC()}}} + seenCommit := &types.Commit{Precommits: []*types.Vote{{Height: 10, + Timestamp: time.Now().UTC()}}} bs.SaveBlock(block, partSet, seenCommit) require.Equal(t, bs.Height(), block.Header.Height, "expecting the new height to be changed") @@ -143,7 +147,8 @@ func TestBlockStoreSaveLoadBlock(t *testing.T) { // End of setup, test data - commitAtH10 := &types.Commit{Precommits: []*types.Vote{{Height: 10, Timestamp: time.Now().UTC()}}} + commitAtH10 := &types.Commit{Precommits: []*types.Vote{{Height: 10, + Timestamp: time.Now().UTC()}}} tuples := []struct { block *types.Block parts *types.PartSet @@ -263,7 +268,8 @@ func TestBlockStoreSaveLoadBlock(t *testing.T) { db.Set(calcBlockCommitKey(commitHeight), []byte("foo-bogus")) } bCommit := bs.LoadBlockCommit(commitHeight) - return &quad{block: bBlock, seenCommit: bSeenCommit, commit: bCommit, meta: bBlockMeta}, nil + return &quad{block: bBlock, seenCommit: bSeenCommit, commit: bCommit, + meta: bBlockMeta}, nil }) if subStr := tuple.wantPanic; subStr != "" { @@ -290,10 +296,12 @@ func TestBlockStoreSaveLoadBlock(t *testing.T) { continue } if tuple.eraseSeenCommitInDB { - assert.Nil(t, qua.seenCommit, "erased the seenCommit in the DB hence we should get back a nil seenCommit") + assert.Nil(t, qua.seenCommit, + "erased the seenCommit in the DB hence we should get back a nil seenCommit") } if tuple.eraseCommitInDB { - assert.Nil(t, qua.commit, "erased the commit in the DB hence we should get back a nil commit") + assert.Nil(t, qua.commit, + "erased the commit in the DB hence we should get back a nil commit") } } } @@ -331,7 +339,8 @@ func TestLoadBlockPart(t *testing.T) { gotPart, _, panicErr := doFn(loadPart) require.Nil(t, panicErr, "an existent and proper block should not panic") require.Nil(t, res, "a properly saved block should return a proper block") - require.Equal(t, gotPart.(*types.Part).Hash(), part1.Hash(), "expecting successful retrieval of previously saved block") + require.Equal(t, gotPart.(*types.Part).Hash(), part1.Hash(), + "expecting successful retrieval of previously saved block") } func TestLoadBlockMeta(t *testing.T) { @@ -360,7 +369,8 @@ func TestLoadBlockMeta(t *testing.T) { gotMeta, _, panicErr := doFn(loadMeta) require.Nil(t, panicErr, "an existent and proper block should not panic") require.Nil(t, res, "a properly saved blockMeta should return a proper blocMeta ") - require.Equal(t, binarySerializeIt(meta), binarySerializeIt(gotMeta), "expecting successful retrieval of previously saved blockMeta") + require.Equal(t, binarySerializeIt(meta), binarySerializeIt(gotMeta), + "expecting successful retrieval of previously saved blockMeta") } func TestBlockFetchAtHeight(t *testing.T) { @@ -369,13 +379,15 @@ func TestBlockFetchAtHeight(t *testing.T) { block := makeBlock(bs.Height()+1, state) partSet := block.MakePartSet(2) - seenCommit := &types.Commit{Precommits: []*types.Vote{{Height: 10, Timestamp: time.Now().UTC()}}} + seenCommit := &types.Commit{Precommits: []*types.Vote{{Height: 10, + Timestamp: time.Now().UTC()}}} bs.SaveBlock(block, partSet, seenCommit) require.Equal(t, bs.Height(), block.Header.Height, "expecting the new height to be changed") blockAtHeight := bs.LoadBlock(bs.Height()) - require.Equal(t, block.Hash(), blockAtHeight.Hash(), "expecting a successful load of the last saved block") + require.Equal(t, block.Hash(), blockAtHeight.Hash(), + "expecting a successful load of the last saved block") blockAtHeightPlus1 := bs.LoadBlock(bs.Height() + 1) require.Nil(t, blockAtHeightPlus1, "expecting an unsuccessful load of Height()+1") diff --git a/docs/.python-version b/docs/.python-version new file mode 100644 index 000000000..9bbf49249 --- /dev/null +++ b/docs/.python-version @@ -0,0 +1 @@ +2.7.14 diff --git a/docs/Makefile b/docs/Makefile index f8d1790de..442c9be65 100644 --- a/docs/Makefile +++ b/docs/Makefile @@ -12,6 +12,9 @@ BUILDDIR = _build help: @$(SPHINXBUILD) -M help "$(SOURCEDIR)" "$(BUILDDIR)" $(SPHINXOPTS) $(O) +install: + @pip install -r requirements.txt + .PHONY: help Makefile # Catch-all target: route all unknown targets to Sphinx using the new diff --git a/docs/specification/new-spec/blockchain.md b/docs/specification/new-spec/blockchain.md index ce2529f80..f029d7d47 100644 --- a/docs/specification/new-spec/blockchain.md +++ b/docs/specification/new-spec/blockchain.md @@ -2,7 +2,7 @@ Here we describe the data structures in the Tendermint blockchain and the rules for validating them. -# Data Structures +## Data Structures The Tendermint blockchains consists of a short list of basic data types: `Block`, `Header`, `Vote`, `BlockID`, `Signature`, and `Evidence`. @@ -12,7 +12,7 @@ The Tendermint blockchains consists of a short list of basic data types: A block consists of a header, a list of transactions, a list of votes (the commit), and a list of evidence if malfeasance (ie. signing conflicting votes). -``` +```go type Block struct { Header Header Txs [][]byte @@ -26,7 +26,7 @@ type Block struct { A block header contains metadata about the block and about the consensus, as well as commitments to the data in the current block, the previous block, and the results returned by the application: -``` +```go type Header struct { // block metadata Version string // Version string @@ -66,7 +66,7 @@ the block during consensus, is the Merkle root of the complete serialized block cut into parts. The `BlockID` includes these two hashes, as well as the number of parts. -``` +```go type BlockID struct { Hash []byte Parts PartsHeader @@ -83,7 +83,7 @@ type PartsHeader struct { A vote is a signed message from a validator for a particular block. The vote includes information about the validator signing it. -``` +```go type Vote struct { Timestamp int64 Address []byte @@ -96,7 +96,6 @@ type Vote struct { } ``` - There are two types of votes: a prevote has `vote.Type == 1` and a precommit has `vote.Type == 2`. @@ -111,7 +110,7 @@ Currently, Tendermint supports Ed25519 and Secp256k1. An ED25519 signature has `Type == 0x1`. It looks like: -``` +```go // Implements Signature type Ed25519Signature struct { Type int8 = 0x1 @@ -125,7 +124,7 @@ where `Signature` is the 64 byte signature. A `Secp256k1` signature has `Type == 0x2`. It looks like: -``` +```go // Implements Signature type Secp256k1Signature struct { Type int8 = 0x2 @@ -135,7 +134,7 @@ type Secp256k1Signature struct { where `Signature` is the DER encoded signature, ie: -``` +```hex 0x30 <0x02> 0x2 . ``` @@ -143,7 +142,7 @@ where `Signature` is the DER encoded signature, ie: TODO -# Validation +## Validation Here we describe the validation rules for every element in a block. Blocks which do not satisfy these rules are considered invalid. @@ -159,7 +158,7 @@ and other results from the application. Elements of an object are accessed as expected, ie. `block.Header`. See [here](state.md) for the definition of `state`. -## Header +### Header A Header is valid if its corresponding fields are valid. @@ -173,7 +172,7 @@ Arbitrary constant string. ### Height -``` +```go block.Header.Height > 0 block.Header.Height == prevBlock.Header.Height + 1 ``` @@ -190,7 +189,7 @@ block being voted on. ### NumTxs -``` +```go block.Header.NumTxs == len(block.Txs) ``` @@ -198,7 +197,7 @@ Number of transactions included in the block. ### TxHash -``` +```go block.Header.TxHash == SimpleMerkleRoot(block.Txs) ``` @@ -206,7 +205,7 @@ Simple Merkle root of the transactions in the block. ### LastCommitHash -``` +```go block.Header.LastCommitHash == SimpleMerkleRoot(block.LastCommit) ``` @@ -217,7 +216,7 @@ The first block has `block.Header.LastCommitHash == []byte{}` ### TotalTxs -``` +```go block.Header.TotalTxs == prevBlock.Header.TotalTxs + block.Header.NumTxs ``` @@ -227,7 +226,7 @@ The first block has `block.Header.TotalTxs = block.Header.NumberTxs`. ### LastBlockID -``` +```go prevBlockParts := MakeParts(prevBlock, state.LastConsensusParams.BlockGossip.BlockPartSize) block.Header.LastBlockID == BlockID { Hash: SimpleMerkleRoot(prevBlock.Header), @@ -245,7 +244,7 @@ The first block has `block.Header.LastBlockID == BlockID{}`. ### ResultsHash -``` +```go block.ResultsHash == SimpleMerkleRoot(state.LastResults) ``` @@ -255,7 +254,7 @@ The first block has `block.Header.ResultsHash == []byte{}`. ### AppHash -``` +```go block.AppHash == state.AppHash ``` @@ -265,7 +264,7 @@ The first block has `block.Header.AppHash == []byte{}`. ### ValidatorsHash -``` +```go block.ValidatorsHash == SimpleMerkleRoot(state.Validators) ``` @@ -275,7 +274,7 @@ May be updated by the application. ### ConsensusParamsHash -``` +```go block.ConsensusParamsHash == SimpleMerkleRoot(state.ConsensusParams) ``` @@ -284,7 +283,7 @@ May be updated by the application. ### Proposer -``` +```go block.Header.Proposer in state.Validators ``` @@ -296,7 +295,7 @@ and we do not track the initial round the block was proposed. ### EvidenceHash -``` +```go block.EvidenceHash == SimpleMerkleRoot(block.Evidence) ``` @@ -310,7 +309,7 @@ Arbitrary length array of arbitrary length byte-arrays. The first height is an exception - it requires the LastCommit to be empty: -``` +```go if block.Header.Height == 1 { len(b.LastCommit) == 0 } @@ -318,7 +317,7 @@ if block.Header.Height == 1 { Otherwise, we require: -``` +```go len(block.LastCommit) == len(state.LastValidators) talliedVotingPower := 0 for i, vote := range block.LastCommit{ @@ -356,7 +355,7 @@ For signing, votes are encoded in JSON, and the ChainID is included, in the form We define a method `Verify` that returns `true` if the signature verifies against the pubkey for the CanonicalSignBytes using the given ChainID: -``` +```go func (v Vote) Verify(chainID string, pubKey PubKey) bool { return pubKey.Verify(v.Signature, CanonicalSignBytes(chainID, v)) } @@ -384,7 +383,7 @@ Once a block is validated, it can be executed against the state. The state follows the recursive equation: -``` +```go app = NewABCIApp state(1) = InitialState state(h+1) <- Execute(state(h), app, block(h)) diff --git a/docs/specification/new-spec/blockchain_reactor.md b/docs/specification/new-spec/blockchain_reactor.md new file mode 100644 index 000000000..5633eb050 --- /dev/null +++ b/docs/specification/new-spec/blockchain_reactor.md @@ -0,0 +1,91 @@ +# Blockchain Reactor + +The Blockchain Reactor's high level responsibility is to enable peers who are +far behind the current state of the consensus to quickly catch up by downloading +many blocks in parallel, verifying their commits, and executing them against the +ABCI application. + +Tendermint full nodes run the Blockchain Reactor as a service to provide blocks +to new nodes. New nodes run the Blockchain Reactor in "fast_sync" mode, +where they actively make requests for more blocks until they sync up. +Once caught up, they disable "fast_sync" mode, and turn on the Consensus Reactor. + +## Message Types + +```go +const ( + msgTypeBlockRequest = byte(0x10) + msgTypeBlockResponse = byte(0x11) + msgTypeNoBlockResponse = byte(0x12) + msgTypeStatusResponse = byte(0x20) + msgTypeStatusRequest = byte(0x21) +) +``` + +```go +type bcBlockRequestMessage struct { + Height int64 +} + +type bcNoBlockResponseMessage struct { + Height int64 +} + +type bcBlockResponseMessage struct { + Block Block +} + +type bcStatusRequestMessage struct { + Height int64 + +type bcStatusResponseMessage struct { + Height int64 +} +``` + +## Blockchain Reactor + +* coordinates the pool for syncing +* coordinates the store for persistence +* coordinates the playing of blocks towards the app using a sm.BlockExecutor +* handles switching between fastsync and consensus +* it is a p2p.BaseReactor +* starts the pool.Start() and its poolRoutine() +* registers all the concrete types and interfaces for serialisation + +### poolRoutine + +* listens to these channels: + * pool requests blocks from a specific peer by posting to requestsCh, block reactor then sends + a &bcBlockRequestMessage for a specific height + * pool signals timeout of a specific peer by posting to timeoutsCh + * switchToConsensusTicker to periodically try and switch to consensus + * trySyncTicker to periodically check if we have fallen behind and then catch-up sync + * if there aren't any new blocks available on the pool it skips syncing +* tries to sync the app by taking downloaded blocks from the pool, gives them to the app and stores + them on disk +* implements Receive which is called by the switch/peer + * calls AddBlock on the pool when it receives a new block from a peer + +## Block Pool + +* responsible for downloading blocks from peers +* makeRequestersRoutine() + * removes timeout peers + * starts new requesters by calling makeNextRequester() +* requestRoutine(): + * picks a peer and sends the request, then blocks until: + * pool is stopped by listening to pool.Quit + * requester is stopped by listening to Quit + * request is redone + * we receive a block + * gotBlockCh is strange + +## Block Store + +* persists blocks to disk + +# TODO + +* How does the switch from bcR to conR happen? Does conR persist blocks to disk too? +* What is the interaction between the consensus and blockchain reactors? diff --git a/docs/specification/new-spec/consensus.md b/docs/specification/new-spec/consensus.md index 5c6810565..1f311c44e 100644 --- a/docs/specification/new-spec/consensus.md +++ b/docs/specification/new-spec/consensus.md @@ -1,197 +1,212 @@ -# Tendermint Consensus +# Tendermint Consensus Reactor -Tendermint consensus is a distributed protocol executed by validator processes to agree on -the next block to be added to the Tendermint blockchain. The protocol proceeds in rounds, where -each round is a try to reach agreement on the next block. A round starts by having a dedicated -process (called proposer) suggesting to other processes what should be the next block with +Tendermint Consensus is a distributed protocol executed by validator processes to agree on +the next block to be added to the Tendermint blockchain. The protocol proceeds in rounds, where +each round is a try to reach agreement on the next block. A round starts by having a dedicated +process (called proposer) suggesting to other processes what should be the next block with the `ProposalMessage`. -The processes respond by voting for a block with `VoteMessage` (there are two kinds of vote messages, prevote -and precommit votes). Note that a proposal message is just a suggestion what the next block should be; a -validator might vote with a `VoteMessage` for a different block. If in some round, enough number -of processes vote for the same block, then this block is committed and later added to the blockchain. -`ProposalMessage` and `VoteMessage` are signed by the private key of the validator. -The internals of the protocol and how it ensures safety and liveness properties are +The processes respond by voting for a block with `VoteMessage` (there are two kinds of vote +messages, prevote and precommit votes). Note that a proposal message is just a suggestion what the +next block should be; a validator might vote with a `VoteMessage` for a different block. If in some +round, enough number of processes vote for the same block, then this block is committed and later +added to the blockchain. `ProposalMessage` and `VoteMessage` are signed by the private key of the +validator. The internals of the protocol and how it ensures safety and liveness properties are explained [here](https://github.com/tendermint/spec). -For efficiency reasons, validators in Tendermint consensus protocol do not agree directly on the block -as the block size is big, i.e., they don't embed the block inside `Proposal` and `VoteMessage`. Instead, they -reach agreement on the `BlockID` (see `BlockID` definition in [Blockchain](blockchain.md) section) -that uniquely identifies each block. The block itself is disseminated to validator processes using -peer-to-peer gossiping protocol. It starts by having a proposer first splitting a block into a -number of block parts, that are then gossiped between processes using `BlockPartMessage`. - -Validators in Tendermint communicate by peer-to-peer gossiping protocol. Each validator is connected -only to a subset of processes called peers. By the gossiping protocol, a validator send to its peers -all needed information (`ProposalMessage`, `VoteMessage` and `BlockPartMessage`) so they can -reach agreement on some block, and also obtain the content of the chosen block (block parts). As part of -the gossiping protocol, processes also send auxiliary messages that inform peers about the -executed steps of the core consensus algorithm (`NewRoundStepMessage` and `CommitStepMessage`), and +For efficiency reasons, validators in Tendermint consensus protocol do not agree directly on the +block as the block size is big, i.e., they don't embed the block inside `Proposal` and +`VoteMessage`. Instead, they reach agreement on the `BlockID` (see `BlockID` definition in +[Blockchain](blockchain.md) section) that uniquely identifies each block. The block itself is +disseminated to validator processes using peer-to-peer gossiping protocol. It starts by having a +proposer first splitting a block into a number of block parts, that are then gossiped between +processes using `BlockPartMessage`. + +Validators in Tendermint communicate by peer-to-peer gossiping protocol. Each validator is connected +only to a subset of processes called peers. By the gossiping protocol, a validator send to its peers +all needed information (`ProposalMessage`, `VoteMessage` and `BlockPartMessage`) so they can +reach agreement on some block, and also obtain the content of the chosen block (block parts). As +part of the gossiping protocol, processes also send auxiliary messages that inform peers about the +executed steps of the core consensus algorithm (`NewRoundStepMessage` and `CommitStepMessage`), and also messages that inform peers what votes the process has seen (`HasVoteMessage`, -`VoteSetMaj23Message` and `VoteSetBitsMessage`). These messages are then used in the gossiping protocol -to determine what messages a process should send to its peers. - +`VoteSetMaj23Message` and `VoteSetBitsMessage`). These messages are then used in the gossiping +protocol to determine what messages a process should send to its peers. + We now describe the content of each message exchanged during Tendermint consensus protocol. ## ProposalMessage -ProposalMessage is sent when a new block is proposed. It is a suggestion of what the + +ProposalMessage is sent when a new block is proposed. It is a suggestion of what the next block in the blockchain should be. -``` + +```go type ProposalMessage struct { - Proposal Proposal + Proposal Proposal } ``` + ### Proposal -Proposal contains height and round for which this proposal is made, BlockID as a unique identifier of proposed -block, timestamp, and two fields (POLRound and POLBlockID) that are needed for termination of the consensus. -The message is signed by the validator private key. -``` +Proposal contains height and round for which this proposal is made, BlockID as a unique identifier +of proposed block, timestamp, and two fields (POLRound and POLBlockID) that are needed for +termination of the consensus. The message is signed by the validator private key. + +```go type Proposal struct { - Height int64 - Round int - Timestamp Time - BlockID BlockID - POLRound int - POLBlockID BlockID - Signature Signature + Height int64 + Round int + Timestamp Time + BlockID BlockID + POLRound int + POLBlockID BlockID + Signature Signature } ``` NOTE: In the current version of the Tendermint, the consensus value in proposal is represented with -PartSetHeader, and with BlockID in vote message. It should be aligned as suggested in this spec as -BlockID contains PartSetHeader. +PartSetHeader, and with BlockID in vote message. It should be aligned as suggested in this spec as +BlockID contains PartSetHeader. ## VoteMessage -VoteMessage is sent to vote for some block (or to inform others that a process does not vote in the current round). -Vote is defined in [Blockchain](blockchain.md) section and contains validator's information (validator address -and index), height and round for which the vote is sent, vote type, blockID if process vote for some -block (`nil` otherwise) and a timestamp when the vote is sent. The message is signed by the validator private key. -``` + +VoteMessage is sent to vote for some block (or to inform others that a process does not vote in the +current round). Vote is defined in [Blockchain](blockchain.md) section and contains validator's +information (validator address and index), height and round for which the vote is sent, vote type, +blockID if process vote for some block (`nil` otherwise) and a timestamp when the vote is sent. The +message is signed by the validator private key. + +```go type VoteMessage struct { - Vote Vote + Vote Vote } ``` ## BlockPartMessage -BlockPartMessage is sent when gossipping a piece of the proposed block. It contains height, round -and the block part. -``` +BlockPartMessage is sent when gossipping a piece of the proposed block. It contains height, round +and the block part. + +```go type BlockPartMessage struct { - Height int64 - Round int - Part Part + Height int64 + Round int + Part Part } ``` -## ProposalHeartbeatMessage -ProposalHeartbeatMessage is sent to signal that a node is alive and waiting for transactions +## ProposalHeartbeatMessage + +ProposalHeartbeatMessage is sent to signal that a node is alive and waiting for transactions to be able to create a next block proposal. -``` +```go type ProposalHeartbeatMessage struct { Heartbeat Heartbeat } ``` ### Heartbeat + Heartbeat contains validator information (address and index), -height, round and sequence number. It is signed by the private key of the validator. +height, round and sequence number. It is signed by the private key of the validator. -``` +```go type Heartbeat struct { - ValidatorAddress []byte - ValidatorIndex int - Height int64 - Round int - Sequence int - Signature Signature + ValidatorAddress []byte + ValidatorIndex int + Height int64 + Round int + Sequence int + Signature Signature } ``` ## NewRoundStepMessage -NewRoundStepMessage is sent for every step transition during the core consensus algorithm execution. It is -used in the gossip part of the Tendermint protocol to inform peers about a current height/round/step -a process is in. -``` +NewRoundStepMessage is sent for every step transition during the core consensus algorithm execution. +It is used in the gossip part of the Tendermint protocol to inform peers about a current +height/round/step a process is in. + +```go type NewRoundStepMessage struct { - Height int64 - Round int - Step RoundStepType - SecondsSinceStartTime int - LastCommitRound int + Height int64 + Round int + Step RoundStepType + SecondsSinceStartTime int + LastCommitRound int } ``` ## CommitStepMessage -CommitStepMessage is sent when an agreement on some block is reached. It contains height for which agreement -is reached, block parts header that describes the decided block and is used to obtain all block parts, -and a bit array of the block parts a process currently has, so its peers can know what parts -it is missing so they can send them. -``` +CommitStepMessage is sent when an agreement on some block is reached. It contains height for which +agreement is reached, block parts header that describes the decided block and is used to obtain all +block parts, and a bit array of the block parts a process currently has, so its peers can know what +parts it is missing so they can send them. + +```go type CommitStepMessage struct { - Height int64 - BlockID BlockID - BlockParts BitArray + Height int64 + BlockID BlockID + BlockParts BitArray } ``` -TODO: We use BlockID instead of BlockPartsHeader (in current implementation) for symmetry. +TODO: We use BlockID instead of BlockPartsHeader (in current implementation) for symmetry. ## ProposalPOLMessage + ProposalPOLMessage is sent when a previous block is re-proposed. It is used to inform peers in what round the process learned for this block (ProposalPOLRound), -and what prevotes for the re-proposed block the process has. - -``` +and what prevotes for the re-proposed block the process has. + +```go type ProposalPOLMessage struct { - Height int64 - ProposalPOLRound int - ProposalPOL BitArray + Height int64 + ProposalPOLRound int + ProposalPOL BitArray } ``` - ## HasVoteMessage -HasVoteMessage is sent to indicate that a particular vote has been received. It contains height, -round, vote type and the index of the validator that is the originator of the corresponding vote. -``` +HasVoteMessage is sent to indicate that a particular vote has been received. It contains height, +round, vote type and the index of the validator that is the originator of the corresponding vote. + +```go type HasVoteMessage struct { - Height int64 - Round int - Type byte - Index int + Height int64 + Round int + Type byte + Index int } ``` ## VoteSetMaj23Message + VoteSetMaj23Message is sent to indicate that a process has seen +2/3 votes for some BlockID. -It contains height, round, vote type and the BlockID. +It contains height, round, vote type and the BlockID. -``` +```go type VoteSetMaj23Message struct { - Height int64 - Round int - Type byte - BlockID BlockID + Height int64 + Round int + Type byte + BlockID BlockID } ``` ## VoteSetBitsMessage -VoteSetBitsMessage is sent to communicate the bit-array of votes a process has seen for a given -BlockID. It contains height, round, vote type, BlockID and a bit array of + +VoteSetBitsMessage is sent to communicate the bit-array of votes a process has seen for a given +BlockID. It contains height, round, vote type, BlockID and a bit array of the votes a process has. -``` +```go type VoteSetBitsMessage struct { - Height int64 - Round int - Type byte - BlockID BlockID - Votes BitArray + Height int64 + Round int + Type byte + BlockID BlockID + Votes BitArray } ``` - diff --git a/docs/specification/new-spec/encoding.md b/docs/specification/new-spec/encoding.md index f401dde7c..205b8574e 100644 --- a/docs/specification/new-spec/encoding.md +++ b/docs/specification/new-spec/encoding.md @@ -2,9 +2,11 @@ ## Binary Serialization (TMBIN) -Tendermint aims to encode data structures in a manner similar to how the corresponding Go structs are laid out in memory. +Tendermint aims to encode data structures in a manner similar to how the corresponding Go structs +are laid out in memory. Variable length items are length-prefixed. -While the encoding was inspired by Go, it is easily implemented in other languages as well given its intuitive design. +While the encoding was inspired by Go, it is easily implemented in other languages as well given its +intuitive design. XXX: This is changing to use real varints and 4-byte-prefixes. See https://github.com/tendermint/go-wire/tree/sdk2. @@ -19,7 +21,7 @@ Negative integers are encoded via twos-complement. Examples: -``` +```go encode(uint8(6)) == [0x06] encode(uint32(6)) == [0x00, 0x00, 0x00, 0x06] @@ -36,10 +38,9 @@ Negative integers are encoded by flipping the leading bit of the length-prefix t Zero is encoded as `0x00`. It is not length-prefixed. - Examples: -``` +```go encode(uint(6)) == [0x01, 0x06] encode(uint(70000)) == [0x03, 0x01, 0x11, 0x70] @@ -58,7 +59,7 @@ The empty string is encoded as `0x00`. It is not length-prefixed. Examples: -``` +```go encode("") == [0x00] encode("a") == [0x01, 0x01, 0x61] encode("hello") == [0x01, 0x05, 0x68, 0x65, 0x6C, 0x6C, 0x6F] @@ -72,7 +73,7 @@ There is no length-prefix. Examples: -``` +```go encode([4]int8{1, 2, 3, 4}) == [0x01, 0x02, 0x03, 0x04] encode([4]int16{1, 2, 3, 4}) == [0x00, 0x01, 0x00, 0x02, 0x00, 0x03, 0x00, 0x04] encode([4]int{1, 2, 3, 4}) == [0x01, 0x01, 0x01, 0x02, 0x01, 0x03, 0x01, 0x04] @@ -81,14 +82,15 @@ encode([2]string{"abc", "efg"}) == [0x01, 0x03, 0x61, 0x62, 0x63, 0x01, 0x03, 0x ### Slices (variable length) -An encoded variable-length array is a length prefix followed by the concatenation of the encoding of its elements. +An encoded variable-length array is a length prefix followed by the concatenation of the encoding of +its elements. The length-prefix is itself encoded as an `int`. An empty slice is encoded as `0x00`. It is not length-prefixed. Examples: -``` +```go encode([]int8{}) == [0x00] encode([]int8{1, 2, 3, 4}) == [0x01, 0x04, 0x01, 0x02, 0x03, 0x04] encode([]int16{1, 2, 3, 4}) == [0x01, 0x04, 0x00, 0x01, 0x00, 0x02, 0x00, 0x03, 0x00, 0x04] @@ -97,10 +99,11 @@ encode([]string{"abc", "efg"}) == [0x01, 0x02, 0x01, 0x03, 0x61, 0x62, 0x63, 0x ``` ### BitArray + BitArray is encoded as an `int` of the number of bits, and with an array of `uint64` to encode value of each array element. -``` +```go type BitArray struct { Bits int Elems []uint64 @@ -116,7 +119,7 @@ Times before then are invalid. Examples: -``` +```go encode(time.Time("Jan 1 00:00:00 UTC 1970")) == [0x00, 0x00, 0x00, 0x00, 0x00, 0x00, 0x00, 0x00] encode(time.Time("Jan 1 00:00:01 UTC 1970")) == [0x00, 0x00, 0x00, 0x00, 0x3B, 0x9A, 0xCA, 0x00] // 1,000,000,000 ns encode(time.Time("Mon Jan 2 15:04:05 -0700 MST 2006")) == [0x0F, 0xC4, 0xBB, 0xC1, 0x53, 0x03, 0x12, 0x00] @@ -129,7 +132,7 @@ There is no length-prefix. Examples: -``` +```go type MyStruct struct{ A int B string @@ -139,7 +142,6 @@ encode(MyStruct{4, "hello", time.Time("Mon Jan 2 15:04:05 -0700 MST 2006")}) == [0x01, 0x04, 0x01, 0x05, 0x68, 0x65, 0x6C, 0x6C, 0x6F, 0x0F, 0xC4, 0xBB, 0xC1, 0x53, 0x03, 0x12, 0x00] ``` - ## Merkle Trees Simple Merkle trees are used in numerous places in Tendermint to compute a cryptographic digest of a data structure. @@ -148,23 +150,24 @@ RIPEMD160 is always used as the hashing function. The function `SimpleMerkleRoot` is a simple recursive function defined as follows: -``` +```go func SimpleMerkleRoot(hashes [][]byte) []byte{ - switch len(hashes) { - case 0: - return nil - case 1: - return hashes[0] - default: - left := SimpleMerkleRoot(hashes[:(len(hashes)+1)/2]) - right := SimpleMerkleRoot(hashes[(len(hashes)+1)/2:]) - return RIPEMD160(append(left, right)) - } + switch len(hashes) { + case 0: + return nil + case 1: + return hashes[0] + default: + left := SimpleMerkleRoot(hashes[:(len(hashes)+1)/2]) + right := SimpleMerkleRoot(hashes[(len(hashes)+1)/2:]) + return RIPEMD160(append(left, right)) + } } ``` Note we abuse notion and call `SimpleMerkleRoot` with arguments of type `struct` or type `[]struct`. -For `struct` arguments, we compute a `[][]byte` by sorting elements of the `struct` according to field name and then hashing them. +For `struct` arguments, we compute a `[][]byte` by sorting elements of the `struct` according to +field name and then hashing them. For `[]struct` arguments, we compute a `[][]byte` by hashing the individual `struct` elements. ## JSON (TMJSON) @@ -172,10 +175,12 @@ For `[]struct` arguments, we compute a `[][]byte` by hashing the individual `str Signed messages (eg. votes, proposals) in the consensus are encoded in TMJSON, rather than TMBIN. TMJSON is JSON where `[]byte` are encoded as uppercase hex, rather than base64. -When signing, the elements of a message are sorted by key and the sorted message is embedded in an outer JSON that includes a `chain_id` field. -We call this encoding the CanonicalSignBytes. For instance, CanonicalSignBytes for a vote would look like: +When signing, the elements of a message are sorted by key and the sorted message is embedded in an +outer JSON that includes a `chain_id` field. +We call this encoding the CanonicalSignBytes. For instance, CanonicalSignBytes for a vote would look +like: -``` +```json {"chain_id":"my-chain-id","vote":{"block_id":{"hash":DEADBEEF,"parts":{"hash":BEEFDEAD,"total":3}},"height":3,"round":2,"timestamp":1234567890, "type":2} ``` @@ -187,16 +192,16 @@ Note how the fields within each level are sorted. TMBIN encode an object and slice it into parts. -``` +```go MakeParts(object, partSize) ``` ### Part -``` +```go type Part struct { - Index int - Bytes byte[] - Proof byte[] + Index int + Bytes byte[] + Proof byte[] } ``` diff --git a/docs/specification/new-spec/state.md b/docs/specification/new-spec/state.md index 1d7900273..3594de57b 100644 --- a/docs/specification/new-spec/state.md +++ b/docs/specification/new-spec/state.md @@ -2,13 +2,13 @@ ## State -The state contains information whose cryptographic digest is included in block headers, -and thus is necessary for validating new blocks. -For instance, the Merkle root of the results from executing the previous block, or the Merkle root of the current validators. -While neither the results of transactions now the validators are ever included in the blockchain itself, -the Merkle roots are, and hence we need a separate data structure to track them. +The state contains information whose cryptographic digest is included in block headers, and thus is +necessary for validating new blocks. For instance, the Merkle root of the results from executing the +previous block, or the Merkle root of the current validators. While neither the results of +transactions now the validators are ever included in the blockchain itself, the Merkle roots are, +and hence we need a separate data structure to track them. -``` +```go type State struct { LastResults []Result AppHash []byte @@ -22,7 +22,7 @@ type State struct { ### Result -``` +```go type Result struct { Code uint32 Data []byte @@ -46,7 +46,7 @@ represented in the tags. A validator is an active participant in the consensus with a public key and a voting power. Validator's also contain an address which is derived from the PubKey: -``` +```go type Validator struct { Address []byte PubKey PubKey @@ -59,7 +59,7 @@ so that there is a canonical order for computing the SimpleMerkleRoot. We also define a `TotalVotingPower` function, to return the total voting power: -``` +```go func TotalVotingPower(vals []Validators) int64{ sum := 0 for v := range vals{ @@ -82,7 +82,7 @@ TODO: We define an `Execute` function that takes a state and a block, executes the block against the application, and returns an updated state. -``` +```go Execute(s State, app ABCIApp, block Block) State { abciResponses := app.ApplyBlock(block)