Browse Source

Merge remote-tracking branch 'origin/master' into wb/abci-finalize-block-synchronize

pull/7983/head
William Banfield 3 years ago
parent
commit
102bda3dba
No known key found for this signature in database GPG Key ID: EFAD3442BF29E3AC
49 changed files with 1941 additions and 565 deletions
  1. +21
    -0
      CHANGELOG.md
  2. +8
    -3
      Makefile
  3. +4
    -1
      docs/architecture/README.md
  4. +3
    -2
      docs/architecture/adr-075-rpc-subscription.md
  5. +23
    -5
      docs/architecture/adr-template.md
  6. +94
    -0
      docs/tendermint-core/consensus/proposer-based-timestamps.md
  7. +1
    -0
      docs/versions
  8. +25
    -19
      internal/blocksync/reactor.go
  9. +1
    -1
      internal/blocksync/reactor_test.go
  10. +21
    -11
      internal/consensus/byzantine_test.go
  11. +15
    -5
      internal/consensus/common_test.go
  12. +4
    -3
      internal/consensus/reactor_test.go
  13. +17
    -12
      internal/consensus/replay_file.go
  14. +36
    -9
      internal/consensus/state.go
  15. +27
    -19
      internal/consensus/wal_generator.go
  16. +9
    -10
      internal/consensus/wal_test.go
  17. +0
    -7
      internal/eventbus/event_bus.go
  18. +13
    -0
      internal/libs/autofile/group.go
  19. +1
    -2
      internal/mempool/mempool.go
  20. +1
    -1
      internal/mempool/mempool_test.go
  21. +8
    -11
      internal/p2p/pex/reactor_test.go
  22. +0
    -20
      internal/pubsub/pubsub.go
  23. +7
    -8
      internal/rpc/core/env.go
  24. +6
    -3
      internal/rpc/core/events.go
  25. +2
    -2
      internal/rpc/core/net.go
  26. +1
    -1
      internal/rpc/core/status.go
  27. +2
    -2
      internal/state/execution.go
  28. +73
    -10
      internal/state/tx_filter.go
  29. +1
    -1
      internal/state/tx_filter_test.go
  30. +20
    -37
      node/node.go
  31. +0
    -3
      node/node_test.go
  32. +11
    -9
      node/setup.go
  33. +93
    -49
      proto/tendermint/abci/types.proto
  34. +476
    -0
      proto/tendermint/abci/types.proto.intermediate
  35. +2
    -0
      proto/tendermint/types/types.proto
  36. +192
    -0
      proto/tendermint/types/types.proto.intermediate
  37. +49
    -0
      rpc/client/eventstream/eventstream_test.go
  38. +31
    -0
      scripts/abci-gen.sh
  39. +1
    -2
      spec/abci++/README.md
  40. +3
    -2
      spec/abci++/abci++_basic_concepts_002_draft.md
  41. +98
    -61
      spec/abci++/abci++_methods_002_draft.md
  42. +2
    -1
      spec/abci++/abci++_tmint_expected_behavior_002_draft.md
  43. +13
    -0
      spec/abci/apps.md
  44. +109
    -0
      spec/consensus/proposer-based-timestamp/tla/Apalache.tla
  45. +77
    -0
      spec/consensus/proposer-based-timestamp/tla/MC_PBT.tla
  46. +323
    -217
      spec/consensus/proposer-based-timestamp/tla/TendermintPBT_002_draft.tla
  47. +2
    -3
      spec/consensus/proposer-based-timestamp/tla/typedefs.tla
  48. +15
    -12
      spec/core/data_structures.md
  49. +0
    -1
      test/fuzz/mempool/checktx.go

+ 21
- 0
CHANGELOG.md View File

@ -2,6 +2,27 @@
Friendly reminder: We have a [bug bounty program](https://hackerone.com/cosmos).
## v0.35.2
February 28, 2022
Special thanks to external contributors on this release: @ashcherbakov, @yihuang, @waelsy123
### IMPROVEMENTS
- [consensus] [\#7875](https://github.com/tendermint/tendermint/pull/7875) additional timing metrics. (@williambanfield)
### BUG FIXES
- [abci] [\#7990](https://github.com/tendermint/tendermint/pull/7990) revert buffer limit change. (@williambanfield)
- [cli] [#7837](https://github.com/tendermint/tendermint/pull/7837) fix app hash in state rollback. (@yihuang)
- [cli] [\#7869](https://github.com/tendermint/tendermint/pull/7869) Update unsafe-reset-all command to match release v35. (waelsy123)
- [light] [\#7640](https://github.com/tendermint/tendermint/pull/7640) Light Client: fix absence proof verification (@ashcherbakov)
- [light] [\#7641](https://github.com/tendermint/tendermint/pull/7641) Light Client: fix querying against the latest height (@ashcherbakov)
- [mempool] [\#7718](https://github.com/tendermint/tendermint/pull/7718) return duplicate tx errors more consistently. (@tychoish)
- [rpc] [\#7744](https://github.com/tendermint/tendermint/pull/7744) fix layout of endpoint list. (@creachadair)
- [statesync] [\#7886](https://github.com/tendermint/tendermint/pull/7886) assert app version matches. (@cmwaters)
## v0.35.1
January 26, 2022


+ 8
- 3
Makefile View File

@ -92,6 +92,12 @@ proto-gen:
@$(DOCKER_PROTO_BUILDER) buf generate --template=./buf.gen.yaml --config ./buf.yaml
.PHONY: proto-gen
# TODO: Should be removed when work on ABCI++ is complete.
# For more information, see https://github.com/tendermint/tendermint/issues/8066
abci-proto-gen:
./scripts/abci-gen.sh
.PHONY: abci-proto-gen
proto-lint:
@$(DOCKER_PROTO_BUILDER) buf lint --error-format=json --config ./buf.yaml
.PHONY: proto-lint
@ -222,9 +228,7 @@ build-docs:
mkdir -p ~/output/$${path_prefix} ; \
cp -r .vuepress/dist/* ~/output/$${path_prefix}/ ; \
cp ~/output/$${path_prefix}/index.html ~/output ; \
done < versions ; \
mkdir -p ~/output/master ; \
cp -r .vuepress/dist/* ~/output/master/
done < versions ;
.PHONY: build-docs
###############################################################################
@ -331,3 +335,4 @@ split-test-packages:$(BUILDDIR)/packages.txt
split -d -n l/$(NUM_SPLIT) $< $<.
test-group-%:split-test-packages
cat $(BUILDDIR)/packages.txt.$* | xargs go test -mod=readonly -timeout=5m -race -coverprofile=$(BUILDDIR)/$*.profile.out

+ 4
- 1
docs/architecture/README.md View File

@ -86,13 +86,16 @@ Note the context/background should be written in the present tense.
- [ADR-075: RPC Event Subscription Interface](./adr-075-rpc-subscription.md)
- [ADR-076: Combine Spec and Tendermint Repositories](./adr-076-combine-spec-repo.md)
### Deprecated
None
### Rejected
- [ADR-023: ABCI-Propose-tx](./adr-023-ABCI-propose-tx.md)
- [ADR-029: Check-Tx-Consensus](./adr-029-check-tx-consensus.md)
- [ADR-058: Event-Hashing](./adr-058-event-hashing.md)
### Proposed
- [ADR-007: Trust-Metric-Usage](./adr-007-trust-metric-usage.md)


+ 3
- 2
docs/architecture/adr-075-rpc-subscription.md View File

@ -2,6 +2,7 @@
## Changelog
- 01-Mar-2022: Update long-polling interface (@creachadair).
- 10-Feb-2022: Updates to reflect implementation.
- 26-Jan-2022: Marked accepted.
- 22-Jan-2022: Updated and expanded (@creachadair).
@ -347,8 +348,8 @@ limit.
The `wait_time` parameter is used to effect polling. If `before` is empty and
no items are available, the server will wait for up to `wait_time` for matching
items to arrive at the head of the log. If `wait_time` is zero, the server will
return whatever eligible items are available immediately.
items to arrive at the head of the log. If `wait_time` is zero or negative, the
server will wait for a default (positive) interval.
If `before` non-empty, `wait_time` is ignored: new results are only added to
the head of the log, so there is no need to wait. This allows the client to


+ 23
- 5
docs/architecture/adr-template.md View File

@ -6,12 +6,30 @@
## Status
> A decision may be "proposed" if it hasn't been agreed upon yet, or "accepted"
> once it is agreed upon. Once the ADR has been implemented mark the ADR as
> "implemented". If a later ADR changes or reverses a decision, it may be marked
> as "deprecated" or "superseded" with a reference to its replacement.
> An architecture decision is considered "proposed" when a PR containing the ADR
> is submitted. When merged, an ADR must have a status associated with it, which
> must be one of: "Accepted", "Rejected", "Deprecated" or "Superseded".
>
> An accepted ADR's implementation status must be tracked via a tracking issue,
> milestone or project board (only one of these is necessary). For example:
>
> Accepted
>
> [Tracking issue](https://github.com/tendermint/tendermint/issues/123)
> [Milestone](https://github.com/tendermint/tendermint/milestones/123)
> [Project board](https://github.com/orgs/tendermint/projects/123)
>
> Rejected ADRs are captured as a record of recommendations that we specifically
> do not (and possibly never) want to implement. The ADR itself must, for
> posterity, include reasoning as to why it was rejected.
>
> If an ADR is deprecated, simply write "Deprecated" in this section. If an ADR
> is superseded by one or more other ADRs, provide local a reference to those
> ADRs, e.g.:
>
> Superseded by [ADR 123](./adr-123.md)
{Deprecated|Declined|Accepted|Implemented}
Accepted | Rejected | Deprecated | Superseded by
## Context


+ 94
- 0
docs/tendermint-core/consensus/proposer-based-timestamps.md View File

@ -0,0 +1,94 @@
--- order: 3 ---
# PBTS
This document provides an overview of the Proposer-Based Timestamp (PBTS)
algorithm added to Tendermint in the v0.36 release. It outlines the core
functionality as well as the parameters and constraints of the this algorithm.
## Algorithm Overview
The PBTS algorithm defines a way for a Tendermint blockchain to create block
timestamps that are within a reasonable bound of the clocks of the validators on
the network. This replaces the original BFTTime algorithm for timestamp
assignment that relied on the timestamps included in precommit messages.
## Algorithm Parameters
The functionality of the PBTS algorithm is governed by two parameters within
Tendermint. These two parameters are [consensus
parameters](https://github.com/tendermint/tendermint/blob/master/spec/abci/apps.md#L291),
meaning they are configured by the ABCI application and are expected to be the
same across all nodes on the network.
### `Precision`
The `Precision` parameter configures the acceptable upper-bound of clock drift
among all of the nodes on a Tendermint network. Any two nodes on a Tendermint
network are expected to have clocks that differ by at most `Precision`
milliseconds any given instant.
### `MessageDelay`
The `MessageDelay` parameter configures the acceptable upper-bound for
transmitting a `Proposal` message from the proposer to _all_ of the validators
on the network.
Networks should choose as small a value for `MessageDelay` as is practical,
provided it is large enough that messages can reach all participants with high
probability given the number of participants and latency of their connections.
## Algorithm Concepts
### Block timestamps
Each block produced by the Tendermint consensus engine contains a timestamp.
The timestamp produced in each block is a meaningful representation of time that is
useful for the protocols and applications built on top of Tendermint.
The following protocols and application features require a reliable source of time:
* Tendermint Light Clients [rely on correspondence between their known time](https://github.com/tendermint/tendermint/blob/master/spec/light-client/verification/README.md#definitions-1) and the block time for block verification.
* Tendermint Evidence validity is determined [either in terms of heights or in terms of time](https://github.com/tendermint/tendermint/blob/master/spec/consensus/evidence.md#verification).
* Unbonding of staked assets in the Cosmos Hub [occurs after a period of 21
days](https://github.com/cosmos/governance/blob/master/params-change/Staking.md#unbondingtime).
* IBC packets can use either a [timestamp or a height to timeout packet
delivery](https://docs.cosmos.network/v0.44/ibc/overview.html#acknowledgements)
### Proposer Selects a Block Timestamp
When the proposer node creates a new block proposal, the node reads the time
from its local clock and uses this reading as the timestamp for the proposed
block.
### Timeliness
When each validator on a Tendermint network receives a proposed block, it
performs a series of checks to ensure that the block can be considered valid as
a candidate to be the next block in the chain.
The PBTS algorithm performs a validity check on the timestamp of proposed
blocks. When a validator receives a proposal it ensures that the timestamp in
the proposal is within a bound of the validator's local clock. Specifically, the
algorithm checks that the timestamp is no more than `Precision` greater than the
node's local clock and no less than `Precision` + `MessageDelay` behind than the
node's local clock. This creates range of acceptable timestamps around the
node's local time. If the timestamp is within this range, the PBTS algorithm
considers the block **timely**. If a block is not **timely**, the node will
issue a `nil` `prevote` for this block, signaling to the rest of the network
that the node does not consider the block to be valid.
### Clock Synchronization
The PBTS algorithm requires the clocks of the validators on a Tendermint network
are within `Precision` of each other. In practice, this means that validators
should periodically synchronize to a reliable NTP server. Validators that drift
too far away from the rest of the network will no longer propose blocks with
valid timestamps. Additionally they will not view the timestamps of blocks
proposed by their peers to be valid either.
## See Also
* [The PBTS specification](https://github.com/tendermint/tendermint/blob/master/spec/consensus/proposer-based-timestamp/README.md)
contains all of the details of the algorithm.

+ 1
- 0
docs/versions View File

@ -1,3 +1,4 @@
master master
v0.33.x v0.33
v0.34.x v0.34
v0.35.x v0.35

+ 25
- 19
internal/blocksync/reactor.go View File

@ -70,6 +70,8 @@ type Reactor struct {
// immutable
initialState sm.State
// store
stateStore sm.Store
blockExec *sm.BlockExecutor
store *store.BlockStore
@ -101,7 +103,7 @@ type Reactor struct {
func NewReactor(
ctx context.Context,
logger log.Logger,
state sm.State,
stateStore sm.Store,
blockExec *sm.BlockExecutor,
store *store.BlockStore,
consReactor consensusReactor,
@ -111,19 +113,6 @@ func NewReactor(
metrics *consensus.Metrics,
eventBus *eventbus.EventBus,
) (*Reactor, error) {
if state.LastBlockHeight != store.Height() {
return nil, fmt.Errorf("state (%v) and store (%v) height mismatch", state.LastBlockHeight, store.Height())
}
startHeight := store.Height() + 1
if startHeight == 1 {
startHeight = state.InitialHeight
}
requestsCh := make(chan BlockRequest, maxTotalRequesters)
errorsCh := make(chan peerError, maxPeerErrBuffer) // NOTE: The capacity should be larger than the peer count.
blockSyncCh, err := channelCreator(ctx, GetChannelDescriptor())
if err != nil {
return nil, err
@ -131,20 +120,16 @@ func NewReactor(
r := &Reactor{
logger: logger,
initialState: state,
stateStore: stateStore,
blockExec: blockExec,
store: store,
pool: NewBlockPool(logger, startHeight, requestsCh, errorsCh),
consReactor: consReactor,
blockSync: newAtomicBool(blockSync),
requestsCh: requestsCh,
errorsCh: errorsCh,
blockSyncCh: blockSyncCh,
blockSyncOutBridgeCh: make(chan p2p.Envelope),
peerUpdates: peerUpdates,
metrics: metrics,
eventBus: eventBus,
syncStartTime: time.Time{},
}
r.BaseService = *service.NewBaseService(logger, "BlockSync", r)
@ -159,6 +144,27 @@ func NewReactor(
// If blockSync is enabled, we also start the pool and the pool processing
// goroutine. If the pool fails to start, an error is returned.
func (r *Reactor) OnStart(ctx context.Context) error {
state, err := r.stateStore.Load()
if err != nil {
return err
}
r.initialState = state
if state.LastBlockHeight != r.store.Height() {
return fmt.Errorf("state (%v) and store (%v) height mismatch", state.LastBlockHeight, r.store.Height())
}
startHeight := r.store.Height() + 1
if startHeight == 1 {
startHeight = state.InitialHeight
}
requestsCh := make(chan BlockRequest, maxTotalRequesters)
errorsCh := make(chan peerError, maxPeerErrBuffer) // NOTE: The capacity should be larger than the peer count.
r.pool = NewBlockPool(r.logger, startHeight, requestsCh, errorsCh)
r.requestsCh = requestsCh
r.errorsCh = errorsCh
if r.blockSync.IsSet() {
if err := r.pool.Start(ctx); err != nil {
return err


+ 1
- 1
internal/blocksync/reactor_test.go View File

@ -176,7 +176,7 @@ func (rts *reactorTestSuite) addNode(
rts.reactors[nodeID], err = NewReactor(
ctx,
rts.logger.With("nodeID", nodeID),
state.Copy(),
stateStore,
blockExec,
blockStore,
nil,


+ 21
- 11
internal/consensus/byzantine_test.go View File

@ -82,7 +82,6 @@ func TestByzantinePrevoteEquivocation(t *testing.T) {
log.TestingLogger().With("module", "mempool"),
thisConfig.Mempool,
proxyAppConnMem,
0,
)
if thisConfig.Consensus.WaitForTxs() {
mempool.EnableTxsAvailable()
@ -95,7 +94,8 @@ func TestByzantinePrevoteEquivocation(t *testing.T) {
// Make State
blockExec := sm.NewBlockExecutor(stateStore, log.TestingLogger(), proxyAppConnCon, mempool, evpool, blockStore)
cs := NewState(ctx, logger, thisConfig.Consensus, state, blockExec, blockStore, mempool, evpool)
cs, err := NewState(ctx, logger, thisConfig.Consensus, stateStore, blockExec, blockStore, mempool, evpool)
require.NoError(t, err)
// set private validator
pv := privVals[i]
cs.SetPrivValidator(ctx, pv)
@ -105,14 +105,13 @@ func TestByzantinePrevoteEquivocation(t *testing.T) {
require.NoError(t, err)
cs.SetEventBus(eventBus)
evpool.SetEventBus(eventBus)
cs.SetTimeoutTicker(tickerFunc())
states[i] = cs
}()
}
rts := setup(ctx, t, nValidators, states, 100) // buffer must be large enough to not deadlock
rts := setup(ctx, t, nValidators, states, 512) // buffer must be large enough to not deadlock
var bzNodeID types.NodeID
@ -238,8 +237,7 @@ func TestByzantinePrevoteEquivocation(t *testing.T) {
}
for _, reactor := range rts.reactors {
state := reactor.state.GetState()
reactor.SwitchToConsensus(ctx, state, false)
reactor.SwitchToConsensus(ctx, reactor.state.GetState(), false)
}
// Evidence should be submitted and committed at the third height but
@ -248,20 +246,26 @@ func TestByzantinePrevoteEquivocation(t *testing.T) {
var wg sync.WaitGroup
i := 0
subctx, subcancel := context.WithCancel(ctx)
defer subcancel()
for _, sub := range rts.subs {
wg.Add(1)
go func(j int, s eventbus.Subscription) {
defer wg.Done()
for {
if ctx.Err() != nil {
if subctx.Err() != nil {
return
}
msg, err := s.Next(subctx)
if subctx.Err() != nil {
return
}
msg, err := s.Next(ctx)
assert.NoError(t, err)
if err != nil {
cancel()
t.Errorf("waiting for subscription: %v", err)
subcancel()
return
}
@ -273,12 +277,18 @@ func TestByzantinePrevoteEquivocation(t *testing.T) {
}
}
}(i, sub)
i++
}
wg.Wait()
// don't run more assertions if we've encountered a timeout
select {
case <-subctx.Done():
t.Fatal("encountered timeout")
default:
}
pubkey, err := bzNodeState.privValidator.GetPubKey(ctx)
require.NoError(t, err)


+ 15
- 5
internal/consensus/common_test.go View File

@ -370,7 +370,11 @@ func subscribeToVoter(ctx context.Context, t *testing.T, cs *State, addr []byte)
vote := msg.Data().(types.EventDataVote)
// we only fire for our own votes
if bytes.Equal(addr, vote.Vote.ValidatorAddress) {
ch <- msg
select {
case <-ctx.Done():
return ctx.Err()
case ch <- msg:
}
}
return nil
}, types.EventQueryVote); err != nil {
@ -401,7 +405,10 @@ func subscribeToVoterBuffered(ctx context.Context, t *testing.T, cs *State, addr
vote := msg.Data().(types.EventDataVote)
// we only fire for our own votes
if bytes.Equal(addr, vote.Vote.ValidatorAddress) {
ch <- msg
select {
case <-ctx.Done():
case ch <- msg:
}
}
}
}()
@ -462,7 +469,6 @@ func newStateWithConfigAndBlockStore(
logger.With("module", "mempool"),
thisConfig.Mempool,
proxyAppConnMem,
0,
)
if thisConfig.Consensus.WaitForTxs() {
@ -477,15 +483,19 @@ func newStateWithConfigAndBlockStore(
require.NoError(t, stateStore.Save(state))
blockExec := sm.NewBlockExecutor(stateStore, logger, proxyAppConnCon, mempool, evpool, blockStore)
cs := NewState(ctx,
cs, err := NewState(ctx,
logger.With("module", "consensus"),
thisConfig.Consensus,
state,
stateStore,
blockExec,
blockStore,
mempool,
evpool,
)
if err != nil {
t.Fatal(err)
}
cs.SetPrivValidator(ctx, pv)
eventBus := eventbus.NewDefault(logger.With("module", "events"))


+ 4
- 3
internal/consensus/reactor_test.go View File

@ -461,6 +461,7 @@ func TestReactorWithEvidence(t *testing.T) {
stateStore := sm.NewStore(stateDB)
state, err := sm.MakeGenesisState(genDoc)
require.NoError(t, err)
require.NoError(t, stateStore.Save(state))
thisConfig, err := ResetConfig(t.TempDir(), fmt.Sprintf("%s_%d", testName, i))
require.NoError(t, err)
@ -483,7 +484,6 @@ func TestReactorWithEvidence(t *testing.T) {
log.TestingLogger().With("module", "mempool"),
thisConfig.Mempool,
proxyAppConnMem,
0,
)
if thisConfig.Consensus.WaitForTxs() {
@ -506,8 +506,9 @@ func TestReactorWithEvidence(t *testing.T) {
blockExec := sm.NewBlockExecutor(stateStore, log.TestingLogger(), proxyAppConnCon, mempool, evpool, blockStore)
cs := NewState(ctx, logger.With("validator", i, "module", "consensus"),
thisConfig.Consensus, state, blockExec, blockStore, mempool, evpool2)
cs, err := NewState(ctx, logger.With("validator", i, "module", "consensus"),
thisConfig.Consensus, stateStore, blockExec, blockStore, mempool, evpool2)
require.NoError(t, err)
cs.SetPrivValidator(ctx, pv)
eventBus := eventbus.NewDefault(log.TestingLogger().With("module", "events"))


+ 17
- 12
internal/consensus/replay_file.go View File

@ -84,7 +84,7 @@ func (cs *State) ReplayFile(ctx context.Context, file string, console bool) erro
return err
}
pb := newPlayback(file, fp, cs, cs.state.Copy())
pb := newPlayback(file, fp, cs, cs.stateStore)
defer pb.fp.Close()
var nextN int // apply N msgs in a row
@ -126,17 +126,17 @@ type playback struct {
count int // how many lines/msgs into the file are we
// replays can be reset to beginning
fileName string // so we can close/reopen the file
genesisState sm.State // so the replay session knows where to restart from
fileName string // so we can close/reopen the file
stateStore sm.Store
}
func newPlayback(fileName string, fp *os.File, cs *State, genState sm.State) *playback {
func newPlayback(fileName string, fp *os.File, cs *State, store sm.Store) *playback {
return &playback{
cs: cs,
fp: fp,
fileName: fileName,
genesisState: genState,
dec: NewWALDecoder(fp),
cs: cs,
fp: fp,
fileName: fileName,
stateStore: store,
dec: NewWALDecoder(fp),
}
}
@ -145,8 +145,11 @@ func (pb *playback) replayReset(ctx context.Context, count int, newStepSub event
pb.cs.Stop()
pb.cs.Wait()
newCS := NewState(ctx, pb.cs.logger, pb.cs.config, pb.genesisState.Copy(), pb.cs.blockExec,
newCS, err := NewState(ctx, pb.cs.logger, pb.cs.config, pb.stateStore, pb.cs.blockExec,
pb.cs.blockStore, pb.cs.txNotifier, pb.cs.evpool)
if err != nil {
return err
}
newCS.SetEventBus(pb.cs.eventBus)
newCS.startForReplay()
@ -345,9 +348,11 @@ func newConsensusStateForReplay(
mempool, evpool := emptyMempool{}, sm.EmptyEvidencePool{}
blockExec := sm.NewBlockExecutor(stateStore, logger, proxyApp.Consensus(), mempool, evpool, blockStore)
consensusState := NewState(ctx, logger, csConfig, state.Copy(), blockExec,
consensusState, err := NewState(ctx, logger, csConfig, stateStore, blockExec,
blockStore, mempool, evpool)
if err != nil {
return nil, err
}
consensusState.SetEventBus(eventBus)
return consensusState, nil
}

+ 36
- 9
internal/consensus/state.go View File

@ -121,6 +121,9 @@ type State struct {
// store blocks and commits
blockStore sm.BlockStore
stateStore sm.Store
initialStatePopulated bool
// create and execute blocks
blockExec *sm.BlockExecutor
@ -189,18 +192,19 @@ func NewState(
ctx context.Context,
logger log.Logger,
cfg *config.ConsensusConfig,
state sm.State,
store sm.Store,
blockExec *sm.BlockExecutor,
blockStore sm.BlockStore,
txNotifier txNotifier,
evpool evidencePool,
options ...StateOption,
) *State {
) (*State, error) {
cs := &State{
logger: logger,
config: cfg,
blockExec: blockExec,
blockStore: blockStore,
stateStore: store,
txNotifier: txNotifier,
peerMsgQueue: make(chan msgInfo, msgQueueSize),
internalMsgQueue: make(chan msgInfo, msgQueueSize),
@ -220,21 +224,40 @@ func NewState(
cs.doPrevote = cs.defaultDoPrevote
cs.setProposal = cs.defaultSetProposal
// We have no votes, so reconstruct LastCommit from SeenCommit.
if state.LastBlockHeight > 0 {
cs.reconstructLastCommit(state)
if err := cs.updateStateFromStore(ctx); err != nil {
return nil, err
}
cs.updateToState(ctx, state)
// NOTE: we do not call scheduleRound0 yet, we do that upon Start()
cs.BaseService = *service.NewBaseService(logger, "State", cs)
for _, option := range options {
option(cs)
}
return cs
return cs, nil
}
func (cs *State) updateStateFromStore(ctx context.Context) error {
if cs.initialStatePopulated {
return nil
}
state, err := cs.stateStore.Load()
if err != nil {
return fmt.Errorf("loading state: %w", err)
}
if state.IsEmpty() {
return nil
}
// We have no votes, so reconstruct LastCommit from SeenCommit.
if state.LastBlockHeight > 0 {
cs.reconstructLastCommit(state)
}
cs.updateToState(ctx, state)
cs.initialStatePopulated = true
return nil
}
// SetEventBus sets event bus.
@ -365,6 +388,10 @@ func (cs *State) LoadCommit(height int64) *types.Commit {
// OnStart loads the latest state via the WAL, and starts the timeout and
// receive routines.
func (cs *State) OnStart(ctx context.Context) error {
if err := cs.updateStateFromStore(ctx); err != nil {
return err
}
// We may set the WAL in testing before calling Start, so only OpenWAL if its
// still the nilWAL.
if _, ok := cs.wal.(nilWAL); ok {


+ 27
- 19
internal/consensus/wal_generator.go View File

@ -30,8 +30,10 @@ import (
// stripped down version of node (proxy app, event bus, consensus state) with a
// persistent kvstore application and special consensus wal instance
// (byteBufferWAL) and waits until numBlocks are created.
// If the node fails to produce given numBlocks, it returns an error.
func WALGenerateNBlocks(ctx context.Context, t *testing.T, logger log.Logger, wr io.Writer, numBlocks int) (err error) {
// If the node fails to produce given numBlocks, it fails the test.
func WALGenerateNBlocks(ctx context.Context, t *testing.T, logger log.Logger, wr io.Writer, numBlocks int) {
t.Helper()
cfg := getConfig(t)
app := kvstore.NewPersistentKVStoreApplication(logger, filepath.Join(cfg.DBDir(), "wal_generator"))
@ -46,40 +48,46 @@ func WALGenerateNBlocks(ctx context.Context, t *testing.T, logger log.Logger, wr
privValidatorStateFile := cfg.PrivValidator.StateFile()
privValidator, err := privval.LoadOrGenFilePV(privValidatorKeyFile, privValidatorStateFile)
if err != nil {
return err
t.Fatal(err)
}
genDoc, err := types.GenesisDocFromFile(cfg.GenesisFile())
if err != nil {
return fmt.Errorf("failed to read genesis file: %w", err)
t.Fatal(fmt.Errorf("failed to read genesis file: %w", err))
}
blockStoreDB := dbm.NewMemDB()
stateDB := blockStoreDB
stateStore := sm.NewStore(stateDB)
state, err := sm.MakeGenesisState(genDoc)
if err != nil {
return fmt.Errorf("failed to make genesis state: %w", err)
t.Fatal(fmt.Errorf("failed to make genesis state: %w", err))
}
state.Version.Consensus.App = kvstore.ProtocolVersion
if err = stateStore.Save(state); err != nil {
t.Error(err)
t.Fatal(err)
}
blockStore := store.NewBlockStore(blockStoreDB)
proxyApp := proxy.NewAppConns(abciclient.NewLocalCreator(app), logger.With("module", "proxy"), proxy.NopMetrics())
if err := proxyApp.Start(ctx); err != nil {
return fmt.Errorf("failed to start proxy app connections: %w", err)
t.Fatal(fmt.Errorf("failed to start proxy app connections: %w", err))
}
t.Cleanup(proxyApp.Wait)
eventBus := eventbus.NewDefault(logger.With("module", "events"))
if err := eventBus.Start(ctx); err != nil {
return fmt.Errorf("failed to start event bus: %w", err)
t.Fatal(fmt.Errorf("failed to start event bus: %w", err))
}
t.Cleanup(func() { eventBus.Stop(); eventBus.Wait() })
mempool := emptyMempool{}
evpool := sm.EmptyEvidencePool{}
blockExec := sm.NewBlockExecutor(stateStore, log.TestingLogger(), proxyApp.Consensus(), mempool, evpool, blockStore)
consensusState := NewState(ctx, logger, cfg.Consensus, state.Copy(), blockExec, blockStore, mempool, evpool)
consensusState, err := NewState(ctx, logger, cfg.Consensus, stateStore, blockExec, blockStore, mempool, evpool)
if err != nil {
t.Fatal(err)
}
consensusState.SetEventBus(eventBus)
if privValidator != nil && privValidator != (*privval.FilePV)(nil) {
consensusState.SetPrivValidator(ctx, privValidator)
@ -91,22 +99,24 @@ func WALGenerateNBlocks(ctx context.Context, t *testing.T, logger log.Logger, wr
wal := newByteBufferWAL(logger, NewWALEncoder(wr), int64(numBlocks), numBlocksWritten)
// see wal.go#103
if err := wal.Write(EndHeightMessage{0}); err != nil {
t.Error(err)
t.Fatal(err)
}
consensusState.wal = wal
if err := consensusState.Start(ctx); err != nil {
return fmt.Errorf("failed to start consensus state: %w", err)
t.Fatal(fmt.Errorf("failed to start consensus state: %w", err))
}
t.Cleanup(consensusState.Wait)
defer consensusState.Stop()
timer := time.NewTimer(time.Minute)
defer timer.Stop()
select {
case <-numBlocksWritten:
consensusState.Stop()
return nil
case <-time.After(1 * time.Minute):
consensusState.Stop()
return fmt.Errorf("waited too long for tendermint to produce %d blocks (grep logs for `wal_generator`)", numBlocks)
case <-timer.C:
t.Fatal(fmt.Errorf("waited too long for tendermint to produce %d blocks (grep logs for `wal_generator`)", numBlocks))
}
}
@ -115,9 +125,7 @@ func WALWithNBlocks(ctx context.Context, t *testing.T, logger log.Logger, numBlo
var b bytes.Buffer
wr := bufio.NewWriter(&b)
if err := WALGenerateNBlocks(ctx, t, logger, wr, numBlocks); err != nil {
return []byte{}, err
}
WALGenerateNBlocks(ctx, t, logger, wr, numBlocks)
wr.Flush()
return b.Bytes(), nil


+ 9
- 10
internal/consensus/wal_test.go View File

@ -3,6 +3,7 @@ package consensus
import (
"bytes"
"context"
"os"
"path/filepath"
"testing"
@ -41,13 +42,12 @@ func TestWALTruncate(t *testing.T) {
require.NoError(t, err)
err = wal.Start(ctx)
require.NoError(t, err)
t.Cleanup(wal.Wait)
t.Cleanup(func() { wal.Stop(); wal.Group().Stop(); wal.Group().Wait(); wal.Wait() })
// 60 block's size nearly 70K, greater than group's headBuf size(4096 * 10),
// when headBuf is full, truncate content will Flush to the file. at this
// time, RotateFile is called, truncate content exist in each file.
err = WALGenerateNBlocks(ctx, t, logger, wal.Group(), 60)
require.NoError(t, err)
WALGenerateNBlocks(ctx, t, logger, wal.Group(), 60)
// put the leakcheck here so it runs after other cleanup
// functions.
@ -112,7 +112,7 @@ func TestWALWrite(t *testing.T) {
require.NoError(t, err)
err = wal.Start(ctx)
require.NoError(t, err)
t.Cleanup(wal.Wait)
t.Cleanup(func() { wal.Stop(); wal.Group().Stop(); wal.Group().Wait(); wal.Wait() })
// 1) Write returns an error if msg is too big
msg := &BlockPartMessage{
@ -151,7 +151,6 @@ func TestWALSearchForEndHeight(t *testing.T) {
wal, err := NewWAL(ctx, logger, walFile)
require.NoError(t, err)
t.Cleanup(func() { wal.Stop(); wal.Wait() })
h := int64(3)
gr, found, err := wal.SearchForEndHeight(h, &WALSearchOptions{})
@ -176,24 +175,24 @@ func TestWALPeriodicSync(t *testing.T) {
walDir := t.TempDir()
walFile := filepath.Join(walDir, "wal")
wal, err := NewWAL(ctx, log.TestingLogger(), walFile, autofile.GroupCheckDuration(1*time.Millisecond))
defer os.RemoveAll(walFile)
wal, err := NewWAL(ctx, log.TestingLogger(), walFile, autofile.GroupCheckDuration(250*time.Millisecond))
require.NoError(t, err)
wal.SetFlushInterval(walTestFlushInterval)
logger := log.NewNopLogger()
// Generate some data
err = WALGenerateNBlocks(ctx, t, logger, wal.Group(), 5)
require.NoError(t, err)
WALGenerateNBlocks(ctx, t, logger, wal.Group(), 5)
// We should have data in the buffer now
assert.NotZero(t, wal.Group().Buffered())
require.NoError(t, wal.Start(ctx))
t.Cleanup(func() { wal.Stop(); wal.Wait() })
t.Cleanup(func() { wal.Stop(); wal.Group().Stop(); wal.Group().Wait(); wal.Wait() })
time.Sleep(walTestFlushInterval + (10 * time.Millisecond))
time.Sleep(walTestFlushInterval + (20 * time.Millisecond))
// The data should have been flushed by the periodic sync
assert.Zero(t, wal.Group().Buffered())


+ 0
- 7
internal/eventbus/event_bus.go View File

@ -50,13 +50,6 @@ func (b *EventBus) NumClientSubscriptions(clientID string) int {
return b.pubsub.NumClientSubscriptions(clientID)
}
// Deprecated: Use SubscribeWithArgs instead.
func (b *EventBus) Subscribe(ctx context.Context,
clientID string, query *tmquery.Query, capacities ...int) (Subscription, error) {
return b.pubsub.Subscribe(ctx, clientID, query, capacities...)
}
func (b *EventBus) SubscribeWithArgs(ctx context.Context, args tmpubsub.SubscribeArgs) (Subscription, error) {
return b.pubsub.SubscribeWithArgs(ctx, args)
}


+ 13
- 0
internal/libs/autofile/group.go View File

@ -274,6 +274,10 @@ func (g *Group) checkTotalSizeLimit(ctx context.Context) {
g.mtx.Lock()
defer g.mtx.Unlock()
if err := ctx.Err(); err != nil {
return
}
if g.totalSizeLimit == 0 {
return
}
@ -290,6 +294,11 @@ func (g *Group) checkTotalSizeLimit(ctx context.Context) {
g.logger.Error("Group's head may grow without bound", "head", g.Head.Path)
return
}
if ctx.Err() != nil {
return
}
pathToRemove := filePathForIndex(g.Head.Path, index, gInfo.MaxIndex)
fInfo, err := os.Stat(pathToRemove)
if err != nil {
@ -314,6 +323,10 @@ func (g *Group) rotateFile(ctx context.Context) {
g.mtx.Lock()
defer g.mtx.Unlock()
if err := ctx.Err(); err != nil {
return
}
headPath := g.Head.Path
if err := g.headBuf.Flush(); err != nil {


+ 1
- 2
internal/mempool/mempool.go View File

@ -94,7 +94,6 @@ func NewTxMempool(
logger log.Logger,
cfg *config.MempoolConfig,
proxyAppConn proxy.AppConnMempool,
height int64,
options ...TxMempoolOption,
) *TxMempool {
@ -102,7 +101,7 @@ func NewTxMempool(
logger: logger,
config: cfg,
proxyAppConn: proxyAppConn,
height: height,
height: -1,
cache: NopTxCache{},
metrics: NopMetrics(),
txStore: NewTxStore(),


+ 1
- 1
internal/mempool/mempool_test.go View File

@ -95,7 +95,7 @@ func setup(ctx context.Context, t testing.TB, cacheSize int, options ...TxMempoo
appConnMem.Wait()
})
return NewTxMempool(logger.With("test", t.Name()), cfg.Mempool, appConnMem, 0, options...)
return NewTxMempool(logger.With("test", t.Name()), cfg.Mempool, appConnMem, options...)
}
func checkTxs(ctx context.Context, t *testing.T, txmp *TxMempool, numTxs int, peerID uint16) []testTx {


+ 8
- 11
internal/p2p/pex/reactor_test.go View File

@ -1,6 +1,4 @@
// Temporarily disabled pending ttps://github.com/tendermint/tendermint/issues/7626.
//go:build issue7626
//nolint:unused
package pex_test
import (
@ -103,6 +101,7 @@ func TestReactorSendsRequestsTooOften(t *testing.T) {
}
func TestReactorSendsResponseWithoutRequest(t *testing.T) {
t.Skip("This test needs updated https://github.com/tendermint/tendermint/issue/7634")
ctx, cancel := context.WithCancel(context.Background())
defer cancel()
@ -124,6 +123,7 @@ func TestReactorSendsResponseWithoutRequest(t *testing.T) {
}
func TestReactorNeverSendsTooManyPeers(t *testing.T) {
t.Skip("This test needs updated https://github.com/tendermint/tendermint/issue/7634")
ctx, cancel := context.WithCancel(context.Background())
defer cancel()
@ -235,6 +235,7 @@ func TestReactorLargePeerStoreInASmallNetwork(t *testing.T) {
}
func TestReactorWithNetworkGrowth(t *testing.T) {
t.Skip("This test needs updated https://github.com/tendermint/tendermint/issue/7634")
ctx, cancel := context.WithCancel(context.Background())
defer cancel()
@ -686,20 +687,16 @@ func (r *reactorTestSuite) connectPeers(ctx context.Context, t *testing.T, sourc
select {
case peerUpdate := <-targetSub.Updates():
require.Equal(t, p2p.PeerUpdate{
NodeID: node1,
Status: p2p.PeerStatusUp,
}, peerUpdate)
require.Equal(t, peerUpdate.NodeID, node1)
require.Equal(t, peerUpdate.Status, p2p.PeerStatusUp)
case <-time.After(2 * time.Second):
require.Fail(t, "timed out waiting for peer", "%v accepting %v",
targetNode, sourceNode)
}
select {
case peerUpdate := <-sourceSub.Updates():
require.Equal(t, p2p.PeerUpdate{
NodeID: node2,
Status: p2p.PeerStatusUp,
}, peerUpdate)
require.Equal(t, peerUpdate.NodeID, node2)
require.Equal(t, peerUpdate.Status, p2p.PeerStatusUp)
case <-time.After(2 * time.Second):
require.Fail(t, "timed out waiting for peer", "%v dialing %v",
sourceNode, targetNode)


+ 0
- 20
internal/pubsub/pubsub.go View File

@ -153,26 +153,6 @@ func BufferCapacity(cap int) Option {
// BufferCapacity returns capacity of the publication queue.
func (s *Server) BufferCapacity() int { return cap(s.queue) }
// Subscribe creates a subscription for the given client ID and query.
// If len(capacities) > 0, its first value is used as the queue capacity.
//
// Deprecated: Use SubscribeWithArgs. This method will be removed in v0.36.
func (s *Server) Subscribe(ctx context.Context, clientID string, query *query.Query, capacities ...int) (*Subscription, error) {
args := SubscribeArgs{
ClientID: clientID,
Query: query,
Limit: 1,
}
if len(capacities) > 0 {
args.Limit = capacities[0]
if len(capacities) > 1 {
args.Quota = capacities[1]
}
// bounds are checked below
}
return s.SubscribeWithArgs(ctx, args)
}
// Observe registers an observer function that will be called synchronously
// with each published message matching any of the given queries, prior to it
// being forwarded to any subscriber. If no queries are specified, all


+ 7
- 8
internal/rpc/core/env.go View File

@ -57,12 +57,6 @@ type consensusState interface {
GetRoundStateSimpleJSON() ([]byte, error)
}
type transport interface {
Listeners() []string
IsListening() bool
NodeInfo() types.NodeInfo
}
type peerManager interface {
Peers() []types.NodeID
Addresses(types.NodeID) []p2p.NodeAddress
@ -84,8 +78,9 @@ type Environment struct {
ConsensusReactor *consensus.Reactor
BlockSyncReactor *blocksync.Reactor
// Legacy p2p stack
P2PTransport transport
IsListening bool
Listeners []string
NodeInfo types.NodeInfo
// interfaces for new p2p interfaces
PeerManager peerManager
@ -226,6 +221,10 @@ func (env *Environment) StartService(ctx context.Context, conf *config.Config) (
return nil, err
}
env.Listeners = []string{
fmt.Sprintf("Listener(@%v)", conf.P2P.ExternalAddress),
}
listenAddrs := strings.SplitAndTrimEmpty(conf.RPC.ListenAddress, ",", " ")
routes := NewRoutesMap(env, &RouteOptions{
Unsafe: conf.RPC.Unsafe,


+ 6
- 3
internal/rpc/core/events.go View File

@ -165,8 +165,11 @@ func (env *Environment) Events(ctx context.Context,
maxItems = 100
}
const minWaitTime = 1 * time.Second
const maxWaitTime = 30 * time.Second
if waitTime > maxWaitTime {
if waitTime < minWaitTime {
waitTime = minWaitTime
} else if waitTime > maxWaitTime {
waitTime = maxWaitTime
}
@ -185,7 +188,7 @@ func (env *Environment) Events(ctx context.Context,
accept := func(itm *eventlog.Item) error {
// N.B. We accept up to one item more than requested, so we can tell how
// to set the "more" flag in the response.
if len(items) > maxItems {
if len(items) > maxItems || itm.Cursor.Before(after) {
return eventlog.ErrStopScan
}
if cursorInRange(itm.Cursor, before, after) && query.Matches(itm.Events) {
@ -194,7 +197,7 @@ func (env *Environment) Events(ctx context.Context,
return nil
}
if waitTime > 0 && before.IsZero() {
if before.IsZero() {
ctx, cancel := context.WithTimeout(ctx, waitTime)
defer cancel()


+ 2
- 2
internal/rpc/core/net.go View File

@ -27,8 +27,8 @@ func (env *Environment) NetInfo(ctx context.Context) (*coretypes.ResultNetInfo,
}
return &coretypes.ResultNetInfo{
Listening: env.P2PTransport.IsListening(),
Listeners: env.P2PTransport.Listeners(),
Listening: env.IsListening,
Listeners: env.Listeners,
NPeers: len(peers),
Peers: peers,
}, nil


+ 1
- 1
internal/rpc/core/status.go View File

@ -66,7 +66,7 @@ func (env *Environment) Status(ctx context.Context) (*coretypes.ResultStatus, er
}
result := &coretypes.ResultStatus{
NodeInfo: env.P2PTransport.NodeInfo(),
NodeInfo: env.NodeInfo,
ApplicationInfo: applicationInfo,
SyncInfo: coretypes.SyncInfo{
LatestBlockHash: latestBlockHash,


+ 2
- 2
internal/state/execution.go View File

@ -367,8 +367,8 @@ func (blockExec *BlockExecutor) Commit(
block.Height,
block.Txs,
txResults,
TxPreCheck(state),
TxPostCheck(state),
TxPreCheckForState(state),
TxPostCheckForState(state),
)
return res.Data, res.RetainHeight, err


+ 73
- 10
internal/state/tx_filter.go View File

@ -1,22 +1,85 @@
package state
import (
"sync"
"time"
abci "github.com/tendermint/tendermint/abci/types"
"github.com/tendermint/tendermint/internal/mempool"
"github.com/tendermint/tendermint/types"
)
// TxPreCheck returns a function to filter transactions before processing.
// The function limits the size of a transaction to the block's maximum data size.
func TxPreCheck(state State) mempool.PreCheckFunc {
maxDataBytes := types.MaxDataBytesNoEvidence(
state.ConsensusParams.Block.MaxBytes,
state.Validators.Size(),
func cachingStateFetcher(store Store) func() (State, error) {
const ttl = time.Second
var (
last time.Time
mutex = &sync.Mutex{}
cache State
err error
)
return mempool.PreCheckMaxBytes(maxDataBytes)
return func() (State, error) {
mutex.Lock()
defer mutex.Unlock()
if time.Since(last) < ttl && cache.ChainID != "" {
return cache, nil
}
cache, err = store.Load()
if err != nil {
return State{}, err
}
last = time.Now()
return cache, nil
}
}
// TxPostCheck returns a function to filter transactions after processing.
// TxPreCheckFromStore returns a function to filter transactions before processing.
// The function limits the size of a transaction to the block's maximum data size.
func TxPreCheckFromStore(store Store) mempool.PreCheckFunc {
fetch := cachingStateFetcher(store)
return func(tx types.Tx) error {
state, err := fetch()
if err != nil {
return err
}
return TxPreCheckForState(state)(tx)
}
}
func TxPreCheckForState(state State) mempool.PreCheckFunc {
return func(tx types.Tx) error {
maxDataBytes := types.MaxDataBytesNoEvidence(
state.ConsensusParams.Block.MaxBytes,
state.Validators.Size(),
)
return mempool.PreCheckMaxBytes(maxDataBytes)(tx)
}
}
// TxPostCheckFromStore returns a function to filter transactions after processing.
// The function limits the gas wanted by a transaction to the block's maximum total gas.
func TxPostCheck(state State) mempool.PostCheckFunc {
return mempool.PostCheckMaxGas(state.ConsensusParams.Block.MaxGas)
func TxPostCheckFromStore(store Store) mempool.PostCheckFunc {
fetch := cachingStateFetcher(store)
return func(tx types.Tx, resp *abci.ResponseCheckTx) error {
state, err := fetch()
if err != nil {
return err
}
return mempool.PostCheckMaxGas(state.ConsensusParams.Block.MaxGas)(tx, resp)
}
}
func TxPostCheckForState(state State) mempool.PostCheckFunc {
return func(tx types.Tx, resp *abci.ResponseCheckTx) error {
return mempool.PostCheckMaxGas(state.ConsensusParams.Block.MaxGas)(tx, resp)
}
}

+ 1
- 1
internal/state/tx_filter_test.go View File

@ -31,7 +31,7 @@ func TestTxFilter(t *testing.T) {
state, err := sm.MakeGenesisState(genDoc)
require.NoError(t, err)
f := sm.TxPreCheck(state)
f := sm.TxPreCheckForState(state)
if tc.isErr {
assert.NotNil(t, f(tc.tx), "#%v", i)
} else {


+ 20
- 37
node/node.go View File

@ -58,7 +58,6 @@ type nodeImpl struct {
router *p2p.Router
nodeInfo types.NodeInfo
nodeKey types.NodeKey // our node privkey
isListening bool
// services
eventSinks []indexer.EventSink
@ -144,11 +143,8 @@ func makeNode(
return nil, combineCloseError(err, makeCloser(closers))
}
err = genDoc.ValidateAndComplete()
if err != nil {
return nil, combineCloseError(
fmt.Errorf("error in genesis doc: %w", err),
makeCloser(closers))
if err = genDoc.ValidateAndComplete(); err != nil {
return nil, combineCloseError(fmt.Errorf("error in genesis doc: %w", err), makeCloser(closers))
}
state, err := loadStateFromDBOrGenesisDocProvider(stateStore, genDoc)
@ -242,10 +238,6 @@ func makeNode(
}
}
// Determine whether we should do block sync. This must happen after the handshake, since the
// app may modify the validator set, specifying ourself as the only validator.
blockSync := !onlyValidatorIsUs(state, pubKey)
logNodeStartupInfo(state, pubKey, logger, cfg.Mode)
// TODO: Fetch and provide real options and do proper p2p bootstrapping.
@ -272,14 +264,14 @@ func makeNode(
}
mpReactor, mp, err := createMempoolReactor(ctx,
cfg, proxyApp, state, nodeMetrics.mempool, peerManager, router, logger,
cfg, proxyApp, stateStore, nodeMetrics.mempool, peerManager, router, logger,
)
if err != nil {
return nil, combineCloseError(err, makeCloser(closers))
}
evReactor, evPool, err := createEvidenceReactor(ctx,
cfg, dbProvider, stateDB, blockStore, peerManager, router, logger, nodeMetrics.evidence, eventBus,
cfg, dbProvider, stateStore, blockStore, peerManager, router, logger, nodeMetrics.evidence, eventBus,
)
if err != nil {
return nil, combineCloseError(err, makeCloser(closers))
@ -296,8 +288,12 @@ func makeNode(
sm.BlockExecutorWithMetrics(nodeMetrics.state),
)
// Determine whether we should do block sync. This must happen after the handshake, since the
// app may modify the validator set, specifying ourself as the only validator.
blockSync := !onlyValidatorIsUs(state, pubKey)
csReactor, csState, err := createConsensusReactor(ctx,
cfg, state, blockExec, blockStore, mp, evPool,
cfg, stateStore, blockExec, blockStore, mp, evPool,
privValidator, nodeMetrics.consensus, stateSync || blockSync, eventBus,
peerManager, router, logger,
)
@ -309,7 +305,7 @@ func makeNode(
// doing a state sync first.
bcReactor, err := blocksync.NewReactor(ctx,
logger.With("module", "blockchain"),
state.Copy(),
stateStore,
blockExec,
blockStore,
csReactor,
@ -421,8 +417,6 @@ func makeNode(
node.rpcEnv.PubKey = pubKey
}
node.rpcEnv.P2PTransport = node
node.BaseService = *service.NewBaseService(logger, "Node", node)
return node, nil
@ -467,6 +461,7 @@ func (n *nodeImpl) OnStart(ctx context.Context) error {
}
}
n.rpcEnv.NodeInfo = n.nodeInfo
// Start the RPC server before the P2P server
// so we can eg. receive txs for the first block
if n.config.RPC.ListenAddress != "" {
@ -485,7 +480,7 @@ func (n *nodeImpl) OnStart(ctx context.Context) error {
if err := n.router.Start(ctx); err != nil {
return err
}
n.isListening = true
n.rpcEnv.IsListening = true
for _, reactor := range n.services {
if err := reactor.Start(ctx); err != nil {
@ -580,7 +575,7 @@ func (n *nodeImpl) OnStop() {
n.stateSyncReactor.Wait()
n.router.Wait()
n.isListening = false
n.rpcEnv.IsListening = false
// finally stop the listeners / external services
for _, l := range n.rpcListeners {
@ -669,21 +664,6 @@ func (n *nodeImpl) RPCEnvironment() *rpccore.Environment {
//------------------------------------------------------------------------------
func (n *nodeImpl) Listeners() []string {
return []string{
fmt.Sprintf("Listener(@%v)", n.config.P2P.ExternalAddress),
}
}
func (n *nodeImpl) IsListening() bool {
return n.isListening
}
// NodeInfo returns the Node's Info from the Switch.
func (n *nodeImpl) NodeInfo() types.NodeInfo {
return n.nodeInfo
}
// genesisDocProvider returns a GenesisDoc.
// It allows the GenesisDoc to be pulled from sources other than the
// filesystem, for instance from a distributed key-value store cluster.
@ -747,10 +727,7 @@ func defaultMetricsProvider(cfg *config.InstrumentationConfig) metricsProvider {
// loadStateFromDBOrGenesisDocProvider attempts to load the state from the
// database, or creates one using the given genesisDocProvider. On success this also
// returns the genesis doc loaded through the given provider.
func loadStateFromDBOrGenesisDocProvider(
stateStore sm.Store,
genDoc *types.GenesisDoc,
) (sm.State, error) {
func loadStateFromDBOrGenesisDocProvider(stateStore sm.Store, genDoc *types.GenesisDoc) (sm.State, error) {
// 1. Attempt to load state form the database
state, err := stateStore.Load()
@ -764,6 +741,12 @@ func loadStateFromDBOrGenesisDocProvider(
if err != nil {
return sm.State{}, err
}
// 3. save the gensis document to the state store so
// its fetchable by other callers.
if err := stateStore.Save(state); err != nil {
return sm.State{}, err
}
}
return state, nil


+ 0
- 3
node/node_test.go View File

@ -292,7 +292,6 @@ func TestCreateProposalBlock(t *testing.T) {
logger.With("module", "mempool"),
cfg.Mempool,
proxyApp.Mempool(),
state.LastBlockHeight,
)
// Make EvidencePool
@ -392,7 +391,6 @@ func TestMaxTxsProposalBlockSize(t *testing.T) {
logger.With("module", "mempool"),
cfg.Mempool,
proxyApp.Mempool(),
state.LastBlockHeight,
)
// fill the mempool with one txs just below the maximum size
@ -457,7 +455,6 @@ func TestMaxProposalBlockSize(t *testing.T) {
logger.With("module", "mempool"),
cfg.Mempool,
proxyApp.Mempool(),
state.LastBlockHeight,
)
// fill the mempool with one txs just below the maximum size


+ 11
- 9
node/setup.go View File

@ -172,7 +172,7 @@ func createMempoolReactor(
ctx context.Context,
cfg *config.Config,
proxyApp proxy.AppConns,
state sm.State,
store sm.Store,
memplMetrics *mempool.Metrics,
peerManager *p2p.PeerManager,
router *p2p.Router,
@ -184,10 +184,9 @@ func createMempoolReactor(
logger,
cfg.Mempool,
proxyApp.Mempool(),
state.LastBlockHeight,
mempool.WithMetrics(memplMetrics),
mempool.WithPreCheck(sm.TxPreCheck(state)),
mempool.WithPostCheck(sm.TxPostCheck(state)),
mempool.WithPreCheck(sm.TxPreCheckFromStore(store)),
mempool.WithPostCheck(sm.TxPostCheckFromStore(store)),
)
reactor, err := mempool.NewReactor(
@ -214,7 +213,7 @@ func createEvidenceReactor(
ctx context.Context,
cfg *config.Config,
dbProvider config.DBProvider,
stateDB dbm.DB,
store sm.Store,
blockStore *store.BlockStore,
peerManager *p2p.PeerManager,
router *p2p.Router,
@ -229,7 +228,7 @@ func createEvidenceReactor(
logger = logger.With("module", "evidence")
evidencePool, err := evidence.NewPool(logger, evidenceDB, sm.NewStore(stateDB), blockStore, metrics)
evidencePool, err := evidence.NewPool(logger, evidenceDB, store, blockStore, metrics)
if err != nil {
return nil, nil, fmt.Errorf("creating evidence pool: %w", err)
}
@ -253,7 +252,7 @@ func createEvidenceReactor(
func createConsensusReactor(
ctx context.Context,
cfg *config.Config,
state sm.State,
store sm.Store,
blockExec *sm.BlockExecutor,
blockStore sm.BlockStore,
mp mempool.Mempool,
@ -268,16 +267,19 @@ func createConsensusReactor(
) (*consensus.Reactor, *consensus.State, error) {
logger = logger.With("module", "consensus")
consensusState := consensus.NewState(ctx,
consensusState, err := consensus.NewState(ctx,
logger,
cfg.Consensus,
state.Copy(),
store,
blockExec,
blockStore,
mp,
evidencePool,
consensus.StateMetrics(csMetrics),
)
if err != nil {
return nil, nil, err
}
if privValidator != nil && cfg.Mode == config.ModeValidator {
consensusState.SetPrivValidator(ctx, privValidator)


+ 93
- 49
proto/tendermint/abci/types.proto View File

@ -21,26 +21,26 @@ import "gogoproto/gogo.proto";
message Request {
oneof value {
RequestEcho echo = 1;
RequestFlush flush = 2;
RequestInfo info = 3;
RequestInitChain init_chain = 4;
RequestQuery query = 5;
RequestBeginBlock begin_block = 6 [deprecated = true];
RequestCheckTx check_tx = 7;
RequestDeliverTx deliver_tx = 8 [deprecated = true];
RequestEndBlock end_block = 9 [deprecated = true];
RequestCommit commit = 10;
RequestListSnapshots list_snapshots = 11;
RequestOfferSnapshot offer_snapshot = 12;
RequestLoadSnapshotChunk load_snapshot_chunk = 13;
RequestApplySnapshotChunk apply_snapshot_chunk = 14;
RequestPrepareProposal prepare_proposal = 15;
RequestProcessProposal process_proposal = 16;
RequestFinalizeBlock finalize_block = 19;
RequestEcho echo = 1;
RequestFlush flush = 2;
RequestInfo info = 3;
RequestInitChain init_chain = 4;
RequestQuery query = 5;
RequestBeginBlock begin_block = 6 [deprecated = true];
RequestCheckTx check_tx = 7;
RequestDeliverTx deliver_tx = 8 [deprecated = true];
RequestEndBlock end_block = 9 [deprecated = true];
RequestCommit commit = 10;
RequestListSnapshots list_snapshots = 11;
RequestOfferSnapshot offer_snapshot = 12;
RequestLoadSnapshotChunk load_snapshot_chunk = 13;
RequestApplySnapshotChunk apply_snapshot_chunk = 14;
RequestPrepareProposal prepare_proposal = 15;
RequestProcessProposal process_proposal = 16;
RequestExtendVote extend_vote = 17;
RequestVerifyVoteExtension verify_vote_extension = 18;
RequestFinalizeBlock finalize_block = 19;
}
reserved 17; // Placeholder for RequestExtendVote in v0.37
reserved 18; // Placeholder for RequestVerifyVoteExtension in v0.37
}
message RequestEcho {
@ -75,7 +75,7 @@ message RequestQuery {
message RequestBeginBlock {
bytes hash = 1;
tendermint.types.Header header = 2 [(gogoproto.nullable) = false];
LastCommitInfo last_commit_info = 3 [(gogoproto.nullable) = false];
CommitInfo last_commit_info = 3 [(gogoproto.nullable) = false];
repeated Evidence byzantine_validators = 4 [(gogoproto.nullable) = false];
}
@ -127,9 +127,9 @@ message RequestPrepareProposal {
tendermint.types.Header header = 2 [(gogoproto.nullable) = false];
// txs is an array of transactions that will be included in a block,
// sent to the app for possible modifications.
repeated bytes txs = 3;
LastCommitInfo last_commit_info = 4 [(gogoproto.nullable) = false];
repeated Evidence byzantine_validators = 5 [(gogoproto.nullable) = false];
repeated bytes txs = 3;
ExtendedCommitInfo local_last_commit = 4 [(gogoproto.nullable) = false];
repeated Evidence byzantine_validators = 5 [(gogoproto.nullable) = false];
// the modified transactions cannot exceed this size.
int64 max_tx_bytes = 6;
}
@ -138,15 +138,29 @@ message RequestProcessProposal {
bytes hash = 1;
tendermint.types.Header header = 2 [(gogoproto.nullable) = false];
repeated bytes txs = 3;
LastCommitInfo last_commit_info = 4 [(gogoproto.nullable) = false];
CommitInfo proposed_last_commit = 4 [(gogoproto.nullable) = false];
repeated Evidence byzantine_validators = 5 [(gogoproto.nullable) = false];
}
// Extends a vote with application-side injection
message RequestExtendVote {
bytes hash = 1;
int64 height = 2;
}
// Verify the vote extension
message RequestVerifyVoteExtension {
bytes hash = 1;
bytes validator_address = 2;
int64 height = 3;
bytes vote_extension = 4;
}
message RequestFinalizeBlock {
bytes hash = 1;
tendermint.types.Header header = 2 [(gogoproto.nullable) = false];
repeated bytes txs = 3;
LastCommitInfo last_commit_info = 4 [(gogoproto.nullable) = false];
CommitInfo decided_last_commit = 4 [(gogoproto.nullable) = false];
repeated Evidence byzantine_validators = 5 [(gogoproto.nullable) = false];
}
@ -155,27 +169,27 @@ message RequestFinalizeBlock {
message Response {
oneof value {
ResponseException exception = 1;
ResponseEcho echo = 2;
ResponseFlush flush = 3;
ResponseInfo info = 4;
ResponseInitChain init_chain = 5;
ResponseQuery query = 6;
ResponseBeginBlock begin_block = 7 [deprecated = true];
ResponseCheckTx check_tx = 8;
ResponseDeliverTx deliver_tx = 9 [deprecated = true];
ResponseEndBlock end_block = 10 [deprecated = true];
ResponseCommit commit = 11;
ResponseListSnapshots list_snapshots = 12;
ResponseOfferSnapshot offer_snapshot = 13;
ResponseLoadSnapshotChunk load_snapshot_chunk = 14;
ResponseApplySnapshotChunk apply_snapshot_chunk = 15;
ResponsePrepareProposal prepare_proposal = 16;
ResponseProcessProposal process_proposal = 17;
ResponseFinalizeBlock finalize_block = 20;
ResponseException exception = 1;
ResponseEcho echo = 2;
ResponseFlush flush = 3;
ResponseInfo info = 4;
ResponseInitChain init_chain = 5;
ResponseQuery query = 6;
ResponseBeginBlock begin_block = 7 [deprecated = true];
ResponseCheckTx check_tx = 8;
ResponseDeliverTx deliver_tx = 9 [deprecated = true];
ResponseEndBlock end_block = 10 [deprecated = true];
ResponseCommit commit = 11;
ResponseListSnapshots list_snapshots = 12;
ResponseOfferSnapshot offer_snapshot = 13;
ResponseLoadSnapshotChunk load_snapshot_chunk = 14;
ResponseApplySnapshotChunk apply_snapshot_chunk = 15;
ResponsePrepareProposal prepare_proposal = 16;
ResponseProcessProposal process_proposal = 17;
ResponseExtendVote extend_vote = 18;
ResponseVerifyVoteExtension verify_vote_extension = 19;
ResponseFinalizeBlock finalize_block = 20;
}
reserved 18; // Placeholder for ResponseExtendVote in v0.37
reserved 19; // Placeholder for ResponseVerifyVoteExtension in v0.37
}
// nondeterministic
@ -308,7 +322,7 @@ message ResponsePrepareProposal {
repeated ExecTxResult tx_results = 4;
repeated ValidatorUpdate validator_updates = 5;
tendermint.types.ConsensusParams consensus_param_updates = 6;
reserved 7; // Placeholder for app_signed_updates in v0.37
repeated bytes app_signed_updates = 7;
}
message ResponseProcessProposal {
@ -319,6 +333,14 @@ message ResponseProcessProposal {
tendermint.types.ConsensusParams consensus_param_updates = 5;
}
message ResponseExtendVote {
bytes vote_extension = 1;
}
message ResponseVerifyVoteExtension {
bool accept = 1;
}
message ResponseFinalizeBlock {
repeated Event block_events = 1
[(gogoproto.nullable) = false, (gogoproto.jsontag) = "events,omitempty"];
@ -332,11 +354,16 @@ message ResponseFinalizeBlock {
//----------------------------------------
// Misc.
message LastCommitInfo {
message CommitInfo {
int32 round = 1;
repeated VoteInfo votes = 2 [(gogoproto.nullable) = false];
}
message ExtendedCommitInfo {
int32 round = 1;
repeated ExtendedVoteInfo votes = 2 [(gogoproto.nullable) = false];
}
// Event allows application developers to attach additional information to
// ResponseBeginBlock, ResponseEndBlock, ResponseCheckTx and ResponseDeliverTx.
// Later, transactions may be queried using these events.
@ -410,8 +437,23 @@ message ValidatorUpdate {
message VoteInfo {
Validator validator = 1 [(gogoproto.nullable) = false];
bool signed_last_block = 2;
reserved 3; // Placeholder for tendermint_signed_extension in v0.37
reserved 4; // Placeholder for app_signed_extension in v0.37
}
// ExtendedVoteInfo
message ExtendedVoteInfo {
Validator validator = 1 [(gogoproto.nullable) = false];
bool signed_last_block = 2;
bytes vote_extension = 3;
}
// CanonicalVoteExtension
// TODO: move this to core Tendermint data structures
message CanonicalVoteExtension {
bytes extension = 1;
int64 height = 2;
int32 round = 3;
string chain_id = 4;
bytes address = 5;
}
enum EvidenceType {
@ -462,5 +504,7 @@ service ABCIApplication {
rpc ApplySnapshotChunk(RequestApplySnapshotChunk) returns (ResponseApplySnapshotChunk);
rpc PrepareProposal(RequestPrepareProposal) returns (ResponsePrepareProposal);
rpc ProcessProposal(RequestProcessProposal) returns (ResponseProcessProposal);
rpc ExtendVote(RequestExtendVote) returns (ResponseExtendVote);
rpc VerifyVoteExtension(RequestVerifyVoteExtension) returns (ResponseVerifyVoteExtension);
rpc FinalizeBlock(RequestFinalizeBlock) returns (ResponseFinalizeBlock);
}

+ 476
- 0
proto/tendermint/abci/types.proto.intermediate View File

@ -0,0 +1,476 @@
syntax = "proto3";
package tendermint.abci;
import "tendermint/crypto/proof.proto";
import "tendermint/types/types.proto";
import "tendermint/crypto/keys.proto";
import "tendermint/types/params.proto";
import "google/protobuf/timestamp.proto";
import "gogoproto/gogo.proto";
// This file is a temporary workaround to enable development during the ABCI++
// project. This file should be deleted and any references to it removed when
// the ongoing work on ABCI++ is completed.
//
// For the duration of ABCI++, this file should be able to build the `abci/types/types.pb.go`
// file. Any changes that update that file must come as a result of a change in
// this .proto file.
// For more information, see https://github.com/tendermint/tendermint/issues/8066
//----------------------------------------
// Request types
message Request {
oneof value {
RequestEcho echo = 1;
RequestFlush flush = 2;
RequestInfo info = 3;
RequestInitChain init_chain = 4;
RequestQuery query = 5;
RequestBeginBlock begin_block = 6 [deprecated = true];
RequestCheckTx check_tx = 7;
RequestDeliverTx deliver_tx = 8 [deprecated = true];
RequestEndBlock end_block = 9 [deprecated = true];
RequestCommit commit = 10;
RequestListSnapshots list_snapshots = 11;
RequestOfferSnapshot offer_snapshot = 12;
RequestLoadSnapshotChunk load_snapshot_chunk = 13;
RequestApplySnapshotChunk apply_snapshot_chunk = 14;
RequestPrepareProposal prepare_proposal = 15;
RequestProcessProposal process_proposal = 16;
RequestExtendVote extend_vote = 17;
RequestVerifyVoteExtension verify_vote_extension = 18;
RequestFinalizeBlock finalize_block = 19;
}
}
message RequestEcho {
string message = 1;
}
message RequestFlush {}
message RequestInfo {
string version = 1;
uint64 block_version = 2;
uint64 p2p_version = 3;
string abci_version = 4;
}
message RequestInitChain {
google.protobuf.Timestamp time = 1 [(gogoproto.nullable) = false, (gogoproto.stdtime) = true];
string chain_id = 2;
tendermint.types.ConsensusParams consensus_params = 3;
repeated ValidatorUpdate validators = 4 [(gogoproto.nullable) = false];
bytes app_state_bytes = 5;
int64 initial_height = 6;
}
message RequestQuery {
bytes data = 1;
string path = 2;
int64 height = 3;
bool prove = 4;
}
message RequestBeginBlock {
bytes hash = 1;
tendermint.types.Header header = 2 [(gogoproto.nullable) = false];
LastCommitInfo last_commit_info = 3 [(gogoproto.nullable) = false];
repeated Evidence byzantine_validators = 4 [(gogoproto.nullable) = false];
}
enum CheckTxType {
NEW = 0 [(gogoproto.enumvalue_customname) = "New"];
RECHECK = 1 [(gogoproto.enumvalue_customname) = "Recheck"];
}
message RequestCheckTx {
bytes tx = 1;
CheckTxType type = 2;
}
message RequestDeliverTx {
bytes tx = 1;
}
message RequestEndBlock {
int64 height = 1;
}
message RequestCommit {}
// lists available snapshots
message RequestListSnapshots {}
// offers a snapshot to the application
message RequestOfferSnapshot {
Snapshot snapshot = 1; // snapshot offered by peers
bytes app_hash = 2; // light client-verified app hash for snapshot height
}
// loads a snapshot chunk
message RequestLoadSnapshotChunk {
uint64 height = 1;
uint32 format = 2;
uint32 chunk = 3;
}
// Applies a snapshot chunk
message RequestApplySnapshotChunk {
uint32 index = 1;
bytes chunk = 2;
string sender = 3;
}
// Extends a vote with application-side injection
message RequestExtendVote {
types.Vote vote = 1;
}
// Verify the vote extension
message RequestVerifyVoteExtension {
types.Vote vote = 1;
}
message RequestPrepareProposal {
// block_data is an array of transactions that will be included in a block,
// sent to the app for possible modifications.
// applications can not exceed the size of the data passed to it.
repeated bytes block_data = 1;
// If an application decides to populate block_data with extra information, they can not exceed this value.
int64 block_data_size = 2;
// votes includes all votes from the previous block. This contains vote extension data that can be used in proposal
// preparation. The votes here will then form the last commit that gets sent in the proposed block.
repeated tendermint.types.Vote votes = 3;
}
message RequestProcessProposal {
bytes hash = 1;
tendermint.types.Header header = 2 [(gogoproto.nullable) = false];
repeated bytes txs = 3;
LastCommitInfo last_commit_info = 4 [(gogoproto.nullable) = false];
repeated Evidence byzantine_validators = 5 [(gogoproto.nullable) = false];
}
message RequestFinalizeBlock {
repeated bytes txs = 1;
bytes hash = 2;
int64 height = 3;
tendermint.types.Header header = 4 [(gogoproto.nullable) = false];
LastCommitInfo last_commit_info = 5 [(gogoproto.nullable) = false];
repeated Evidence byzantine_validators = 6 [(gogoproto.nullable) = false];
}
//----------------------------------------
// Response types
message Response {
oneof value {
ResponseException exception = 1;
ResponseEcho echo = 2;
ResponseFlush flush = 3;
ResponseInfo info = 4;
ResponseInitChain init_chain = 5;
ResponseQuery query = 6;
ResponseBeginBlock begin_block = 7 [deprecated = true];
ResponseCheckTx check_tx = 8;
ResponseDeliverTx deliver_tx = 9 [deprecated = true];
ResponseEndBlock end_block = 10 [deprecated = true];
ResponseCommit commit = 11;
ResponseListSnapshots list_snapshots = 12;
ResponseOfferSnapshot offer_snapshot = 13;
ResponseLoadSnapshotChunk load_snapshot_chunk = 14;
ResponseApplySnapshotChunk apply_snapshot_chunk = 15;
ResponsePrepareProposal prepare_proposal = 16;
ResponseProcessProposal process_proposal = 17;
ResponseExtendVote extend_vote = 18;
ResponseVerifyVoteExtension verify_vote_extension = 19;
ResponseFinalizeBlock finalize_block = 20;
}
}
// nondeterministic
message ResponseException {
string error = 1;
}
message ResponseEcho {
string message = 1;
}
message ResponseFlush {}
message ResponseInfo {
string data = 1;
// this is the software version of the application. TODO: remove?
string version = 2;
uint64 app_version = 3;
int64 last_block_height = 4;
bytes last_block_app_hash = 5;
}
message ResponseInitChain {
tendermint.types.ConsensusParams consensus_params = 1;
repeated ValidatorUpdate validators = 2 [(gogoproto.nullable) = false];
bytes app_hash = 3;
}
message ResponseQuery {
uint32 code = 1;
// bytes data = 2; // use "value" instead.
string log = 3; // nondeterministic
string info = 4; // nondeterministic
int64 index = 5;
bytes key = 6;
bytes value = 7;
tendermint.crypto.ProofOps proof_ops = 8;
int64 height = 9;
string codespace = 10;
}
message ResponseBeginBlock {
repeated Event events = 1 [(gogoproto.nullable) = false, (gogoproto.jsontag) = "events,omitempty"];
}
message ResponseCheckTx {
uint32 code = 1;
bytes data = 2;
string log = 3; // nondeterministic
string info = 4; // nondeterministic
int64 gas_wanted = 5;
int64 gas_used = 6;
repeated Event events = 7 [(gogoproto.nullable) = false, (gogoproto.jsontag) = "events,omitempty"];
string codespace = 8;
string sender = 9;
int64 priority = 10;
// mempool_error is set by Tendermint.
// ABCI applications creating a ResponseCheckTX should not set mempool_error.
string mempool_error = 11;
}
message ResponseDeliverTx {
uint32 code = 1;
bytes data = 2;
string log = 3; // nondeterministic
string info = 4; // nondeterministic
int64 gas_wanted = 5 [json_name = "gas_wanted"];
int64 gas_used = 6 [json_name = "gas_used"];
repeated Event events = 7
[(gogoproto.nullable) = false, (gogoproto.jsontag) = "events,omitempty"]; // nondeterministic
string codespace = 8;
}
message ResponseEndBlock {
repeated ValidatorUpdate validator_updates = 1 [(gogoproto.nullable) = false];
tendermint.types.ConsensusParams consensus_param_updates = 2;
repeated Event events = 3 [(gogoproto.nullable) = false, (gogoproto.jsontag) = "events,omitempty"];
}
message ResponseCommit {
// reserve 1
bytes data = 2;
int64 retain_height = 3;
}
message ResponseListSnapshots {
repeated Snapshot snapshots = 1;
}
message ResponseOfferSnapshot {
Result result = 1;
enum Result {
UNKNOWN = 0; // Unknown result, abort all snapshot restoration
ACCEPT = 1; // Snapshot accepted, apply chunks
ABORT = 2; // Abort all snapshot restoration
REJECT = 3; // Reject this specific snapshot, try others
REJECT_FORMAT = 4; // Reject all snapshots of this format, try others
REJECT_SENDER = 5; // Reject all snapshots from the sender(s), try others
}
}
message ResponseLoadSnapshotChunk {
bytes chunk = 1;
}
message ResponseApplySnapshotChunk {
Result result = 1;
repeated uint32 refetch_chunks = 2; // Chunks to refetch and reapply
repeated string reject_senders = 3; // Chunk senders to reject and ban
enum Result {
UNKNOWN = 0; // Unknown result, abort all snapshot restoration
ACCEPT = 1; // Chunk successfully accepted
ABORT = 2; // Abort all snapshot restoration
RETRY = 3; // Retry chunk (combine with refetch and reject)
RETRY_SNAPSHOT = 4; // Retry snapshot (combine with refetch and reject)
REJECT_SNAPSHOT = 5; // Reject this snapshot, try others
}
}
message ResponseExtendVote {
tendermint.types.VoteExtension vote_extension = 1;
}
message ResponseVerifyVoteExtension {
Result result = 1;
enum Result {
UNKNOWN = 0; // Unknown result, reject vote extension
ACCEPT = 1; // Vote extension verified, include the vote
SLASH = 2; // Vote extension verification aborted, continue but slash validator
REJECT = 3; // Vote extension invalidated
}
}
message ResponsePrepareProposal {
repeated bytes block_data = 1;
}
message ResponseProcessProposal {
bool accept = 1;
bytes app_hash = 2;
repeated ExecTxResult tx_results = 3;
repeated ValidatorUpdate validator_updates = 4;
tendermint.types.ConsensusParams consensus_param_updates = 5;
}
message ResponseFinalizeBlock {
repeated ResponseDeliverTx txs = 1;
repeated ValidatorUpdate validator_updates = 2 [(gogoproto.nullable) = false];
tendermint.types.ConsensusParams consensus_param_updates = 3;
repeated Event events = 4
[(gogoproto.nullable) = false, (gogoproto.jsontag) = "events,omitempty"];
}
//----------------------------------------
// Misc.
message LastCommitInfo {
int32 round = 1;
repeated VoteInfo votes = 2 [(gogoproto.nullable) = false];
}
// Event allows application developers to attach additional information to
// ResponseBeginBlock, ResponseEndBlock, ResponseCheckTx and ResponseDeliverTx.
// Later, transactions may be queried using these events.
message Event {
string type = 1;
repeated EventAttribute attributes = 2 [(gogoproto.nullable) = false, (gogoproto.jsontag) = "attributes,omitempty"];
}
// EventAttribute is a single key-value pair, associated with an event.
message EventAttribute {
string key = 1;
string value = 2;
bool index = 3; // nondeterministic
}
// ExecTxResult contains results of executing one individual transaction.
//
// * Its structure is equivalent to #ResponseDeliverTx which will be deprecated/deleted
message ExecTxResult {
uint32 code = 1;
bytes data = 2;
string log = 3; // nondeterministic
string info = 4; // nondeterministic
int64 gas_wanted = 5;
int64 gas_used = 6;
repeated Event tx_events = 7
[(gogoproto.nullable) = false, (gogoproto.jsontag) = "events,omitempty"]; // nondeterministic
string codespace = 8;
}
// TxResult contains results of executing the transaction.
//
// One usage is indexing transaction results.
message TxResult {
int64 height = 1;
uint32 index = 2;
bytes tx = 3;
ResponseDeliverTx result = 4 [(gogoproto.nullable) = false];
}
//----------------------------------------
// Blockchain Types
// Validator
message Validator {
bytes address = 1; // The first 20 bytes of SHA256(public key)
// PubKey pub_key = 2 [(gogoproto.nullable)=false];
int64 power = 3; // The voting power
}
// ValidatorUpdate
message ValidatorUpdate {
tendermint.crypto.PublicKey pub_key = 1 [(gogoproto.nullable) = false];
int64 power = 2;
}
// VoteInfo
message VoteInfo {
Validator validator = 1 [(gogoproto.nullable) = false];
bool signed_last_block = 2;
reserved 3; // Placeholder for tendermint_signed_extension in v0.37
reserved 4; // Placeholder for app_signed_extension in v0.37
}
enum EvidenceType {
UNKNOWN = 0;
DUPLICATE_VOTE = 1;
LIGHT_CLIENT_ATTACK = 2;
}
message Evidence {
EvidenceType type = 1;
// The offending validator
Validator validator = 2 [(gogoproto.nullable) = false];
// The height when the offense occurred
int64 height = 3;
// The corresponding time where the offense occurred
google.protobuf.Timestamp time = 4 [(gogoproto.nullable) = false, (gogoproto.stdtime) = true];
// Total voting power of the validator set in case the ABCI application does
// not store historical validators.
// https://github.com/tendermint/tendermint/issues/4581
int64 total_voting_power = 5;
}
//----------------------------------------
// State Sync Types
message Snapshot {
uint64 height = 1; // The height at which the snapshot was taken
uint32 format = 2; // The application-specific snapshot format
uint32 chunks = 3; // Number of chunks in the snapshot
bytes hash = 4; // Arbitrary snapshot hash, equal only if identical
bytes metadata = 5; // Arbitrary application metadata
}
//----------------------------------------
// Service Definition
service ABCIApplication {
rpc Echo(RequestEcho) returns (ResponseEcho);
rpc Flush(RequestFlush) returns (ResponseFlush);
rpc Info(RequestInfo) returns (ResponseInfo);
rpc CheckTx(RequestCheckTx) returns (ResponseCheckTx);
rpc Query(RequestQuery) returns (ResponseQuery);
rpc Commit(RequestCommit) returns (ResponseCommit);
rpc InitChain(RequestInitChain) returns (ResponseInitChain);
rpc ListSnapshots(RequestListSnapshots) returns (ResponseListSnapshots);
rpc OfferSnapshot(RequestOfferSnapshot) returns (ResponseOfferSnapshot);
rpc LoadSnapshotChunk(RequestLoadSnapshotChunk) returns (ResponseLoadSnapshotChunk);
rpc ApplySnapshotChunk(RequestApplySnapshotChunk) returns (ResponseApplySnapshotChunk);
rpc PrepareProposal(RequestPrepareProposal) returns (ResponsePrepareProposal);
rpc ProcessProposal(RequestProcessProposal) returns (ResponseProcessProposal);
rpc ExtendVote(RequestExtendVote) returns (ResponseExtendVote);
rpc VerifyVoteExtension(RequestVerifyVoteExtension) returns (ResponseVerifyVoteExtension);
rpc FinalizeBlock(RequestFinalizeBlock) returns (ResponseFinalizeBlock);
}

+ 2
- 0
proto/tendermint/types/types.proto View File

@ -1,6 +1,8 @@
syntax = "proto3";
package tendermint.types;
option go_package = "github.com/tendermint/tendermint/types";
import "gogoproto/gogo.proto";
import "google/protobuf/timestamp.proto";
import "tendermint/crypto/proof.proto";


+ 192
- 0
proto/tendermint/types/types.proto.intermediate View File

@ -0,0 +1,192 @@
syntax = "proto3";
package tendermint.types;
import "gogoproto/gogo.proto";
import "google/protobuf/timestamp.proto";
import "tendermint/crypto/proof.proto";
import "tendermint/version/types.proto";
import "tendermint/types/validator.proto";
// This file is a temporary workaround to enable development during the ABCI++
// project. This file should be deleted and any references to it removed when
// the ongoing work on ABCI++ is completed.
//
// This file supports building of the `tendermint.abci` proto package.
// For more information, see https://github.com/tendermint/tendermint/issues/8066
// BlockIdFlag indicates which BlockID the signature is for
enum BlockIDFlag {
option (gogoproto.goproto_enum_stringer) = true;
option (gogoproto.goproto_enum_prefix) = false;
BLOCK_ID_FLAG_UNKNOWN = 0
[(gogoproto.enumvalue_customname) = "BlockIDFlagUnknown"];
BLOCK_ID_FLAG_ABSENT = 1
[(gogoproto.enumvalue_customname) = "BlockIDFlagAbsent"];
BLOCK_ID_FLAG_COMMIT = 2
[(gogoproto.enumvalue_customname) = "BlockIDFlagCommit"];
BLOCK_ID_FLAG_NIL = 3 [(gogoproto.enumvalue_customname) = "BlockIDFlagNil"];
}
// SignedMsgType is a type of signed message in the consensus.
enum SignedMsgType {
option (gogoproto.goproto_enum_stringer) = true;
option (gogoproto.goproto_enum_prefix) = false;
SIGNED_MSG_TYPE_UNKNOWN = 0
[(gogoproto.enumvalue_customname) = "UnknownType"];
// Votes
SIGNED_MSG_TYPE_PREVOTE = 1
[(gogoproto.enumvalue_customname) = "PrevoteType"];
SIGNED_MSG_TYPE_PRECOMMIT = 2
[(gogoproto.enumvalue_customname) = "PrecommitType"];
// Proposals
SIGNED_MSG_TYPE_PROPOSAL = 32
[(gogoproto.enumvalue_customname) = "ProposalType"];
}
// PartsetHeader
message PartSetHeader {
uint32 total = 1;
bytes hash = 2;
}
message Part {
uint32 index = 1;
bytes bytes = 2;
tendermint.crypto.Proof proof = 3 [(gogoproto.nullable) = false];
}
// BlockID
message BlockID {
bytes hash = 1;
PartSetHeader part_set_header = 2 [(gogoproto.nullable) = false];
}
// --------------------------------
// Header defines the structure of a Tendermint block header.
message Header {
// basic block info
tendermint.version.Consensus version = 1 [(gogoproto.nullable) = false];
string chain_id = 2 [(gogoproto.customname) = "ChainID"];
int64 height = 3;
google.protobuf.Timestamp time = 4
[(gogoproto.nullable) = false, (gogoproto.stdtime) = true];
// prev block info
BlockID last_block_id = 5 [(gogoproto.nullable) = false];
// hashes of block data
bytes last_commit_hash = 6; // commit from validators from the last block
bytes data_hash = 7; // transactions
// hashes from the app output from the prev block
bytes validators_hash = 8; // validators for the current block
bytes next_validators_hash = 9; // validators for the next block
bytes consensus_hash = 10; // consensus params for current block
bytes app_hash = 11; // state after txs from the previous block
bytes last_results_hash =
12; // root hash of all results from the txs from the previous block
// consensus info
bytes evidence_hash = 13; // evidence included in the block
bytes proposer_address = 14; // original proposer of the block
}
// Data contains the set of transactions included in the block
message Data {
// Txs that will be applied by state @ block.Height+1.
// NOTE: not all txs here are valid. We're just agreeing on the order first.
// This means that block.AppHash does not include these txs.
repeated bytes txs = 1;
}
// Vote represents a prevote, precommit, or commit vote from validators for
// consensus.
message Vote {
SignedMsgType type = 1;
int64 height = 2;
int32 round = 3;
BlockID block_id = 4 [
(gogoproto.nullable) = false,
(gogoproto.customname) = "BlockID"
]; // zero if vote is nil.
google.protobuf.Timestamp timestamp = 5
[(gogoproto.nullable) = false, (gogoproto.stdtime) = true];
bytes validator_address = 6;
int32 validator_index = 7;
bytes signature = 8;
VoteExtension vote_extension = 9;
}
// VoteExtension is app-defined additional information to the validator votes.
message VoteExtension {
bytes app_data_to_sign = 1;
bytes app_data_self_authenticating = 2;
}
// VoteExtensionToSign is a subset of VoteExtension that is signed by the validators private key.
// VoteExtensionToSign is extracted from an existing VoteExtension.
message VoteExtensionToSign {
bytes app_data_to_sign = 1;
}
// Commit contains the evidence that a block was committed by a set of
// validators.
message Commit {
int64 height = 1;
int32 round = 2;
BlockID block_id = 3
[(gogoproto.nullable) = false, (gogoproto.customname) = "BlockID"];
repeated CommitSig signatures = 4 [(gogoproto.nullable) = false];
}
// CommitSig is a part of the Vote included in a Commit.
message CommitSig {
BlockIDFlag block_id_flag = 1;
bytes validator_address = 2;
google.protobuf.Timestamp timestamp = 3
[(gogoproto.nullable) = false, (gogoproto.stdtime) = true];
bytes signature = 4;
VoteExtensionToSign vote_extension = 5;
}
message Proposal {
SignedMsgType type = 1;
int64 height = 2;
int32 round = 3;
int32 pol_round = 4;
BlockID block_id = 5
[(gogoproto.customname) = "BlockID", (gogoproto.nullable) = false];
google.protobuf.Timestamp timestamp = 6
[(gogoproto.nullable) = false, (gogoproto.stdtime) = true];
bytes signature = 7;
}
message SignedHeader {
Header header = 1;
Commit commit = 2;
}
message LightBlock {
SignedHeader signed_header = 1;
tendermint.types.ValidatorSet validator_set = 2;
}
message BlockMeta {
BlockID block_id = 1
[(gogoproto.customname) = "BlockID", (gogoproto.nullable) = false];
int64 block_size = 2;
Header header = 3 [(gogoproto.nullable) = false];
int64 num_txs = 4;
}
// TxProof represents a Merkle proof of the presence of a transaction in the
// Merkle tree.
message TxProof {
bytes root_hash = 1;
bytes data = 2;
tendermint.crypto.Proof proof = 3;
}

+ 49
- 0
rpc/client/eventstream/eventstream_test.go View File

@ -90,6 +90,55 @@ func TestStream_lostItem(t *testing.T) {
s.stopWait()
}
func TestMinPollTime(t *testing.T) {
defer leaktest.Check(t)
s := newStreamTester(t, ``, eventlog.LogSettings{
WindowSize: 30 * time.Second,
}, nil)
s.publish("bad", "whatever")
// Waiting for an item on a log with no matching events incurs a minimum
// wait time and reports no events.
ctx := context.Background()
filter := &coretypes.EventFilter{Query: `tm.event = 'good'`}
var zero cursor.Cursor
t.Run("NoneMatch", func(t *testing.T) {
start := time.Now()
// Request a very short delay, and affirm we got the server's minimum.
rsp, err := s.env.Events(ctx, filter, 1, zero, zero, 10*time.Millisecond)
if err != nil {
t.Fatalf("Events failed: %v", err)
} else if elapsed := time.Since(start); elapsed < time.Second {
t.Errorf("Events returned too quickly: got %v, wanted 1s", elapsed)
} else if len(rsp.Items) != 0 {
t.Errorf("Events returned %d items, expected none", len(rsp.Items))
}
})
s.publish("good", "whatever")
// Waiting for an available matching item incurs no delay.
t.Run("SomeMatch", func(t *testing.T) {
start := time.Now()
// Request a long-ish delay and affirm we don't block for it.
// Check for this by ensuring we return sooner than the minimum delay,
// since we don't know the exact timing.
rsp, err := s.env.Events(ctx, filter, 1, zero, zero, 10*time.Second)
if err != nil {
t.Fatalf("Events failed: %v", err)
} else if elapsed := time.Since(start); elapsed > 500*time.Millisecond {
t.Errorf("Events returned too slowly: got %v, wanted immediate", elapsed)
} else if len(rsp.Items) == 0 {
t.Error("Events returned no items, wanted at least 1")
}
})
}
// testItem is a wrapper for comparing item results in a friendly output format
// for the cmp package.
type testItem struct {


+ 31
- 0
scripts/abci-gen.sh View File

@ -0,0 +1,31 @@
#!/usr/bin/env bash
# This file was added during development of ABCI++. This file is a script to allow
# the intermediate proto files to be built while active development proceeds
# on ABCI++.
# This file should be removed when work on ABCI++ is complete.
# For more information, see https://github.com/tendermint/tendermint/issues/8066.
set -euo pipefail
cp ./proto/tendermint/abci/types.proto.intermediate ./proto/tendermint/abci/types.proto
cp ./proto/tendermint/types/types.proto.intermediate ./proto/tendermint/types/types.proto
MODNAME="$(go list -m)"
find ./proto/tendermint -name '*.proto' -not -path "./proto/tendermint/abci/types.proto" \
-exec sh ./scripts/protopackage.sh {} "$MODNAME" ';'
sh ./scripts/protopackage.sh ./proto/tendermint/abci/types.proto $MODNAME "abci/types"
make proto-gen
mv ./proto/tendermint/abci/types.pb.go ./abci/types
echo "proto files have been compiled"
echo "checking out copied files"
find proto/tendermint/ -name '*.proto' -not -path "*.intermediate"\
| xargs -I {} git checkout {}
find proto/tendermint/ -name '*.pb.go' \
| xargs -I {} git checkout {}

+ 1
- 2
spec/abci++/README.md View File

@ -20,8 +20,7 @@ for handling all ABCI++ methods.
Thus, Tendermint always sends the `Request*` messages and receives the `Response*` messages
in return.
All ABCI++ messages and methods are defined in
[protocol buffers](https://github.com/tendermint/tendermint/blob/master/proto/spec/abci/types.proto).
All ABCI++ messages and methods are defined in [protocol buffers](../../proto/tendermint/abci/types.proto).
This allows Tendermint to run with applications written in many programming languages.
This specification is split as follows:


+ 3
- 2
spec/abci++/abci++_basic_concepts_002_draft.md View File

@ -206,8 +206,9 @@ the local process is the proposer of the round.
When Tendermint's consensus is about to send a non-`nil` precommit message, it calls
method `ExtendVote`, which gives the Application the opportunity to include
non-deterministic data, opaque to Tendermint, that will be attached to the precommit
message. The data, called _vote extension_, will also be part of the proposed block
in the next height, along with the vote it is extending.
message. The data, called _vote extension_, will also be made available to the
application in the next height, along with the vote it is extending, in the rounds
where the local process is the proposer.
The vote extension data is split into two parts, one signed by Tendermint as part
of the vote data structure, and the other (optionally) signed by the Application.


+ 98
- 61
spec/abci++/abci++_methods_002_draft.md View File

@ -290,15 +290,10 @@ title: Methods
| hash | bytes | The block header's hash of the block to propose. Present for convenience (can be derived from the block header). | 1 |
| header | [Header](../core/data_structures.md#header) | The header of the block to propose. | 2 |
| txs | repeated bytes | Preliminary list of transactions that have been picked as part of the block to propose. | 3 |
| last_commit_info | [LastCommitInfo](#lastcommitinfo) | Info about the last commit, including the round, the validator list, and which ones signed the last block. | 4 |
| local_last_commit | [ExtendedCommitInfo](#extendedcommitinfo) | Info about the last commit, obtained locally from Tendermint's data structures. | 4 |
| byzantine_validators | repeated [Evidence](#evidence) | List of evidence of validators that acted maliciously. | 5 |
| max_tx_bytes | int64 | Currently configured maximum size in bytes taken by the modified transactions. | 6 |
>**TODO**: Add the changes needed in LastCommitInfo for vote extensions
>**TODO**: DISCUSS: We need to make clear whether a proposer is also running the logic of a non-proposer node (in particular "ProcessProposal")
From the App's perspective, they'll probably skip ProcessProposal
* **Response**:
| Name | Type | Description | Field Number |
@ -340,7 +335,7 @@ From the App's perspective, they'll probably skip ProcessProposal
for blocks `H+1`, and `H+2`. Heights following a validator update are affected in the following way:
* `H`: `NextValidatorsHash` includes the new `validator_updates` value.
* `H+1`: The validator set change takes effect and `ValidatorsHash` is updated.
* `H+2`: `last_commit_info` is changed to include the altered validator set.
* `H+2`: `local_last_commit` now includes the altered validator set.
* `ResponseFinalizeBlock.consensus_param_updates` returned for block `H` apply to the consensus
params for block `H+1` even if the change is agreed in block `H`.
For more information on the consensus parameters,
@ -414,7 +409,7 @@ Note that, if _p_ has a non-`nil` _validValue_, Tendermint will use it as propos
| hash | bytes | The block header's hash of the proposed block. Present for convenience (can be derived from the block header). | 1 |
| header | [Header](../core/data_structures.md#header) | The proposed block's header. | 2 |
| txs | repeated bytes | List of transactions that have been picked as part of the proposed block. | 3 |
| last_commit_info | [LastCommitInfo](#lastcommitinfo) | Info about the last commit, including the round , the validator list, and which ones signed the last block. | 4 |
| proposed_last_commit | [CommitInfo](#commitinfo) | Info about the last commit, obtained from the information in the proposed block. | 4 |
| byzantine_validators | repeated [Evidence](#evidence) | List of evidence of validators that acted maliciously. | 5 |
* **Response**:
@ -495,18 +490,17 @@ When a validator _p_ enters Tendermint consensus round _r_, height _h_, in which
* **Response**:
| Name | Type | Description | Field Number |
|-------------------|-------|---------------------------------------------------------------------|--------------|
| app_signed | bytes | Optional information signed by the Application (not by Tendermint). | 1 |
| tendermint_signed | bytes | Optional information signed by Tendermint. | 2 |
| Name | Type | Description | Field Number |
|-------------------|-------|-----------------------------------------------|--------------|
| vote_extension | bytes | Optional information signed by by Tendermint. | 1 |
* **Usage**:
* Both `ResponseExtendVote.app_signed` and `ResponseExtendVote.tendermint_signed` are optional information that will
be attached to the Precommit message.
* `ResponseExtendVote.vote_extension` is optional information that, if present, will be signed by Tendermint and
attached to the Precommit message.
* `RequestExtendVote.hash` corresponds to the hash of a proposed block that was made available to the application
in a previous call to `ProcessProposal` or `PrepareProposal` for the current height.
* `ResponseExtendVote.app_signed` and `ResponseExtendVote.tendermint_signed` will always be attached to a non-`nil`
Precommit message. If Tendermint is to precommit `nil`, it will not call `RequestExtendVote`.
* `ResponseExtendVote.vote_extension` will only be attached to a non-`nil` Precommit message. If Tendermint is to
precommit `nil`, it will not call `RequestExtendVote`.
* The Application logic that creates the extension can be non-deterministic.
#### When does Tendermint call it?
@ -520,11 +514,18 @@ then _p_'s Tendermint locks _v_ and sends a Precommit message in the following
1. _p_'s Tendermint sets _lockedValue_ and _validValue_ to _v_, and sets _lockedRound_ and _validRound_ to _r_
2. _p_'s Tendermint calls `RequestExtendVote` with _id(v)_ (`RequestExtendVote.hash`). The call is synchronous.
3. The Application returns an array of bytes, `ResponseExtendVote.extension`, which is not interpreted by Tendermint.
4. _p_'s Tendermint includes `ResponseExtendVote.extension` as a new field in the Precommit message.
5. _p_'s Tendermint signs and broadcasts the Precommit message.
In the cases when _p_'s Tendermint is to broadcast `precommit nil` messages (either _2f+1_ `prevote nil` messages received, or _timeoutPrevote_ triggered), _p_'s Tendermint does **not** call `RequestExtendVote` and will include an empty byte array as vote extension in the `precommit nil` message.
3. The Application optionally returns an array of bytes, `ResponseExtendVote.extension`, which is not interpreted by Tendermint.
4. _p_'s Tendermint includes `ResponseExtendVote.extension` in a field of type [CanonicalVoteExtension](#canonicalvoteextension),
it then populates the other fields in [CanonicalVoteExtension](#canonicalvoteextension), and signs the populated
data structure.
5. _p_'s Tendermint constructs and signs the [CanonicalVote](../core/data_structures.md#canonicalvote) structure.
6. _p_'s Tendermint constructs the Precommit message (i.e. [Vote](../core/data_structures.md#vote) structure)
using [CanonicalVoteExtension](#canonicalvoteextension) and [CanonicalVote](../core/data_structures.md#canonicalvote).
7. _p_'s Tendermint broadcasts the Precommit message.
In the cases when _p_'s Tendermint is to broadcast `precommit nil` messages (either _2f+1_ `prevote nil` messages received,
or _timeoutPrevote_ triggered), _p_'s Tendermint does **not** call `RequestExtendVote` and will not include
a [CanonicalVoteExtension](#canonicalvoteextension) field in the `precommit nil` message.
### VerifyVoteExtension
@ -534,11 +535,10 @@ In the cases when _p_'s Tendermint is to broadcast `precommit nil` messages (eit
| Name | Type | Description | Field Number |
|-------------------|-------|------------------------------------------------------------------------------------------|--------------|
| app_signed | bytes | Optional information signed by the Application (not by Tendermint). | 1 |
| tendermint_signed | bytes | Optional information signed by Tendermint. | 2 |
| hash | bytes | The header hash of the propsed block that the vote extension refers to. | 3 |
| validator_address | bytes | [Address](../core/data_structures.md#address) of the validator that signed the extension | 4 |
| height | int64 | Height of the block (for sanity check). | 5 |
| hash | bytes | The header hash of the propsed block that the vote extension refers to. | 1 |
| validator_address | bytes | [Address](../core/data_structures.md#address) of the validator that signed the extension | 2 |
| height | int64 | Height of the block (for sanity check). | 3 |
| vote_extension | bytes | Optional information signed by Tendermint. | 4 |
* **Response**:
@ -566,11 +566,8 @@ from this condition, but not sure), and _p_ receives a Precommit message for rou
2. The Application returns _accept_ or _reject_ via `ResponseVerifyVoteExtension.accept`.
3. If the Application returns
* _accept_, _p_'s Tendermint will keep the received vote, together with its corresponding
vote extension in its internal data structures. It will be used to:
* calculate field _LastCommitHash_ in the header of the block proposed for height _h + 1_
(in the rounds where _p_ will be proposer).
* populate _LastCommitInfo_ in calls to `RequestPrepareProposal`, `RequestProcessProposal`,
and `RequestFinalizeBlock` in height _h + 1_.
vote extension in its internal data structures. It will be used to populate the [ExtendedCommitInfo](#extendedcommitinfo)
structure in calls to `RequestPrepareProposal`, in rounds of height _h + 1_ where _p_ is the proposer.
* _reject_, _p_'s Tendermint will deem the Precommit message invalid and discard it.
### FinalizeBlock
@ -579,13 +576,13 @@ from this condition, but not sure), and _p_ receives a Precommit message for rou
* **Request**:
| Name | Type | Description | Field Number |
|----------------------|---------------------------------------------|-------------------------------------------------------------------------------------------------------------------|--------------|
| hash | bytes | The block header's hash. Present for convenience (can be derived from the block header). | 1 |
| header | [Header](../core/data_structures.md#header) | The block header. | 2 |
| txs | repeated bytes | List of transactions committed as part of the block. | 3 |
| last_commit_info | [LastCommitInfo](#lastcommitinfo) | Info about the last commit, including the round, and the list of validators and which ones signed the last block. | 4 |
| byzantine_validators | repeated [Evidence](#evidence) | List of evidence of validators that acted maliciously. | 5 |
| Name | Type | Description | Field Number |
|----------------------|---------------------------------------------|------------------------------------------------------------------------------------------|--------------|
| hash | bytes | The block header's hash. Present for convenience (can be derived from the block header). | 1 |
| header | [Header](../core/data_structures.md#header) | The block header. | 2 |
| txs | repeated bytes | List of transactions committed as part of the block. | 3 |
| decided_last_commit | [CommitInfo](#commitinfo) | Info about the last commit, obtained from the block that was just decided. | 4 |
| byzantine_validators | repeated [Evidence](#evidence) | List of evidence of validators that acted maliciously. | 5 |
* **Response**:
@ -603,7 +600,7 @@ from this condition, but not sure), and _p_ receives a Precommit message for rou
* This method is equivalent to the call sequence `BeginBlock`, [`DeliverTx`],
`EndBlock`, `Commit` in the previous version of ABCI.
* The header exactly matches the Tendermint header of the proposed block.
* The Application can use `RequestFinalizeBlock.last_commit_info` and `RequestFinalizeBlock.byzantine_validators`
* The Application can use `RequestFinalizeBlock.decided_last_commit` and `RequestFinalizeBlock.byzantine_validators`
to determine rewards and punishments for the validators.
* The application must execute the transactions in full, in the order they appear in `RequestFinalizeBlock.txs`,
before returning control to Tendermint. Alternatively, it can commit the candidate state corresponding to the same block
@ -619,7 +616,7 @@ from this condition, but not sure), and _p_ receives a Precommit message for rou
for blocks `H+1`, `H+2`, and `H+3`. Heights following a validator update are affected in the following way:
- Height `H+1`: `NextValidatorsHash` includes the new `validator_updates` value.
- Height `H+2`: The validator set change takes effect and `ValidatorsHash` is updated.
- Height `H+3`: `last_commit_info` is changed to include the altered validator set.
- Height `H+3`: `decided_last_commit` now includes the altered validator set.
* `ResponseFinalizeBlock.consensus_param_updates` returned for block `H` apply to the consensus
params for block `H+1`. For more information on the consensus parameters,
see the [application spec entry on consensus parameters](../abci/apps.md#consensus-parameters).
@ -728,25 +725,16 @@ Most of the data structures used in ABCI are shared [common data structures](../
| DUPLICATE_VOTE | 1 |
| LIGHT_CLIENT_ATTACK | 2 |
### LastCommitInfo
* **Fields**:
| Name | Type | Description | Field Number |
|-------|--------------------------------|-----------------------------------------------------------------------------------------------------------------------|--------------|
| round | int32 | Commit round. Reflects the total amount of rounds it took to come to consensus for the current block. | 1 |
| votes | repeated [VoteInfo](#voteinfo) | List of validators addresses in the last validator set with their voting power and whether or not they signed a vote. | 2 |
### ConsensusParams
* **Fields**:
| Name | Type | Description | Field Number |
|-----------|---------------------------------------------------------------|------------------------------------------------------------------------------|--------------|
| block | [BlockParams](../core/data_structures.md#blockparams) | Parameters limiting the size of a block and time between consecutive blocks. | 1 |
| block | [BlockParams](../core/data_structures.md#blockparams) | Parameters limiting the size of a block and time between consecutive blocks. | 1 |
| evidence | [EvidenceParams](../core/data_structures.md#evidenceparams) | Parameters limiting the validity of evidence of byzantine behaviour. | 2 |
| validator | [ValidatorParams](../core/data_structures.md#validatorparams) | Parameters limiting the types of public keys validators can use. | 3 |
| version | [VersionsParams](../core/data_structures.md#versionparams) | The ABCI application version. | 4 |
| version | [VersionsParams](../core/data_structures.md#versionparams) | The ABCI application version. | 4 |
### ProofOps
@ -790,18 +778,47 @@ Most of the data structures used in ABCI are shared [common data structures](../
* **Fields**:
| Name | Type | Description | Field Number |
|-----------------------------|-------------------------|---------------------------------------------------------------|--------------|
| validator | [Validator](#validator) | A validator | 1 |
| signed_last_block | bool | Indicates whether or not the validator signed the last block | 2 |
| tendermint_signed_extension | bytes | Indicates whether or not the validator signed the last block | 3 |
| app_signed_extension | bytes | Indicates whether or not the validator signed the last block | 3 |
| Name | Type | Description | Field Number |
|-----------------------------|-------------------------|----------------------------------------------------------------|--------------|
| validator | [Validator](#validator) | The validator that sent the vote. | 1 |
| signed_last_block | bool | Indicates whether or not the validator signed the last block. | 2 |
* **Usage**:
* Indicates whether a validator signed the last block, allowing for rewards
based on validator availability
* `tendermint_signed_extension` conveys the part of the validator's vote extension that was signed by Tendermint.
* `app_signed_extension` conveys the optional *app_signed* part of the validator's vote extension.
* Indicates whether a validator signed the last block, allowing for rewards based on validator availability.
* This information is typically extracted from a proposed or decided block.
### ExtendedVoteInfo
* **Fields**:
| Name | Type | Description | Field Number |
|-------------------|-------------------------|------------------------------------------------------------------------------|--------------|
| validator | [Validator](#validator) | The validator that sent the vote. | 1 |
| signed_last_block | bool | Indicates whether or not the validator signed the last block. | 2 |
| vote_extension | bytes | Non-deterministic extension provided by the sending validator's Application. | 3 |
* **Usage**:
* Indicates whether a validator signed the last block, allowing for rewards based on validator availability.
* This information is extracted from Tendermint's data structures in the local process.
* `vote_extension` contains the sending validator's vote extension, which is signed by Tendermint. It can be empty
### CommitInfo
* **Fields**:
| Name | Type | Description | Field Number |
|-------|--------------------------------|----------------------------------------------------------------------------------------------|--------------|
| round | int32 | Commit round. Reflects the round at which the block proposer decided in the previous height. | 1 |
| votes | repeated [VoteInfo](#voteinfo) | List of validators' addresses in the last validator set with their voting information. | 2 |
### ExtendedCommitInfo
* **Fields**:
| Name | Type | Description | Field Number |
|-------|------------------------------------------------|-------------------------------------------------------------------------------------------------------------------|--------------|
| round | int32 | Commit round. Reflects the round at which the block proposer decided in the previous height. | 1 |
| votes | repeated [ExtendedVoteInfo](#extendedvoteinfo) | List of validators' addresses in the last validator set with their voting information, including vote extensions. | 2 |
### ExecTxResult
@ -843,3 +860,23 @@ Most of the data structures used in ABCI are shared [common data structures](../
|------------|-----------------------|------------------------------------------------------------------|--------------|
| action | [TxAction](#txaction) | What should Tendermint do with this transaction? | 1 |
| tx | bytes | Transaction contents | 2 |
### CanonicalVoteExtension
>**TODO**: This protobuf message definition is not part of the ABCI++ interface, but rather belongs to the
> Precommit message which is broadcast via P2P. So it is to be moved to the relevant section of the spec.
* **Fields**:
| Name | Type | Description | Field Number |
|-----------|--------|--------------------------------------------------------------------------------------------|--------------|
| extension | bytes | Vote extension provided by the Application. | 1 |
| height | int64 | Height in which the extension was provided. | 2 |
| round | int32 | Round in which the extension was provided. | 3 |
| chain_id | string | ID of the blockchain running consensus. | 4 |
| address | bytes | [Address](../core/data_structures.md#address) of the validator that provided the extension | 5 |
* **Usage**:
* Tendermint is to sign the whole data structure and attach it to a Precommit message
* Upon reception, Tendermint validates the sender's signature and sanity-checks the values of `height`, `round`, and `chain_id`.
Then it sends `extension` to the Application via `RequestVerifyVoteExtension` for verification.

+ 2
- 1
spec/abci++/abci++_tmint_expected_behavior_002_draft.md View File

@ -10,7 +10,8 @@ title: Tendermint's expected behavior
This section describes what the Application can expect from Tendermint.
The Tendermint consensus algorithm is designed to protect safety under any network conditions, as long as
less than 1/3 of validators' voting power is byzantine. Most of the time, though, the network will behave synchronously and there will be no byzantine process. In these frequent, benign conditions:
less than 1/3 of validators' voting power is byzantine. Most of the time, though, the network will behave
synchronously and there will be no byzantine process. In these frequent, benign conditions:
* Tendermint will decide in round 0;
* `PrepareProposal` will be called exactly once at the proposer process of round 0, height _h_;


+ 13
- 0
spec/abci/apps.md View File

@ -346,6 +346,19 @@ a block minus it's overhead ( ~ `MaxBytes`).
Must have `MaxNum > 0`.
### SynchronyParams.Precision
`SynchronyParams.Precision` is a parameter of the Proposer-Based Timestamps algorithm.
that configures the acceptable upper-bound of clock drift among
all of the nodes on a Tendermint network. Any two nodes on a Tendermint network
are expected to have clocks that differ by at most `Precision`.
### SynchronyParams.MessageDelay
`SynchronyParams.MessageDelay` is a parameter of the Proposer-Based Timestamps
algorithm that configures the acceptable upper-bound for transmitting a `Proposal`
message from the proposer to all of the validators on the network.
### Updates
The application may set the ConsensusParams during InitChain, and update them during


+ 109
- 0
spec/consensus/proposer-based-timestamp/tla/Apalache.tla View File

@ -0,0 +1,109 @@
--------------------------- MODULE Apalache -----------------------------------
(*
* This is a standard module for use with the Apalache model checker.
* The meaning of the operators is explained in the comments.
* Many of the operators serve as additional annotations of their arguments.
* As we like to preserve compatibility with TLC and TLAPS, we define the
* operator bodies by erasure. The actual interpretation of the operators is
* encoded inside Apalache. For the moment, these operators are mirrored in
* the class at.forsyte.apalache.tla.lir.oper.ApalacheOper.
*
* Igor Konnov, Jure Kukovec, Informal Systems 2020-2021
*)
(**
* An assignment of an expression e to a state variable x. Typically, one
* uses the non-primed version of x in the initializing predicate Init and
* the primed version of x (that is, x') in the transition predicate Next.
* Although TLA+ does not have a concept of a variable assignment, we find
* this concept extremely useful for symbolic model checking. In pure TLA+,
* one would simply write x = e, or x \in {e}.
*
* Apalache automatically converts some expressions of the form
* x = e or x \in {e} into assignments. However, if you like to annotate
* assignments by hand, you can use this operator.
*
* For a further discussion on that matter, see:
* https://github.com/informalsystems/apalache/blob/ik/idiomatic-tla/docs/idiomatic/assignments.md
*)
x := e == x = e
(**
* A generator of a data structure. Given a positive integer `bound`, and
* assuming that the type of the operator application is known, we
* recursively generate a TLA+ data structure as a tree, whose width is
* bound by the number `bound`.
*
* The body of this operator is redefined by Apalache.
*)
Gen(size) == {}
(**
* Convert a set of pairs S to a function F. Note that if S contains at least
* two pairs <<x, y>> and <<u, v>> such that x = u and y /= v,
* then F is not uniquely defined. We use CHOOSE to resolve this ambiguity.
* Apalache implements a more efficient encoding of this operator
* than the default one.
*
* @type: Set(<<a, b>>) => (a -> b);
*)
SetAsFun(S) ==
LET Dom == { x: <<x, y>> \in S }
Rng == { y: <<x, y>> \in S }
IN
[ x \in Dom |-> CHOOSE y \in Rng: <<x, y>> \in S ]
(**
* As TLA+ is untyped, one can use function- and sequence-specific operators
* interchangeably. However, to maintain correctness w.r.t. our type-system,
* an explicit cast is needed when using functions as sequences.
*)
LOCAL INSTANCE Sequences
FunAsSeq(fn, maxSeqLen) == SubSeq(fn, 1, maxSeqLen)
(**
* Annotating an expression \E x \in S: P as Skolemizable. That is, it can
* be replaced with an expression c \in S /\ P(c) for a fresh constant c.
* Not every exisential can be replaced with a constant, this should be done
* with care. Apalache detects Skolemizable expressions by static analysis.
*)
Skolem(e) == e
(**
* A hint to the model checker to expand a set S, instead of dealing
* with it symbolically. Apalache finds out which sets have to be expanded
* by static analysis.
*)
Expand(S) == S
(**
* A hint to the model checker to replace its argument Cardinality(S) >= k
* with a series of existential quantifiers for a constant k.
* Similar to Skolem, this has to be done carefully. Apalache automatically
* places this hint by static analysis.
*)
ConstCardinality(cardExpr) == cardExpr
(**
* The folding operator, used to implement computation over a set.
* Apalache implements a more efficient encoding than the one below.
* (from the community modules).
*)
RECURSIVE FoldSet(_,_,_)
FoldSet( Op(_,_), v, S ) == IF S = {}
THEN v
ELSE LET w == CHOOSE x \in S: TRUE
IN LET T == S \ {w}
IN FoldSet( Op, Op(v,w), T )
(**
* The folding operator, used to implement computation over a sequence.
* Apalache implements a more efficient encoding than the one below.
* (from the community modules).
*)
RECURSIVE FoldSeq(_,_,_)
FoldSeq( Op(_,_), v, seq ) == IF seq = <<>>
THEN v
ELSE FoldSeq( Op, Op(v,Head(seq)), Tail(seq) )
===============================================================================

+ 77
- 0
spec/consensus/proposer-based-timestamp/tla/MC_PBT.tla View File

@ -0,0 +1,77 @@
----------------------------- MODULE MC_PBT -------------------------------
CONSTANT
\* @type: ROUND -> PROCESS;
Proposer
VARIABLES
\* @type: PROCESS -> ROUND;
round, \* a process round number
\* @type: PROCESS -> STEP;
step, \* a process step
\* @type: PROCESS -> DECISION;
decision, \* process decision
\* @type: PROCESS -> VALUE;
lockedValue, \* a locked value
\* @type: PROCESS -> ROUND;
lockedRound, \* a locked round
\* @type: PROCESS -> PROPOSAL;
validValue, \* a valid value
\* @type: PROCESS -> ROUND;
validRound \* a valid round
\* time-related variables
VARIABLES
\* @type: PROCESS -> TIME;
localClock, \* a process local clock: Corr -> Ticks
\* @type: TIME;
realTime \* a reference Newtonian real time
\* book-keeping variables
VARIABLES
\* @type: ROUND -> Set(PROPMESSAGE);
msgsPropose, \* PROPOSE messages broadcast in the system, Rounds -> Messages
\* @type: ROUND -> Set(PREMESSAGE);
msgsPrevote, \* PREVOTE messages broadcast in the system, Rounds -> Messages
\* @type: ROUND -> Set(PREMESSAGE);
msgsPrecommit, \* PRECOMMIT messages broadcast in the system, Rounds -> Messages
\* @type: Set(MESSAGE);
evidence, \* the messages that were used by the correct processes to make transitions
\* @type: ACTION;
action, \* we use this variable to see which action was taken
\* @type: PROCESS -> Set(PROPMESSAGE);
receivedTimelyProposal, \* used to keep track when a process receives a timely VALUE message
\* @type: <<ROUND,PROCESS>> -> TIME;
inspectedProposal \* used to keep track when a process tries to receive a message
\* Invariant support
VARIABLES
\* @type: ROUND -> TIME;
beginRound, \* the minimum of the local clocks at the time any process entered a new round
\* @type: PROCESS -> TIME;
endConsensus, \* the local time when a decision is made
\* @type: ROUND -> TIME;
lastBeginRound, \* the maximum of the local clocks in each round
\* @type: ROUND -> TIME;
proposalTime, \* the real time when a proposer proposes in a round
\* @type: ROUND -> TIME;
proposalReceivedTime \* the real time when a correct process first receives a proposal message in a round
INSTANCE TendermintPBT_002_draft WITH
Corr <- {"c1", "c2"},
Faulty <- {"f3", "f4"},
N <- 4,
T <- 1,
ValidValues <- { "v0", "v1" },
InvalidValues <- {"v2"},
MaxRound <- 5,
MaxTimestamp <- 10,
MinTimestamp <- 2,
Delay <- 2,
Precision <- 2
\* run Apalache with --cinit=CInit
CInit == \* the proposer is arbitrary -- works for safety
Proposer \in [Rounds -> AllProcs]
=============================================================================

+ 323
- 217
spec/consensus/proposer-based-timestamp/tla/TendermintPBT_002_draft.tla View File

@ -5,10 +5,11 @@
the Tendermint TLA+ specification for fork accountability:
https://github.com/tendermint/spec/blob/master/spec/light-client/accountability/TendermintAcc_004_draft.tla
* Version 1. A preliminary specification.
* Version 2. A preliminary specification.
Zarko Milosevic, Igor Konnov, Informal Systems, 2019-2020.
Ilina Stoilkovska, Josef Widder, Informal Systems, 2021.
Jure Kukovec, Informal Systems, 2022.
*)
EXTENDS Integers, FiniteSets, Apalache, typedefs
@ -38,13 +39,11 @@ CONSTANTS
\* @type: TIME;
MaxTimestamp, \* the maximal value of the clock tick
\* @type: TIME;
Delay, \* message delay
MinTimestamp, \* the minimal value of the clock tick
\* @type: TIME;
Precision, \* clock precision: the maximal difference between two local clocks
Delay, \* message delay
\* @type: TIME;
Accuracy, \* clock accuracy: the maximal difference between a local clock and the real time
\* @type: Bool;
ClockDrift \* is there clock drift between the local clocks and the global clock
Precision \* clock precision: the maximal difference between two local clocks
ASSUME(N = Cardinality(Corr \union Faulty))
@ -66,24 +65,39 @@ Values == ValidValues \union InvalidValues \* the set of all values
\* @type: VALUE;
NilValue == "None" \* a special value for a nil round, outside of Values
\* @type: Set(PROPOSAL);
Proposals == Values \X Timestamps
Proposals == Values \X Timestamps \X Rounds
\* @type: PROPOSAL;
NilProposal == <<NilValue, NilTimestamp>>
NilProposal == <<NilValue, NilTimestamp, NilRound>>
\* @type: Set(VALUE);
ValuesOrNil == Values \union {NilValue}
\* @type: Set(DECISION);
Decisions == Values \X Timestamps \X Rounds
Decisions == Proposals \X Rounds
\* @type: DECISION;
NilDecision == <<NilValue, NilTimestamp, NilRound>>
NilDecision == <<NilProposal, NilRound>>
ValidProposals == ValidValues \X (MinTimestamp..MaxTimestamp) \X Rounds
\* a value hash is modeled as identity
\* @type: (t) => t;
Id(v) == v
\* The validity predicate
\* @type: (VALUE) => Bool;
IsValid(v) == v \in ValidValues
\* @type: (PROPOSAL) => Bool;
IsValid(p) == p \in ValidProposals
\* Time validity check. If we want MaxTimestamp = \infty, set ValidTime(t) == TRUE
ValidTime(t) == t < MaxTimestamp
\* @type: (PROPMESSAGE) => VALUE;
MessageValue(msg) == msg.proposal[1]
\* @type: (PROPMESSAGE) => TIME;
MessageTime(msg) == msg.proposal[2]
\* @type: (PROPMESSAGE) => ROUND;
MessageRound(msg) == msg.proposal[3]
\* @type: (TIME, TIME) => Bool;
IsTimely(processTime, messageTime) ==
/\ processTime >= messageTime - Precision
/\ processTime <= messageTime + Precision + Delay
\* the two thresholds that are used in the algorithm
\* @type: Int;
@ -91,23 +105,24 @@ THRESHOLD1 == T + 1 \* at least one process is not faulty
\* @type: Int;
THRESHOLD2 == 2 * T + 1 \* a quorum when having N > 3 * T
\* @type: (TIME, TIME) => TIME;
Min2(a,b) == IF a <= b THEN a ELSE b
\* @type: (Set(TIME)) => TIME;
Min(S) == CHOOSE x \in S : \A y \in S : x <= y
Min(S) == FoldSet( Min2, MaxTimestamp, S )
\* Min(S) == CHOOSE x \in S : \A y \in S : x <= y
\* @type: (TIME, TIME) => TIME;
Max2(a,b) == IF a >= b THEN a ELSE b
\* @type: (Set(TIME)) => TIME;
Max(S) == CHOOSE x \in S : \A y \in S : y <= x
(********************* TYPE ANNOTATIONS FOR APALACHE **************************)
\* a type annotation for an empty set of messages
\* @type: Set(MESSAGE);
EmptyMsgSet == {}
\* @type: Set(RCVPROP);
EmptyRcvProp == {}
Max(S) == FoldSet( Max2, NilTimestamp, S )
\* Max(S) == CHOOSE x \in S : \A y \in S : y <= x
\* @type: Set(PROCESS);
EmptyProcSet == {}
\* @type: (Set(MESSAGE)) => Int;
Card(S) ==
LET
\* @type: (Int, MESSAGE) => Int;
PlusOne(i, m) == i + 1
IN FoldSet( PlusOne, 0, S )
(********************* PROTOCOL STATE VARIABLES ******************************)
VARIABLES
@ -121,11 +136,15 @@ VARIABLES
lockedValue, \* a locked value
\* @type: PROCESS -> ROUND;
lockedRound, \* a locked round
\* @type: PROCESS -> VALUE;
\* @type: PROCESS -> PROPOSAL;
validValue, \* a valid value
\* @type: PROCESS -> ROUND;
validRound \* a valid round
coreVars ==
<<round, step, decision, lockedValue,
lockedRound, validValue, validRound>>
\* time-related variables
VARIABLES
\* @type: PROCESS -> TIME;
@ -133,6 +152,8 @@ VARIABLES
\* @type: TIME;
realTime \* a reference Newtonian real time
temporalVars == <<localClock, realTime>>
\* book-keeping variables
VARIABLES
\* @type: ROUND -> Set(PROPMESSAGE);
@ -145,28 +166,35 @@ VARIABLES
evidence, \* the messages that were used by the correct processes to make transitions
\* @type: ACTION;
action, \* we use this variable to see which action was taken
\* @type: Set(RCVPROP);
\* @type: PROCESS -> Set(PROPMESSAGE);
receivedTimelyProposal, \* used to keep track when a process receives a timely PROPOSAL message
\* @type: ROUND -> Set(PROCESS);
inspectedProposal, \* used to keep track when a process tries to receive a message
\* @type: TIME;
beginConsensus, \* the minimum of the local clocks in the initial state
\* @type: <<ROUND,PROCESS>> -> TIME;
inspectedProposal \* used to keep track when a process tries to receive a message
\* Action is excluded from the tuple, because it always changes
bookkeepingVars ==
<<msgsPropose, msgsPrevote, msgsPrecommit,
evidence, (*action,*) receivedTimelyProposal,
inspectedProposal>>
\* Invariant support
VARIABLES
\* @type: ROUND -> TIME;
beginRound, \* the minimum of the local clocks at the time any process entered a new round
\* @type: PROCESS -> TIME;
endConsensus, \* the local time when a decision is made
\* @type: TIME;
lastBeginConsensus, \* the maximum of the local clocks in the initial state
\* @type: ROUND -> TIME;
lastBeginRound, \* the maximum of the local clocks in each round
\* @type: ROUND -> TIME;
proposalTime, \* the real time when a proposer proposes in a round
\* @type: ROUND -> TIME;
proposalReceivedTime \* the real time when a correct process first receives a proposal message in a round
invariantVars ==
<<beginRound, endConsensus, lastBeginRound,
proposalTime, proposalReceivedTime>>
(* to see a type invariant, check TendermintAccInv3 *)
\* a handy definition used in UNCHANGED
vars == <<round, step, decision, lockedValue, lockedRound,
validValue, validRound, evidence, msgsPropose, msgsPrevote, msgsPrecommit,
localClock, realTime, receivedTimelyProposal, inspectedProposal, action,
beginConsensus, endConsensus, lastBeginConsensus, proposalTime, proposalReceivedTime>>
(********************* PROTOCOL INITIALIZATION ******************************)
\* @type: (ROUND) => Set(PROPMESSAGE);
@ -255,30 +283,37 @@ BenignRoundsInMessages(msgfun) ==
\* The initial states of the protocol. Some faults can be in the system already.
Init ==
/\ round = [p \in Corr |-> 0]
/\ \/ /\ ~ClockDrift
/\ localClock \in [Corr -> 0..Accuracy]
\/ /\ ClockDrift
/\ localClock = [p \in Corr |-> 0]
/\ localClock \in [Corr -> MinTimestamp..(MinTimestamp + Precision)]
/\ realTime = 0
/\ step = [p \in Corr |-> "PROPOSE"]
/\ decision = [p \in Corr |-> NilDecision]
/\ lockedValue = [p \in Corr |-> NilValue]
/\ lockedRound = [p \in Corr |-> NilRound]
/\ validValue = [p \in Corr |-> NilValue]
/\ validValue = [p \in Corr |-> NilProposal]
/\ validRound = [p \in Corr |-> NilRound]
/\ msgsPropose \in [Rounds -> SUBSET AllFaultyProposals]
/\ msgsPrevote \in [Rounds -> SUBSET AllFaultyPrevotes]
/\ msgsPrecommit \in [Rounds -> SUBSET AllFaultyPrecommits]
/\ receivedTimelyProposal = EmptyRcvProp
/\ inspectedProposal = [r \in Rounds |-> EmptyProcSet]
/\ receivedTimelyProposal = [p \in Corr |-> {}]
/\ inspectedProposal = [r \in Rounds, p \in Corr |-> NilTimestamp]
/\ BenignRoundsInMessages(msgsPropose)
/\ BenignRoundsInMessages(msgsPrevote)
/\ BenignRoundsInMessages(msgsPrecommit)
/\ evidence = EmptyMsgSet
/\ evidence = {}
/\ action' = "Init"
/\ beginConsensus = Min({localClock[p] : p \in Corr})
/\ beginRound =
[r \in Rounds |->
IF r = 0
THEN Min({localClock[p] : p \in Corr})
ELSE MaxTimestamp
]
/\ endConsensus = [p \in Corr |-> NilTimestamp]
/\ lastBeginConsensus = Max({localClock[p] : p \in Corr})
/\ lastBeginRound =
[r \in Rounds |->
IF r = 0
THEN Max({localClock[p] : p \in Corr})
ELSE NilTimestamp
]
/\ proposalTime = [r \in Rounds |-> NilTimestamp]
/\ proposalReceivedTime = [r \in Rounds |-> NilTimestamp]
@ -296,7 +331,7 @@ BroadcastProposal(pSrc, pRound, pProposal, pValidRound) ==
validRound |-> pValidRound
]
IN
msgsPropose' = [msgsPropose EXCEPT ![pRound] = msgsPropose[pRound] \union {newMsg}]
/\ msgsPropose' = [msgsPropose EXCEPT ![pRound] = msgsPropose[pRound] \union {newMsg}]
\* @type: (PROCESS, ROUND, PROPOSAL) => Bool;
BroadcastPrevote(pSrc, pRound, pId) ==
@ -310,7 +345,7 @@ BroadcastPrevote(pSrc, pRound, pId) ==
id |-> pId
]
IN
msgsPrevote' = [msgsPrevote EXCEPT ![pRound] = msgsPrevote[pRound] \union {newMsg}]
/\ msgsPrevote' = [msgsPrevote EXCEPT ![pRound] = msgsPrevote[pRound] \union {newMsg}]
\* @type: (PROCESS, ROUND, PROPOSAL) => Bool;
BroadcastPrecommit(pSrc, pRound, pId) ==
@ -324,7 +359,7 @@ BroadcastPrecommit(pSrc, pRound, pId) ==
id |-> pId
]
IN
msgsPrecommit' = [msgsPrecommit EXCEPT ![pRound] = msgsPrecommit[pRound] \union {newMsg}]
/\ msgsPrecommit' = [msgsPrecommit EXCEPT ![pRound] = msgsPrecommit[pRound] \union {newMsg}]
(***************************** TIME **************************************)
@ -339,14 +374,14 @@ SynchronizedLocalClocks ==
/\ localClock[q] - localClock[p] < Precision
\* [PBTS-PROPOSE.0]
\* @type: (VALUE, TIME) => PROPOSAL;
Proposal(v, t) ==
<<v, t>>
\* @type: (VALUE, TIME, ROUND) => PROPOSAL;
Proposal(v, t, r) ==
<<v, t, r>>
\* [PBTS-DECISION-ROUND.0]
\* @type: (VALUE, TIME, ROUND) => DECISION;
Decision(v, t, r) ==
<<v, t, r>>
\* @type: (PROPOSAL, ROUND) => DECISION;
Decision(p, r) ==
<<p, r>>
(**************** MESSAGE PROCESSING TRANSITIONS *************************)
\* lines 12-13
@ -354,7 +389,10 @@ Decision(v, t, r) ==
StartRound(p, r) ==
/\ step[p] /= "DECIDED" \* a decided process does not participate in consensus
/\ round' = [round EXCEPT ![p] = r]
/\ step' = [step EXCEPT ![p] = "PROPOSE"]
/\ step' = [step EXCEPT ![p] = "PROPOSE"]
\* We only need to update (last)beginRound[r] once a process enters round `r`
/\ beginRound' = [beginRound EXCEPT ![r] = Min2(@, localClock[p])]
/\ lastBeginRound' = [lastBeginRound EXCEPT ![r] = Max2(@, localClock[p])]
\* lines 14-19, a proposal may be sent later
\* @type: (PROCESS) => Bool;
@ -365,20 +403,22 @@ InsertProposal(p) ==
\* if the proposer is sending a proposal, then there are no other proposals
\* by the correct processes for the same round
/\ \A m \in msgsPropose[r]: m.src /= p
\* /\ localClock[p] >
/\ \E v \in ValidValues:
LET value ==
IF validValue[p] /= NilValue
LET proposal ==
IF validValue[p] /= NilProposal
THEN validValue[p]
ELSE v
IN LET
proposal == Proposal(value, localClock[p])
ELSE Proposal(v, localClock[p], r)
IN
/\ BroadcastProposal(p, round[p], proposal, validRound[p])
/\ BroadcastProposal(p, r, proposal, validRound[p])
/\ proposalTime' = [proposalTime EXCEPT ![r] = realTime]
/\ UNCHANGED <<evidence, round, decision, lockedValue, lockedRound,
validValue, step, validRound, msgsPrevote, msgsPrecommit,
localClock, realTime, receivedTimelyProposal, inspectedProposal,
beginConsensus, endConsensus, lastBeginConsensus, proposalReceivedTime>>
/\ UNCHANGED <<temporalVars, coreVars>>
/\ UNCHANGED
<<(*msgsPropose,*) msgsPrevote, msgsPrecommit,
evidence, receivedTimelyProposal, inspectedProposal>>
/\ UNCHANGED
<<beginRound, endConsensus, lastBeginRound,
(*proposalTime,*) proposalReceivedTime>>
/\ action' = "InsertProposal"
\* a new action used to filter messages that are not on time
@ -394,92 +434,120 @@ ReceiveProposal(p) ==
type |-> "PROPOSAL",
src |-> Proposer[round[p]],
round |-> round[p],
proposal |-> Proposal(v, t),
proposal |-> Proposal(v, t, r),
validRound |-> NilRound
]
IN
/\ msg \in msgsPropose[round[p]]
/\ p \notin inspectedProposal[r]
/\ <<p, msg>> \notin receivedTimelyProposal
/\ inspectedProposal' = [inspectedProposal EXCEPT ![r] = @ \union {p}]
/\ \/ /\ localClock[p] - Precision < t
/\ t < localClock[p] + Precision + Delay
/\ receivedTimelyProposal' = receivedTimelyProposal \union {<<p, msg>>}
/\ \/ /\ proposalReceivedTime[r] = NilTimestamp
/\ proposalReceivedTime' = [proposalReceivedTime EXCEPT ![r] = realTime]
\/ /\ proposalReceivedTime[r] /= NilTimestamp
/\ UNCHANGED proposalReceivedTime
\/ /\ \/ localClock[p] - Precision >= t
\/ t >= localClock[p] + Precision + Delay
/\ UNCHANGED <<receivedTimelyProposal, proposalReceivedTime>>
/\ UNCHANGED <<round, step, decision, lockedValue, lockedRound,
validValue, validRound, evidence, msgsPropose, msgsPrevote, msgsPrecommit,
localClock, realTime, beginConsensus, endConsensus, lastBeginConsensus, proposalTime>>
/\ inspectedProposal[r,p] = NilTimestamp
/\ msg \notin receivedTimelyProposal[p]
/\ inspectedProposal' = [inspectedProposal EXCEPT ![r,p] = localClock[p]]
/\ LET
isTimely == IsTimely(localClock[p], t)
IN
\/ /\ isTimely
/\ receivedTimelyProposal' = [receivedTimelyProposal EXCEPT ![p] = @ \union {msg}]
/\ LET
isNilTimestamp == proposalReceivedTime[r] = NilTimestamp
IN
\/ /\ isNilTimestamp
/\ proposalReceivedTime' = [proposalReceivedTime EXCEPT ![r] = realTime]
\/ /\ ~isNilTimestamp
/\ UNCHANGED proposalReceivedTime
\/ /\ ~isTimely
/\ UNCHANGED <<receivedTimelyProposal, proposalReceivedTime>>
/\ UNCHANGED <<temporalVars, coreVars>>
/\ UNCHANGED
<<msgsPropose, msgsPrevote, msgsPrecommit,
evidence(*, receivedTimelyProposal, inspectedProposal*)>>
/\ UNCHANGED
<<beginRound, endConsensus, lastBeginRound,
proposalTime(*, proposalReceivedTime*)>>
/\ action' = "ReceiveProposal"
\* lines 22-27
\* @type: (PROCESS) => Bool;
UponProposalInPropose(p) ==
\E v \in Values, t \in Timestamps:
LET
r == round[p]
IN LET
\* @type: PROPOSAL;
prop == Proposal(v,t,r)
IN
/\ step[p] = "PROPOSE" (* line 22 *)
/\ LET
\* @type: PROPMESSAGE;
msg ==
[
type |-> "PROPOSAL",
src |-> Proposer[round[p]],
round |-> round[p],
proposal |-> Proposal(v, t),
src |-> Proposer[r],
round |-> r,
proposal |-> prop,
validRound |-> NilRound
]
IN
/\ <<p, msg>> \in receivedTimelyProposal \* updated line 22
/\ msg \in receivedTimelyProposal[p] \* updated line 22
/\ evidence' = {msg} \union evidence
/\ LET mid == (* line 23 *)
IF IsValid(v) /\ (lockedRound[p] = NilRound \/ lockedValue[p] = v)
THEN Id(Proposal(v, t))
IF IsValid(prop) /\ (lockedRound[p] = NilRound \/ lockedValue[p] = v)
THEN Id(prop)
ELSE NilProposal
IN
BroadcastPrevote(p, round[p], mid) \* lines 24-26
BroadcastPrevote(p, r, mid) \* lines 24-26
/\ step' = [step EXCEPT ![p] = "PREVOTE"]
/\ UNCHANGED <<round, decision, lockedValue, lockedRound,
validValue, validRound, msgsPropose, msgsPrecommit,
localClock, realTime, receivedTimelyProposal, inspectedProposal,
beginConsensus, endConsensus, lastBeginConsensus, proposalTime, proposalReceivedTime>>
/\ UNCHANGED <<temporalVars, invariantVars>>
/\ UNCHANGED
<<round, (*step,*) decision, lockedValue,
lockedRound, validValue, validRound>>
/\ UNCHANGED
<<msgsPropose, (*msgsPrevote,*) msgsPrecommit,
(*evidence,*) receivedTimelyProposal, inspectedProposal>>
/\ action' = "UponProposalInPropose"
\* lines 28-33
\* [PBTS-ALG-OLD-PREVOTE.0]
\* @type: (PROCESS) => Bool;
UponProposalInProposeAndPrevote(p) ==
\E v \in Values, t1 \in Timestamps, t2 \in Timestamps, vr \in Rounds:
/\ step[p] = "PROPOSE" /\ 0 <= vr /\ vr < round[p] \* line 28, the while part
/\ LET
\E v \in Values, t \in Timestamps, vr \in Rounds, pr \in Rounds:
LET
r == round[p]
IN LET
\* @type: PROPOSAL;
prop == Proposal(v,t,pr)
IN
/\ step[p] = "PROPOSE" /\ 0 <= vr /\ vr < r \* line 28, the while part
/\ pr <= vr
/\ LET
\* @type: PROPMESSAGE;
msg ==
[
type |-> "PROPOSAL",
src |-> Proposer[round[p]],
round |-> round[p],
proposal |-> Proposal(v, t1),
src |-> Proposer[r],
round |-> r,
proposal |-> prop,
validRound |-> vr
]
IN
/\ <<p, msg>> \in receivedTimelyProposal \* updated line 28
/\ LET PV == { m \in msgsPrevote[vr]: m.id = Id(Proposal(v, t2)) } IN
\* Changed from 001: no need to re-check timeliness
/\ msg \in msgsPropose[r] \* line 28
/\ LET PV == { m \in msgsPrevote[vr]: m.id = Id(prop) } IN
/\ Cardinality(PV) >= THRESHOLD2 \* line 28
/\ evidence' = PV \union {msg} \union evidence
/\ LET mid == (* line 29 *)
IF IsValid(v) /\ (lockedRound[p] <= vr \/ lockedValue[p] = v)
THEN Id(Proposal(v, t1))
IF IsValid(prop) /\ (lockedRound[p] <= vr \/ lockedValue[p] = v)
THEN Id(prop)
ELSE NilProposal
IN
BroadcastPrevote(p, round[p], mid) \* lines 24-26
BroadcastPrevote(p, r, mid) \* lines 24-26
/\ step' = [step EXCEPT ![p] = "PREVOTE"]
/\ UNCHANGED <<round, decision, lockedValue, lockedRound,
validValue, validRound, msgsPropose, msgsPrecommit,
localClock, realTime, receivedTimelyProposal, inspectedProposal,
beginConsensus, endConsensus, lastBeginConsensus, proposalTime, proposalReceivedTime>>
/\ UNCHANGED <<temporalVars, invariantVars>>
/\ UNCHANGED
<<round, (*step,*) decision, lockedValue,
lockedRound, validValue, validRound>>
/\ UNCHANGED
<<msgsPropose, (*msgsPrevote,*) msgsPrecommit,
(*evidence,*) receivedTimelyProposal, inspectedProposal>>
/\ action' = "UponProposalInProposeAndPrevote"
\* lines 34-35 + lines 61-64 (onTimeoutPrevote)
@ -494,10 +562,13 @@ UponQuorumOfPrevotesAny(p) ==
/\ evidence' = MyEvidence \union evidence
/\ BroadcastPrecommit(p, round[p], NilProposal)
/\ step' = [step EXCEPT ![p] = "PRECOMMIT"]
/\ UNCHANGED <<round, decision, lockedValue, lockedRound,
validValue, validRound, msgsPropose, msgsPrevote,
localClock, realTime, receivedTimelyProposal, inspectedProposal,
beginConsensus, endConsensus, lastBeginConsensus, proposalTime, proposalReceivedTime>>
/\ UNCHANGED <<temporalVars, invariantVars>>
/\ UNCHANGED
<<round, (*step,*) decision, lockedValue,
lockedRound, validValue, validRound>>
/\ UNCHANGED
<<msgsPropose, msgsPrevote, (*msgsPrecommit, *)
(*evidence,*) receivedTimelyProposal, inspectedProposal>>
/\ action' = "UponQuorumOfPrevotesAny"
\* lines 36-46
@ -505,36 +576,47 @@ UponQuorumOfPrevotesAny(p) ==
\* @type: (PROCESS) => Bool;
UponProposalInPrevoteOrCommitAndPrevote(p) ==
\E v \in ValidValues, t \in Timestamps, vr \in RoundsOrNil:
LET
r == round[p]
IN LET
\* @type: PROPOSAL;
prop == Proposal(v,t,r)
IN
/\ step[p] \in {"PREVOTE", "PRECOMMIT"} \* line 36
/\ LET
\* @type: PROPMESSAGE;
msg ==
[
type |-> "PROPOSAL",
src |-> Proposer[round[p]],
round |-> round[p],
proposal |-> Proposal(v, t),
src |-> Proposer[r],
round |-> r,
proposal |-> prop,
validRound |-> vr
]
IN
/\ <<p, msg>> \in receivedTimelyProposal \* updated line 36
/\ LET PV == { m \in msgsPrevote[round[p]]: m.id = Id(Proposal(v, t)) } IN
\* Changed from 001: no need to re-check timeliness
/\ msg \in msgsPropose[r] \* line 36
/\ LET PV == { m \in msgsPrevote[r]: m.id = Id(prop) } IN
/\ Cardinality(PV) >= THRESHOLD2 \* line 36
/\ evidence' = PV \union {msg} \union evidence
/\ IF step[p] = "PREVOTE"
THEN \* lines 38-41:
/\ lockedValue' = [lockedValue EXCEPT ![p] = v]
/\ lockedRound' = [lockedRound EXCEPT ![p] = round[p]]
/\ BroadcastPrecommit(p, round[p], Id(Proposal(v, t)))
/\ lockedRound' = [lockedRound EXCEPT ![p] = r]
/\ BroadcastPrecommit(p, r, Id(prop))
/\ step' = [step EXCEPT ![p] = "PRECOMMIT"]
ELSE
UNCHANGED <<lockedValue, lockedRound, msgsPrecommit, step>>
\* lines 42-43
/\ validValue' = [validValue EXCEPT ![p] = v]
/\ validRound' = [validRound EXCEPT ![p] = round[p]]
/\ UNCHANGED <<round, decision, msgsPropose, msgsPrevote,
localClock, realTime, receivedTimelyProposal, inspectedProposal,
beginConsensus, endConsensus, lastBeginConsensus, proposalTime, proposalReceivedTime>>
/\ validValue' = [validValue EXCEPT ![p] = prop]
/\ validRound' = [validRound EXCEPT ![p] = r]
/\ UNCHANGED <<temporalVars, invariantVars>>
/\ UNCHANGED
<<round, (*step,*) decision(*, lockedValue,
lockedRound, validValue, validRound*)>>
/\ UNCHANGED
<<msgsPropose, msgsPrevote, (*msgsPrecommit, *)
(*evidence,*) receivedTimelyProposal, inspectedProposal>>
/\ action' = "UponProposalInPrevoteOrCommitAndPrevote"
\* lines 47-48 + 65-67 (onTimeoutPrecommit)
@ -547,11 +629,17 @@ UponQuorumOfPrecommitsAny(p) ==
/\ Cardinality(Committers) >= THRESHOLD2 \* line 47
/\ evidence' = MyEvidence \union evidence
/\ round[p] + 1 \in Rounds
/\ StartRound(p, round[p] + 1)
/\ UNCHANGED <<decision, lockedValue, lockedRound, validValue,
validRound, msgsPropose, msgsPrevote, msgsPrecommit,
localClock, realTime, receivedTimelyProposal, inspectedProposal,
beginConsensus, endConsensus, lastBeginConsensus, proposalTime, proposalReceivedTime>>
/\ StartRound(p, round[p] + 1)
/\ UNCHANGED temporalVars
/\ UNCHANGED
<<(*beginRound,*) endConsensus, (*lastBeginRound,*)
proposalTime, proposalReceivedTime>>
/\ UNCHANGED
<<(*round, step,*) decision, lockedValue,
lockedRound, validValue, validRound>>
/\ UNCHANGED
<<msgsPropose, msgsPrevote, msgsPrecommit,
(*evidence,*) receivedTimelyProposal, inspectedProposal>>
/\ action' = "UponQuorumOfPrecommitsAny"
\* lines 49-54
@ -559,7 +647,11 @@ UponQuorumOfPrecommitsAny(p) ==
\* @type: (PROCESS) => Bool;
UponProposalInPrecommitNoDecision(p) ==
/\ decision[p] = NilDecision \* line 49
/\ \E v \in ValidValues, t \in Timestamps (* line 50*) , r \in Rounds, vr \in RoundsOrNil:
/\ \E v \in ValidValues, t \in Timestamps (* line 50*) , r \in Rounds, pr \in Rounds, vr \in RoundsOrNil:
LET
\* @type: PROPOSAL;
prop == Proposal(v,t,pr)
IN
/\ LET
\* @type: PROPMESSAGE;
msg ==
@ -567,24 +659,30 @@ UponProposalInPrecommitNoDecision(p) ==
type |-> "PROPOSAL",
src |-> Proposer[r],
round |-> r,
proposal |-> Proposal(v, t),
proposal |-> prop,
validRound |-> vr
]
IN
/\ msg \in msgsPropose[r] \* line 49
/\ p \in inspectedProposal[r]
/\ LET PV == { m \in msgsPrecommit[r]: m.id = Id(Proposal(v, t)) } IN
/\ inspectedProposal[r,p] /= NilTimestamp \* Keep?
/\ LET PV == { m \in msgsPrecommit[r]: m.id = Id(prop) } IN
/\ Cardinality(PV) >= THRESHOLD2 \* line 49
/\ evidence' = PV \union {msg} \union evidence
/\ decision' = [decision EXCEPT ![p] = Decision(v, t, round[p])] \* update the decision, line 51
/\ decision' = [decision EXCEPT ![p] = Decision(prop, r)] \* update the decision, line 51
\* The original algorithm does not have 'DECIDED', but it increments the height.
\* We introduced 'DECIDED' here to prevent the process from changing its decision.
/\ endConsensus' = [endConsensus EXCEPT ![p] = localClock[p]]
/\ step' = [step EXCEPT ![p] = "DECIDED"]
/\ UNCHANGED <<round, lockedValue, lockedRound, validValue,
validRound, msgsPropose, msgsPrevote, msgsPrecommit,
localClock, realTime, receivedTimelyProposal, inspectedProposal,
beginConsensus, lastBeginConsensus, proposalTime, proposalReceivedTime>>
/\ UNCHANGED temporalVars
/\ UNCHANGED
<<round, (*step, decision,*) lockedValue,
lockedRound, validValue, validRound>>
/\ UNCHANGED
<<msgsPropose, msgsPrevote, msgsPrecommit,
(*evidence,*) receivedTimelyProposal, inspectedProposal>>
/\ UNCHANGED
<<beginRound, (*endConsensus,*) lastBeginRound,
proposalTime, proposalReceivedTime>>
/\ action' = "UponProposalInPrecommitNoDecision"
\* the actions below are not essential for safety, but added for completeness
@ -596,10 +694,13 @@ OnTimeoutPropose(p) ==
/\ p /= Proposer[round[p]]
/\ BroadcastPrevote(p, round[p], NilProposal)
/\ step' = [step EXCEPT ![p] = "PREVOTE"]
/\ UNCHANGED <<round, lockedValue, lockedRound, validValue,
validRound, decision, evidence, msgsPropose, msgsPrecommit,
localClock, realTime, receivedTimelyProposal, inspectedProposal,
beginConsensus, endConsensus, lastBeginConsensus, proposalTime, proposalReceivedTime>>
/\ UNCHANGED <<temporalVars, invariantVars>>
/\ UNCHANGED
<<round, (*step,*) decision, lockedValue,
lockedRound, validValue, validRound>>
/\ UNCHANGED
<<msgsPropose, (*msgsPrevote,*) msgsPrecommit,
evidence, receivedTimelyProposal, inspectedProposal>>
/\ action' = "OnTimeoutPropose"
\* lines 44-46
@ -611,10 +712,13 @@ OnQuorumOfNilPrevotes(p) ==
/\ evidence' = PV \union evidence
/\ BroadcastPrecommit(p, round[p], Id(NilProposal))
/\ step' = [step EXCEPT ![p] = "PRECOMMIT"]
/\ UNCHANGED <<round, lockedValue, lockedRound, validValue,
validRound, decision, msgsPropose, msgsPrevote,
localClock, realTime, receivedTimelyProposal, inspectedProposal,
beginConsensus, endConsensus, lastBeginConsensus, proposalTime, proposalReceivedTime>>
/\ UNCHANGED <<temporalVars, invariantVars>>
/\ UNCHANGED
<<round, (*step,*) decision, lockedValue,
lockedRound, validValue, validRound>>
/\ UNCHANGED
<<msgsPropose, msgsPrevote, (*msgsPrecommit,*)
(*evidence,*) receivedTimelyProposal, inspectedProposal>>
/\ action' = "OnQuorumOfNilPrevotes"
\* lines 55-56
@ -627,10 +731,16 @@ OnRoundCatchup(p) ==
/\ Cardinality(Faster) >= THRESHOLD1
/\ evidence' = MyEvidence \union evidence
/\ StartRound(p, r)
/\ UNCHANGED <<decision, lockedValue, lockedRound, validValue,
validRound, msgsPropose, msgsPrevote, msgsPrecommit,
localClock, realTime, receivedTimelyProposal, inspectedProposal,
beginConsensus, endConsensus, lastBeginConsensus, proposalTime, proposalReceivedTime>>
/\ UNCHANGED temporalVars
/\ UNCHANGED
<<(*beginRound,*) endConsensus, (*lastBeginRound,*)
proposalTime, proposalReceivedTime>>
/\ UNCHANGED
<<(*round, step,*) decision, lockedValue,
lockedRound, validValue, validRound>>
/\ UNCHANGED
<<msgsPropose, msgsPrevote, msgsPrecommit,
(*evidence,*) receivedTimelyProposal, inspectedProposal>>
/\ action' = "OnRoundCatchup"
@ -638,28 +748,24 @@ OnRoundCatchup(p) ==
\* advance the global clock
\* @type: Bool;
AdvanceRealTime ==
/\ realTime < MaxTimestamp
/\ realTime' = realTime + 1
/\ \/ /\ ~ClockDrift
/\ localClock' = [p \in Corr |-> localClock[p] + 1]
\/ /\ ClockDrift
/\ UNCHANGED localClock
/\ UNCHANGED <<round, step, decision, lockedValue, lockedRound,
validValue, validRound, evidence, msgsPropose, msgsPrevote, msgsPrecommit,
localClock, receivedTimelyProposal, inspectedProposal,
beginConsensus, endConsensus, lastBeginConsensus, proposalTime, proposalReceivedTime>>
/\ ValidTime(realTime)
/\ \E t \in Timestamps:
/\ t > realTime
/\ realTime' = t
/\ localClock' = [p \in Corr |-> localClock[p] + (t - realTime)]
/\ UNCHANGED <<coreVars, bookkeepingVars, invariantVars>>
/\ action' = "AdvanceRealTime"
\* advance the local clock of node p
\* @type: (PROCESS) => Bool;
AdvanceLocalClock(p) ==
/\ localClock[p] < MaxTimestamp
/\ localClock' = [localClock EXCEPT ![p] = @ + 1]
/\ UNCHANGED <<round, step, decision, lockedValue, lockedRound,
validValue, validRound, evidence, msgsPropose, msgsPrevote, msgsPrecommit,
realTime, receivedTimelyProposal, inspectedProposal,
beginConsensus, endConsensus, lastBeginConsensus, proposalTime, proposalReceivedTime>>
/\ action' = "AdvanceLocalClock"
\* advance the local clock of node p to some larger time t, not necessarily by 1
\* #type: (PROCESS) => Bool;
\* AdvanceLocalClock(p) ==
\* /\ ValidTime(localClock[p])
\* /\ \E t \in Timestamps:
\* /\ t > localClock[p]
\* /\ localClock' = [localClock EXCEPT ![p] = t]
\* /\ UNCHANGED <<coreVars, bookkeepingVars, invariantVars>>
\* /\ UNCHANGED realTime
\* /\ action' = "AdvanceLocalClock"
\* process timely messages
\* @type: (PROCESS) => Bool;
@ -684,10 +790,8 @@ MessageProcessing(p) ==
* A system transition. In this specificatiom, the system may eventually deadlock,
* e.g., when all processes decide. This is expected behavior, as we focus on safety.
*)
Next ==
Next ==
\/ AdvanceRealTime
\/ /\ ClockDrift
/\ \E p \in Corr: AdvanceLocalClock(p)
\/ /\ SynchronizedLocalClocks
/\ \E p \in Corr: MessageProcessing(p)
@ -700,59 +804,62 @@ AgreementOnValue ==
\A p, q \in Corr:
/\ decision[p] /= NilDecision
/\ decision[q] /= NilDecision
=> \E v \in ValidValues, t1 \in Timestamps, t2 \in Timestamps, r1 \in Rounds, r2 \in Rounds :
/\ decision[p] = Decision(v, t1, r1)
/\ decision[q] = Decision(v, t2, r2)
\* [PBTS-INV-TIME-AGR.0]
AgreementOnTime ==
\A p, q \in Corr:
\A v1 \in ValidValues, v2 \in ValidValues, t1 \in Timestamps, t2 \in Timestamps, r \in Rounds :
/\ decision[p] = Decision(v1, t1, r)
/\ decision[q] = Decision(v2, t2, r)
=> t1 = t2
=> \E v \in ValidValues, t \in Timestamps, pr \in Rounds, r1 \in Rounds, r2 \in Rounds :
LET prop == Proposal(v,t,pr)
IN
/\ decision[p] = Decision(prop, r1)
/\ decision[q] = Decision(prop, r2)
\* [PBTS-CONSENSUS-TIME-VALID.0]
ConsensusTimeValid ==
\A p \in Corr, t \in Timestamps :
\A p \in Corr:
\* if a process decides on v and t
(\E v \in ValidValues, r \in Rounds : decision[p] = Decision(v, t, r))
\E v \in ValidValues, t \in Timestamps, pr \in Rounds, dr \in Rounds :
decision[p] = Decision(Proposal(v,t,pr), dr)
\* then
=> /\ beginConsensus - Precision <= t
/\ t < endConsensus[p] + Precision + Delay
\* TODO: consider tighter bound where beginRound[pr] is replaced
\* w/ MedianOfRound[pr]
=> (/\ beginRound[pr] - Precision - Delay <= t
/\ t <= endConsensus[p] + Precision)
\* [PBTS-CONSENSUS-SAFE-VALID-CORR-PROP.0]
ConsensusSafeValidCorrProp ==
\A v \in ValidValues, t \in Timestamps :
\* if the proposer in the first round is correct
(/\ Proposer[0] \in Corr
\* and there exists a process that decided on v, t
/\ \E p \in Corr, r \in Rounds : decision[p] = Decision(v, t, r))
\* then t is between the minimal and maximal initial local time
=> /\ beginConsensus <= t
/\ t <= lastBeginConsensus
\A v \in ValidValues:
\* and there exists a process that decided on v, t
/\ \E p \in Corr, t \in Timestamps, pr \in Rounds, dr \in Rounds :
\* if the proposer in the round is correct
(/\ Proposer[pr] \in Corr
/\ decision[p] = Decision(Proposal(v,t,pr), dr))
\* then t is between the minimal and maximal initial local time
=> /\ beginRound[pr] <= t
/\ t <= lastBeginRound[pr]
\* [PBTS-CONSENSUS-REALTIME-VALID-CORR.0]
ConsensusRealTimeValidCorr ==
\A t \in Timestamps, r \in Rounds :
(/\ \E p \in Corr, v \in ValidValues : decision[p] = Decision(v, t, r)
/\ proposalTime[r] /= NilTimestamp)
=> /\ proposalTime[r] - Accuracy < t
/\ t < proposalTime[r] + Accuracy
\A r \in Rounds :
\E p \in Corr, v \in ValidValues, t \in Timestamps, pr \in Rounds:
(/\ decision[p] = Decision(Proposal(v,t,pr), r)
/\ proposalTime[r] /= NilTimestamp)
=> (/\ proposalTime[r] - Precision <= t
/\ t <= proposalTime[r] + Precision)
\* [PBTS-CONSENSUS-REALTIME-VALID.0]
ConsensusRealTimeValid ==
\A t \in Timestamps, r \in Rounds :
(\E p \in Corr, v \in ValidValues : decision[p] = Decision(v, t, r))
=> /\ proposalReceivedTime[r] - Accuracy - Precision < t
/\ t < proposalReceivedTime[r] + Accuracy + Precision + Delay
(\E p \in Corr, v \in ValidValues, pr \in Rounds :
decision[p] = Decision(Proposal(v,t,pr), r))
=> /\ proposalReceivedTime[r] - Precision < t
/\ t < proposalReceivedTime[r] + Precision + Delay
DecideAfterMin == TRUE
\* if decide => time > min
\* [PBTS-MSG-FAIR.0]
BoundedDelay ==
\A r \in Rounds :
(/\ proposalTime[r] /= NilTimestamp
/\ proposalTime[r] + Delay < realTime)
=> inspectedProposal[r] = Corr
=> \A p \in Corr: inspectedProposal[r,p] /= NilTimestamp
\* [PBTS-CONSENSUS-TIME-LIVE.0]
ConsensusTimeLive ==
@ -761,19 +868,18 @@ ConsensusTimeLive ==
/\ proposalTime[r] + Delay < realTime
/\ Proposer[r] \in Corr
/\ round[p] <= r)
=> \E msg \in RoundProposals(r) : <<p, msg>> \in receivedTimelyProposal
=> \E msg \in RoundProposals(r) : msg \in receivedTimelyProposal[p]
\* a conjunction of all invariants
Inv ==
/\ AgreementOnValue
/\ AgreementOnTime
/\ ConsensusTimeValid
/\ ConsensusSafeValidCorrProp
/\ ConsensusRealTimeValid
/\ ConsensusRealTimeValidCorr
/\ BoundedDelay
\* /\ ConsensusRealTimeValid
\* /\ ConsensusRealTimeValidCorr
\* /\ BoundedDelay
Liveness ==
ConsensusTimeLive
\* Liveness ==
\* ConsensusTimeLive
=============================================================================

+ 2
- 3
spec/consensus/proposer-based-timestamp/tla/typedefs.tla View File

@ -7,9 +7,8 @@
@typeAlias: ACTION = Str;
@typeAlias: TRACE = Seq(Str);
@typeAlias: TIME = Int;
@typeAlias: PROPOSAL = <<VALUE, TIME>>;
@typeAlias: DECISION = <<VALUE, TIME, ROUND>>;
@typeAlias: RCVPROP = <<PROCESS, PROPMESSAGE>>;
@typeAlias: PROPOSAL = <<VALUE, TIME, ROUND>>;
@typeAlias: DECISION = <<PROPOSAL, ROUND>>;
@typeAlias: PROPMESSAGE =
[
type: STEP,


+ 15
- 12
spec/core/data_structures.md View File

@ -230,17 +230,20 @@ enum BlockIDFlag {
A vote is a signed message from a validator for a particular block.
The vote includes information about the validator signing it. When stored in the blockchain or propagated over the network, votes are encoded in Protobuf.
| Name | Type | Description | Validation |
|------------------|---------------------------------|---------------------------------------------------------------------------------------------|------------------------------------------------------------------------------------------------------|
| Type | [SignedMsgType](#signedmsgtype) | Either prevote or precommit. [SignedMsgType](#signedmsgtype) | A Vote is valid if its corresponding fields are included in the enum [signedMsgType](#signedmsgtype) |
| Height | uint64 | Height for which this vote was created for | Must be > 0 |
| Round | int32 | Round that the commit corresponds to. | Must be > 0 |
| BlockID | [BlockID](#blockid) | The blockID of the corresponding block. | [BlockID](#blockid) |
| Timestamp | [Time](#Time) | Timestamp represents the time at which a validator signed. | [Time](#time) |
| ValidatorAddress | slice of bytes (`[]byte`) | Address of the validator | Length must be equal to 20 |
| ValidatorIndex | int32 | Index at a specific block height that corresponds to the Index of the validator in the set. | must be > 0 |
| Signature | slice of bytes (`[]byte`) | Signature by the validator if they participated in consensus for the associated bock. | Length of signature must be > 0 and < 64 |
The vote extension is not part of the [`CanonicalVote`](#canonicalvote).
| Name | Type | Description | Validation |
|--------------------|---------------------------------|---------------------------------------------------------------------------------------------|------------------------------------------------------------------------------------------------------|
| Type | [SignedMsgType](#signedmsgtype) | Either prevote or precommit. [SignedMsgType](#signedmsgtype) | A Vote is valid if its corresponding fields are included in the enum [signedMsgType](#signedmsgtype) |
| Height | uint64 | Height for which this vote was created. | Must be > 0 |
| Round | int32 | Round that the commit corresponds to. | Must be > 0 |
| BlockID | [BlockID](#blockid) | The blockID of the corresponding block. | [BlockID](#blockid) |
| Timestamp | [Time](#Time) | The time at which a validator signed. | [Time](#time) |
| ValidatorAddress | slice of bytes (`[]byte`) | Address of the validator | Length must be equal to 20 |
| ValidatorIndex | int32 | Index at a specific block height that corresponds to the Index of the validator in the set. | must be > 0 |
| Signature | slice of bytes (`[]byte`) | Signature by the validator if they participated in consensus for the associated bock. | Length of signature must be > 0 and < 64 |
| Extension | slice of bytes (`[]byte`) | The vote extension provided by the Application. Only valid for precommit messages. | Length must be 0 if Type != `SIGNED_MSG_TYPE_PRECOMMIT` |
| ExtensionSignature | slice of bytes (`[]byte`) | Signature by the validator if they participated in consensus for the associated bock. | Length must be 0 if Type != `SIGNED_MSG_TYPE_PRECOMMIT`; else length must be > 0 and < 64 |
## CanonicalVote
@ -250,7 +253,7 @@ the fields.
```proto
message CanonicalVote {
SignedMsgType type = 1;
fixed64 height = 2;
fixed64 height = 2;
sfixed64 round = 3;
CanonicalBlockID block_id = 4;
google.protobuf.Timestamp timestamp = 5;


+ 0
- 1
test/fuzz/mempool/checktx.go View File

@ -31,7 +31,6 @@ func init() {
log.NewNopLogger(),
cfg,
appConnMem,
0,
)
}


Loading…
Cancel
Save