Browse Source

TxSelector add CoordIdxDB, SelectionConfig, abstract filtering

TxTypeToEthAddr & TxTypeToBJJ

- TxSelector
	- Add SelectionConfig for each batch
	- Add CoordIdxDB key-value where the CoordinatorIdxs are stored
	- Separated method for filtering TxTypeToEthAddr & TxTypeToBJJ
feature/sql-semaphore1
arnaucube 3 years ago
parent
commit
004f852b44
5 changed files with 359 additions and 160 deletions
  1. +32
    -11
      coordinator/coordinator.go
  2. +21
    -3
      coordinator/coordinator_test.go
  3. +7
    -1
      node/node.go
  4. +245
    -127
      txselector/txselector.go
  5. +54
    -18
      txselector/txselector_test.go

+ 32
- 11
coordinator/coordinator.go

@ -13,6 +13,7 @@ import (
"github.com/hermeznetwork/hermez-node/common" "github.com/hermeznetwork/hermez-node/common"
"github.com/hermeznetwork/hermez-node/db/historydb" "github.com/hermeznetwork/hermez-node/db/historydb"
"github.com/hermeznetwork/hermez-node/db/l2db" "github.com/hermeznetwork/hermez-node/db/l2db"
"github.com/hermeznetwork/hermez-node/db/statedb"
"github.com/hermeznetwork/hermez-node/eth" "github.com/hermeznetwork/hermez-node/eth"
"github.com/hermeznetwork/hermez-node/log" "github.com/hermeznetwork/hermez-node/log"
"github.com/hermeznetwork/hermez-node/prover" "github.com/hermeznetwork/hermez-node/prover"
@ -202,18 +203,31 @@ func (c *Coordinator) canForge(stats *synchronizer.Stats) bool {
func (c *Coordinator) syncStats(ctx context.Context, stats *synchronizer.Stats) error { func (c *Coordinator) syncStats(ctx context.Context, stats *synchronizer.Stats) error {
c.txManager.SetLastBlock(stats.Eth.LastBlock.Num) c.txManager.SetLastBlock(stats.Eth.LastBlock.Num)
// TMP
//nolint:gomnd
selectionConfig := &txselector.SelectionConfig{
MaxL1UserTxs: 32,
MaxL1CoordinatorTxs: 32,
ProcessTxsConfig: statedb.ProcessTxsConfig{
NLevels: 32,
MaxFeeTx: 64,
MaxTx: 512,
MaxL1Tx: 64,
},
}
canForge := c.canForge(stats) canForge := c.canForge(stats)
if c.pipeline == nil { if c.pipeline == nil {
if canForge { if canForge {
log.Infow("Coordinator: forging state begin", "block", stats.Eth.LastBlock.Num,
"batch", stats.Sync.LastBatch)
log.Infow("Coordinator: forging state begin", "block",
stats.Eth.LastBlock.Num, "batch", stats.Sync.LastBatch)
batchNum := common.BatchNum(stats.Sync.LastBatch) batchNum := common.BatchNum(stats.Sync.LastBatch)
var err error var err error
if c.pipeline, err = c.newPipeline(ctx); err != nil { if c.pipeline, err = c.newPipeline(ctx); err != nil {
return tracerr.Wrap(err) return tracerr.Wrap(err)
} }
if err := c.pipeline.Start(batchNum, stats.Sync.LastForgeL1TxsNum, if err := c.pipeline.Start(batchNum, stats.Sync.LastForgeL1TxsNum,
stats, &c.vars); err != nil {
stats, &c.vars, selectionConfig); err != nil {
c.pipeline = nil c.pipeline = nil
return tracerr.Wrap(err) return tracerr.Wrap(err)
} }
@ -510,6 +524,7 @@ const longWaitDuration = 999 * time.Hour
func (t *TxManager) Run(ctx context.Context) { func (t *TxManager) Run(ctx context.Context) {
next := 0 next := 0
waitDuration := time.Duration(longWaitDuration) waitDuration := time.Duration(longWaitDuration)
for { for {
select { select {
case <-ctx.Done(): case <-ctx.Done():
@ -659,7 +674,8 @@ func (p *Pipeline) reset(batchNum common.BatchNum, lastForgeL1TxsNum int64,
// Start the forging pipeline // Start the forging pipeline
func (p *Pipeline) Start(batchNum common.BatchNum, lastForgeL1TxsNum int64, func (p *Pipeline) Start(batchNum common.BatchNum, lastForgeL1TxsNum int64,
syncStats *synchronizer.Stats, initSCVars *synchronizer.SCVariables) error {
syncStats *synchronizer.Stats, initSCVars *synchronizer.SCVariables,
selectionConfig *txselector.SelectionConfig) error {
if p.started { if p.started {
log.Fatal("Pipeline already started") log.Fatal("Pipeline already started")
} }
@ -685,7 +701,7 @@ func (p *Pipeline) Start(batchNum common.BatchNum, lastForgeL1TxsNum int64,
p.stats = syncStats p.stats = syncStats
default: default:
batchNum = p.batchNum + 1 batchNum = p.batchNum + 1
batchInfo, err := p.forgeBatch(p.ctx, batchNum)
batchInfo, err := p.forgeBatch(p.ctx, batchNum, selectionConfig)
if common.IsErrDone(err) { if common.IsErrDone(err) {
continue continue
} else if err != nil { } else if err != nil {
@ -777,7 +793,7 @@ func (p *Pipeline) sendServerProof(ctx context.Context, batchInfo *BatchInfo) er
} }
// forgeBatch the next batch. // forgeBatch the next batch.
func (p *Pipeline) forgeBatch(ctx context.Context, batchNum common.BatchNum) (*BatchInfo, error) {
func (p *Pipeline) forgeBatch(ctx context.Context, batchNum common.BatchNum, selectionConfig *txselector.SelectionConfig) (*BatchInfo, error) {
// remove transactions from the pool that have been there for too long // remove transactions from the pool that have been there for too long
_, err := p.purger.InvalidateMaybe(p.l2DB, p.txSelector.LocalAccountsDB(), _, err := p.purger.InvalidateMaybe(p.l2DB, p.txSelector.LocalAccountsDB(),
p.stats.Sync.LastBlock.Num, int64(batchNum)) p.stats.Sync.LastBlock.Num, int64(batchNum))
@ -794,6 +810,7 @@ func (p *Pipeline) forgeBatch(ctx context.Context, batchNum common.BatchNum) (*B
var poolL2Txs []common.PoolL2Tx var poolL2Txs []common.PoolL2Tx
// var feesInfo // var feesInfo
var l1UserTxsExtra, l1CoordTxs []common.L1Tx var l1UserTxsExtra, l1CoordTxs []common.L1Tx
var coordIdxs []common.Idx
// 1. Decide if we forge L2Tx or L1+L2Tx // 1. Decide if we forge L2Tx or L1+L2Tx
if p.shouldL1L2Batch() { if p.shouldL1L2Batch() {
p.lastScheduledL1BatchBlockNum = p.stats.Eth.LastBlock.Num p.lastScheduledL1BatchBlockNum = p.stats.Eth.LastBlock.Num
@ -803,13 +820,16 @@ func (p *Pipeline) forgeBatch(ctx context.Context, batchNum common.BatchNum) (*B
if err != nil { if err != nil {
return nil, tracerr.Wrap(err) return nil, tracerr.Wrap(err)
} }
l1UserTxsExtra, l1CoordTxs, poolL2Txs, err = p.txSelector.GetL1L2TxSelection([]common.Idx{}, batchNum, l1UserTxs) // TODO once feesInfo is added to method return, add the var
// TODO once feesInfo is added to method return, add the var
coordIdxs, l1UserTxsExtra, l1CoordTxs, poolL2Txs, err =
p.txSelector.GetL1L2TxSelection(selectionConfig, batchNum, l1UserTxs)
if err != nil { if err != nil {
return nil, tracerr.Wrap(err) return nil, tracerr.Wrap(err)
} }
} else { } else {
// 2b: only L2 txs // 2b: only L2 txs
l1CoordTxs, poolL2Txs, err = p.txSelector.GetL2TxSelection([]common.Idx{}, batchNum)
coordIdxs, l1CoordTxs, poolL2Txs, err =
p.txSelector.GetL2TxSelection(selectionConfig, batchNum)
if err != nil { if err != nil {
return nil, tracerr.Wrap(err) return nil, tracerr.Wrap(err)
} }
@ -830,7 +850,8 @@ func (p *Pipeline) forgeBatch(ctx context.Context, batchNum common.BatchNum) (*B
// the poolL2Txs selected. Will mark as invalid the txs that have a // the poolL2Txs selected. Will mark as invalid the txs that have a
// (fromIdx, nonce) which already appears in the selected txs (includes // (fromIdx, nonce) which already appears in the selected txs (includes
// all the nonces smaller than the current one) // all the nonces smaller than the current one)
err = poolMarkInvalidOldNoncesFromL2Txs(p.l2DB, idxsNonceFromPoolL2Txs(poolL2Txs), batchInfo.BatchNum)
err = poolMarkInvalidOldNoncesFromL2Txs(p.l2DB, idxsNonceFromPoolL2Txs(poolL2Txs),
batchInfo.BatchNum)
if err != nil { if err != nil {
return nil, tracerr.Wrap(err) return nil, tracerr.Wrap(err)
} }
@ -839,8 +860,8 @@ func (p *Pipeline) forgeBatch(ctx context.Context, batchNum common.BatchNum) (*B
configBatch := &batchbuilder.ConfigBatch{ configBatch := &batchbuilder.ConfigBatch{
ForgerAddress: p.cfg.ForgerAddress, ForgerAddress: p.cfg.ForgerAddress,
} }
zkInputs, err := p.batchBuilder.BuildBatch([]common.Idx{}, configBatch,
l1UserTxsExtra, l1CoordTxs, poolL2Txs, nil) // TODO []common.TokenID --> feesInfo
zkInputs, err := p.batchBuilder.BuildBatch(coordIdxs, configBatch, l1UserTxsExtra,
l1CoordTxs, poolL2Txs, nil) // TODO []common.TokenID --> feesInfo
if err != nil { if err != nil {
return nil, tracerr.Wrap(err) return nil, tracerr.Wrap(err)
} }

+ 21
- 3
coordinator/coordinator_test.go

@ -111,7 +111,13 @@ func newTestModules(t *testing.T) modules {
txSelDBPath, err = ioutil.TempDir("", "tmpTxSelDB") txSelDBPath, err = ioutil.TempDir("", "tmpTxSelDB")
require.NoError(t, err) require.NoError(t, err)
deleteme = append(deleteme, txSelDBPath) deleteme = append(deleteme, txSelDBPath)
txSelector, err := txselector.NewTxSelector(txSelDBPath, syncStateDB, l2DB, maxL1UserTxs, maxL1CoordinatorTxs, maxTxs)
coordAccount := &txselector.CoordAccount{ // TODO TMP
Addr: ethCommon.HexToAddress("0xc58d29fA6e86E4FAe04DDcEd660d45BCf3Cb2370"),
BJJ: nil,
AccountCreationAuth: nil,
}
txSelector, err := txselector.NewTxSelector(coordAccount, txSelDBPath, syncStateDB, l2DB)
assert.NoError(t, err) assert.NoError(t, err)
batchBuilderDBPath, err = ioutil.TempDir("", "tmpBatchBuilderDB") batchBuilderDBPath, err = ioutil.TempDir("", "tmpBatchBuilderDB")
@ -603,12 +609,24 @@ PoolTransfer(0) User2-User3: 300 (126)
pipeline.batchBuilder.LocalStateDB().MerkleTree().Root()) pipeline.batchBuilder.LocalStateDB().MerkleTree().Root())
batchNum++ batchNum++
batchInfo, err := pipeline.forgeBatch(ctx, batchNum)
selectionConfig := &txselector.SelectionConfig{
MaxL1UserTxs: maxL1UserTxs,
MaxL1CoordinatorTxs: maxL1CoordinatorTxs,
ProcessTxsConfig: statedb.ProcessTxsConfig{
NLevels: nLevels,
MaxFeeTx: maxFeeTxs,
MaxTx: uint32(maxTxs),
MaxL1Tx: uint32(maxL1Txs),
},
}
batchInfo, err := pipeline.forgeBatch(ctx, batchNum, selectionConfig)
require.NoError(t, err) require.NoError(t, err)
assert.Equal(t, 3, len(batchInfo.L2Txs)) assert.Equal(t, 3, len(batchInfo.L2Txs))
batchNum++ batchNum++
batchInfo, err = pipeline.forgeBatch(ctx, batchNum)
batchInfo, err = pipeline.forgeBatch(ctx, batchNum, selectionConfig)
require.NoError(t, err) require.NoError(t, err)
assert.Equal(t, 0, len(batchInfo.L2Txs)) assert.Equal(t, 0, len(batchInfo.L2Txs))
} }

+ 7
- 1
node/node.go

@ -7,6 +7,7 @@ import (
"sync" "sync"
"time" "time"
ethCommon "github.com/ethereum/go-ethereum/common"
"github.com/ethereum/go-ethereum/ethclient" "github.com/ethereum/go-ethereum/ethclient"
"github.com/gin-contrib/cors" "github.com/gin-contrib/cors"
"github.com/gin-gonic/gin" "github.com/gin-gonic/gin"
@ -144,7 +145,12 @@ func NewNode(mode Mode, cfg *config.Node) (*Node, error) {
cfg.Coordinator.L2DB.TTL.Duration, cfg.Coordinator.L2DB.TTL.Duration,
) )
// TODO: Get (maxL1UserTxs, maxL1OperatorTxs, maxTxs) from the smart contract // TODO: Get (maxL1UserTxs, maxL1OperatorTxs, maxTxs) from the smart contract
txSelector, err := txselector.NewTxSelector(cfg.Coordinator.TxSelector.Path, stateDB, l2DB, 10, 10, 10)
coordAccount := &txselector.CoordAccount{ // TODO TMP
Addr: ethCommon.HexToAddress("0xc58d29fA6e86E4FAe04DDcEd660d45BCf3Cb2370"),
BJJ: nil,
AccountCreationAuth: nil,
}
txSelector, err := txselector.NewTxSelector(coordAccount, cfg.Coordinator.TxSelector.Path, stateDB, l2DB)
if err != nil { if err != nil {
return nil, tracerr.Wrap(err) return nil, tracerr.Wrap(err)
} }

+ 245
- 127
txselector/txselector.go

@ -4,6 +4,7 @@ package txselector
import ( import (
"bytes" "bytes"
"fmt"
"math/big" "math/big"
"sort" "sort"
@ -14,6 +15,13 @@ import (
"github.com/hermeznetwork/hermez-node/log" "github.com/hermeznetwork/hermez-node/log"
"github.com/hermeznetwork/tracerr" "github.com/hermeznetwork/tracerr"
"github.com/iden3/go-iden3-crypto/babyjub" "github.com/iden3/go-iden3-crypto/babyjub"
"github.com/iden3/go-merkletree/db/pebble"
)
const (
// PathCoordIdxsDB defines the path of the key-value db where the
// CoordIdxs will be stored
PathCoordIdxsDB = "/coordidxs"
) )
// txs implements the interface Sort for an array of Tx // txs implements the interface Sort for an array of Tx
@ -29,32 +37,56 @@ func (t txs) Less(i, j int) bool {
return t[i].AbsoluteFee > t[j].AbsoluteFee return t[i].AbsoluteFee > t[j].AbsoluteFee
} }
// TxSelector implements all the functionalities to select the txs for the next batch
type TxSelector struct {
// CoordAccount contains the data of the Coordinator account, that will be used
// to create new transactions of CreateAccountDeposit type to add new TokenID
// accounts for the Coordinator to receive the fees.
type CoordAccount struct {
Addr ethCommon.Address
BJJ *babyjub.PublicKey
AccountCreationAuth []byte
}
// SelectionConfig contains the parameters of configuration of the selection of
// transactions for the next batch
type SelectionConfig struct {
// MaxL1UserTxs is the maximum L1-user-tx for a batch // MaxL1UserTxs is the maximum L1-user-tx for a batch
MaxL1UserTxs uint64 MaxL1UserTxs uint64
// MaxL1OperatorTxs is the maximum L1-operator-tx for a batch
MaxL1OperatorTxs uint64
// MaxTxs is the maximum txs for a batch
MaxTxs uint64
// MaxL1CoordinatorTxs is the maximum L1-coordinator-tx for a batch
MaxL1CoordinatorTxs uint64
// ProcessTxsConfig contains the config for ProcessTxs
ProcessTxsConfig statedb.ProcessTxsConfig
}
// TxSelector implements all the functionalities to select the txs for the next
// batch
type TxSelector struct {
l2db *l2db.L2DB l2db *l2db.L2DB
localAccountsDB *statedb.LocalStateDB localAccountsDB *statedb.LocalStateDB
coordAccount *CoordAccount
coordIdxsDB *pebble.PebbleStorage
} }
// NewTxSelector returns a *TxSelector // NewTxSelector returns a *TxSelector
func NewTxSelector(dbpath string, synchronizerStateDB *statedb.StateDB, l2 *l2db.L2DB, maxL1UserTxs, maxL1OperatorTxs, maxTxs uint64) (*TxSelector, error) {
localAccountsDB, err := statedb.NewLocalStateDB(dbpath, synchronizerStateDB, statedb.TypeTxSelector, 0) // without merkletree
func NewTxSelector(coordAccount *CoordAccount, dbpath string,
synchronizerStateDB *statedb.StateDB, l2 *l2db.L2DB) (*TxSelector, error) {
localAccountsDB, err := statedb.NewLocalStateDB(dbpath,
synchronizerStateDB, statedb.TypeTxSelector, 0) // without merkletree
if err != nil {
return nil, tracerr.Wrap(err)
}
coordIdxsDB, err := pebble.NewPebbleStorage(dbpath+PathCoordIdxsDB, false)
if err != nil { if err != nil {
return nil, tracerr.Wrap(err) return nil, tracerr.Wrap(err)
} }
return &TxSelector{ return &TxSelector{
MaxL1UserTxs: maxL1UserTxs,
MaxL1OperatorTxs: maxL1OperatorTxs,
MaxTxs: maxTxs,
l2db: l2,
localAccountsDB: localAccountsDB,
l2db: l2,
localAccountsDB: localAccountsDB,
coordAccount: coordAccount,
coordIdxsDB: coordIdxsDB,
}, nil }, nil
} }
@ -73,141 +105,114 @@ func (txsel *TxSelector) Reset(batchNum common.BatchNum) error {
return nil return nil
} }
// AddCoordIdxs stores the given TokenID with the correspondent Idx to the
// CoordIdxsDB
func (txsel *TxSelector) AddCoordIdxs(idxs map[common.TokenID]common.Idx) error {
tx, err := txsel.coordIdxsDB.NewTx()
if err != nil {
return tracerr.Wrap(err)
}
for tokenID, idx := range idxs {
idxBytes, err := idx.Bytes()
if err != nil {
return tracerr.Wrap(err)
}
err = tx.Put(tokenID.Bytes(), idxBytes[:])
if err != nil {
return tracerr.Wrap(err)
}
}
if err := tx.Commit(); err != nil {
return tracerr.Wrap(err)
}
return nil
}
// GetCoordIdxs returns a map with the stored TokenID with the correspondent
// Coordinator Idx
func (txsel *TxSelector) GetCoordIdxs() (map[common.TokenID]common.Idx, error) {
r := make(map[common.TokenID]common.Idx)
err := txsel.coordIdxsDB.Iterate(func(tokenIDBytes []byte, idxBytes []byte) (bool, error) {
idx, err := common.IdxFromBytes(idxBytes)
if err != nil {
return false, tracerr.Wrap(err)
}
tokenID, err := common.TokenIDFromBytes(tokenIDBytes)
if err != nil {
return false, tracerr.Wrap(err)
}
r[tokenID] = idx
return true, nil
})
return r, tracerr.Wrap(err)
}
// GetL2TxSelection returns the L1CoordinatorTxs and a selection of the L2Txs // GetL2TxSelection returns the L1CoordinatorTxs and a selection of the L2Txs
// for the next batch, from the L2DB pool // for the next batch, from the L2DB pool
func (txsel *TxSelector) GetL2TxSelection(coordIdxs []common.Idx, batchNum common.BatchNum) ([]common.L1Tx, []common.PoolL2Tx, error) {
_, l1CoordinatorTxs, l2Txs, err := txsel.GetL1L2TxSelection(coordIdxs, batchNum, []common.L1Tx{})
return l1CoordinatorTxs, l2Txs, tracerr.Wrap(err)
func (txsel *TxSelector) GetL2TxSelection(selectionConfig *SelectionConfig,
batchNum common.BatchNum) ([]common.Idx, []common.L1Tx, []common.PoolL2Tx, error) {
coordIdxs, _, l1CoordinatorTxs, l2Txs, err := txsel.GetL1L2TxSelection(selectionConfig, batchNum,
[]common.L1Tx{})
return coordIdxs, l1CoordinatorTxs, l2Txs, tracerr.Wrap(err)
} }
// GetL1L2TxSelection returns the selection of L1 + L2 txs // GetL1L2TxSelection returns the selection of L1 + L2 txs
func (txsel *TxSelector) GetL1L2TxSelection(coordIdxs []common.Idx, batchNum common.BatchNum, l1Txs []common.L1Tx) ([]common.L1Tx, []common.L1Tx, []common.PoolL2Tx, error) {
func (txsel *TxSelector) GetL1L2TxSelection(selectionConfig *SelectionConfig,
batchNum common.BatchNum, l1Txs []common.L1Tx) ([]common.Idx, []common.L1Tx, []common.L1Tx,
[]common.PoolL2Tx, error) {
// apply l1-user-tx to localAccountDB // apply l1-user-tx to localAccountDB
// create new leaves // create new leaves
// update balances // update balances
// update nonces // update nonces
// get existing CoordIdxs
coordIdxsMap, err := txsel.GetCoordIdxs()
if err != nil {
return nil, nil, nil, nil, tracerr.Wrap(err)
}
var coordIdxs []common.Idx
for tokenID := range coordIdxsMap {
coordIdxs = append(coordIdxs, coordIdxsMap[tokenID])
}
// get pending l2-tx from tx-pool // get pending l2-tx from tx-pool
l2TxsRaw, err := txsel.l2db.GetPendingTxs() // (batchID) l2TxsRaw, err := txsel.l2db.GetPendingTxs() // (batchID)
if err != nil { if err != nil {
return nil, nil, nil, tracerr.Wrap(err)
return nil, nil, nil, nil, tracerr.Wrap(err)
} }
var validTxs txs var validTxs txs
var l1CoordinatorTxs []common.L1Tx var l1CoordinatorTxs []common.L1Tx
positionL1 := len(l1Txs) positionL1 := len(l1Txs)
// if tx.ToIdx>=256, tx.ToIdx should exist to localAccountsDB, if so,
// tx is used. if tx.ToIdx==0, check if tx.ToEthAddr/tx.ToBJJ exist in
// localAccountsDB, if yes tx is used; if not, check if tx.ToEthAddr is
// in AccountCreationAuthDB, if so, tx is used and L1CoordinatorTx of
// CreateAccountAndDeposit is created.
for i := 0; i < len(l2TxsRaw); i++ { for i := 0; i < len(l2TxsRaw); i++ {
if l2TxsRaw[i].ToIdx == 0 {
if checkAlreadyPendingToCreate(l1CoordinatorTxs, l2TxsRaw[i].ToEthAddr, l2TxsRaw[i].ToBJJ) {
// if L2Tx needs a new L1CoordinatorTx of CreateAccount type,
// and a previous L2Tx in the current process already created
// a L1CoordinatorTx of this type, in the DB there still seem
// that needs to create a new L1CoordinatorTx, but as is already
// created, the tx is valid
validTxs = append(validTxs, l2TxsRaw[i])
continue
}
if !bytes.Equal(l2TxsRaw[i].ToEthAddr.Bytes(), common.EmptyAddr.Bytes()) &&
!bytes.Equal(l2TxsRaw[i].ToEthAddr.Bytes(), common.FFAddr.Bytes()) {
// case: ToEthAddr != 0x00 neither 0xff
var accAuth *common.AccountCreationAuth
if l2TxsRaw[i].ToBJJ != nil {
// case: ToBJJ!=0:
// if idx exist for EthAddr&BJJ use it
_, err := txsel.localAccountsDB.GetIdxByEthAddrBJJ(l2TxsRaw[i].ToEthAddr, l2TxsRaw[i].ToBJJ, l2TxsRaw[i].TokenID)
if err == nil {
// account for ToEthAddr&ToBJJ already exist,
// there is no need to create a new one.
// tx valid, StateDB will use the ToIdx==0 to define the AuxToIdx
validTxs = append(validTxs, l2TxsRaw[i])
continue
}
// if not, check if AccountCreationAuth exist for that ToEthAddr&BJJ
// accAuth, err = txsel.l2db.GetAccountCreationAuthBJJ(l2TxsRaw[i].ToEthAddr, l2TxsRaw[i].ToBJJ)
accAuth, err = txsel.l2db.GetAccountCreationAuth(l2TxsRaw[i].ToEthAddr)
if err != nil {
// not found, l2Tx will not be added in the selection
log.Debugw("invalid L2Tx: ToIdx not found in StateDB, neither ToEthAddr & ToBJJ found in AccountCreationAuths L2DB", "ToIdx", l2TxsRaw[i].ToIdx, "ToEthAddr", l2TxsRaw[i].ToEthAddr)
continue
}
if accAuth.BJJ != l2TxsRaw[i].ToBJJ {
// if AccountCreationAuth.BJJ is not the same than in the tx, tx is not accepted
log.Debugw("invalid L2Tx: ToIdx not found in StateDB, neither ToEthAddr & ToBJJ found in AccountCreationAuths L2DB", "ToIdx", l2TxsRaw[i].ToIdx, "ToEthAddr", l2TxsRaw[i].ToEthAddr, "ToBJJ", l2TxsRaw[i].ToBJJ)
continue
}
validTxs = append(validTxs, l2TxsRaw[i])
} else {
// case: ToBJJ==0:
// if idx exist for EthAddr use it
_, err := txsel.localAccountsDB.GetIdxByEthAddr(l2TxsRaw[i].ToEthAddr, l2TxsRaw[i].TokenID)
if err == nil {
// account for ToEthAddr already exist,
// there is no need to create a new one.
// tx valid, StateDB will use the ToIdx==0 to define the AuxToIdx
validTxs = append(validTxs, l2TxsRaw[i])
continue
}
// if not, check if AccountCreationAuth exist for that ToEthAddr
accAuth, err = txsel.l2db.GetAccountCreationAuth(l2TxsRaw[i].ToEthAddr)
if err != nil {
// not found, l2Tx will not be added in the selection
log.Debugw("invalid L2Tx: ToIdx not found in StateDB, neither ToEthAddr found in AccountCreationAuths L2DB", "ToIdx", l2TxsRaw[i].ToIdx, "ToEthAddr", l2TxsRaw[i].ToEthAddr)
continue
}
validTxs = append(validTxs, l2TxsRaw[i])
}
// create L1CoordinatorTx for the accountCreation
l1CoordinatorTx := common.L1Tx{
Position: positionL1,
UserOrigin: false,
FromEthAddr: accAuth.EthAddr,
FromBJJ: accAuth.BJJ,
TokenID: l2TxsRaw[i].TokenID,
DepositAmount: big.NewInt(0),
Type: common.TxTypeCreateAccountDeposit,
}
positionL1++
l1CoordinatorTxs = append(l1CoordinatorTxs, l1CoordinatorTx)
} else if bytes.Equal(l2TxsRaw[i].ToEthAddr.Bytes(), common.FFAddr.Bytes()) && l2TxsRaw[i].ToBJJ != nil {
// if idx exist for EthAddr&BJJ use it
_, err := txsel.localAccountsDB.GetIdxByEthAddrBJJ(l2TxsRaw[i].ToEthAddr, l2TxsRaw[i].ToBJJ, l2TxsRaw[i].TokenID)
if err == nil {
// account for ToEthAddr&ToBJJ already exist, (where ToEthAddr==0xff)
// there is no need to create a new one.
// tx valid, StateDB will use the ToIdx==0 to define the AuxToIdx
validTxs = append(validTxs, l2TxsRaw[i])
continue
}
// if idx don't exist for EthAddr&BJJ,
// coordinator can create a new account without
// L1Authorization, as ToEthAddr==0xff
// create L1CoordinatorTx for the accountCreation
l1CoordinatorTx := common.L1Tx{
Position: positionL1,
UserOrigin: false,
FromEthAddr: l2TxsRaw[i].ToEthAddr,
FromBJJ: l2TxsRaw[i].ToBJJ,
TokenID: l2TxsRaw[i].TokenID,
DepositAmount: big.NewInt(0),
Type: common.TxTypeCreateAccountDeposit,
}
positionL1++
l1CoordinatorTxs = append(l1CoordinatorTxs, l1CoordinatorTx)
// If tx.ToIdx>=256, tx.ToIdx should exist to localAccountsDB,
// if so, tx is used. If tx.ToIdx==0, for an L2Tx will be the
// case of TxToEthAddr or TxToBJJ, check if
// tx.ToEthAddr/tx.ToBJJ exist in localAccountsDB, if yes tx is
// used; if not, check if tx.ToEthAddr is in
// AccountCreationAuthDB, if so, tx is used and L1CoordinatorTx
// of CreateAccountAndDeposit is created. If tx.ToIdx==1, is a
// Exit type and is used.
if l2TxsRaw[i].ToIdx == 0 { // ToEthAddr/ToBJJ case
validTxs, l1CoordinatorTxs, positionL1, err =
txsel.processTxToEthAddrBJJ(validTxs, l1CoordinatorTxs,
positionL1, l2TxsRaw[i])
if err != nil {
log.Debug(err)
} }
} else if l2TxsRaw[i].ToIdx >= common.IdxUserThreshold { } else if l2TxsRaw[i].ToIdx >= common.IdxUserThreshold {
_, err = txsel.localAccountsDB.GetAccount(l2TxsRaw[i].ToIdx) _, err = txsel.localAccountsDB.GetAccount(l2TxsRaw[i].ToIdx)
if err != nil { if err != nil {
// tx not valid // tx not valid
log.Debugw("invalid L2Tx: ToIdx not found in StateDB", "ToIdx", l2TxsRaw[i].ToIdx)
log.Debugw("invalid L2Tx: ToIdx not found in StateDB",
"ToIdx", l2TxsRaw[i].ToIdx)
continue continue
} }
// TODO if EthAddr!=0 or BJJ!=0, check that ToIdxAccount.EthAddr or BJJ
// Account found in the DB, include the l2Tx in the selection // Account found in the DB, include the l2Tx in the selection
validTxs = append(validTxs, l2TxsRaw[i]) validTxs = append(validTxs, l2TxsRaw[i])
} else if l2TxsRaw[i].ToIdx == common.Idx(1) { } else if l2TxsRaw[i].ToIdx == common.Idx(1) {
@ -217,7 +222,7 @@ func (txsel *TxSelector) GetL1L2TxSelection(coordIdxs []common.Idx, batchNum com
} }
// get most profitable L2-tx // get most profitable L2-tx
maxL2Txs := txsel.MaxTxs - uint64(len(l1CoordinatorTxs)) // - len(l1UserTxs) // TODO if there are L1UserTxs take them in to account
maxL2Txs := selectionConfig.ProcessTxsConfig.MaxTx - uint32(len(l1CoordinatorTxs)) // - len(l1UserTxs) // TODO if there are L1UserTxs take them in to account
l2Txs := txsel.getL2Profitable(validTxs, maxL2Txs) l2Txs := txsel.getL2Profitable(validTxs, maxL2Txs)
//nolint:gomnd //nolint:gomnd
@ -230,17 +235,130 @@ func (txsel *TxSelector) GetL1L2TxSelection(coordIdxs []common.Idx, batchNum com
// process the txs in the local AccountsDB // process the txs in the local AccountsDB
_, err = txsel.localAccountsDB.ProcessTxs(ptc, coordIdxs, l1Txs, l1CoordinatorTxs, l2Txs) _, err = txsel.localAccountsDB.ProcessTxs(ptc, coordIdxs, l1Txs, l1CoordinatorTxs, l2Txs)
if err != nil { if err != nil {
return nil, nil, nil, tracerr.Wrap(err)
return nil, nil, nil, nil, tracerr.Wrap(err)
} }
err = txsel.localAccountsDB.MakeCheckpoint() err = txsel.localAccountsDB.MakeCheckpoint()
if err != nil { if err != nil {
return nil, nil, nil, tracerr.Wrap(err)
return nil, nil, nil, nil, tracerr.Wrap(err)
}
return nil, l1Txs, l1CoordinatorTxs, l2Txs, nil
}
// processTxsToEthAddrBJJ process the common.PoolL2Tx in the case where
// ToIdx==0, which can be the tx type of ToEthAddr or ToBJJ. If the receiver
// does not have an account yet, a new L1CoordinatorTx of type
// CreateAccountDeposit (with 0 as DepositAmount) is created and added to the
// l1CoordinatorTxs array, and then the PoolL2Tx is added into the validTxs
// array.
func (txsel *TxSelector) processTxToEthAddrBJJ(validTxs txs, l1CoordinatorTxs []common.L1Tx,
positionL1 int, l2Tx common.PoolL2Tx) (txs, []common.L1Tx, int, error) {
// if L2Tx needs a new L1CoordinatorTx of CreateAccount type, and a
// previous L2Tx in the current process already created a
// L1CoordinatorTx of this type, in the DB there still seem that needs
// to create a new L1CoordinatorTx, but as is already created, the tx
// is valid
if checkAlreadyPendingToCreate(l1CoordinatorTxs, l2Tx.ToEthAddr, l2Tx.ToBJJ) {
validTxs = append(validTxs, l2Tx)
return validTxs, l1CoordinatorTxs, positionL1, nil
}
if !bytes.Equal(l2Tx.ToEthAddr.Bytes(), common.EmptyAddr.Bytes()) &&
!bytes.Equal(l2Tx.ToEthAddr.Bytes(), common.FFAddr.Bytes()) {
// case: ToEthAddr != 0x00 neither 0xff
var accAuth *common.AccountCreationAuth
if l2Tx.ToBJJ != nil {
// case: ToBJJ!=0:
// if idx exist for EthAddr&BJJ use it
_, err := txsel.localAccountsDB.GetIdxByEthAddrBJJ(l2Tx.ToEthAddr,
l2Tx.ToBJJ, l2Tx.TokenID)
if err == nil {
// account for ToEthAddr&ToBJJ already exist,
// there is no need to create a new one.
// tx valid, StateDB will use the ToIdx==0 to define the AuxToIdx
validTxs = append(validTxs, l2Tx)
return validTxs, l1CoordinatorTxs, positionL1, nil
}
// if not, check if AccountCreationAuth exist for that
// ToEthAddr
accAuth, err = txsel.l2db.GetAccountCreationAuth(l2Tx.ToEthAddr)
if err != nil {
// not found, l2Tx will not be added in the selection
return validTxs, l1CoordinatorTxs, positionL1, tracerr.Wrap(fmt.Errorf("invalid L2Tx: ToIdx not found in StateDB, neither ToEthAddr found in AccountCreationAuths L2DB. ToIdx: %d, ToEthAddr: %s",
l2Tx.ToIdx, l2Tx.ToEthAddr.Hex()))
}
if accAuth.BJJ != l2Tx.ToBJJ {
// if AccountCreationAuth.BJJ is not the same
// than in the tx, tx is not accepted
return validTxs, l1CoordinatorTxs, positionL1, tracerr.Wrap(fmt.Errorf("invalid L2Tx: ToIdx not found in StateDB, neither ToEthAddr & ToBJJ found in AccountCreationAuths L2DB. ToIdx: %d, ToEthAddr: %s, ToBJJ: %s",
l2Tx.ToIdx, l2Tx.ToEthAddr.Hex(), l2Tx.ToBJJ.String()))
}
validTxs = append(validTxs, l2Tx)
} else {
// case: ToBJJ==0:
// if idx exist for EthAddr use it
_, err := txsel.localAccountsDB.GetIdxByEthAddr(l2Tx.ToEthAddr, l2Tx.TokenID)
if err == nil {
// account for ToEthAddr already exist,
// there is no need to create a new one.
// tx valid, StateDB will use the ToIdx==0 to define the AuxToIdx
validTxs = append(validTxs, l2Tx)
return validTxs, l1CoordinatorTxs, positionL1, nil
}
// if not, check if AccountCreationAuth exist for that ToEthAddr
accAuth, err = txsel.l2db.GetAccountCreationAuth(l2Tx.ToEthAddr)
if err != nil {
// not found, l2Tx will not be added in the selection
return validTxs, l1CoordinatorTxs, positionL1, tracerr.Wrap(fmt.Errorf("invalid L2Tx: ToIdx not found in StateDB, neither ToEthAddr found in AccountCreationAuths L2DB. ToIdx: %d, ToEthAddr: %s",
l2Tx.ToIdx, l2Tx.ToEthAddr))
}
validTxs = append(validTxs, l2Tx)
}
// create L1CoordinatorTx for the accountCreation
l1CoordinatorTx := common.L1Tx{
Position: positionL1,
UserOrigin: false,
FromEthAddr: accAuth.EthAddr,
FromBJJ: accAuth.BJJ,
TokenID: l2Tx.TokenID,
DepositAmount: big.NewInt(0),
Type: common.TxTypeCreateAccountDeposit,
}
positionL1++
l1CoordinatorTxs = append(l1CoordinatorTxs, l1CoordinatorTx)
} else if bytes.Equal(l2Tx.ToEthAddr.Bytes(), common.FFAddr.Bytes()) && l2Tx.ToBJJ != nil {
// if idx exist for EthAddr&BJJ use it
_, err := txsel.localAccountsDB.GetIdxByEthAddrBJJ(l2Tx.ToEthAddr, l2Tx.ToBJJ,
l2Tx.TokenID)
if err == nil {
// account for ToEthAddr&ToBJJ already exist, (where ToEthAddr==0xff)
// there is no need to create a new one.
// tx valid, StateDB will use the ToIdx==0 to define the AuxToIdx
validTxs = append(validTxs, l2Tx)
return validTxs, l1CoordinatorTxs, positionL1, nil
}
// if idx don't exist for EthAddr&BJJ,
// coordinator can create a new account without
// L1Authorization, as ToEthAddr==0xff
// create L1CoordinatorTx for the accountCreation
l1CoordinatorTx := common.L1Tx{
Position: positionL1,
UserOrigin: false,
FromEthAddr: l2Tx.ToEthAddr,
FromBJJ: l2Tx.ToBJJ,
TokenID: l2Tx.TokenID,
DepositAmount: big.NewInt(0),
Type: common.TxTypeCreateAccountDeposit,
}
positionL1++
l1CoordinatorTxs = append(l1CoordinatorTxs, l1CoordinatorTx)
} }
return l1Txs, l1CoordinatorTxs, l2Txs, nil
return validTxs, l1CoordinatorTxs, positionL1, nil
} }
func checkAlreadyPendingToCreate(l1CoordinatorTxs []common.L1Tx, addr ethCommon.Address, bjj *babyjub.PublicKey) bool {
func checkAlreadyPendingToCreate(l1CoordinatorTxs []common.L1Tx,
addr ethCommon.Address, bjj *babyjub.PublicKey) bool {
for i := 0; i < len(l1CoordinatorTxs); i++ { for i := 0; i < len(l1CoordinatorTxs); i++ {
if bytes.Equal(l1CoordinatorTxs[i].FromEthAddr.Bytes(), addr.Bytes()) { if bytes.Equal(l1CoordinatorTxs[i].FromEthAddr.Bytes(), addr.Bytes()) {
if bjj == nil { if bjj == nil {
@ -255,7 +373,7 @@ func checkAlreadyPendingToCreate(l1CoordinatorTxs []common.L1Tx, addr ethCommon.
} }
// getL2Profitable returns the profitable selection of L2Txssorted by Nonce // getL2Profitable returns the profitable selection of L2Txssorted by Nonce
func (txsel *TxSelector) getL2Profitable(txs txs, max uint64) txs {
func (txsel *TxSelector) getL2Profitable(txs txs, max uint32) txs {
sort.Sort(txs) sort.Sort(txs)
if len(txs) < int(max) { if len(txs) < int(max) {
return txs return txs

+ 54
- 18
txselector/txselector_test.go

@ -20,54 +20,84 @@ import (
"github.com/stretchr/testify/require" "github.com/stretchr/testify/require"
) )
func initTest(t *testing.T, testSet string, maxL1UserTxs, maxL1OperatorTxs, maxTxs uint64) *TxSelector {
func initTest(t *testing.T, testSet string) *TxSelector {
pass := os.Getenv("POSTGRES_PASS") pass := os.Getenv("POSTGRES_PASS")
db, err := dbUtils.InitSQLDB(5432, "localhost", "hermez", pass, "hermez") db, err := dbUtils.InitSQLDB(5432, "localhost", "hermez", pass, "hermez")
require.Nil(t, err)
require.NoError(t, err)
l2DB := l2db.NewL2DB(db, 10, 100, 24*time.Hour) l2DB := l2db.NewL2DB(db, 10, 100, 24*time.Hour)
dir, err := ioutil.TempDir("", "tmpdb") dir, err := ioutil.TempDir("", "tmpdb")
require.Nil(t, err)
require.NoError(t, err)
defer assert.Nil(t, os.RemoveAll(dir)) defer assert.Nil(t, os.RemoveAll(dir))
sdb, err := statedb.NewStateDB(dir, statedb.TypeTxSelector, 0) sdb, err := statedb.NewStateDB(dir, statedb.TypeTxSelector, 0)
require.Nil(t, err)
require.NoError(t, err)
txselDir, err := ioutil.TempDir("", "tmpTxSelDB") txselDir, err := ioutil.TempDir("", "tmpTxSelDB")
require.Nil(t, err)
require.NoError(t, err)
defer assert.Nil(t, os.RemoveAll(dir)) defer assert.Nil(t, os.RemoveAll(dir))
txsel, err := NewTxSelector(txselDir, sdb, l2DB, maxL1UserTxs, maxL1OperatorTxs, maxTxs)
require.Nil(t, err)
coordAccount := &CoordAccount{ // TODO TMP
Addr: ethCommon.HexToAddress("0xc58d29fA6e86E4FAe04DDcEd660d45BCf3Cb2370"),
BJJ: nil,
AccountCreationAuth: nil,
}
txsel, err := NewTxSelector(coordAccount, txselDir, sdb, l2DB)
require.NoError(t, err)
return txsel return txsel
} }
func addL2Txs(t *testing.T, txsel *TxSelector, poolL2Txs []common.PoolL2Tx) { func addL2Txs(t *testing.T, txsel *TxSelector, poolL2Txs []common.PoolL2Tx) {
for i := 0; i < len(poolL2Txs); i++ { for i := 0; i < len(poolL2Txs); i++ {
err := txsel.l2db.AddTxTest(&poolL2Txs[i]) err := txsel.l2db.AddTxTest(&poolL2Txs[i])
require.Nil(t, err)
require.NoError(t, err)
} }
} }
func addTokens(t *testing.T, tokens []common.Token, db *sqlx.DB) { func addTokens(t *testing.T, tokens []common.Token, db *sqlx.DB) {
hdb := historydb.NewHistoryDB(db) hdb := historydb.NewHistoryDB(db)
test.WipeDB(hdb.DB()) test.WipeDB(hdb.DB())
assert.Nil(t, hdb.AddBlock(&common.Block{
assert.NoError(t, hdb.AddBlock(&common.Block{
Num: 1, Num: 1,
})) }))
assert.Nil(t, hdb.AddTokens(tokens))
assert.NoError(t, hdb.AddTokens(tokens))
}
func TestCoordIdxsDB(t *testing.T) {
txsel := initTest(t, til.SetPool0)
test.WipeDB(txsel.l2db.DB())
coordIdxs := make(map[common.TokenID]common.Idx)
coordIdxs[common.TokenID(0)] = common.Idx(256)
coordIdxs[common.TokenID(1)] = common.Idx(257)
coordIdxs[common.TokenID(2)] = common.Idx(258)
err := txsel.AddCoordIdxs(coordIdxs)
assert.NoError(t, err)
r, err := txsel.GetCoordIdxs()
assert.NoError(t, err)
assert.Equal(t, coordIdxs, r)
} }
func TestGetL2TxSelection(t *testing.T) { func TestGetL2TxSelection(t *testing.T) {
txsel := initTest(t, til.SetPool0, 5, 5, 10)
txsel := initTest(t, til.SetPool0)
test.WipeDB(txsel.l2db.DB()) test.WipeDB(txsel.l2db.DB())
tc := til.NewContext(common.RollupConstMaxL1UserTx) tc := til.NewContext(common.RollupConstMaxL1UserTx)
// generate test transactions // generate test transactions
blocks, err := tc.GenerateBlocks(til.SetBlockchain0) blocks, err := tc.GenerateBlocks(til.SetBlockchain0)
assert.Nil(t, err)
assert.NoError(t, err)
// poolL2Txs, err := tc.GeneratePoolL2Txs(til.SetPool0) // poolL2Txs, err := tc.GeneratePoolL2Txs(til.SetPool0)
// assert.Nil(t, err) // assert.Nil(t, err)
coordIdxs := []common.Idx{256, 257, 258, 259}
coordIdxs := make(map[common.TokenID]common.Idx)
coordIdxs[common.TokenID(0)] = common.Idx(256)
coordIdxs[common.TokenID(1)] = common.Idx(257)
coordIdxs[common.TokenID(2)] = common.Idx(258)
coordIdxs[common.TokenID(3)] = common.Idx(259)
err = txsel.AddCoordIdxs(coordIdxs)
assert.NoError(t, err)
// add tokens to HistoryDB to avoid breaking FK constrains // add tokens to HistoryDB to avoid breaking FK constrains
var tokens []common.Token var tokens []common.Token
@ -89,21 +119,27 @@ func TestGetL2TxSelection(t *testing.T) {
MaxTx: 512, MaxTx: 512,
MaxL1Tx: 64, MaxL1Tx: 64,
} }
selectionConfig := &SelectionConfig{
MaxL1UserTxs: 32,
MaxL1CoordinatorTxs: 32,
ProcessTxsConfig: ptc,
}
// Process the 1st batch, which contains the L1CoordinatorTxs necessary // Process the 1st batch, which contains the L1CoordinatorTxs necessary
// to create the Coordinator accounts to receive the fees // to create the Coordinator accounts to receive the fees
_, err = txsel.localAccountsDB.ProcessTxs(ptc, nil, nil, blocks[0].Rollup.Batches[0].L1CoordinatorTxs, nil) _, err = txsel.localAccountsDB.ProcessTxs(ptc, nil, nil, blocks[0].Rollup.Batches[0].L1CoordinatorTxs, nil)
require.Nil(t, err)
require.NoError(t, err)
// add the 1st batch of transactions to the TxSelector // add the 1st batch of transactions to the TxSelector
addL2Txs(t, txsel, common.L2TxsToPoolL2Txs(blocks[0].Rollup.Batches[0].L2Txs)) addL2Txs(t, txsel, common.L2TxsToPoolL2Txs(blocks[0].Rollup.Batches[0].L2Txs))
l1CoordTxs, l2Txs, err := txsel.GetL2TxSelection(coordIdxs, 0)
assert.Nil(t, err)
_, l1CoordTxs, l2Txs, err := txsel.GetL2TxSelection(selectionConfig, 0)
assert.NoError(t, err)
assert.Equal(t, 0, len(l2Txs)) assert.Equal(t, 0, len(l2Txs))
assert.Equal(t, 0, len(l1CoordTxs)) assert.Equal(t, 0, len(l1CoordTxs))
_, _, _, err = txsel.GetL1L2TxSelection(coordIdxs, 0, blocks[0].Rollup.L1UserTxs)
assert.Nil(t, err)
_, _, _, _, err = txsel.GetL1L2TxSelection(selectionConfig, 0, blocks[0].Rollup.L1UserTxs)
assert.NoError(t, err)
// TODO once L2DB is updated to return error in case that AddTxTest // TODO once L2DB is updated to return error in case that AddTxTest
// fails, and the Til is updated, update this test, checking that the // fails, and the Til is updated, update this test, checking that the

Loading…
Cancel
Save