refactor(mempool): use account nonce to sort transactions

This commit is contained in:
zhourong 2021-04-09 16:16:53 +08:00
parent 448f8d181c
commit fcb7529e70
20 changed files with 265 additions and 246 deletions

2
go.mod
View File

@ -27,7 +27,7 @@ require (
github.com/magiconair/properties v1.8.4
github.com/meshplus/bitxhub-core v0.1.0-rc1.0.20210330035001-b327cf056572
github.com/meshplus/bitxhub-kit v1.1.2-0.20210112075018-319e668d6359
github.com/meshplus/bitxhub-model v1.1.2-0.20210309053945-afaea82e9fe1
github.com/meshplus/bitxhub-model v1.1.2-0.20210409071219-0526019e06c4
github.com/meshplus/did-registry v0.0.0-20210407092831-8da970934f93
github.com/meshplus/go-libp2p-cert v0.0.0-20210125063330-7c25fd5b7a49
github.com/meshplus/go-lightp2p v0.0.0-20210120082108-df5a536a6192

2
go.sum
View File

@ -696,6 +696,8 @@ github.com/meshplus/bitxhub-model v1.1.2-0.20201021152621-0b3c17c54b23/go.mod h1
github.com/meshplus/bitxhub-model v1.1.2-0.20210120083349-c7a006b03fcb/go.mod h1:x3H+TL24wcByzHegenLfs+5PQkQGNsk8eCm31QJMa+Q=
github.com/meshplus/bitxhub-model v1.1.2-0.20210309053945-afaea82e9fe1 h1:ziae0L0cbCMKp66OYzjZuU1WtNoB2TgfFhNIVWOTod4=
github.com/meshplus/bitxhub-model v1.1.2-0.20210309053945-afaea82e9fe1/go.mod h1:x3H+TL24wcByzHegenLfs+5PQkQGNsk8eCm31QJMa+Q=
github.com/meshplus/bitxhub-model v1.1.2-0.20210409071219-0526019e06c4 h1:i+ytsSGPPo6QKKOjT2DEvKf9z1PJz8E5hDIaTeDNYhk=
github.com/meshplus/bitxhub-model v1.1.2-0.20210409071219-0526019e06c4/go.mod h1:x3H+TL24wcByzHegenLfs+5PQkQGNsk8eCm31QJMa+Q=
github.com/meshplus/bitxid v0.0.0-20210331062815-ef07c54e5ab0 h1:SK2eve3MfqiX7WTz6Mt62Zw+HdJDGqs4WPax4zOO1Ls=
github.com/meshplus/bitxid v0.0.0-20210331062815-ef07c54e5ab0/go.mod h1:vAldSRfDe2Qo7exsSTbchVmZWXPY7fhWQrRw18QJHho=
github.com/meshplus/did-registry v0.0.0-20210331065856-fb7f8bc2f8a2/go.mod h1:s8sqpmd+1N8Y+O3WYOKj7zFU3lajZ4MDd75J1h0yUj8=

View File

@ -64,6 +64,7 @@ func NewBitXHub(rep *repo.Repo) (*BitXHub, error) {
order.WithDigest(chainMeta.BlockHash.String()),
order.WithGetChainMetaFunc(bxh.Ledger.GetChainMeta),
order.WithGetBlockByHeightFunc(bxh.Ledger.GetBlock),
order.WithGetAccountNonceFunc(bxh.Ledger.GetNonce),
)
if err != nil {
return nil, err

View File

@ -200,6 +200,8 @@ func TestBlockExecutor_ApplyReadonlyTransactions(t *testing.T) {
mockLedger.EXPECT().PersistExecutionResult(gomock.Any(), gomock.Any(), gomock.Any()).Return(nil).AnyTimes()
mockLedger.EXPECT().FlushDirtyDataAndComputeJournal().Return(make(map[string]*ledger.Account), &ledger.BlockJournal{}).AnyTimes()
mockLedger.EXPECT().PersistBlockData(gomock.Any()).AnyTimes()
mockLedger.EXPECT().GetNonce(gomock.Any()).Return(uint64(0)).AnyTimes()
mockLedger.EXPECT().SetNonce(gomock.Any(), gomock.Any()).AnyTimes()
logger := log.NewWithModule("executor")
exec, err := New(mockLedger, logger, executorType)

View File

@ -271,6 +271,9 @@ func (exec *BlockExecutor) postBlockEvent(block *pb.Block, interchainMeta *pb.In
}
func (exec *BlockExecutor) applyTransaction(i int, tx *pb.Transaction, opt *agency.TxOpt) ([]byte, error) {
curNonce := exec.ledger.GetNonce(tx.From)
defer exec.ledger.SetNonce(tx.From, curNonce+1)
if tx.IsIBTP() {
ctx := vm.NewContext(tx, uint64(i), nil, exec.ledger, exec.logger)
instance := boltvm.New(ctx, exec.validationEngine, exec.getContracts(opt))

View File

@ -4,25 +4,27 @@ import (
"fmt"
"github.com/meshplus/bitxhub-kit/crypto"
"github.com/meshplus/bitxhub-kit/types"
"github.com/meshplus/bitxhub-model/pb"
"github.com/meshplus/bitxhub/pkg/peermgr"
"github.com/sirupsen/logrus"
)
type Config struct {
ID uint64
IsNew bool
RepoRoot string
StoragePath string
PluginPath string
PeerMgr peermgr.PeerManager
PrivKey crypto.PrivateKey
Logger logrus.FieldLogger
Nodes map[uint64]*pb.VpInfo
Applied uint64
Digest string
GetChainMetaFunc func() *pb.ChainMeta
GetBlockByHeight func(height uint64) (*pb.Block, error)
ID uint64
IsNew bool
RepoRoot string
StoragePath string
PluginPath string
PeerMgr peermgr.PeerManager
PrivKey crypto.PrivateKey
Logger logrus.FieldLogger
Nodes map[uint64]*pb.VpInfo
Applied uint64
Digest string
GetChainMetaFunc func() *pb.ChainMeta
GetBlockByHeight func(height uint64) (*pb.Block, error)
GetAccountNonce func(address *types.Address) uint64
}
type Option func(*Config)
@ -105,6 +107,12 @@ func WithGetBlockByHeightFunc(f func(height uint64) (*pb.Block, error)) Option {
}
}
func WithGetAccountNonceFunc(f func(address *types.Address) uint64) Option {
return func(config *Config) {
config.GetAccountNonce = f
}
}
func checkConfig(config *Config) error {
if config.Logger == nil {
return fmt.Errorf("logger is nil")

View File

@ -93,10 +93,11 @@ func NewNode(opts ...order.Option) (order.Order, error) {
return nil, fmt.Errorf("generate raft txpool config: %w", err)
}
mempoolConf := &mempool.Config{
ID: config.ID,
ChainHeight: config.Applied,
Logger: config.Logger,
StoragePath: config.StoragePath,
ID: config.ID,
ChainHeight: config.Applied,
Logger: config.Logger,
StoragePath: config.StoragePath,
GetAccountNonce: config.GetAccountNonce,
BatchSize: raftConfig.RAFT.MempoolConfig.BatchSize,
PoolSize: raftConfig.RAFT.MempoolConfig.PoolSize,

View File

@ -55,6 +55,9 @@ func TestNode_Start(t *testing.T) {
order.WithStoragePath(repo.GetStoragePath(repoRoot, "order")),
order.WithLogger(log.NewWithModule("consensus")),
order.WithApplied(1),
order.WithGetAccountNonceFunc(func(address *types.Address) uint64 {
return 0
}),
)
require.Nil(t, err)
@ -111,6 +114,9 @@ func TestMulti_Node_Start(t *testing.T) {
order.WithLogger(log.NewWithModule("consensus")),
order.WithGetBlockByHeightFunc(nil),
order.WithApplied(1),
order.WithGetAccountNonceFunc(func(address *types.Address) uint64 {
return 0
}),
)
require.Nil(t, err)
err = order.Start()
@ -166,6 +172,9 @@ func TestMulti_Node_Start_Without_Cert_Verification(t *testing.T) {
order.WithLogger(log.NewWithModule("consensus")),
order.WithGetBlockByHeightFunc(nil),
order.WithApplied(1),
order.WithGetAccountNonceFunc(func(address *types.Address) uint64 {
return 0
}),
)
require.Nil(t, err)
err = order.Start()

View File

@ -69,6 +69,9 @@ func mockRaftNode(t *testing.T) (*Node, error) {
TxSliceSize: raftConfig.RAFT.MempoolConfig.TxSliceSize,
TxSliceTimeout: raftConfig.RAFT.MempoolConfig.TxSliceTimeout,
StoragePath: filepath.Join(repoRoot, "storage"),
GetAccountNonce: func(address *types.Address) uint64 {
return 0
},
}
mempoolInst, err := mempool.NewMempool(mempoolConf)
if err != nil {

View File

@ -1,10 +1,8 @@
package mempool
import (
"fmt"
"math"
"os"
"path/filepath"
"sync"
"time"
@ -29,17 +27,13 @@ type mempoolImpl struct {
}
func newMempoolImpl(config *Config) (*mempoolImpl, error) {
db, err := loadOrCreateStorage(filepath.Join(config.StoragePath, "mempool"))
if err != nil {
return nil, fmt.Errorf("load or create mem pool storage :%w", err)
}
mpi := &mempoolImpl{
localID: config.ID,
batchSeqNo: config.ChainHeight,
logger: config.Logger,
txSliceSize: config.TxSliceSize,
}
mpi.txStore = newTransactionStore(db, config.Logger)
mpi.txStore = newTransactionStore(config.GetAccountNonce, config.Logger)
if config.BatchSize == 0 {
mpi.batchSize = DefaultBatchSize
} else {
@ -162,7 +156,7 @@ func (mpi *mempoolImpl) generateBlock() (*raftproto.RequestBatch, error) {
}
// include transaction if it's "next" for given account or
// we've already sent its ancestor to Consensus
if seenPrevious || (txSeq == commitNonce) {
if seenPrevious || (txSeq == commitNonce+1) {
ptr := orderedIndexKey{account: tx.account, nonce: txSeq}
mpi.txStore.batchedTxs[ptr] = true
result = append(result, ptr)
@ -229,7 +223,7 @@ func (mpi *mempoolImpl) processCommitTransactions(state *ChainState) {
continue
}
preCommitNonce := mpi.txStore.nonceCache.getCommitNonce(txPointer.account)
newCommitNonce := txPointer.nonce + 1
newCommitNonce := txPointer.nonce
if updateAccounts[txPointer.account] < newCommitNonce && preCommitNonce < newCommitNonce {
updateAccounts[txPointer.account] = newCommitNonce
}
@ -245,7 +239,7 @@ func (mpi *mempoolImpl) processCommitTransactions(state *ChainState) {
commitNonce := mpi.txStore.nonceCache.getCommitNonce(account)
if list, ok := mpi.txStore.allTxs[account]; ok {
// remove all previous seq number txs for this account.
removedTxs := list.forward(commitNonce)
removedTxs := list.forward(commitNonce + 1)
// remove index smaller than commitNonce delete index.
var wg sync.WaitGroup
wg.Add(4)

View File

@ -153,9 +153,9 @@ func TestProcessTransactions(t *testing.T) {
ast.Equal(5, len(mpi.txStore.txHashMap))
ast.Equal(2, mpi.txStore.allTxs[account1.String()].index.size())
ast.Equal(3, mpi.txStore.allTxs[account2.String()].index.size())
ast.Equal(uint64(1), mpi.txStore.nonceCache.getCommitNonce(account1.String()))
ast.Equal(uint64(0), mpi.txStore.nonceCache.getCommitNonce(account1.String()))
ast.Equal(uint64(3), mpi.txStore.nonceCache.getPendingNonce(account1.String()))
ast.Equal(uint64(1), mpi.txStore.nonceCache.getCommitNonce(account2.String()))
ast.Equal(uint64(0), mpi.txStore.nonceCache.getCommitNonce(account2.String()))
ast.Equal(uint64(3), mpi.txStore.nonceCache.getPendingNonce(account2.String()))
mpi.batchSize = 4
@ -231,9 +231,9 @@ func TestUnorderedIncomingTxs(t *testing.T) {
ast.Equal(6, len(mpi.txStore.txHashMap))
ast.Equal(3, mpi.txStore.allTxs[account1.String()].index.size())
ast.Equal(3, mpi.txStore.allTxs[account2.String()].index.size())
ast.Equal(uint64(1), mpi.txStore.nonceCache.getCommitNonce(account1.String()))
ast.Equal(uint64(0), mpi.txStore.nonceCache.getCommitNonce(account1.String()))
ast.Equal(uint64(3), mpi.txStore.nonceCache.getPendingNonce(account1.String()))
ast.Equal(uint64(1), mpi.txStore.nonceCache.getCommitNonce(account2.String()))
ast.Equal(uint64(0), mpi.txStore.nonceCache.getCommitNonce(account2.String()))
ast.Equal(uint64(3), mpi.txStore.nonceCache.getPendingNonce(account2.String()))
tx7 := constructTx(uint64(3), &privKey1)
@ -275,7 +275,7 @@ func TestUnorderedIncomingTxs(t *testing.T) {
ast.Equal(1, mpi.txStore.allTxs[account1.String()].index.size())
ast.Equal(2, mpi.txStore.allTxs[account2.String()].index.size())
ast.Equal(uint64(5), mpi.txStore.nonceCache.getPendingNonce(account1.String()))
ast.Equal(uint64(3), mpi.txStore.nonceCache.getCommitNonce(account2.String()))
ast.Equal(uint64(2), mpi.txStore.nonceCache.getCommitNonce(account2.String()))
ast.Equal(uint64(3), mpi.txStore.nonceCache.getPendingNonce(account2.String()))
// generate block3
@ -298,9 +298,9 @@ func TestUnorderedIncomingTxs(t *testing.T) {
ast.Equal(7, len(mpi.txStore.txHashMap))
ast.Equal(3, mpi.txStore.allTxs[account1.String()].index.size())
ast.Equal(4, mpi.txStore.allTxs[account2.String()].index.size())
ast.Equal(uint64(4), mpi.txStore.nonceCache.getCommitNonce(account1.String()))
ast.Equal(uint64(3), mpi.txStore.nonceCache.getCommitNonce(account1.String()))
ast.Equal(uint64(7), mpi.txStore.nonceCache.getPendingNonce(account1.String()))
ast.Equal(uint64(3), mpi.txStore.nonceCache.getCommitNonce(account2.String()))
ast.Equal(uint64(2), mpi.txStore.nonceCache.getCommitNonce(account2.String()))
ast.Equal(uint64(7), mpi.txStore.nonceCache.getPendingNonce(account2.String()))
}
@ -348,9 +348,9 @@ func TestGetTimeoutTransaction(t *testing.T) {
ast.Equal(8, len(mpi.txStore.txHashMap))
ast.Equal(4, mpi.txStore.allTxs[account1.String()].index.size())
ast.Equal(4, mpi.txStore.allTxs[account2.String()].index.size())
ast.Equal(uint64(1), mpi.txStore.nonceCache.getCommitNonce(account1.String()))
ast.Equal(uint64(0), mpi.txStore.nonceCache.getCommitNonce(account1.String()))
ast.Equal(uint64(3), mpi.txStore.nonceCache.getPendingNonce(account1.String()))
ast.Equal(uint64(1), mpi.txStore.nonceCache.getCommitNonce(account2.String()))
ast.Equal(uint64(0), mpi.txStore.nonceCache.getCommitNonce(account2.String()))
ast.Equal(uint64(3), mpi.txStore.nonceCache.getPendingNonce(account2.String()))
tx1 := constructTx(3, &privKey1)
@ -407,7 +407,7 @@ func TestRestore(t *testing.T) {
nonce := mpi.GetPendingNonceByAccount(account1.String())
ast.Equal(uint64(1), nonce)
privKey2 := genPrivKey()
account2, _ := privKey1.PublicKey().Address()
//account2, _ := privKey1.PublicKey().Address()
tx1 := constructTx(uint64(1), &privKey1)
tx2 := constructTx(uint64(2), &privKey1)
tx3 := constructTx(uint64(1), &privKey2)
@ -443,10 +443,10 @@ func TestRestore(t *testing.T) {
ast.Equal(1, mpi.txStore.parkingLotIndex.size())
// stop and restore
ast.Nil(mpi.txStore.nonceCache.fallback.Close())
newMpi, _ := mockMempoolImpl(storePath)
ast.Equal(uint64(3), newMpi.txStore.nonceCache.getCommitNonce(account1.String()))
ast.Equal(uint64(3), newMpi.txStore.nonceCache.getCommitNonce(account2.String()))
ast.Equal(uint64(3), newMpi.txStore.nonceCache.getPendingNonce(account1.String()))
ast.Equal(uint64(3), newMpi.txStore.nonceCache.getPendingNonce(account2.String()))
//ast.Nil(mpi.txStore.nonceCache.fallback.Close())
//newMpi, _ := mockMempoolImpl(storePath)
//ast.Equal(uint64(3), newMpi.txStore.nonceCache.getCommitNonce(account1.String()))
//ast.Equal(uint64(3), newMpi.txStore.nonceCache.getCommitNonce(account2.String()))
//ast.Equal(uint64(3), newMpi.txStore.nonceCache.getPendingNonce(account1.String()))
//ast.Equal(uint64(3), newMpi.txStore.nonceCache.getPendingNonce(account2.String()))
}

View File

@ -21,16 +21,21 @@ const (
DefaultTestTxSetSize = uint64(1)
)
func mockGetAccountNonce(address *types.Address) uint64 {
return 0
}
func mockMempoolImpl(path string) (*mempoolImpl, chan *raftproto.Ready) {
config := &Config{
ID: 1,
ChainHeight: DefaultTestChainHeight,
BatchSize: DefaultTestBatchSize,
PoolSize: DefaultPoolSize,
TxSliceSize: DefaultTestTxSetSize,
TxSliceTimeout: DefaultTxSetTick,
Logger: log.NewWithModule("consensus"),
StoragePath: path,
ID: 1,
ChainHeight: DefaultTestChainHeight,
BatchSize: DefaultTestBatchSize,
PoolSize: DefaultPoolSize,
TxSliceSize: DefaultTestTxSetSize,
TxSliceTimeout: DefaultTxSetTick,
Logger: log.NewWithModule("consensus"),
StoragePath: path,
GetAccountNonce: mockGetAccountNonce,
}
proposalC := make(chan *raftproto.Ready)
mempool, _ := newMempoolImpl(config)

View File

@ -1,16 +1,13 @@
package mempool
import (
"fmt"
"math"
"strconv"
"sync"
"github.com/sirupsen/logrus"
"github.com/google/btree"
"github.com/meshplus/bitxhub-kit/storage"
"github.com/meshplus/bitxhub-kit/types"
"github.com/meshplus/bitxhub-model/pb"
"github.com/sirupsen/logrus"
)
type transactionStore struct {
@ -35,7 +32,7 @@ type transactionStore struct {
priorityNonBatchSize uint64
}
func newTransactionStore(db storage.Storage, logger logrus.FieldLogger) *transactionStore {
func newTransactionStore(f GetAccountNonceFunc, logger logrus.FieldLogger) *transactionStore {
return &transactionStore{
txHashMap: make(map[string]*orderedIndexKey, 0),
allTxs: make(map[string]*txSortedMap),
@ -43,7 +40,7 @@ func newTransactionStore(db storage.Storage, logger logrus.FieldLogger) *transac
parkingLotIndex: newBtreeIndex(),
priorityIndex: newBtreeIndex(),
ttlIndex: newTxLiveTimeMap(),
nonceCache: newNonceCache(db, logger),
nonceCache: newNonceCache(f, logger),
}
}
@ -159,26 +156,27 @@ type nonceCache struct {
commitNonces map[string]uint64
// pendingNonces records each account's latest nonce which has been included in
// priority queue. Invariant: pendingNonces[account] >= commitNonces[account]
pendingNonces map[string]uint64
pendingMu sync.RWMutex
// falling back to reading from local database if an account is unknown.
fallback storage.Storage
logger logrus.FieldLogger
pendingNonces map[string]uint64
pendingMu sync.RWMutex
getAccountNonce GetAccountNonceFunc
logger logrus.FieldLogger
}
func newNonceCache(store storage.Storage, logger logrus.FieldLogger) *nonceCache {
func newNonceCache(f GetAccountNonceFunc, logger logrus.FieldLogger) *nonceCache {
return &nonceCache{
commitNonces: make(map[string]uint64),
pendingNonces: make(map[string]uint64),
fallback: store,
logger: logger,
commitNonces: make(map[string]uint64),
pendingNonces: make(map[string]uint64),
getAccountNonce: f,
logger: logger,
}
}
func (nc *nonceCache) getCommitNonce(account string) uint64 {
nonce, ok := nc.commitNonces[account]
if !ok {
return nc.getNonceFromDB(committedNonceKey(account))
cn := nc.getAccountNonce(types.NewAddressByStr(account))
nc.commitNonces[account] = cn
return cn
}
return nonce
}
@ -194,7 +192,7 @@ func (nc *nonceCache) getPendingNonce(account string) uint64 {
if !ok {
// if nonce is unknown, check if there is committed nonce persisted in db
// cause there aer no pending txs in mempool now, pending nonce is equal to committed nonce
return nc.getNonceFromDB(committedNonceKey(account))
return nc.getCommitNonce(account) + 1
}
return nonce
}
@ -212,31 +210,11 @@ func (nc *nonceCache) updatePendingNonce(newPending map[string]uint64) {
}
func (nc *nonceCache) updateCommittedNonce(newCommitted map[string]uint64) {
storageBatch := nc.fallback.NewBatch()
defer storageBatch.Commit()
for account, committedNonce := range newCommitted {
nc.setCommitNonce(account, committedNonce)
storageBatch.Put(committedNonceKey(account), []byte(strconv.FormatUint(committedNonce, 10)))
}
}
func (nc *nonceCache) getNonceFromDB(key []byte) uint64 {
var value []byte
if value = nc.fallback.Get(key); value == nil {
return 1
}
nonce, err := strconv.ParseUint(string(value), 10, 64)
if err != nil {
nc.logger.Errorf("Parse invalid nonce from db %s", err.Error())
return 1
}
return nonce
}
func committedNonceKey(account string) []byte {
return []byte(fmt.Sprintf("committed-%s", account))
}
// since the live time field in sortedTtlKey may vary during process
// we need to track the latest live time since its latest broadcast.
type txLiveTimeMap struct {

View File

@ -20,6 +20,8 @@ const (
DefaultTxSetTick = 100 * time.Millisecond
)
type GetAccountNonceFunc func(address *types.Address) uint64
type Config struct {
ID uint64
BatchSize uint64
@ -30,6 +32,7 @@ type Config struct {
ChainHeight uint64
Logger logrus.FieldLogger
StoragePath string // db for persist mem pool meta data
GetAccountNonce GetAccountNonceFunc
}
type txItem struct {

View File

@ -98,10 +98,11 @@ func NewNode(opts ...order.Option) (order.Order, error) {
batchTimeout, memConfig, err := generateSoloConfig(config.RepoRoot)
mempoolConf := &mempool.Config{
ID: config.ID,
ChainHeight: config.Applied,
Logger: config.Logger,
StoragePath: config.StoragePath,
ID: config.ID,
ChainHeight: config.Applied,
Logger: config.Logger,
StoragePath: config.StoragePath,
GetAccountNonce: config.GetAccountNonce,
BatchSize: memConfig.BatchSize,
PoolSize: memConfig.PoolSize,

View File

@ -54,6 +54,9 @@ func TestNode_Start(t *testing.T) {
order.WithID(1),
order.WithNodes(nodes),
order.WithApplied(1),
order.WithGetAccountNonceFunc(func(address *types.Address) uint64 {
return 0
}),
)
require.Nil(t, err)

View File

@ -5,6 +5,8 @@
package mock_peermgr
import (
reflect "reflect"
event "github.com/ethereum/go-ethereum/event"
gomock "github.com/golang/mock/gomock"
peer "github.com/libp2p/go-libp2p-core/peer"
@ -12,61 +14,44 @@ import (
events "github.com/meshplus/bitxhub/internal/model/events"
peermgr "github.com/meshplus/bitxhub/pkg/peermgr"
network "github.com/meshplus/go-lightp2p"
reflect "reflect"
)
// MockPeerManager is a mock of PeerManager interface
// MockPeerManager is a mock of PeerManager interface.
type MockPeerManager struct {
ctrl *gomock.Controller
recorder *MockPeerManagerMockRecorder
}
// MockPeerManagerMockRecorder is the mock recorder for MockPeerManager
// MockPeerManagerMockRecorder is the mock recorder for MockPeerManager.
type MockPeerManagerMockRecorder struct {
mock *MockPeerManager
}
// NewMockPeerManager creates a new mock instance
// NewMockPeerManager creates a new mock instance.
func NewMockPeerManager(ctrl *gomock.Controller) *MockPeerManager {
mock := &MockPeerManager{ctrl: ctrl}
mock.recorder = &MockPeerManagerMockRecorder{mock}
return mock
}
// EXPECT returns an object that allows the caller to indicate expected use
// EXPECT returns an object that allows the caller to indicate expected use.
func (m *MockPeerManager) EXPECT() *MockPeerManagerMockRecorder {
return m.recorder
}
// Start mocks base method
func (m *MockPeerManager) Start() error {
// AddNode mocks base method.
func (m *MockPeerManager) AddNode(newNodeID uint64, vpInfo *pb.VpInfo) {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "Start")
ret0, _ := ret[0].(error)
return ret0
m.ctrl.Call(m, "AddNode", newNodeID, vpInfo)
}
// Start indicates an expected call of Start
func (mr *MockPeerManagerMockRecorder) Start() *gomock.Call {
// AddNode indicates an expected call of AddNode.
func (mr *MockPeerManagerMockRecorder) AddNode(newNodeID, vpInfo interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Start", reflect.TypeOf((*MockPeerManager)(nil).Start))
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "AddNode", reflect.TypeOf((*MockPeerManager)(nil).AddNode), newNodeID, vpInfo)
}
// Stop mocks base method
func (m *MockPeerManager) Stop() error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "Stop")
ret0, _ := ret[0].(error)
return ret0
}
// Stop indicates an expected call of Stop
func (mr *MockPeerManagerMockRecorder) Stop() *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Stop", reflect.TypeOf((*MockPeerManager)(nil).Stop))
}
// AsyncSend mocks base method
// AsyncSend mocks base method.
func (m *MockPeerManager) AsyncSend(arg0 uint64, arg1 *pb.Message) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "AsyncSend", arg0, arg1)
@ -74,27 +59,107 @@ func (m *MockPeerManager) AsyncSend(arg0 uint64, arg1 *pb.Message) error {
return ret0
}
// AsyncSend indicates an expected call of AsyncSend
// AsyncSend indicates an expected call of AsyncSend.
func (mr *MockPeerManagerMockRecorder) AsyncSend(arg0, arg1 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "AsyncSend", reflect.TypeOf((*MockPeerManager)(nil).AsyncSend), arg0, arg1)
}
// SendWithStream mocks base method
func (m *MockPeerManager) SendWithStream(arg0 network.Stream, arg1 *pb.Message) error {
// Broadcast mocks base method.
func (m *MockPeerManager) Broadcast(arg0 *pb.Message) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "SendWithStream", arg0, arg1)
ret := m.ctrl.Call(m, "Broadcast", arg0)
ret0, _ := ret[0].(error)
return ret0
}
// SendWithStream indicates an expected call of SendWithStream
func (mr *MockPeerManagerMockRecorder) SendWithStream(arg0, arg1 interface{}) *gomock.Call {
// Broadcast indicates an expected call of Broadcast.
func (mr *MockPeerManagerMockRecorder) Broadcast(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SendWithStream", reflect.TypeOf((*MockPeerManager)(nil).SendWithStream), arg0, arg1)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Broadcast", reflect.TypeOf((*MockPeerManager)(nil).Broadcast), arg0)
}
// Send mocks base method
// CountConnectedPeers mocks base method.
func (m *MockPeerManager) CountConnectedPeers() uint64 {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "CountConnectedPeers")
ret0, _ := ret[0].(uint64)
return ret0
}
// CountConnectedPeers indicates an expected call of CountConnectedPeers.
func (mr *MockPeerManagerMockRecorder) CountConnectedPeers() *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "CountConnectedPeers", reflect.TypeOf((*MockPeerManager)(nil).CountConnectedPeers))
}
// DelNode mocks base method.
func (m *MockPeerManager) DelNode(delID uint64) {
m.ctrl.T.Helper()
m.ctrl.Call(m, "DelNode", delID)
}
// DelNode indicates an expected call of DelNode.
func (mr *MockPeerManagerMockRecorder) DelNode(delID interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "DelNode", reflect.TypeOf((*MockPeerManager)(nil).DelNode), delID)
}
// Disconnect mocks base method.
func (m *MockPeerManager) Disconnect(vpInfos map[uint64]*pb.VpInfo) {
m.ctrl.T.Helper()
m.ctrl.Call(m, "Disconnect", vpInfos)
}
// Disconnect indicates an expected call of Disconnect.
func (mr *MockPeerManagerMockRecorder) Disconnect(vpInfos interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Disconnect", reflect.TypeOf((*MockPeerManager)(nil).Disconnect), vpInfos)
}
// OtherPeers mocks base method.
func (m *MockPeerManager) OtherPeers() map[uint64]*peer.AddrInfo {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "OtherPeers")
ret0, _ := ret[0].(map[uint64]*peer.AddrInfo)
return ret0
}
// OtherPeers indicates an expected call of OtherPeers.
func (mr *MockPeerManagerMockRecorder) OtherPeers() *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "OtherPeers", reflect.TypeOf((*MockPeerManager)(nil).OtherPeers))
}
// Peers mocks base method.
func (m *MockPeerManager) Peers() map[uint64]*pb.VpInfo {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "Peers")
ret0, _ := ret[0].(map[uint64]*pb.VpInfo)
return ret0
}
// Peers indicates an expected call of Peers.
func (mr *MockPeerManagerMockRecorder) Peers() *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Peers", reflect.TypeOf((*MockPeerManager)(nil).Peers))
}
// PierManager mocks base method.
func (m *MockPeerManager) PierManager() peermgr.PierManager {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "PierManager")
ret0, _ := ret[0].(peermgr.PierManager)
return ret0
}
// PierManager indicates an expected call of PierManager.
func (mr *MockPeerManagerMockRecorder) PierManager() *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "PierManager", reflect.TypeOf((*MockPeerManager)(nil).PierManager))
}
// Send mocks base method.
func (m *MockPeerManager) Send(arg0 uint64, arg1 *pb.Message) (*pb.Message, error) {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "Send", arg0, arg1)
@ -103,69 +168,55 @@ func (m *MockPeerManager) Send(arg0 uint64, arg1 *pb.Message) (*pb.Message, erro
return ret0, ret1
}
// Send indicates an expected call of Send
// Send indicates an expected call of Send.
func (mr *MockPeerManagerMockRecorder) Send(arg0, arg1 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Send", reflect.TypeOf((*MockPeerManager)(nil).Send), arg0, arg1)
}
// Broadcast mocks base method
func (m *MockPeerManager) Broadcast(arg0 *pb.Message) error {
// SendWithStream mocks base method.
func (m *MockPeerManager) SendWithStream(arg0 network.Stream, arg1 *pb.Message) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "Broadcast", arg0)
ret := m.ctrl.Call(m, "SendWithStream", arg0, arg1)
ret0, _ := ret[0].(error)
return ret0
}
// Broadcast indicates an expected call of Broadcast
func (mr *MockPeerManagerMockRecorder) Broadcast(arg0 interface{}) *gomock.Call {
// SendWithStream indicates an expected call of SendWithStream.
func (mr *MockPeerManagerMockRecorder) SendWithStream(arg0, arg1 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Broadcast", reflect.TypeOf((*MockPeerManager)(nil).Broadcast), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SendWithStream", reflect.TypeOf((*MockPeerManager)(nil).SendWithStream), arg0, arg1)
}
// CountConnectedPeers mocks base method
func (m *MockPeerManager) CountConnectedPeers() uint64 {
// Start mocks base method.
func (m *MockPeerManager) Start() error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "CountConnectedPeers")
ret0, _ := ret[0].(uint64)
ret := m.ctrl.Call(m, "Start")
ret0, _ := ret[0].(error)
return ret0
}
// CountConnectedPeers indicates an expected call of CountConnectedPeers
func (mr *MockPeerManagerMockRecorder) CountConnectedPeers() *gomock.Call {
// Start indicates an expected call of Start.
func (mr *MockPeerManagerMockRecorder) Start() *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "CountConnectedPeers", reflect.TypeOf((*MockPeerManager)(nil).CountConnectedPeers))
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Start", reflect.TypeOf((*MockPeerManager)(nil).Start))
}
// Peers mocks base method
func (m *MockPeerManager) Peers() map[uint64]*pb.VpInfo {
// Stop mocks base method.
func (m *MockPeerManager) Stop() error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "Peers")
ret0, _ := ret[0].(map[uint64]*pb.VpInfo)
ret := m.ctrl.Call(m, "Stop")
ret0, _ := ret[0].(error)
return ret0
}
// Peers indicates an expected call of Peers
func (mr *MockPeerManagerMockRecorder) Peers() *gomock.Call {
// Stop indicates an expected call of Stop.
func (mr *MockPeerManagerMockRecorder) Stop() *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Peers", reflect.TypeOf((*MockPeerManager)(nil).Peers))
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Stop", reflect.TypeOf((*MockPeerManager)(nil).Stop))
}
// OtherPeers mocks base method
func (m *MockPeerManager) OtherPeers() map[uint64]*peer.AddrInfo {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "OtherPeers")
ret0, _ := ret[0].(map[uint64]*peer.AddrInfo)
return ret0
}
// OtherPeers indicates an expected call of OtherPeers
func (mr *MockPeerManagerMockRecorder) OtherPeers() *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "OtherPeers", reflect.TypeOf((*MockPeerManager)(nil).OtherPeers))
}
// SubscribeOrderMessage mocks base method
// SubscribeOrderMessage mocks base method.
func (m *MockPeerManager) SubscribeOrderMessage(ch chan<- events.OrderMessageEvent) event.Subscription {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "SubscribeOrderMessage", ch)
@ -173,37 +224,13 @@ func (m *MockPeerManager) SubscribeOrderMessage(ch chan<- events.OrderMessageEve
return ret0
}
// SubscribeOrderMessage indicates an expected call of SubscribeOrderMessage
// SubscribeOrderMessage indicates an expected call of SubscribeOrderMessage.
func (mr *MockPeerManagerMockRecorder) SubscribeOrderMessage(ch interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SubscribeOrderMessage", reflect.TypeOf((*MockPeerManager)(nil).SubscribeOrderMessage), ch)
}
// AddNode mocks base method
func (m *MockPeerManager) AddNode(newNodeID uint64, vpInfo *pb.VpInfo) {
m.ctrl.T.Helper()
m.ctrl.Call(m, "AddNode", newNodeID, vpInfo)
}
// AddNode indicates an expected call of AddNode
func (mr *MockPeerManagerMockRecorder) AddNode(newNodeID, vpInfo interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "AddNode", reflect.TypeOf((*MockPeerManager)(nil).AddNode), newNodeID, vpInfo)
}
// DelNode mocks base method
func (m *MockPeerManager) DelNode(delID uint64) {
m.ctrl.T.Helper()
m.ctrl.Call(m, "DelNode", delID)
}
// DelNode indicates an expected call of DelNode
func (mr *MockPeerManagerMockRecorder) DelNode(delID interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "DelNode", reflect.TypeOf((*MockPeerManager)(nil).DelNode), delID)
}
// UpdateRouter mocks base method
// UpdateRouter mocks base method.
func (m *MockPeerManager) UpdateRouter(vpInfos map[uint64]*pb.VpInfo, isNew bool) bool {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "UpdateRouter", vpInfos, isNew)
@ -211,76 +238,36 @@ func (m *MockPeerManager) UpdateRouter(vpInfos map[uint64]*pb.VpInfo, isNew bool
return ret0
}
// UpdateRouter indicates an expected call of UpdateRouter
// UpdateRouter indicates an expected call of UpdateRouter.
func (mr *MockPeerManagerMockRecorder) UpdateRouter(vpInfos, isNew interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "UpdateRouter", reflect.TypeOf((*MockPeerManager)(nil).UpdateRouter), vpInfos, isNew)
}
// Disconnect mocks base method
func (m *MockPeerManager) Disconnect(vpInfos map[uint64]*pb.VpInfo) {
m.ctrl.T.Helper()
m.ctrl.Call(m, "Disconnect", vpInfos)
}
// Disconnect indicates an expected call of Disconnect
func (mr *MockPeerManagerMockRecorder) Disconnect(vpInfos interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Disconnect", reflect.TypeOf((*MockPeerManager)(nil).Disconnect), vpInfos)
}
// PierManager mocks base method
func (m *MockPeerManager) PierManager() peermgr.PierManager {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "PierManager")
ret0, _ := ret[0].(peermgr.PierManager)
return ret0
}
// PierManager indicates an expected call of PierManager
func (mr *MockPeerManagerMockRecorder) PierManager() *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "PierManager", reflect.TypeOf((*MockPeerManager)(nil).PierManager))
}
// MockPierManager is a mock of PierManager interface
// MockPierManager is a mock of PierManager interface.
type MockPierManager struct {
ctrl *gomock.Controller
recorder *MockPierManagerMockRecorder
}
// MockPierManagerMockRecorder is the mock recorder for MockPierManager
// MockPierManagerMockRecorder is the mock recorder for MockPierManager.
type MockPierManagerMockRecorder struct {
mock *MockPierManager
}
// NewMockPierManager creates a new mock instance
// NewMockPierManager creates a new mock instance.
func NewMockPierManager(ctrl *gomock.Controller) *MockPierManager {
mock := &MockPierManager{ctrl: ctrl}
mock.recorder = &MockPierManagerMockRecorder{mock}
return mock
}
// EXPECT returns an object that allows the caller to indicate expected use
// EXPECT returns an object that allows the caller to indicate expected use.
func (m *MockPierManager) EXPECT() *MockPierManagerMockRecorder {
return m.recorder
}
// Piers mocks base method
func (m *MockPierManager) Piers() *peermgr.Piers {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "Piers")
ret0, _ := ret[0].(*peermgr.Piers)
return ret0
}
// Piers indicates an expected call of Piers
func (mr *MockPierManagerMockRecorder) Piers() *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Piers", reflect.TypeOf((*MockPierManager)(nil).Piers))
}
// AskPierMaster mocks base method
// AskPierMaster mocks base method.
func (m *MockPierManager) AskPierMaster(arg0 string) (bool, error) {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "AskPierMaster", arg0)
@ -289,8 +276,22 @@ func (m *MockPierManager) AskPierMaster(arg0 string) (bool, error) {
return ret0, ret1
}
// AskPierMaster indicates an expected call of AskPierMaster
// AskPierMaster indicates an expected call of AskPierMaster.
func (mr *MockPierManagerMockRecorder) AskPierMaster(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "AskPierMaster", reflect.TypeOf((*MockPierManager)(nil).AskPierMaster), arg0)
}
// Piers mocks base method.
func (m *MockPierManager) Piers() *peermgr.Piers {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "Piers")
ret0, _ := ret[0].(*peermgr.Piers)
return ret0
}
// Piers indicates an expected call of Piers.
func (mr *MockPierManagerMockRecorder) Piers() *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Piers", reflect.TypeOf((*MockPierManager)(nil).Piers))
}

View File

@ -174,8 +174,9 @@ func (suite *Interchain) TestHandleIBTP() {
proof := []byte("true")
proofHash := sha256.Sum256(proof)
ib := &pb.IBTP{From: f.String(), To: t.String(), Index: ibtpNonce, Timestamp: time.Now().UnixNano(), Proof: proofHash[:]}
tx, err := genIBTPTransaction(k1, ib)
tx, err := genIBTPTransaction(k1, ib, k1Nonce)
suite.Require().Nil(err)
k1Nonce++
tx.Extra = proof
ret, err = sendTransactionWithReceipt(suite.api, tx)
@ -337,30 +338,33 @@ func (suite *Interchain) TestGetIBTPByID() {
proofHash := sha256.Sum256(proof)
ib := &pb.IBTP{From: f.String(), To: t.String(), Index: ibtpNonce, Payload: []byte("111"), Timestamp: time.Now().UnixNano(), Proof: proofHash[:]}
tx, err := genIBTPTransaction(k1, ib)
tx, err := genIBTPTransaction(k1, ib, k1Nonce)
suite.Require().Nil(err)
tx.Extra = proof
receipt, err := sendTransactionWithReceipt(suite.api, tx)
suite.Require().Nil(err)
suite.Require().EqualValues(true, receipt.IsSuccess(), string(receipt.Ret))
ibtpNonce++
k1Nonce++
ib2 := &pb.IBTP{From: f.String(), To: t.String(), Index: ibtpNonce, Payload: []byte("111"), Timestamp: time.Now().UnixNano(), Proof: proofHash[:]}
tx, err = genIBTPTransaction(k1, ib2)
tx, err = genIBTPTransaction(k1, ib2, k1Nonce)
suite.Require().Nil(err)
tx.Extra = proof
receipt, err = sendTransactionWithReceipt(suite.api, tx)
suite.Require().Nil(err)
suite.Require().EqualValues(true, receipt.IsSuccess(), string(receipt.Ret))
ibtpNonce++
k1Nonce++
ib3 := &pb.IBTP{From: f.String(), To: t.String(), Index: ibtpNonce, Payload: []byte("111"), Timestamp: time.Now().UnixNano(), Proof: proofHash[:]}
tx, err = genIBTPTransaction(k1, ib3)
tx, err = genIBTPTransaction(k1, ib3, k1Nonce)
suite.Require().Nil(err)
tx.Extra = proof
receipt, err = sendTransactionWithReceipt(suite.api, tx)
suite.Assert().Nil(err)
ibtpNonce++
k1Nonce++
ib.Index = 2
ret, err = invokeBVMContract(suite.api, k1, k1Nonce, constant.InterchainContractAddr.Address(), "GetIBTPByID", pb.String(ib.ID()))

View File

@ -21,7 +21,7 @@ func genXVMContractTransaction(privateKey crypto.PrivateKey, nonce uint64, addre
return genContractTransaction(pb.TransactionData_XVM, privateKey, nonce, address, method, args...)
}
func genIBTPTransaction(privateKey crypto.PrivateKey, ibtp *pb.IBTP) (*pb.Transaction, error) {
func genIBTPTransaction(privateKey crypto.PrivateKey, ibtp *pb.IBTP, nonce uint64) (*pb.Transaction, error) {
from, err := privateKey.PublicKey().Address()
if err != nil {
return nil, err
@ -58,7 +58,7 @@ func genIBTPTransaction(privateKey crypto.PrivateKey, ibtp *pb.IBTP) (*pb.Transa
To: constant.InterchainContractAddr.Address(),
Payload: payload,
Timestamp: time.Now().UnixNano(),
Nonce: ibtp.Index,
Nonce: nonce,
IBTP: ibtp,
}

View File

@ -95,6 +95,7 @@ func newTesterBitXHub(rep *repo.Repo) (*app.BitXHub, error) {
order.WithDigest(chainMeta.BlockHash.String()),
order.WithGetChainMetaFunc(bxh.Ledger.GetChainMeta),
order.WithGetBlockByHeightFunc(bxh.Ledger.GetBlock),
order.WithGetAccountNonceFunc(bxh.Ledger.GetNonce),
)
if err != nil {