feat(BB): Setting up the file directory (#156)

This commit is contained in:
David Terpay 2023-06-01 15:56:58 -04:00 committed by GitHub
parent c9bc56f449
commit 2e3883adc2
No known key found for this signature in database
GPG Key ID: 4AEE18F83AFDEB23
11 changed files with 837 additions and 353 deletions

View File

@ -1,17 +1,18 @@
package blockbuster
import (
"github.com/cometbft/cometbft/libs/log"
sdk "github.com/cosmos/cosmos-sdk/types"
sdkmempool "github.com/cosmos/cosmos-sdk/types/mempool"
"github.com/skip-mev/pob/mempool"
)
type (
// LaneConfig defines the configuration for a lane.
LaneConfig[C comparable] struct {
// XXX: For now we use the PriorityNonceMempoolConfig as the base config,
// which should be removed once Cosmos SDK v0.48 is released.
mempool.PriorityNonceMempoolConfig[C]
// BaseLaneConfig defines the basic functionality needed for a lane.
BaseLaneConfig struct {
Logger log.Logger
TxEncoder sdk.TxEncoder
TxDecoder sdk.TxDecoder
AnteHandler sdk.AnteHandler
}
// Lane defines an interface used for block construction
@ -40,3 +41,13 @@ type (
ProcessLane(ctx sdk.Context, proposalTxs [][]byte) error
}
)
// NewLaneConfig returns a new LaneConfig. This will be embedded in a lane.
func NewBaseLaneConfig(logger log.Logger, txEncoder sdk.TxEncoder, txDecoder sdk.TxDecoder, anteHandler sdk.AnteHandler) BaseLaneConfig {
return BaseLaneConfig{
Logger: logger,
TxEncoder: txEncoder,
TxDecoder: txDecoder,
AnteHandler: anteHandler,
}
}

View File

@ -0,0 +1,217 @@
package auction
import (
"bytes"
"errors"
"fmt"
sdk "github.com/cosmos/cosmos-sdk/types"
"github.com/skip-mev/pob/blockbuster"
)
// PrepareLane will attempt to select the highest bid transaction that is valid
// and whose bundled transactions are valid and include them in the proposal. It
// will return an empty partial proposal if no valid bids are found.
func (l *TOBLane) PrepareLane(ctx sdk.Context, maxTxBytes int64, selectedTxs map[string][]byte) ([][]byte, error) {
var tmpSelectedTxs [][]byte
bidTxIterator := l.Select(ctx, nil)
txsToRemove := make(map[sdk.Tx]struct{}, 0)
// Attempt to select the highest bid transaction that is valid and whose
// bundled transactions are valid.
selectBidTxLoop:
for ; bidTxIterator != nil; bidTxIterator = bidTxIterator.Next() {
cacheCtx, write := ctx.CacheContext()
tmpBidTx := bidTxIterator.Tx()
// if the transaction is already in the (partial) block proposal, we skip it.
txHash, err := blockbuster.GetTxHashStr(l.cfg.TxEncoder, tmpBidTx)
if err != nil {
return nil, fmt.Errorf("failed to get bid tx hash: %w", err)
}
if _, ok := selectedTxs[txHash]; ok {
continue selectBidTxLoop
}
bidTxBz, err := l.cfg.TxEncoder(tmpBidTx)
if err != nil {
txsToRemove[tmpBidTx] = struct{}{}
continue selectBidTxLoop
}
bidTxSize := int64(len(bidTxBz))
if bidTxSize <= maxTxBytes {
// Verify the bid transaction and all of its bundled transactions.
if err := l.VerifyTx(cacheCtx, tmpBidTx); err != nil {
txsToRemove[tmpBidTx] = struct{}{}
continue selectBidTxLoop
}
// Build the partial proposal by selecting the bid transaction and all of
// its bundled transactions.
bidInfo, err := l.GetAuctionBidInfo(tmpBidTx)
if bidInfo == nil || err != nil {
// Some transactions in the bundle may be malformed or invalid, so we
// remove the bid transaction and try the next top bid.
txsToRemove[tmpBidTx] = struct{}{}
continue selectBidTxLoop
}
// store the bytes of each ref tx as sdk.Tx bytes in order to build a valid proposal
bundledTxBz := make([][]byte, len(bidInfo.Transactions))
for index, rawRefTx := range bidInfo.Transactions {
sdkTx, err := l.WrapBundleTransaction(rawRefTx)
if err != nil {
txsToRemove[tmpBidTx] = struct{}{}
continue selectBidTxLoop
}
sdkTxBz, err := l.cfg.TxEncoder(sdkTx)
if err != nil {
txsToRemove[tmpBidTx] = struct{}{}
continue selectBidTxLoop
}
bundleTxBz := make([]byte, len(sdkTxBz))
copy(bundleTxBz, sdkTxBz)
bundledTxBz[index] = sdkTxBz
}
// At this point, both the bid transaction itself and all the bundled
// transactions are valid. So we select the bid transaction along with
// all the bundled transactions. We also mark these transactions as seen and
// update the total size selected thus far.
tmpSelectedTxs = append(tmpSelectedTxs, bidTxBz)
tmpSelectedTxs = append(tmpSelectedTxs, bundledTxBz...)
// Write the cache context to the original context when we know we have a
// valid top of block bundle.
write()
break selectBidTxLoop
}
txsToRemove[tmpBidTx] = struct{}{}
l.cfg.Logger.Info(
"failed to select auction bid tx; tx size is too large",
"tx_size", bidTxSize,
"max_size", maxTxBytes,
)
}
// remove all invalid transactions from the mempool
for tx := range txsToRemove {
if err := l.Remove(tx); err != nil {
return nil, err
}
}
return tmpSelectedTxs, nil
}
// ProcessLane will ensure that block proposals that include transactions from
// the top-of-block auction lane are valid. It will return an error if the
// block proposal is invalid. The block proposal is invalid if it does not
// respect the ordering of transactions in the bid transaction or if the bid/bundled
// transactions are invalid.
func (l *TOBLane) ProcessLane(ctx sdk.Context, proposalTxs [][]byte) error {
for index, txBz := range proposalTxs {
tx, err := l.cfg.TxDecoder(txBz)
if err != nil {
return err
}
bidInfo, err := l.GetAuctionBidInfo(tx)
if err != nil {
return err
}
// If the transaction is an auction bid, then we need to ensure that it is
// the first transaction in the block proposal and that the order of
// transactions in the block proposal follows the order of transactions in
// the bid.
if bidInfo != nil {
if index != 0 {
return errors.New("auction bid must be the first transaction in the block proposal")
}
bundledTransactions := bidInfo.Transactions
if len(proposalTxs) < len(bundledTransactions)+1 {
return errors.New("block proposal does not contain enough transactions to match the bundled transactions in the auction bid")
}
for i, refTxRaw := range bundledTransactions {
// Wrap and then encode the bundled transaction to ensure that the underlying
// reference transaction can be processed as an sdk.Tx.
wrappedTx, err := l.WrapBundleTransaction(refTxRaw)
if err != nil {
return err
}
refTxBz, err := l.cfg.TxEncoder(wrappedTx)
if err != nil {
return err
}
if !bytes.Equal(refTxBz, proposalTxs[i+1]) {
return errors.New("block proposal does not match the bundled transactions in the auction bid")
}
}
// Verify the bid transaction.
if err = l.VerifyTx(ctx, tx); err != nil {
return err
}
}
}
return nil
}
// VerifyTx will verify that the bid transaction and all of its bundled
// transactions are valid. It will return an error if any of the transactions
// are invalid.
func (l *TOBLane) VerifyTx(ctx sdk.Context, bidTx sdk.Tx) error {
bidInfo, err := l.GetAuctionBidInfo(bidTx)
if err != nil {
return fmt.Errorf("failed to get auction bid info: %w", err)
}
// verify the top-level bid transaction
ctx, err = l.verifyTx(ctx, bidTx)
if err != nil {
return fmt.Errorf("invalid bid tx; failed to execute ante handler: %w", err)
}
// verify all of the bundled transactions
for _, tx := range bidInfo.Transactions {
bundledTx, err := l.WrapBundleTransaction(tx)
if err != nil {
return fmt.Errorf("invalid bid tx; failed to decode bundled tx: %w", err)
}
// bid txs cannot be included in bundled txs
bidInfo, _ := l.GetAuctionBidInfo(bundledTx)
if bidInfo != nil {
return fmt.Errorf("invalid bid tx; bundled tx cannot be a bid tx")
}
if ctx, err = l.verifyTx(ctx, bundledTx); err != nil {
return fmt.Errorf("invalid bid tx; failed to execute bundled transaction: %w", err)
}
}
return nil
}
// verifyTx will execute the ante handler on the transaction and return the
// resulting context and error.
func (l *TOBLane) verifyTx(ctx sdk.Context, tx sdk.Tx) (sdk.Context, error) {
if l.cfg.AnteHandler != nil {
newCtx, err := l.cfg.AnteHandler(ctx, tx, false)
return newCtx, err
}
return ctx, nil
}

View File

@ -0,0 +1,128 @@
package auction
import (
"fmt"
sdk "github.com/cosmos/cosmos-sdk/types"
"github.com/cosmos/cosmos-sdk/x/auth/signing"
)
type (
// BidInfo defines the information about a bid to the auction house.
BidInfo struct {
Bidder sdk.AccAddress
Bid sdk.Coin
Transactions [][]byte
Timeout uint64
Signers []map[string]struct{}
}
// Factory defines the interface for processing auction transactions. It is
// a wrapper around all of the functionality that each application chain must implement
// in order for auction processing to work.
Factory interface {
// WrapBundleTransaction defines a function that wraps a bundle transaction into a sdk.Tx. Since
// this is a potentially expensive operation, we allow each application chain to define how
// they want to wrap the transaction such that it is only called when necessary (i.e. when the
// transaction is being considered in the proposal handlers).
WrapBundleTransaction(tx []byte) (sdk.Tx, error)
// GetAuctionBidInfo defines a function that returns the bid info from an auction transaction.
GetAuctionBidInfo(tx sdk.Tx) (*BidInfo, error)
}
// DefaultAuctionFactory defines a default implmentation for the auction factory interface for processing auction transactions.
DefaultAuctionFactory struct {
txDecoder sdk.TxDecoder
}
// TxWithTimeoutHeight is used to extract timeouts from sdk.Tx transactions. In the case where,
// timeouts are explicitly set on the sdk.Tx, we can use this interface to extract the timeout.
TxWithTimeoutHeight interface {
sdk.Tx
GetTimeoutHeight() uint64
}
)
var _ Factory = (*DefaultAuctionFactory)(nil)
// NewDefaultAuctionFactory returns a default auction factory interface implementation.
func NewDefaultAuctionFactory(txDecoder sdk.TxDecoder) Factory {
return &DefaultAuctionFactory{
txDecoder: txDecoder,
}
}
// WrapBundleTransaction defines a default function that wraps a transaction
// that is included in the bundle into a sdk.Tx. In the default case, the transaction
// that is included in the bundle will be the raw bytes of an sdk.Tx so we can just
// decode it.
func (config *DefaultAuctionFactory) WrapBundleTransaction(tx []byte) (sdk.Tx, error) {
return config.txDecoder(tx)
}
// GetAuctionBidInfo defines a default function that returns the auction bid info from
// an auction transaction. In the default case, the auction bid info is stored in the
// MsgAuctionBid message.
func (config *DefaultAuctionFactory) GetAuctionBidInfo(tx sdk.Tx) (*BidInfo, error) {
msg, err := GetMsgAuctionBidFromTx(tx)
if err != nil {
return nil, err
}
if msg == nil {
return nil, nil
}
bidder, err := sdk.AccAddressFromBech32(msg.Bidder)
if err != nil {
return nil, fmt.Errorf("invalid bidder address (%s): %w", msg.Bidder, err)
}
timeoutTx, ok := tx.(TxWithTimeoutHeight)
if !ok {
return nil, fmt.Errorf("cannot extract timeout; transaction does not implement TxWithTimeoutHeight")
}
signers, err := config.getBundleSigners(msg.Transactions)
if err != nil {
return nil, err
}
return &BidInfo{
Bid: msg.Bid,
Bidder: bidder,
Transactions: msg.Transactions,
Timeout: timeoutTx.GetTimeoutHeight(),
Signers: signers,
}, nil
}
// getBundleSigners defines a default function that returns the signers of all transactions in
// a bundle. In the default case, each bundle transaction will be an sdk.Tx and the
// signers are the signers of each sdk.Msg in the transaction.
func (config *DefaultAuctionFactory) getBundleSigners(bundle [][]byte) ([]map[string]struct{}, error) {
signers := make([]map[string]struct{}, 0)
for _, tx := range bundle {
sdkTx, err := config.txDecoder(tx)
if err != nil {
return nil, err
}
sigTx, ok := sdkTx.(signing.SigVerifiableTx)
if !ok {
return nil, fmt.Errorf("transaction is not valid")
}
txSigners := make(map[string]struct{})
for _, signer := range sigTx.GetSigners() {
txSigners[signer.String()] = struct{}{}
}
signers = append(signers, txSigners)
}
return signers, nil
}

View File

@ -0,0 +1,63 @@
package auction
import (
"github.com/cometbft/cometbft/libs/log"
sdk "github.com/cosmos/cosmos-sdk/types"
"github.com/skip-mev/pob/blockbuster"
)
const (
// LaneName defines the name of the top-of-block auction lane.
LaneName = "tob"
)
var _ blockbuster.Lane = (*TOBLane)(nil)
// TOBLane defines a top-of-block auction lane. The top of block auction lane
// hosts transactions that want to bid for inclusion at the top of the next block.
// The top of block auction lane stores bid transactions that are sorted by
// their bid price. The highest valid bid transaction is selected for inclusion in the
// next block. The bundled transactions of the selected bid transaction are also
// included in the next block.
type TOBLane struct {
// Mempool defines the mempool for the lane.
Mempool
// LaneConfig defines the base lane configuration.
cfg blockbuster.BaseLaneConfig
// Factory defines the API/functionality which is responsible for determining
// if a transaction is a bid transaction and how to extract relevant
// information from the transaction (bid, timeout, bidder, etc.).
Factory
}
// NewTOBLane returns a new TOB lane.
func NewTOBLane(
logger log.Logger,
txDecoder sdk.TxDecoder,
txEncoder sdk.TxEncoder,
maxTx int,
anteHandler sdk.AnteHandler,
af Factory,
) *TOBLane {
logger = logger.With("lane", LaneName)
return &TOBLane{
Mempool: NewMempool(txEncoder, maxTx, af),
cfg: blockbuster.NewBaseLaneConfig(logger, txEncoder, txDecoder, anteHandler),
Factory: af,
}
}
// Match returns true if the transaction is a bid transaction. This is determined
// by the AuctionFactory.
func (l *TOBLane) Match(tx sdk.Tx) bool {
bidInfo, err := l.GetAuctionBidInfo(tx)
return bidInfo != nil && err == nil
}
// Name returns the name of the lane.
func (l *TOBLane) Name() string {
return LaneName
}

View File

@ -0,0 +1,190 @@
package auction
import (
"context"
"errors"
"fmt"
sdk "github.com/cosmos/cosmos-sdk/types"
sdkmempool "github.com/cosmos/cosmos-sdk/types/mempool"
"github.com/skip-mev/pob/blockbuster"
"github.com/skip-mev/pob/mempool"
)
var _ Mempool = (*TOBMempool)(nil)
type (
// Mempool defines the interface of the auction mempool.
Mempool interface {
sdkmempool.Mempool
// GetTopAuctionTx returns the highest bidding transaction in the auction mempool.
GetTopAuctionTx(ctx context.Context) sdk.Tx
// Contains returns true if the transaction is contained in the mempool.
Contains(tx sdk.Tx) (bool, error)
}
// TOBMempool defines an auction mempool. It can be seen as an extension of
// an SDK PriorityNonceMempool, i.e. a mempool that supports <sender, nonce>
// two-dimensional priority ordering, with the additional support of prioritizing
// and indexing auction bids.
TOBMempool struct {
// index defines an index of auction bids.
index sdkmempool.Mempool
// txEncoder defines the sdk.Tx encoder that allows us to encode transactions
// to bytes.
txEncoder sdk.TxEncoder
// txIndex is a map of all transactions in the mempool. It is used
// to quickly check if a transaction is already in the mempool.
txIndex map[string]struct{}
// Factory implements the functionality required to process auction transactions.
Factory
}
)
// TxPriority returns a TxPriority over auction bid transactions only. It
// is to be used in the auction index only.
func TxPriority(config Factory) mempool.TxPriority[string] {
return mempool.TxPriority[string]{
GetTxPriority: func(goCtx context.Context, tx sdk.Tx) string {
bidInfo, err := config.GetAuctionBidInfo(tx)
if err != nil {
panic(err)
}
return bidInfo.Bid.String()
},
Compare: func(a, b string) int {
aCoins, _ := sdk.ParseCoinsNormalized(a)
bCoins, _ := sdk.ParseCoinsNormalized(b)
switch {
case aCoins == nil && bCoins == nil:
return 0
case aCoins == nil:
return -1
case bCoins == nil:
return 1
default:
switch {
case aCoins.IsAllGT(bCoins):
return 1
case aCoins.IsAllLT(bCoins):
return -1
default:
return 0
}
}
},
MinValue: "",
}
}
// NewMempool returns a new auction mempool.
func NewMempool(txEncoder sdk.TxEncoder, maxTx int, config Factory) *TOBMempool {
return &TOBMempool{
index: mempool.NewPriorityMempool(
mempool.PriorityNonceMempoolConfig[string]{
TxPriority: TxPriority(config),
MaxTx: maxTx,
},
),
txEncoder: txEncoder,
txIndex: make(map[string]struct{}),
Factory: config,
}
}
// Insert inserts a transaction into the auction mempool.
func (am *TOBMempool) Insert(ctx context.Context, tx sdk.Tx) error {
bidInfo, err := am.GetAuctionBidInfo(tx)
if err != nil {
return err
}
// This mempool only supports auction bid transactions.
if bidInfo == nil {
return fmt.Errorf("invalid transaction type")
}
if err := am.index.Insert(ctx, tx); err != nil {
return fmt.Errorf("failed to insert tx into auction index: %w", err)
}
txHashStr, err := blockbuster.GetTxHashStr(am.txEncoder, tx)
if err != nil {
return err
}
am.txIndex[txHashStr] = struct{}{}
return nil
}
// Remove removes a transaction from the mempool based.
func (am *TOBMempool) Remove(tx sdk.Tx) error {
bidInfo, err := am.GetAuctionBidInfo(tx)
if err != nil {
return err
}
// This mempool only supports auction bid transactions.
if bidInfo == nil {
return fmt.Errorf("invalid transaction type")
}
am.removeTx(am.index, tx)
return nil
}
// GetTopAuctionTx returns the highest bidding transaction in the auction mempool.
func (am *TOBMempool) GetTopAuctionTx(ctx context.Context) sdk.Tx {
iterator := am.index.Select(ctx, nil)
if iterator == nil {
return nil
}
return iterator.Tx()
}
func (am *TOBMempool) Select(ctx context.Context, txs [][]byte) sdkmempool.Iterator {
return am.index.Select(ctx, txs)
}
func (am *TOBMempool) CountTx() int {
return am.index.CountTx()
}
// Contains returns true if the transaction is contained in the mempool.
func (am *TOBMempool) Contains(tx sdk.Tx) (bool, error) {
txHashStr, err := blockbuster.GetTxHashStr(am.txEncoder, tx)
if err != nil {
return false, fmt.Errorf("failed to get tx hash string: %w", err)
}
_, ok := am.txIndex[txHashStr]
return ok, nil
}
func (am *TOBMempool) removeTx(mp sdkmempool.Mempool, tx sdk.Tx) {
if err := mp.Remove(tx); err != nil && !errors.Is(err, sdkmempool.ErrTxNotFound) {
panic(fmt.Errorf("failed to remove invalid transaction from the mempool: %w", err))
}
txHashStr, err := blockbuster.GetTxHashStr(am.txEncoder, tx)
if err != nil {
panic(fmt.Errorf("failed to get tx hash string: %w", err))
}
delete(am.txIndex, txHashStr)
}

View File

@ -0,0 +1,35 @@
package auction
import (
"errors"
sdk "github.com/cosmos/cosmos-sdk/types"
buildertypes "github.com/skip-mev/pob/x/builder/types"
)
// GetMsgAuctionBidFromTx attempts to retrieve a MsgAuctionBid from an sdk.Tx if
// one exists. If a MsgAuctionBid does exist and other messages are also present,
// an error is returned. If no MsgAuctionBid is present, <nil, nil> is returned.
func GetMsgAuctionBidFromTx(tx sdk.Tx) (*buildertypes.MsgAuctionBid, error) {
auctionBidMsgs := make([]*buildertypes.MsgAuctionBid, 0)
for _, msg := range tx.GetMsgs() {
t, ok := msg.(*buildertypes.MsgAuctionBid)
if ok {
auctionBidMsgs = append(auctionBidMsgs, t)
}
}
switch {
case len(auctionBidMsgs) == 0:
// a normal transaction without a MsgAuctionBid message
return nil, nil
case len(auctionBidMsgs) == 1 && len(tx.GetMsgs()) == 1:
// a single MsgAuctionBid message transaction
return auctionBidMsgs[0], nil
default:
// a transaction with at at least one MsgAuctionBid message
return nil, errors.New("invalid MsgAuctionBid transaction")
}
}

View File

@ -0,0 +1,15 @@
package base
import sdk "github.com/cosmos/cosmos-sdk/types"
func (l *DefaultLane) PrepareLane(sdk.Context, int64, map[string][]byte) ([][]byte, error) {
panic("implement me")
}
func (l *DefaultLane) ProcessLane(sdk.Context, [][]byte) error {
panic("implement me")
}
func (l *DefaultLane) VerifyTx(sdk.Context, sdk.Tx) error {
panic("implement me")
}

View File

@ -0,0 +1,44 @@
package base
import (
"github.com/cometbft/cometbft/libs/log"
sdk "github.com/cosmos/cosmos-sdk/types"
"github.com/skip-mev/pob/blockbuster"
)
const (
// LaneName defines the name of the default lane.
LaneName = "default"
)
var _ blockbuster.Lane = (*DefaultLane)(nil)
// DefaultLane defines a default lane implementation. It contains a priority-nonce
// index along with core lane functionality.
type DefaultLane struct {
// Mempool defines the mempool for the lane.
Mempool
// LaneConfig defines the base lane configuration.
cfg blockbuster.BaseLaneConfig
}
// NewDefaultLane returns a new default lane.
func NewDefaultLane(logger log.Logger, txDecoder sdk.TxDecoder, txEncoder sdk.TxEncoder, anteHandler sdk.AnteHandler) *DefaultLane {
return &DefaultLane{
Mempool: NewDefaultMempool(txEncoder),
cfg: blockbuster.NewBaseLaneConfig(logger, txEncoder, txDecoder, anteHandler),
}
}
// Match returns true if the transaction belongs to this lane. Since
// this is the default lane, it always returns true. This means that
// any transaction can be included in this lane.
func (l *DefaultLane) Match(sdk.Tx) bool {
return true
}
// Name returns the name of the lane.
func (l *DefaultLane) Name() string {
return LaneName
}

View File

@ -0,0 +1,106 @@
package base
import (
"context"
"errors"
"fmt"
sdk "github.com/cosmos/cosmos-sdk/types"
sdkmempool "github.com/cosmos/cosmos-sdk/types/mempool"
"github.com/skip-mev/pob/blockbuster"
"github.com/skip-mev/pob/mempool"
)
var _ sdkmempool.Mempool = (*DefaultMempool)(nil)
type (
// Mempool defines the interface of the default mempool.
Mempool interface {
sdkmempool.Mempool
// Contains returns true if the transaction is contained in the mempool.
Contains(tx sdk.Tx) (bool, error)
}
// DefaultMempool defines the most basic mempool. It can be seen as an extension of
// an SDK PriorityNonceMempool, i.e. a mempool that supports <sender, nonce>
// two-dimensional priority ordering, with the additional support of prioritizing
// and indexing auction bids.
DefaultMempool struct {
// index defines an index transactions.
index sdkmempool.Mempool
// txEncoder defines the sdk.Tx encoder that allows us to encode transactions
// to bytes.
txEncoder sdk.TxEncoder
// txIndex is a map of all transactions in the mempool. It is used
// to quickly check if a transaction is already in the mempool.
txIndex map[string]struct{}
}
)
func NewDefaultMempool(txEncoder sdk.TxEncoder) *DefaultMempool {
return &DefaultMempool{
index: mempool.NewPriorityMempool(
mempool.DefaultPriorityNonceMempoolConfig(),
),
txEncoder: txEncoder,
txIndex: make(map[string]struct{}),
}
}
// Insert inserts a transaction into the mempool based on the transaction type (normal or auction).
func (am *DefaultMempool) Insert(ctx context.Context, tx sdk.Tx) error {
if err := am.index.Insert(ctx, tx); err != nil {
return fmt.Errorf("failed to insert tx into auction index: %w", err)
}
txHashStr, err := blockbuster.GetTxHashStr(am.txEncoder, tx)
if err != nil {
return err
}
am.txIndex[txHashStr] = struct{}{}
return nil
}
// Remove removes a transaction from the mempool based on the transaction type (normal or auction).
func (am *DefaultMempool) Remove(tx sdk.Tx) error {
am.removeTx(am.index, tx)
return nil
}
func (am *DefaultMempool) Select(ctx context.Context, txs [][]byte) sdkmempool.Iterator {
return am.index.Select(ctx, txs)
}
func (am *DefaultMempool) CountTx() int {
return am.index.CountTx()
}
// Contains returns true if the transaction is contained in the mempool.
func (am *DefaultMempool) Contains(tx sdk.Tx) (bool, error) {
txHashStr, err := blockbuster.GetTxHashStr(am.txEncoder, tx)
if err != nil {
return false, fmt.Errorf("failed to get tx hash string: %w", err)
}
_, ok := am.txIndex[txHashStr]
return ok, nil
}
func (am *DefaultMempool) removeTx(mp sdkmempool.Mempool, tx sdk.Tx) {
err := mp.Remove(tx)
if err != nil && !errors.Is(err, sdkmempool.ErrTxNotFound) {
panic(fmt.Errorf("failed to remove invalid transaction from the mempool: %w", err))
}
txHashStr, err := blockbuster.GetTxHashStr(am.txEncoder, tx)
if err != nil {
panic(fmt.Errorf("failed to get tx hash string: %w", err))
}
delete(am.txIndex, txHashStr)
}

View File

@ -1,347 +0,0 @@
package blockbuster
import (
"bytes"
"context"
"crypto/sha256"
"encoding/hex"
"errors"
"fmt"
"github.com/cometbft/cometbft/libs/log"
sdk "github.com/cosmos/cosmos-sdk/types"
sdkmempool "github.com/cosmos/cosmos-sdk/types/mempool"
"github.com/skip-mev/pob/mempool"
)
const (
// LaneNameTOB defines the name of the top-of-block auction lane.
LaneNameTOB = "tob"
)
type (
// AuctionBidInfo defines the information about a bid to the auction house.
AuctionBidInfo struct {
Bidder sdk.AccAddress
Bid sdk.Coin
Transactions [][]byte
Timeout uint64
Signers []map[string]struct{}
}
// AuctionFactory defines the interface for processing auction transactions. It is
// a wrapper around all of the functionality that each application chain must implement
// in order for auction processing to work.
AuctionFactory interface {
// WrapBundleTransaction defines a function that wraps a bundle transaction into a sdk.Tx. Since
// this is a potentially expensive operation, we allow each application chain to define how
// they want to wrap the transaction such that it is only called when necessary (i.e. when the
// transaction is being considered in the proposal handlers).
WrapBundleTransaction(tx []byte) (sdk.Tx, error)
// GetAuctionBidInfo defines a function that returns the bid info from an auction transaction.
GetAuctionBidInfo(tx sdk.Tx) (*AuctionBidInfo, error)
}
)
var _ Lane = (*TOBLane)(nil)
type TOBLane struct {
logger log.Logger
index sdkmempool.Mempool
af AuctionFactory
txEncoder sdk.TxEncoder
txDecoder sdk.TxDecoder
anteHandler sdk.AnteHandler
// txIndex is a map of all transactions in the mempool. It is used
// to quickly check if a transaction is already in the mempool.
txIndex map[string]struct{}
}
func NewTOBLane(logger log.Logger, txDecoder sdk.TxDecoder, txEncoder sdk.TxEncoder, maxTx int, af AuctionFactory, anteHandler sdk.AnteHandler) *TOBLane {
return &TOBLane{
logger: logger,
index: mempool.NewPriorityMempool(
mempool.PriorityNonceMempoolConfig[int64]{
TxPriority: mempool.NewDefaultTxPriority(),
MaxTx: maxTx,
},
),
af: af,
txEncoder: txEncoder,
txDecoder: txDecoder,
anteHandler: anteHandler,
txIndex: make(map[string]struct{}),
}
}
func (l *TOBLane) Name() string {
return LaneNameTOB
}
func (l *TOBLane) Match(tx sdk.Tx) bool {
bidInfo, err := l.af.GetAuctionBidInfo(tx)
return bidInfo != nil && err == nil
}
func (l *TOBLane) Contains(tx sdk.Tx) (bool, error) {
txHashStr, err := l.getTxHashStr(tx)
if err != nil {
return false, fmt.Errorf("failed to get tx hash string: %w", err)
}
_, ok := l.txIndex[txHashStr]
return ok, nil
}
func (l *TOBLane) VerifyTx(ctx sdk.Context, bidTx sdk.Tx) error {
bidInfo, err := l.af.GetAuctionBidInfo(bidTx)
if err != nil {
return fmt.Errorf("failed to get auction bid info: %w", err)
}
// verify the top-level bid transaction
ctx, err = l.verifyTx(ctx, bidTx)
if err != nil {
return fmt.Errorf("invalid bid tx; failed to execute ante handler: %w", err)
}
// verify all of the bundled transactions
for _, tx := range bidInfo.Transactions {
bundledTx, err := l.af.WrapBundleTransaction(tx)
if err != nil {
return fmt.Errorf("invalid bid tx; failed to decode bundled tx: %w", err)
}
// bid txs cannot be included in bundled txs
bidInfo, _ := l.af.GetAuctionBidInfo(bundledTx)
if bidInfo != nil {
return fmt.Errorf("invalid bid tx; bundled tx cannot be a bid tx")
}
if ctx, err = l.verifyTx(ctx, bundledTx); err != nil {
return fmt.Errorf("invalid bid tx; failed to execute bundled transaction: %w", err)
}
}
return nil
}
// PrepareLane which builds a portion of the block. Inputs a cache of transactions
// that have already been included by a previous lane.
func (l *TOBLane) PrepareLane(ctx sdk.Context, maxTxBytes int64, selectedTxs map[string][]byte) ([][]byte, error) {
var tmpSelectedTxs [][]byte
bidTxIterator := l.index.Select(ctx, nil)
txsToRemove := make(map[sdk.Tx]struct{}, 0)
// Attempt to select the highest bid transaction that is valid and whose
// bundled transactions are valid.
selectBidTxLoop:
for ; bidTxIterator != nil; bidTxIterator = bidTxIterator.Next() {
cacheCtx, write := ctx.CacheContext()
tmpBidTx := bidTxIterator.Tx()
// if the transaction is already in the (partial) block proposal, we skip it
txHash, err := l.getTxHashStr(tmpBidTx)
if err != nil {
return nil, fmt.Errorf("failed to get bid tx hash: %w", err)
}
if _, ok := selectedTxs[txHash]; ok {
continue selectBidTxLoop
}
bidTxBz, err := l.txEncoder(tmpBidTx)
if err != nil {
txsToRemove[tmpBidTx] = struct{}{}
continue selectBidTxLoop
}
bidTxSize := int64(len(bidTxBz))
if bidTxSize <= maxTxBytes {
if err := l.VerifyTx(cacheCtx, tmpBidTx); err != nil {
// Some transactions in the bundle may be malformed or invalid, so we
// remove the bid transaction and try the next top bid.
txsToRemove[tmpBidTx] = struct{}{}
continue selectBidTxLoop
}
bidInfo, err := l.af.GetAuctionBidInfo(tmpBidTx)
if bidInfo == nil || err != nil {
// Some transactions in the bundle may be malformed or invalid, so we
// remove the bid transaction and try the next top bid.
txsToRemove[tmpBidTx] = struct{}{}
continue selectBidTxLoop
}
// store the bytes of each ref tx as sdk.Tx bytes in order to build a valid proposal
bundledTxBz := make([][]byte, len(bidInfo.Transactions))
for index, rawRefTx := range bidInfo.Transactions {
bundleTxBz := make([]byte, len(rawRefTx))
copy(bundleTxBz, rawRefTx)
bundledTxBz[index] = rawRefTx
}
// At this point, both the bid transaction itself and all the bundled
// transactions are valid. So we select the bid transaction along with
// all the bundled transactions. We also mark these transactions as seen and
// update the total size selected thus far.
tmpSelectedTxs = append(tmpSelectedTxs, bidTxBz)
tmpSelectedTxs = append(tmpSelectedTxs, bundledTxBz...)
// Write the cache context to the original context when we know we have a
// valid top of block bundle.
write()
break selectBidTxLoop
}
txsToRemove[tmpBidTx] = struct{}{}
l.logger.Info(
"failed to select auction bid tx; tx size is too large",
"tx_size", bidTxSize,
"max_size", maxTxBytes,
)
}
// remove all invalid transactions from the mempool
for tx := range txsToRemove {
if err := l.Remove(tx); err != nil {
return nil, err
}
}
return tmpSelectedTxs, nil
}
// ProcessLane which verifies the lane's portion of a proposed block.
func (l *TOBLane) ProcessLane(ctx sdk.Context, proposalTxs [][]byte) error {
for index, txBz := range proposalTxs {
tx, err := l.txDecoder(txBz)
if err != nil {
return err
}
// skip transaction if it does not match this lane
if !l.Match(tx) {
continue
}
_, err = l.processProposalVerifyTx(ctx, txBz)
if err != nil {
return err
}
bidInfo, err := l.af.GetAuctionBidInfo(tx)
if err != nil {
return err
}
// If the transaction is an auction bid, then we need to ensure that it is
// the first transaction in the block proposal and that the order of
// transactions in the block proposal follows the order of transactions in
// the bid.
if bidInfo != nil {
if index != 0 {
return errors.New("auction bid must be the first transaction in the block proposal")
}
bundledTransactions := bidInfo.Transactions
if len(proposalTxs) < len(bundledTransactions)+1 {
return errors.New("block proposal does not contain enough transactions to match the bundled transactions in the auction bid")
}
for i, refTxRaw := range bundledTransactions {
// Wrap and then encode the bundled transaction to ensure that the underlying
// reference transaction can be processed as an sdk.Tx.
wrappedTx, err := l.af.WrapBundleTransaction(refTxRaw)
if err != nil {
return err
}
refTxBz, err := l.txEncoder(wrappedTx)
if err != nil {
return err
}
if !bytes.Equal(refTxBz, proposalTxs[i+1]) {
return errors.New("block proposal does not match the bundled transactions in the auction bid")
}
}
}
}
return nil
}
func (l *TOBLane) Insert(goCtx context.Context, tx sdk.Tx) error {
txHashStr, err := l.getTxHashStr(tx)
if err != nil {
return err
}
if err := l.index.Insert(goCtx, tx); err != nil {
return fmt.Errorf("failed to insert tx into auction index: %w", err)
}
l.txIndex[txHashStr] = struct{}{}
return nil
}
func (l *TOBLane) Select(goCtx context.Context, txs [][]byte) sdkmempool.Iterator {
return l.index.Select(goCtx, txs)
}
func (l *TOBLane) CountTx() int {
return l.index.CountTx()
}
func (l *TOBLane) Remove(tx sdk.Tx) error {
txHashStr, err := l.getTxHashStr(tx)
if err != nil {
return fmt.Errorf("failed to get tx hash string: %w", err)
}
if err := l.index.Remove(tx); err != nil && !errors.Is(err, sdkmempool.ErrTxNotFound) {
return fmt.Errorf("failed to remove invalid transaction from the mempool: %w", err)
}
delete(l.txIndex, txHashStr)
return nil
}
func (l *TOBLane) processProposalVerifyTx(ctx sdk.Context, txBz []byte) (sdk.Tx, error) {
tx, err := l.txDecoder(txBz)
if err != nil {
return nil, err
}
if _, err := l.verifyTx(ctx, tx); err != nil {
return nil, err
}
return tx, nil
}
func (l *TOBLane) verifyTx(ctx sdk.Context, tx sdk.Tx) (sdk.Context, error) {
if l.anteHandler != nil {
newCtx, err := l.anteHandler(ctx, tx, false)
return newCtx, err
}
return ctx, nil
}
// getTxHashStr returns the transaction hash string for a given transaction.
func (l *TOBLane) getTxHashStr(tx sdk.Tx) (string, error) {
txBz, err := l.txEncoder(tx)
if err != nil {
return "", fmt.Errorf("failed to encode transaction: %w", err)
}
txHash := sha256.Sum256(txBz)
txHashStr := hex.EncodeToString(txHash[:])
return txHashStr, nil
}

22
blockbuster/utils.go Normal file
View File

@ -0,0 +1,22 @@
package blockbuster
import (
"crypto/sha256"
"encoding/hex"
"fmt"
sdk "github.com/cosmos/cosmos-sdk/types"
)
// GetTxHashStr returns the hex-encoded hash of the transaction.
func GetTxHashStr(txEncoder sdk.TxEncoder, tx sdk.Tx) (string, error) {
txBz, err := txEncoder(tx)
if err != nil {
return "", fmt.Errorf("failed to encode transaction: %w", err)
}
txHash := sha256.Sum256(txBz)
txHashStr := hex.EncodeToString(txHash[:])
return txHashStr, nil
}