Browse Source

Formatting and documentation

pull/1051/head
Adrian Brink 6 years ago
committed by Ethan Buchman
parent
commit
a30315276b
8 changed files with 102 additions and 58 deletions
  1. +4
    -5
      blockchain/pool.go
  2. +3
    -2
      blockchain/pool_test.go
  3. +15
    -7
      blockchain/reactor.go
  4. +5
    -2
      blockchain/reactor_test.go
  5. +3
    -1
      blockchain/store.go
  6. +23
    -11
      blockchain/store_test.go
  7. +17
    -0
      docs/specification/new-spec/blockchain_reactor.md
  8. +32
    -30
      docs/specification/new-spec/consensus.md

+ 4
- 5
blockchain/pool.go View File

@ -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() {


+ 3
- 2
blockchain/pool_test.go View File

@ -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() {


+ 15
- 7
blockchain/reactor.go View File

@ -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 (
@ -56,9 +58,12 @@ type BlockchainReactor struct {
}
// 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)
@ -138,7 +143,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}
@ -293,7 +300,7 @@ FOR_LOOP:
// TODO This is bad, are we zombie?
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 +322,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
}


+ 5
- 2
blockchain/reactor_test.go View File

@ -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


+ 3
- 1
blockchain/store.go View File

@ -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"
)
/*


+ 23
- 11
blockchain/store_test.go View File

@ -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")


+ 17
- 0
docs/specification/new-spec/blockchain_reactor.md View File

@ -0,0 +1,17 @@
# Blockchain Reactor
The Blockchain Reactor's high level responsibility is to maintain connection to a reasonable number
of peers in the network, request blocks from them or provide them with blocks, validate and persist
the blocks to disk and play blocks to the ABCI app.
## Block Reactor
* coordinates synching with other peers
## Block Pool
* maintain connections to other peers
## Block Store
* persists blocks to disk

+ 32
- 30
docs/specification/new-spec/consensus.md View File

@ -5,30 +5,31 @@ the next block to be added to the Tendermint blockchain. The protocol proceeds i
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`.
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
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.
@ -45,9 +46,9 @@ type ProposalMessage struct {
### 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 {
@ -67,10 +68,11 @@ 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 {
@ -120,9 +122,9 @@ type Heartbeat struct {
## 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 {
@ -136,10 +138,10 @@ type NewRoundStepMessage struct {
## 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 {


Loading…
Cancel
Save