Compare commits

...

7 Commits

Author SHA1 Message Date
Manu NALEPA
23d7f87cef Peer das core (#13877)
* Bump `c-kzg-4844` lib to the `das` branch.

* Implement `MerkleProofKZGCommitments`.

* Implement `das-core.md`.

* Use `peerdas.CustodyColumnSubnets` and `peerdas.CustodyColumns`.

* `CustodyColumnSubnets`: Include `i` in the for loop.

* Remove `computeSubscribedColumnSubnet`.

* Remove `peerdas.CustodyColumns` out of the for loop.
2024-07-29 13:19:42 +02:00
Nishant Das
a4db00d7ef Add Request And Response RPC Methods For Data Columns (#13909)
* Add RPC Handler

* Add Column Requests

* Update beacon-chain/db/filesystem/blob.go

Co-authored-by: Manu NALEPA <enalepa@offchainlabs.com>

* Update beacon-chain/p2p/rpc_topic_mappings.go

Co-authored-by: Manu NALEPA <enalepa@offchainlabs.com>

* Manu's Review

* Manu's Review

* Interface Fixes

* mock manager

---------

Co-authored-by: Manu NALEPA <enalepa@offchainlabs.com>
2024-07-29 13:19:42 +02:00
Nishant Das
b6a4e1213c Add Data Column Gossip Handlers (#13894)
* Add Data Column Subscriber

* Add Data Column Vaidator

* Wire all Handlers In

* Fix Build

* Fix Test

* Fix IP in Test

* Fix IP in Test
2024-07-29 13:19:42 +02:00
Nishant Das
86088d9c85 Add Support For Discovery Of Column Subnets (#13883)
* Add Support For Discovery Of Column Subnets

* Lint for SubnetsPerNode

* Manu's Review

* Change to a better name
2024-07-29 13:19:42 +02:00
Nishant Das
e10e610589 add in networking params (#13866) 2024-07-29 13:19:42 +02:00
Nishant Das
e16a25b6cc add it (#13865) 2024-07-29 13:19:42 +02:00
Manu NALEPA
f129dfdee6 Add in column sidecars protos (#13862) 2024-07-29 13:18:55 +02:00
53 changed files with 2757 additions and 1148 deletions

View File

@@ -26,6 +26,7 @@ go_library(
"receive_attestation.go",
"receive_blob.go",
"receive_block.go",
"receive_sidecar.go",
"service.go",
"tracked_proposer.go",
"weak_subjectivity_checks.go",

View File

@@ -51,6 +51,12 @@ type BlobReceiver interface {
ReceiveBlob(context.Context, blocks.VerifiedROBlob) error
}
// DataColumnReceiver interface defines the methods of chain service for receiving new
// data columns
type DataColumnReceiver interface {
ReceiveDataColumn(context.Context, *ethpb.DataColumnSidecar) error
}
// SlashingReceiver interface defines the methods of chain service for receiving validated slashing over the wire.
type SlashingReceiver interface {
ReceiveAttesterSlashing(ctx context.Context, slashing ethpb.AttSlashing)

View File

@@ -0,0 +1,12 @@
package blockchain
import (
"context"
ethpb "github.com/prysmaticlabs/prysm/v5/proto/prysm/v1alpha1"
)
func (s *Service) ReceiveDataColumn(ctx context.Context, ds *ethpb.DataColumnSidecar) error {
// TODO
return nil
}

View File

@@ -628,6 +628,11 @@ func (c *ChainService) ReceiveBlob(_ context.Context, b blocks.VerifiedROBlob) e
return nil
}
// ReceiveDataColumn implements the same method in chain service
func (c *ChainService) ReceiveDataColumn(_ context.Context, _ *ethpb.DataColumnSidecar) error {
return nil
}
// TargetRootForEpoch mocks the same method in the chain service
func (c *ChainService) TargetRootForEpoch(_ [32]byte, _ primitives.Epoch) ([32]byte, error) {
return c.TargetRoot, nil

View File

@@ -8,6 +8,7 @@ go_library(
"attestation_data.go",
"balance_cache_key.go",
"checkpoint_state.go",
"column_subnet_ids.go",
"committee.go",
"committee_disabled.go", # keep
"committees.go",

65
beacon-chain/cache/column_subnet_ids.go vendored Normal file
View File

@@ -0,0 +1,65 @@
package cache
import (
"sync"
"time"
"github.com/patrickmn/go-cache"
"github.com/prysmaticlabs/prysm/v5/config/params"
)
type columnSubnetIDs struct {
colSubCache *cache.Cache
colSubLock sync.RWMutex
}
// ColumnSubnetIDs for column subnet participants
var ColumnSubnetIDs = newColumnSubnetIDs()
const columnKey = "columns"
func newColumnSubnetIDs() *columnSubnetIDs {
epochDuration := time.Duration(params.BeaconConfig().SlotsPerEpoch.Mul(params.BeaconConfig().SecondsPerSlot))
// Set the default duration of a column subnet subscription as the column expiry period.
subLength := epochDuration * time.Duration(params.BeaconConfig().MinEpochsForDataColumnSidecarsRequest)
persistentCache := cache.New(subLength*time.Second, epochDuration*time.Second)
return &columnSubnetIDs{colSubCache: persistentCache}
}
// GetColumnSubnets retrieves the data column subnets.
func (s *columnSubnetIDs) GetColumnSubnets() ([]uint64, bool, time.Time) {
s.colSubLock.RLock()
defer s.colSubLock.RUnlock()
id, duration, ok := s.colSubCache.GetWithExpiration(columnKey)
if !ok {
return nil, false, time.Time{}
}
// Retrieve indices from the cache.
idxs, ok := id.([]uint64)
if !ok {
return nil, false, time.Time{}
}
return idxs, ok, duration
}
// AddColumnSubnets adds the relevant data column subnets.
func (s *columnSubnetIDs) AddColumnSubnets(colIdx []uint64) {
s.colSubLock.Lock()
defer s.colSubLock.Unlock()
s.colSubCache.Set(columnKey, colIdx, 0)
}
// EmptyAllCaches empties out all the related caches and flushes any stored
// entries on them. This should only ever be used for testing, in normal
// production, handling of the relevant subnets for each role is done
// separately.
func (s *columnSubnetIDs) EmptyAllCaches() {
// Clear the cache.
s.colSubLock.Lock()
defer s.colSubLock.Unlock()
s.colSubCache.Flush()
}

View File

@@ -96,6 +96,24 @@ func VerifyBlockHeaderSignature(beaconState state.BeaconState, header *ethpb.Sig
return signing.VerifyBlockHeaderSigningRoot(header.Header, proposerPubKey, header.Signature, domain)
}
func VerifyBlockHeaderSignatureUsingCurrentFork(beaconState state.BeaconState, header *ethpb.SignedBeaconBlockHeader) error {
currentEpoch := slots.ToEpoch(header.Header.Slot)
fork, err := forks.Fork(currentEpoch)
if err != nil {
return err
}
domain, err := signing.Domain(fork, currentEpoch, params.BeaconConfig().DomainBeaconProposer, beaconState.GenesisValidatorsRoot())
if err != nil {
return err
}
proposer, err := beaconState.ValidatorAtIndex(header.Header.ProposerIndex)
if err != nil {
return err
}
proposerPubKey := proposer.PublicKey
return signing.VerifyBlockHeaderSigningRoot(header.Header, proposerPubKey, header.Signature, domain)
}
// VerifyBlockSignatureUsingCurrentFork verifies the proposer signature of a beacon block. This differs
// from the above method by not using fork data from the state and instead retrieving it
// via the respective epoch.

View File

@@ -32,6 +32,9 @@ const (
// AttesterSlashingReceived is sent after an attester slashing is received from gossip or rpc
AttesterSlashingReceived = 8
// DataColumnSidecarReceived is sent after a data column sidecar is received from gossip or rpc.
DataColumnSidecarReceived = 9
)
// UnAggregatedAttReceivedData is the data sent with UnaggregatedAttReceived events.
@@ -77,3 +80,7 @@ type ProposerSlashingReceivedData struct {
type AttesterSlashingReceivedData struct {
AttesterSlashing ethpb.AttSlashing
}
type DataColumnSidecarReceivedData struct {
DataColumn *ethpb.DataColumnSidecar
}

View File

@@ -0,0 +1,21 @@
load("@prysm//tools/go:def.bzl", "go_library")
go_library(
name = "go_default_library",
srcs = ["helpers.go"],
importpath = "github.com/prysmaticlabs/prysm/v5/beacon-chain/core/peerdas",
visibility = ["//visibility:public"],
deps = [
"//config/fieldparams:go_default_library",
"//config/params:go_default_library",
"//consensus-types/blocks:go_default_library",
"//consensus-types/interfaces:go_default_library",
"//crypto/hash:go_default_library",
"//encoding/bytesutil:go_default_library",
"//proto/prysm/v1alpha1:go_default_library",
"@com_github_ethereum_c_kzg_4844//bindings/go:go_default_library",
"@com_github_ethereum_go_ethereum//p2p/enode:go_default_library",
"@com_github_holiman_uint256//:go_default_library",
"@com_github_pkg_errors//:go_default_library",
],
)

View File

@@ -0,0 +1,245 @@
package peerdas
import (
"encoding/binary"
cKzg4844 "github.com/ethereum/c-kzg-4844/bindings/go"
"github.com/holiman/uint256"
"github.com/ethereum/go-ethereum/p2p/enode"
errors "github.com/pkg/errors"
fieldparams "github.com/prysmaticlabs/prysm/v5/config/fieldparams"
"github.com/prysmaticlabs/prysm/v5/config/params"
"github.com/prysmaticlabs/prysm/v5/consensus-types/blocks"
"github.com/prysmaticlabs/prysm/v5/consensus-types/interfaces"
"github.com/prysmaticlabs/prysm/v5/crypto/hash"
"github.com/prysmaticlabs/prysm/v5/encoding/bytesutil"
ethpb "github.com/prysmaticlabs/prysm/v5/proto/prysm/v1alpha1"
)
const (
// Number of field elements per extended blob
fieldElementsPerExtBlob = 2 * cKzg4844.FieldElementsPerBlob
// Bytes per cell
bytesPerCell = cKzg4844.FieldElementsPerCell * cKzg4844.BytesPerFieldElement
// Number of cells in the extended matrix
extendedMatrixSize = fieldparams.MaxBlobsPerBlock * cKzg4844.CellsPerExtBlob
)
type (
extendedMatrix []cKzg4844.Cell
cellCoordinate struct {
blobIndex uint64
cellID uint64
}
)
var (
errCustodySubnetCountTooLarge = errors.New("custody subnet count larger than data column sidecar subnet count")
errCellNotFound = errors.New("cell not found (should never happen)")
errCurveOrder = errors.New("could not set bls curve order as big int")
errBlsFieldElementNil = errors.New("bls field element is nil")
errBlsFieldElementBiggerThanCurveOrder = errors.New("bls field element higher than curve order")
errBlsFieldElementDoesNotFit = errors.New("bls field element does not fit in BytesPerFieldElement")
)
// https://github.com/ethereum/consensus-specs/blob/dev/specs/_features/eip7594/das-core.md#helper-functions
func CustodyColumns(nodeId enode.ID, custodySubnetCount uint64) (map[uint64]bool, error) {
dataColumnSidecarSubnetCount := params.BeaconConfig().DataColumnSidecarSubnetCount
// Compute the custodied subnets.
subnetIds, err := CustodyColumnSubnets(nodeId, custodySubnetCount)
if err != nil {
return nil, errors.Wrap(err, "custody subnets")
}
columnsPerSubnet := cKzg4844.CellsPerExtBlob / dataColumnSidecarSubnetCount
// Knowing the subnet ID and the number of columns per subnet, select all the columns the node should custody.
// Columns belonging to the same subnet are contiguous.
columnIndices := make(map[uint64]bool, custodySubnetCount*columnsPerSubnet)
for i := uint64(0); i < columnsPerSubnet; i++ {
for subnetId := range subnetIds {
columnIndex := dataColumnSidecarSubnetCount*i + subnetId
columnIndices[columnIndex] = true
}
}
return columnIndices, nil
}
func CustodyColumnSubnets(nodeId enode.ID, custodySubnetCount uint64) (map[uint64]bool, error) {
dataColumnSidecarSubnetCount := params.BeaconConfig().DataColumnSidecarSubnetCount
// Check if the custody subnet count is larger than the data column sidecar subnet count.
if custodySubnetCount > dataColumnSidecarSubnetCount {
return nil, errCustodySubnetCountTooLarge
}
// First, compute the subnet IDs that the node should participate in.
subnetIds := make(map[uint64]bool, custodySubnetCount)
for i := uint64(0); uint64(len(subnetIds)) < custodySubnetCount; i++ {
nodeIdUInt256, nextNodeIdUInt256 := new(uint256.Int), new(uint256.Int)
nodeIdUInt256.SetBytes(nodeId.Bytes())
nextNodeIdUInt256.Add(nodeIdUInt256, uint256.NewInt(i))
nextNodeIdUInt64 := nextNodeIdUInt256.Uint64()
nextNodeId := bytesutil.Uint64ToBytesLittleEndian(nextNodeIdUInt64)
hashedNextNodeId := hash.Hash(nextNodeId)
subnetId := binary.LittleEndian.Uint64(hashedNextNodeId[:8]) % dataColumnSidecarSubnetCount
if _, exists := subnetIds[subnetId]; !exists {
subnetIds[subnetId] = true
}
}
return subnetIds, nil
}
// computeExtendedMatrix computes the extended matrix from the blobs.
// https://github.com/ethereum/consensus-specs/blob/dev/specs/_features/eip7594/das-core.md#compute_extended_matrix
func computeExtendedMatrix(blobs []cKzg4844.Blob) (extendedMatrix, error) {
matrix := make(extendedMatrix, 0, extendedMatrixSize)
for i := range blobs {
// Chunk a non-extended blob into cells representing the corresponding extended blob.
blob := &blobs[i]
cells, err := cKzg4844.ComputeCells(blob)
if err != nil {
return nil, errors.Wrap(err, "compute cells for blob")
}
matrix = append(matrix, cells[:]...)
}
return matrix, nil
}
// recoverMatrix recovers the extended matrix from some cells.
// https://github.com/ethereum/consensus-specs/blob/dev/specs/_features/eip7594/das-core.md#recover_matrix
func recoverMatrix(cellFromCoordinate map[cellCoordinate]cKzg4844.Cell, blobCount uint64) (extendedMatrix, error) {
matrix := make(extendedMatrix, 0, extendedMatrixSize)
for blobIndex := uint64(0); blobIndex < blobCount; blobIndex++ {
// Filter all cells that belong to the current blob.
cellIds := make([]uint64, 0, cKzg4844.CellsPerExtBlob)
for coordinate := range cellFromCoordinate {
if coordinate.blobIndex == blobIndex {
cellIds = append(cellIds, coordinate.cellID)
}
}
// Retrieve cells corresponding to all `cellIds`.
cellIdsCount := len(cellIds)
cells := make([]cKzg4844.Cell, 0, cellIdsCount)
for _, cellId := range cellIds {
coordinate := cellCoordinate{blobIndex: blobIndex, cellID: cellId}
cell, ok := cellFromCoordinate[coordinate]
if !ok {
return matrix, errCellNotFound
}
cells = append(cells, cell)
}
// Recover all cells.
allCellsForRow, err := cKzg4844.RecoverAllCells(cellIds, cells)
if err != nil {
return matrix, errors.Wrap(err, "recover all cells")
}
matrix = append(matrix, allCellsForRow[:]...)
}
return matrix, nil
}
// https://github.com/ethereum/consensus-specs/blob/dev/specs/_features/eip7594/das-core.md#recover_matrix
func dataColumnSidecars(signedBlock interfaces.SignedBeaconBlock, blobs []cKzg4844.Blob) ([]ethpb.DataColumnSidecar, error) {
blobsCount := len(blobs)
// Get the signed block header.
signedBlockHeader, err := signedBlock.Header()
if err != nil {
return nil, errors.Wrap(err, "signed block header")
}
// Get the block body.
block := signedBlock.Block()
blockBody := block.Body()
// Get the blob KZG commitments.
blobKzgCommitments, err := blockBody.BlobKzgCommitments()
if err != nil {
return nil, errors.Wrap(err, "blob KZG commitments")
}
// Compute the KZG commitments inclusion proof.
kzgCommitmentsInclusionProof, err := blocks.MerkleProofKZGCommitments(blockBody)
if err != nil {
return nil, errors.Wrap(err, "merkle proof ZKG commitments")
}
// Compute cells and proofs.
cells := make([][cKzg4844.CellsPerExtBlob]cKzg4844.Cell, 0, blobsCount)
proofs := make([][cKzg4844.CellsPerExtBlob]cKzg4844.KZGProof, 0, blobsCount)
for i := range blobs {
blob := &blobs[i]
blobCells, blobProofs, err := cKzg4844.ComputeCellsAndProofs(blob)
if err != nil {
return nil, errors.Wrap(err, "compute cells and proofs")
}
cells = append(cells, blobCells)
proofs = append(proofs, blobProofs)
}
// Get the column sidecars.
sidecars := make([]ethpb.DataColumnSidecar, cKzg4844.CellsPerExtBlob)
for columnIndex := uint64(0); columnIndex < cKzg4844.CellsPerExtBlob; columnIndex++ {
column := make([]cKzg4844.Cell, 0, blobsCount)
kzgProofOfColumn := make([]cKzg4844.KZGProof, 0, blobsCount)
for rowIndex := 0; rowIndex < blobsCount; rowIndex++ {
cell := cells[rowIndex][columnIndex]
column = append(column, cell)
kzgProof := proofs[rowIndex][columnIndex]
kzgProofOfColumn = append(kzgProofOfColumn, kzgProof)
}
columnBytes := make([][]byte, 0, blobsCount)
for i := range column {
cell := column[i]
cellBytes := make([]byte, 0, bytesPerCell)
for _, fieldElement := range cell {
cellBytes = append(cellBytes, fieldElement[:]...)
}
columnBytes = append(columnBytes, cellBytes)
}
kzgProofOfColumnBytes := make([][]byte, 0, blobsCount)
for _, kzgProof := range kzgProofOfColumn {
kzgProofOfColumnBytes = append(kzgProofOfColumnBytes, kzgProof[:])
}
sidecars = append(sidecars, ethpb.DataColumnSidecar{
ColumnIndex: columnIndex,
DataColumn: columnBytes,
KzgCommitments: blobKzgCommitments,
KzgProof: kzgProofOfColumnBytes,
SignedBlockHeader: signedBlockHeader,
KzgCommitmentsInclusionProof: kzgCommitmentsInclusionProof,
})
}
return sidecars, nil
}

View File

@@ -246,6 +246,20 @@ func (bs *BlobStorage) Get(root [32]byte, idx uint64) (blocks.VerifiedROBlob, er
return verification.BlobSidecarNoop(ro)
}
// GetColumn retrieves a single DataColumnSidecar by its root and index.
func (bs *BlobStorage) GetColumn(root [32]byte, idx uint64) (*ethpb.DataColumnSidecar, error) {
expected := blobNamer{root: root, index: idx}
encoded, err := afero.ReadFile(bs.fs, expected.path())
if err != nil {
return nil, err
}
s := &ethpb.DataColumnSidecar{}
if err := s.UnmarshalSSZ(encoded); err != nil {
return nil, err
}
return s, nil
}
// Remove removes all blobs for a given root.
func (bs *BlobStorage) Remove(root [32]byte) error {
rootDir := blobNamer{root: root}.dir()

View File

@@ -46,6 +46,7 @@ go_library(
"//beacon-chain/core/altair:go_default_library",
"//beacon-chain/core/feed/state:go_default_library",
"//beacon-chain/core/helpers:go_default_library",
"//beacon-chain/core/peerdas:go_default_library",
"//beacon-chain/core/time:go_default_library",
"//beacon-chain/db:go_default_library",
"//beacon-chain/p2p/encoder:go_default_library",

View File

@@ -47,11 +47,11 @@ type quicProtocol uint16
// quicProtocol is the "quic" key, which holds the QUIC port of the node.
func (quicProtocol) ENRKey() string { return "quic" }
// RefreshENR uses an epoch to refresh the enr entry for our node
// with the tracked committee ids for the epoch, allowing our node
// to be dynamically discoverable by others given our tracked committee ids.
func (s *Service) RefreshENR() {
// return early if discv5 isn't running
// RefreshPersistentSubnets checks that we are tracking our local persistent subnets for a variety of gossip topics.
// This routine checks for our attestation, sync committee and data column subnets and updates them if they have
// been rotated.
func (s *Service) RefreshPersistentSubnets() {
// return early if discv5 isnt running
if s.dv5Listener == nil || !s.isInitialized() {
return
}
@@ -60,6 +60,10 @@ func (s *Service) RefreshENR() {
log.WithError(err).Error("Could not initialize persistent subnets")
return
}
if err := initializePersistentColumnSubnets(s.dv5Listener.LocalNode().ID()); err != nil {
log.WithError(err).Error("Could not initialize persistent column subnets")
return
}
bitV := bitfield.NewBitvector64()
committees := cache.SubnetIDs.GetAllSubnets()

View File

@@ -435,7 +435,7 @@ func addPeer(t *testing.T, p *peers.Status, state peerdata.PeerConnectionState)
return id
}
func TestRefreshENR_ForkBoundaries(t *testing.T) {
func TestRefreshPersistentSubnets_ForkBoundaries(t *testing.T) {
params.SetupTestConfigCleanup(t)
// Clean up caches after usage.
defer cache.SubnetIDs.EmptyAllCaches()
@@ -601,7 +601,7 @@ func TestRefreshENR_ForkBoundaries(t *testing.T) {
for _, tt := range tests {
t.Run(tt.name, func(t *testing.T) {
s := tt.svcBuilder(t)
s.RefreshENR()
s.RefreshPersistentSubnets()
tt.postValidation(t, s)
s.dv5Listener.Close()
cache.SubnetIDs.EmptyAllCaches()

View File

@@ -22,6 +22,7 @@ var gossipTopicMappings = map[string]func() proto.Message{
SyncCommitteeSubnetTopicFormat: func() proto.Message { return &ethpb.SyncCommitteeMessage{} },
BlsToExecutionChangeSubnetTopicFormat: func() proto.Message { return &ethpb.SignedBLSToExecutionChange{} },
BlobSubnetTopicFormat: func() proto.Message { return &ethpb.BlobSidecar{} },
DataColumnSubnetTopicFormat: func() proto.Message { return &ethpb.DataColumnSidecar{} },
}
// GossipTopicMappings is a function to return the assigned data type

View File

@@ -3,6 +3,7 @@ package p2p
import (
"context"
"github.com/ethereum/go-ethereum/p2p/enode"
"github.com/ethereum/go-ethereum/p2p/enr"
pubsub "github.com/libp2p/go-libp2p-pubsub"
"github.com/libp2p/go-libp2p/core/connmgr"
@@ -81,8 +82,9 @@ type PeerManager interface {
PeerID() peer.ID
Host() host.Host
ENR() *enr.Record
NodeID() enode.ID
DiscoveryAddresses() ([]multiaddr.Multiaddr, error)
RefreshENR()
RefreshPersistentSubnets()
FindPeersWithSubnet(ctx context.Context, topic string, subIndex uint64, threshold int) (bool, error)
AddPingMethod(reqFunc func(ctx context.Context, id peer.ID) error)
}

View File

@@ -90,7 +90,7 @@ func TestService_CanSubscribe(t *testing.T) {
formatting := []interface{}{digest}
// Special case for attestation subnets which have a second formatting placeholder.
if topic == AttestationSubnetTopicFormat || topic == SyncCommitteeSubnetTopicFormat || topic == BlobSubnetTopicFormat {
if topic == AttestationSubnetTopicFormat || topic == SyncCommitteeSubnetTopicFormat || topic == BlobSubnetTopicFormat || topic == DataColumnSubnetTopicFormat {
formatting = append(formatting, 0 /* some subnet ID */)
}

View File

@@ -43,6 +43,9 @@ const BlobSidecarsByRangeName = "/blob_sidecars_by_range"
// BlobSidecarsByRootName is the name for the BlobSidecarsByRoot v1 message topic.
const BlobSidecarsByRootName = "/blob_sidecars_by_root"
// DataColumnSidecarsByRootName is the name for the DataColumnSidecarsByRoot v1 message topic.
const DataColumnSidecarsByRootName = "/data_column_sidecars_by_root"
const (
// V1 RPC Topics
// RPCStatusTopicV1 defines the v1 topic for the status rpc method.
@@ -65,6 +68,9 @@ const (
// RPCBlobSidecarsByRootTopicV1 is a topic for requesting blob sidecars by their block root. New in deneb.
// /eth2/beacon_chain/req/blob_sidecars_by_root/1/
RPCBlobSidecarsByRootTopicV1 = protocolPrefix + BlobSidecarsByRootName + SchemaVersionV1
// RPCDataColumnSidecarsByRootTopicV1 is a topic for requesting data column sidecars by their block root. New in PeerDAS.
// /eth2/beacon_chain/req/data_column_sidecars_by_root/1
RPCDataColumnSidecarsByRootTopicV1 = protocolPrefix + DataColumnSidecarsByRootName + SchemaVersionV1
// V2 RPC Topics
// RPCBlocksByRangeTopicV2 defines v2 the topic for the blocks by range rpc method.

View File

@@ -226,7 +226,7 @@ func (s *Service) Start() {
}
// Initialize metadata according to the
// current epoch.
s.RefreshENR()
s.RefreshPersistentSubnets()
// Periodic functions.
async.RunEvery(s.ctx, params.BeaconConfig().TtfbTimeoutDuration(), func() {
@@ -234,7 +234,7 @@ func (s *Service) Start() {
})
async.RunEvery(s.ctx, 30*time.Minute, s.Peers().Prune)
async.RunEvery(s.ctx, time.Duration(params.BeaconConfig().RespTimeout)*time.Second, s.updateMetrics)
async.RunEvery(s.ctx, refreshRate, s.RefreshENR)
async.RunEvery(s.ctx, refreshRate, s.RefreshPersistentSubnets)
async.RunEvery(s.ctx, 1*time.Minute, func() {
inboundQUICCount := len(s.peers.InboundConnectedWithProtocol(peers.QUIC))
inboundTCPCount := len(s.peers.InboundConnectedWithProtocol(peers.TCP))
@@ -358,6 +358,15 @@ func (s *Service) ENR() *enr.Record {
return s.dv5Listener.Self().Record()
}
// NodeID returns the local node's node ID
// for discovery.
func (s *Service) NodeID() enode.ID {
if s.dv5Listener == nil {
return [32]byte{}
}
return s.dv5Listener.Self().ID()
}
// DiscoveryAddresses represents our enr addresses as multiaddresses.
func (s *Service) DiscoveryAddresses() ([]multiaddr.Multiaddr, error) {
if s.dv5Listener == nil {

View File

@@ -13,6 +13,7 @@ import (
"github.com/prysmaticlabs/go-bitfield"
"github.com/prysmaticlabs/prysm/v5/beacon-chain/cache"
"github.com/prysmaticlabs/prysm/v5/beacon-chain/core/helpers"
"github.com/prysmaticlabs/prysm/v5/beacon-chain/core/peerdas"
"github.com/prysmaticlabs/prysm/v5/cmd/beacon-chain/flags"
"github.com/prysmaticlabs/prysm/v5/config/params"
"github.com/prysmaticlabs/prysm/v5/consensus-types/primitives"
@@ -206,6 +207,25 @@ func initializePersistentSubnets(id enode.ID, epoch primitives.Epoch) error {
return nil
}
func initializePersistentColumnSubnets(id enode.ID) error {
_, ok, expTime := cache.ColumnSubnetIDs.GetColumnSubnets()
if ok && expTime.After(time.Now()) {
return nil
}
subsMap, err := peerdas.CustodyColumnSubnets(id, params.BeaconConfig().CustodyRequirement)
if err != nil {
return err
}
subs := make([]uint64, 0, len(subsMap))
for sub := range subsMap {
subs = append(subs, sub)
}
cache.ColumnSubnetIDs.AddColumnSubnets(subs)
return nil
}
// Spec pseudocode definition:
//
// def compute_subscribed_subnets(node_id: NodeID, epoch: Epoch) -> Sequence[SubnetID]:

View File

@@ -3,6 +3,7 @@ package testing
import (
"context"
"github.com/ethereum/go-ethereum/p2p/enode"
"github.com/ethereum/go-ethereum/p2p/enr"
pubsub "github.com/libp2p/go-libp2p-pubsub"
"github.com/libp2p/go-libp2p/core/control"
@@ -55,6 +56,11 @@ func (_ *FakeP2P) ENR() *enr.Record {
return new(enr.Record)
}
// NodeID returns the node id of the local peer.
func (_ *FakeP2P) NodeID() enode.ID {
return [32]byte{}
}
// DiscoveryAddresses -- fake
func (_ *FakeP2P) DiscoveryAddresses() ([]multiaddr.Multiaddr, error) {
return nil, nil
@@ -65,8 +71,8 @@ func (_ *FakeP2P) FindPeersWithSubnet(_ context.Context, _ string, _ uint64, _ i
return false, nil
}
// RefreshENR mocks the p2p func.
func (_ *FakeP2P) RefreshENR() {}
// RefreshPersistentSubnets mocks the p2p func.
func (_ *FakeP2P) RefreshPersistentSubnets() {}
// LeaveTopic -- fake.
func (_ *FakeP2P) LeaveTopic(_ string) error {

View File

@@ -4,6 +4,7 @@ import (
"context"
"errors"
"github.com/ethereum/go-ethereum/p2p/enode"
"github.com/ethereum/go-ethereum/p2p/enr"
"github.com/libp2p/go-libp2p/core/host"
"github.com/libp2p/go-libp2p/core/peer"
@@ -39,6 +40,11 @@ func (m MockPeerManager) ENR() *enr.Record {
return m.Enr
}
// NodeID .
func (m MockPeerManager) NodeID() enode.ID {
return [32]byte{}
}
// DiscoveryAddresses .
func (m MockPeerManager) DiscoveryAddresses() ([]multiaddr.Multiaddr, error) {
if m.FailDiscoveryAddr {
@@ -47,8 +53,8 @@ func (m MockPeerManager) DiscoveryAddresses() ([]multiaddr.Multiaddr, error) {
return m.DiscoveryAddr, nil
}
// RefreshENR .
func (_ MockPeerManager) RefreshENR() {}
// RefreshPersistentSubnets .
func (_ MockPeerManager) RefreshPersistentSubnets() {}
// FindPeersWithSubnet .
func (_ MockPeerManager) FindPeersWithSubnet(_ context.Context, _ string, _ uint64, _ int) (bool, error) {

View File

@@ -10,6 +10,7 @@ import (
"testing"
"time"
"github.com/ethereum/go-ethereum/p2p/enode"
"github.com/ethereum/go-ethereum/p2p/enr"
pubsub "github.com/libp2p/go-libp2p-pubsub"
core "github.com/libp2p/go-libp2p/core"
@@ -263,6 +264,11 @@ func (_ *TestP2P) ENR() *enr.Record {
return new(enr.Record)
}
// NodeID returns the node id of the local peer.
func (_ *TestP2P) NodeID() enode.ID {
return [32]byte{}
}
// DiscoveryAddresses --
func (_ *TestP2P) DiscoveryAddresses() ([]multiaddr.Multiaddr, error) {
return nil, nil
@@ -360,8 +366,8 @@ func (_ *TestP2P) FindPeersWithSubnet(_ context.Context, _ string, _ uint64, _ i
return false, nil
}
// RefreshENR mocks the p2p func.
func (_ *TestP2P) RefreshENR() {}
// RefreshPersistentSubnets mocks the p2p func.
func (_ *TestP2P) RefreshPersistentSubnets() {}
// ForkDigest mocks the p2p func.
func (p *TestP2P) ForkDigest() ([4]byte, error) {

View File

@@ -30,6 +30,9 @@ const (
GossipBlsToExecutionChangeMessage = "bls_to_execution_change"
// GossipBlobSidecarMessage is the name for the blob sidecar message type.
GossipBlobSidecarMessage = "blob_sidecar"
// GossipDataColumnSidecarMessage is the name for the data column sidecar message type.
GossipDataColumnSidecarMessage = "data_column_sidecar"
// Topic Formats
//
// AttestationSubnetTopicFormat is the topic format for the attestation subnet.
@@ -52,4 +55,6 @@ const (
BlsToExecutionChangeSubnetTopicFormat = GossipProtocolAndDigest + GossipBlsToExecutionChangeMessage
// BlobSubnetTopicFormat is the topic format for the blob subnet.
BlobSubnetTopicFormat = GossipProtocolAndDigest + GossipBlobSidecarMessage + "_%d"
// DataColumnSubnetTopicFormat is the topic format for the data column subnet.
DataColumnSubnetTopicFormat = GossipProtocolAndDigest + GossipDataColumnSidecarMessage + "_%d"
)

View File

@@ -9,10 +9,15 @@ var (
ErrInvalidSequenceNum = errors.New("invalid sequence number provided")
ErrGeneric = errors.New("internal service error")
ErrRateLimited = errors.New("rate limited")
ErrIODeadline = errors.New("i/o deadline exceeded")
ErrInvalidRequest = errors.New("invalid range, step or count")
ErrBlobLTMinRequest = errors.New("blob slot < minimum_request_epoch")
ErrMaxBlobReqExceeded = errors.New("requested more than MAX_REQUEST_BLOB_SIDECARS")
ErrRateLimited = errors.New("rate limited")
ErrIODeadline = errors.New("i/o deadline exceeded")
ErrInvalidRequest = errors.New("invalid range, step or count")
ErrBlobLTMinRequest = errors.New("blob epoch < minimum_request_epoch")
ErrDataColumnLTMinRequest = errors.New("data column epoch < minimum_request_epoch")
ErrMaxBlobReqExceeded = errors.New("requested more than MAX_REQUEST_BLOB_SIDECARS")
ErrMaxDataColumnReqExceeded = errors.New("requested more than MAX_REQUEST_DATA_COLUMN_SIDECARS")
ErrResourceUnavailable = errors.New("resource requested unavailable")
ErrInvalidColumnIndex = errors.New("invalid column index requested")
)

View File

@@ -25,6 +25,7 @@ go_library(
"rpc_blob_sidecars_by_range.go",
"rpc_blob_sidecars_by_root.go",
"rpc_chunked_response.go",
"rpc_data_column_sidecars_by_root.go",
"rpc_goodbye.go",
"rpc_metadata.go",
"rpc_ping.go",
@@ -37,6 +38,7 @@ go_library(
"subscriber_beacon_blocks.go",
"subscriber_blob_sidecar.go",
"subscriber_bls_to_execution_change.go",
"subscriber_data_column_sidecar.go",
"subscriber_handlers.go",
"subscriber_sync_committee_message.go",
"subscriber_sync_contribution_proof.go",
@@ -48,6 +50,7 @@ go_library(
"validate_beacon_blocks.go",
"validate_blob.go",
"validate_bls_to_execution_change.go",
"validate_data_column_sidecar.go",
"validate_proposer_slashing.go",
"validate_sync_committee_message.go",
"validate_sync_contribution_proof.go",
@@ -72,6 +75,7 @@ go_library(
"//beacon-chain/core/feed/operation:go_default_library",
"//beacon-chain/core/feed/state:go_default_library",
"//beacon-chain/core/helpers:go_default_library",
"//beacon-chain/core/peerdas:go_default_library",
"//beacon-chain/core/signing:go_default_library",
"//beacon-chain/core/transition:go_default_library",
"//beacon-chain/core/transition/interop:go_default_library",

View File

@@ -45,6 +45,8 @@ func (s *Service) decodePubsubMessage(msg *pubsub.Message) (ssz.Unmarshaler, err
topic = p2p.GossipTypeMapping[reflect.TypeOf(&ethpb.SyncCommitteeMessage{})]
case strings.Contains(topic, p2p.GossipBlobSidecarMessage):
topic = p2p.GossipTypeMapping[reflect.TypeOf(&ethpb.BlobSidecar{})]
case strings.Contains(topic, p2p.GossipDataColumnSidecarMessage):
topic = p2p.GossipTypeMapping[reflect.TypeOf(&ethpb.DataColumnSidecar{})]
}
base := p2p.GossipTopicMappings(topic, 0)

View File

@@ -14,6 +14,7 @@ import (
ssz "github.com/prysmaticlabs/fastssz"
"github.com/prysmaticlabs/prysm/v5/beacon-chain/p2p"
p2ptypes "github.com/prysmaticlabs/prysm/v5/beacon-chain/p2p/types"
"github.com/prysmaticlabs/prysm/v5/config/features"
"github.com/prysmaticlabs/prysm/v5/config/params"
"github.com/prysmaticlabs/prysm/v5/monitoring/tracing"
"github.com/prysmaticlabs/prysm/v5/time/slots"
@@ -99,6 +100,13 @@ func (s *Service) registerRPCHandlersAltair() {
}
func (s *Service) registerRPCHandlersDeneb() {
if features.Get().EnablePeerDAS {
s.registerRPC(
p2p.RPCDataColumnSidecarsByRootTopicV1,
s.dataColumnSidecarByRootRPCHandler,
)
return
}
s.registerRPC(
p2p.RPCBlobSidecarsByRangeTopicV1,
s.blobSidecarsByRangeRPCHandler,

View File

@@ -11,6 +11,7 @@ import (
"github.com/prysmaticlabs/prysm/v5/consensus-types/blocks"
"github.com/prysmaticlabs/prysm/v5/consensus-types/interfaces"
"github.com/prysmaticlabs/prysm/v5/network/forks"
ethpb "github.com/prysmaticlabs/prysm/v5/proto/prysm/v1alpha1"
"github.com/prysmaticlabs/prysm/v5/runtime/version"
"github.com/prysmaticlabs/prysm/v5/time/slots"
)
@@ -155,3 +156,22 @@ func WriteBlobSidecarChunk(stream libp2pcore.Stream, tor blockchain.TemporalOrac
_, err = encoding.EncodeWithMaxLength(stream, sidecar)
return err
}
// WriteDataColumnSidecarChunk writes data column chunk object to stream.
// response_chunk ::= <result> | <context-bytes> | <encoding-dependent-header> | <encoded-payload>
func WriteDataColumnSidecarChunk(stream libp2pcore.Stream, tor blockchain.TemporalOracle, encoding encoder.NetworkEncoding, sidecar *ethpb.DataColumnSidecar) error {
if _, err := stream.Write([]byte{responseCodeSuccess}); err != nil {
return err
}
valRoot := tor.GenesisValidatorsRoot()
ctxBytes, err := forks.ForkDigestFromEpoch(slots.ToEpoch(sidecar.SignedBlockHeader.Header.Slot), valRoot[:])
if err != nil {
return err
}
if err := writeContextToStream(ctxBytes[:], stream); err != nil {
return err
}
_, err = encoding.EncodeWithMaxLength(stream, sidecar)
return err
}

View File

@@ -0,0 +1,149 @@
package sync
import (
"context"
"fmt"
"math"
"sort"
"time"
libp2pcore "github.com/libp2p/go-libp2p/core"
"github.com/pkg/errors"
"github.com/prysmaticlabs/prysm/v5/beacon-chain/core/peerdas"
"github.com/prysmaticlabs/prysm/v5/beacon-chain/db"
"github.com/prysmaticlabs/prysm/v5/beacon-chain/p2p"
"github.com/prysmaticlabs/prysm/v5/beacon-chain/p2p/types"
"github.com/prysmaticlabs/prysm/v5/cmd/beacon-chain/flags"
"github.com/prysmaticlabs/prysm/v5/config/features"
"github.com/prysmaticlabs/prysm/v5/config/params"
"github.com/prysmaticlabs/prysm/v5/consensus-types/primitives"
"github.com/prysmaticlabs/prysm/v5/encoding/bytesutil"
"github.com/prysmaticlabs/prysm/v5/monitoring/tracing"
"github.com/prysmaticlabs/prysm/v5/time/slots"
"github.com/sirupsen/logrus"
"go.opencensus.io/trace"
)
func (s *Service) dataColumnSidecarByRootRPCHandler(ctx context.Context, msg interface{}, stream libp2pcore.Stream) error {
ctx, span := trace.StartSpan(ctx, "sync.dataColumnSidecarByRootRPCHandler")
defer span.End()
ctx, cancel := context.WithTimeout(ctx, ttfbTimeout)
defer cancel()
SetRPCStreamDeadlines(stream)
log := log.WithField("handler", p2p.DataColumnSidecarsByRootName[1:]) // slice the leading slash off the name var
// We use the same type as for blobs as they are the same data structure.
// TODO: Make the type naming more generic to be extensible to data columns
ref, ok := msg.(*types.BlobSidecarsByRootReq)
if !ok {
return errors.New("message is not type BlobSidecarsByRootReq")
}
columnIdents := *ref
if err := validateDataColummnsByRootRequest(columnIdents); err != nil {
s.cfg.p2p.Peers().Scorers().BadResponsesScorer().Increment(stream.Conn().RemotePeer())
s.writeErrorResponseToStream(responseCodeInvalidRequest, err.Error(), stream)
return err
}
// Sort the identifiers so that requests for the same blob root will be adjacent, minimizing db lookups.
sort.Sort(columnIdents)
// TODO: Customize data column batches too
batchSize := flags.Get().BlobBatchLimit
var ticker *time.Ticker
if len(columnIdents) > batchSize {
ticker = time.NewTicker(time.Second)
}
// Compute the oldest slot we'll allow a peer to request, based on the current slot.
cs := s.cfg.clock.CurrentSlot()
minReqSlot, err := DataColumnsRPCMinValidSlot(cs)
if err != nil {
return errors.Wrapf(err, "unexpected error computing min valid blob request slot, current_slot=%d", cs)
}
// Compute all custodied columns.
custodiedColumns, err := peerdas.CustodyColumns(s.cfg.p2p.NodeID(), params.BeaconConfig().CustodyRequirement)
if err != nil {
log.WithError(err).Errorf("unexpected error retrieving the node id")
s.writeErrorResponseToStream(responseCodeServerError, types.ErrGeneric.Error(), stream)
return err
}
for i := range columnIdents {
if err := ctx.Err(); err != nil {
closeStream(stream, log)
return err
}
// Throttle request processing to no more than batchSize/sec.
if ticker != nil && i != 0 && i%batchSize == 0 {
<-ticker.C
}
s.rateLimiter.add(stream, 1)
root, idx := bytesutil.ToBytes32(columnIdents[i].BlockRoot), columnIdents[i].Index
isCustodied := custodiedColumns[idx]
if !isCustodied {
s.cfg.p2p.Peers().Scorers().BadResponsesScorer().Increment(stream.Conn().RemotePeer())
s.writeErrorResponseToStream(responseCodeInvalidRequest, types.ErrInvalidColumnIndex.Error(), stream)
return types.ErrInvalidColumnIndex
}
// TODO: Differentiate between blobs and columns for our storage engine
sc, err := s.cfg.blobStorage.GetColumn(root, idx)
if err != nil {
if db.IsNotFound(err) {
log.WithError(err).WithFields(logrus.Fields{
"root": fmt.Sprintf("%#x", root),
"index": idx,
}).Debugf("Peer requested data column sidecar by root not found in db")
continue
}
log.WithError(err).Errorf("unexpected db error retrieving data column, root=%x, index=%d", root, idx)
s.writeErrorResponseToStream(responseCodeServerError, types.ErrGeneric.Error(), stream)
return err
}
// If any root in the request content references a block earlier than minimum_request_epoch,
// peers MAY respond with error code 3: ResourceUnavailable or not include the data column in the response.
// note: we are deviating from the spec to allow requests for data column that are before minimum_request_epoch,
// up to the beginning of the retention period.
if sc.SignedBlockHeader.Header.Slot < minReqSlot {
s.writeErrorResponseToStream(responseCodeResourceUnavailable, types.ErrDataColumnLTMinRequest.Error(), stream)
log.WithError(types.ErrDataColumnLTMinRequest).
Debugf("requested data column for block %#x before minimum_request_epoch", columnIdents[i].BlockRoot)
return types.ErrDataColumnLTMinRequest
}
SetStreamWriteDeadline(stream, defaultWriteDuration)
if chunkErr := WriteDataColumnSidecarChunk(stream, s.cfg.chain, s.cfg.p2p.Encoding(), sc); chunkErr != nil {
log.WithError(chunkErr).Debug("Could not send a chunked response")
s.writeErrorResponseToStream(responseCodeServerError, types.ErrGeneric.Error(), stream)
tracing.AnnotateError(span, chunkErr)
return chunkErr
}
}
closeStream(stream, log)
return nil
}
func validateDataColummnsByRootRequest(colIdents types.BlobSidecarsByRootReq) error {
if uint64(len(colIdents)) > params.BeaconConfig().MaxRequestDataColumnSidecars {
return types.ErrMaxDataColumnReqExceeded
}
return nil
}
func DataColumnsRPCMinValidSlot(current primitives.Slot) (primitives.Slot, error) {
// Avoid overflow if we're running on a config where deneb is set to far future epoch.
if params.BeaconConfig().DenebForkEpoch == math.MaxUint64 || !features.Get().EnablePeerDAS {
return primitives.Slot(math.MaxUint64), nil
}
minReqEpochs := params.BeaconConfig().MinEpochsForDataColumnSidecarsRequest
currEpoch := slots.ToEpoch(current)
minStart := params.BeaconConfig().DenebForkEpoch
if currEpoch > minReqEpochs && currEpoch-minReqEpochs > minStart {
minStart = currEpoch - minReqEpochs
}
return slots.EpochStart(minStart)
}

View File

@@ -208,6 +208,29 @@ func SendBlobSidecarByRoot(
return readChunkEncodedBlobs(stream, p2pApi.Encoding(), ctxMap, blobValidatorFromRootReq(req), max)
}
func SendDataColumnSidecarByRoot(
ctx context.Context, tor blockchain.TemporalOracle, p2pApi p2p.P2P, pid peer.ID,
ctxMap ContextByteVersions, req *p2ptypes.BlobSidecarsByRootReq,
) ([]blocks.ROBlob, error) {
if uint64(len(*req)) > params.BeaconConfig().MaxRequestDataColumnSidecars {
return nil, errors.Wrapf(p2ptypes.ErrMaxDataColumnReqExceeded, "length=%d", len(*req))
}
topic, err := p2p.TopicFromMessage(p2p.DataColumnSidecarsByRootName, slots.ToEpoch(tor.CurrentSlot()))
if err != nil {
return nil, err
}
log.WithField("topic", topic).Debug("Sending data column sidecar request")
stream, err := p2pApi.Send(ctx, req, topic, pid)
if err != nil {
return nil, err
}
defer closeStream(stream, log)
maxCol := params.BeaconConfig().MaxRequestDataColumnSidecars
return readChunkEncodedBlobs(stream, p2pApi.Encoding(), ctxMap, blobValidatorFromRootReq(req), maxCol)
}
// BlobResponseValidation represents a function that can validate aspects of a single unmarshaled blob
// that was received from a peer in response to an rpc request.
type BlobResponseValidation func(blocks.ROBlob) error

View File

@@ -102,6 +102,7 @@ type config struct {
type blockchainService interface {
blockchain.BlockReceiver
blockchain.BlobReceiver
blockchain.DataColumnReceiver
blockchain.HeadFetcher
blockchain.FinalizationFetcher
blockchain.ForkFetcher

View File

@@ -137,13 +137,32 @@ func (s *Service) registerSubscribers(epoch primitives.Epoch, digest [4]byte) {
// New Gossip Topic in Deneb
if epoch >= params.BeaconConfig().DenebForkEpoch {
s.subscribeStaticWithSubnets(
p2p.BlobSubnetTopicFormat,
s.validateBlob, /* validator */
s.blobSubscriber, /* message handler */
digest,
params.BeaconConfig().BlobsidecarSubnetCount,
)
if features.Get().EnablePeerDAS {
if flags.Get().SubscribeToAllSubnets {
s.subscribeStaticWithSubnets(
p2p.DataColumnSubnetTopicFormat,
s.validateDataColumn, /* validator */
s.dataColumnSubscriber, /* message handler */
digest,
params.BeaconConfig().DataColumnSidecarSubnetCount,
)
} else {
s.subscribeDynamicWithColumnSubnets(
p2p.DataColumnSubnetTopicFormat,
s.validateDataColumn, /* validator */
s.dataColumnSubscriber, /* message handler */
digest,
)
}
} else {
s.subscribeStaticWithSubnets(
p2p.BlobSubnetTopicFormat,
s.validateBlob, /* validator */
s.blobSubscriber, /* message handler */
digest,
params.BeaconConfig().BlobsidecarSubnetCount,
)
}
}
}
@@ -646,6 +665,87 @@ func (s *Service) subscribeDynamicWithSyncSubnets(
}()
}
// subscribe missing subnets for our persistent columns.
func (s *Service) subscribeColumnSubnet(
subscriptions map[uint64]*pubsub.Subscription,
idx uint64,
digest [4]byte,
validate wrappedVal,
handle subHandler,
) {
// do not subscribe if we have no peers in the same
// subnet
topic := p2p.GossipTypeMapping[reflect.TypeOf(&ethpb.DataColumnSidecar{})]
subnetTopic := fmt.Sprintf(topic, digest, idx)
// check if subscription exists and if not subscribe the relevant subnet.
if _, exists := subscriptions[idx]; !exists {
subscriptions[idx] = s.subscribeWithBase(subnetTopic, validate, handle)
}
if !s.validPeersExist(subnetTopic) {
log.Debugf("No peers found subscribed to column gossip subnet with "+
"column index %d. Searching network for peers subscribed to the subnet.", idx)
_, err := s.cfg.p2p.FindPeersWithSubnet(s.ctx, subnetTopic, idx, flags.Get().MinimumPeersPerSubnet)
if err != nil {
log.WithError(err).Debug("Could not search for peers")
}
}
}
func (s *Service) subscribeDynamicWithColumnSubnets(
topicFormat string,
validate wrappedVal,
handle subHandler,
digest [4]byte,
) {
genRoot := s.cfg.clock.GenesisValidatorsRoot()
_, e, err := forks.RetrieveForkDataFromDigest(digest, genRoot[:])
if err != nil {
panic(err)
}
base := p2p.GossipTopicMappings(topicFormat, e)
if base == nil {
panic(fmt.Sprintf("%s is not mapped to any message in GossipTopicMappings", topicFormat))
}
subscriptions := make(map[uint64]*pubsub.Subscription, params.BeaconConfig().DataColumnSidecarSubnetCount)
genesis := s.cfg.clock.GenesisTime()
ticker := slots.NewSlotTicker(genesis, params.BeaconConfig().SecondsPerSlot)
go func() {
for {
select {
case <-s.ctx.Done():
ticker.Done()
return
case <-ticker.C():
if s.chainStarted.IsSet() && s.cfg.initialSync.Syncing() {
continue
}
valid, err := isDigestValid(digest, genesis, genRoot)
if err != nil {
log.Error(err)
continue
}
if !valid {
log.Warnf("Column subnets with digest %#x are no longer valid, unsubscribing from all of them.", digest)
// Unsubscribes from all our current subnets.
s.reValidateSubscriptions(subscriptions, []uint64{}, topicFormat, digest)
ticker.Done()
return
}
wantedSubs := s.retrieveActiveColumnSubnets()
// Resize as appropriate.
s.reValidateSubscriptions(subscriptions, wantedSubs, topicFormat, digest)
// subscribe desired column subnets.
for _, idx := range wantedSubs {
s.subscribeColumnSubnet(subscriptions, idx, digest, validate, handle)
}
}
}
}()
}
// lookup peers for attester specific subnets.
func (s *Service) lookupAttesterSubnets(digest [4]byte, idx uint64) {
topic := p2p.GossipTypeMapping[reflect.TypeOf(&ethpb.Attestation{})]
@@ -697,6 +797,14 @@ func (*Service) retrieveActiveSyncSubnets(currEpoch primitives.Epoch) []uint64 {
return slice.SetUint64(subs)
}
func (*Service) retrieveActiveColumnSubnets() []uint64 {
subs, ok, _ := cache.ColumnSubnetIDs.GetColumnSubnets()
if !ok {
return nil
}
return subs
}
// filters out required peers for the node to function, not
// pruning peers who are in our attestation subnets.
func (s *Service) filterNeededPeers(pids []peer.ID) []peer.ID {

View File

@@ -0,0 +1,34 @@
package sync
import (
"context"
"fmt"
"github.com/prysmaticlabs/prysm/v5/beacon-chain/core/feed"
opfeed "github.com/prysmaticlabs/prysm/v5/beacon-chain/core/feed/operation"
ethpb "github.com/prysmaticlabs/prysm/v5/proto/prysm/v1alpha1"
"google.golang.org/protobuf/proto"
)
func (s *Service) dataColumnSubscriber(ctx context.Context, msg proto.Message) error {
b, ok := msg.(*ethpb.DataColumnSidecar)
if !ok {
return fmt.Errorf("message was not type DataColumnSidecar, type=%T", msg)
}
// TODO:Change to new one for data columns
s.setSeenBlobIndex(b.SignedBlockHeader.Header.Slot, b.SignedBlockHeader.Header.ProposerIndex, b.ColumnIndex)
if err := s.cfg.chain.ReceiveDataColumn(ctx, b); err != nil {
return err
}
s.cfg.operationNotifier.OperationFeed().Send(&feed.Event{
Type: opfeed.DataColumnSidecarReceived,
Data: &opfeed.DataColumnSidecarReceivedData{
DataColumn: b,
},
})
return nil
}

View File

@@ -0,0 +1,138 @@
package sync
import (
"context"
"fmt"
"strings"
pubsub "github.com/libp2p/go-libp2p-pubsub"
"github.com/libp2p/go-libp2p/core/peer"
"github.com/pkg/errors"
"github.com/prysmaticlabs/prysm/v5/beacon-chain/blockchain"
"github.com/prysmaticlabs/prysm/v5/beacon-chain/core/blocks"
"github.com/prysmaticlabs/prysm/v5/beacon-chain/core/helpers"
"github.com/prysmaticlabs/prysm/v5/beacon-chain/core/transition"
"github.com/prysmaticlabs/prysm/v5/config/params"
eth "github.com/prysmaticlabs/prysm/v5/proto/prysm/v1alpha1"
prysmTime "github.com/prysmaticlabs/prysm/v5/time"
"github.com/prysmaticlabs/prysm/v5/time/slots"
"github.com/sirupsen/logrus"
)
func (s *Service) validateDataColumn(ctx context.Context, pid peer.ID, msg *pubsub.Message) (pubsub.ValidationResult, error) {
receivedTime := prysmTime.Now()
if pid == s.cfg.p2p.PeerID() {
return pubsub.ValidationAccept, nil
}
if s.cfg.initialSync.Syncing() {
return pubsub.ValidationIgnore, nil
}
if msg.Topic == nil {
return pubsub.ValidationReject, errInvalidTopic
}
m, err := s.decodePubsubMessage(msg)
if err != nil {
log.WithError(err).Error("Failed to decode message")
return pubsub.ValidationReject, err
}
ds, ok := m.(*eth.DataColumnSidecar)
if !ok {
log.WithField("message", m).Error("Message is not of type *eth.DataColumnSidecar")
return pubsub.ValidationReject, errWrongMessage
}
if ds.ColumnIndex >= params.BeaconConfig().NumberOfColumns {
return pubsub.ValidationReject, errors.Errorf("invalid column index provided, got %d", ds.ColumnIndex)
}
want := fmt.Sprintf("data_column_sidecar_%d", computeSubnetForColumnSidecar(ds.ColumnIndex))
if !strings.Contains(*msg.Topic, want) {
log.Debug("Column Sidecar index does not match topic")
return pubsub.ValidationReject, fmt.Errorf("wrong topic name: %s", *msg.Topic)
}
if err := slots.VerifyTime(uint64(s.cfg.clock.GenesisTime().Unix()), ds.SignedBlockHeader.Header.Slot, params.BeaconConfig().MaximumGossipClockDisparityDuration()); err != nil {
log.WithError(err).Debug("Ignored sidecar: could not verify slot time")
return pubsub.ValidationIgnore, nil
}
cp := s.cfg.chain.FinalizedCheckpt()
startSlot, err := slots.EpochStart(cp.Epoch)
if err != nil {
log.WithError(err).Debug("Ignored column sidecar: could not calculate epoch start slot")
return pubsub.ValidationIgnore, nil
}
if startSlot >= ds.SignedBlockHeader.Header.Slot {
err := fmt.Errorf("finalized slot %d greater or equal to block slot %d", startSlot, ds.SignedBlockHeader.Header.Slot)
log.Debug(err)
return pubsub.ValidationIgnore, err
}
// Handle sidecar when the parent is unknown.
if !s.cfg.chain.HasBlock(ctx, [32]byte(ds.SignedBlockHeader.Header.ParentRoot)) {
err := errors.Errorf("unknown parent for data column sidecar with slot %d and parent root %#x", ds.SignedBlockHeader.Header.Slot, ds.SignedBlockHeader.Header.ParentRoot)
log.WithError(err).Debug("Could not identify parent for data column sidecar")
return pubsub.ValidationIgnore, err
}
if s.hasBadBlock([32]byte(ds.SignedBlockHeader.Header.ParentRoot)) {
bRoot, err := ds.SignedBlockHeader.Header.HashTreeRoot()
if err != nil {
return pubsub.ValidationIgnore, err
}
s.setBadBlock(ctx, bRoot)
return pubsub.ValidationReject, errors.Errorf("column sidecar with bad parent provided")
}
parentSlot, err := s.cfg.chain.RecentBlockSlot([32]byte(ds.SignedBlockHeader.Header.ParentRoot))
if err != nil {
return pubsub.ValidationIgnore, err
}
if ds.SignedBlockHeader.Header.Slot <= parentSlot {
return pubsub.ValidationReject, errors.Errorf("invalid column sidecar slot: %d", ds.SignedBlockHeader.Header.Slot)
}
if !s.cfg.chain.InForkchoice([32]byte(ds.SignedBlockHeader.Header.ParentRoot)) {
return pubsub.ValidationReject, blockchain.ErrNotDescendantOfFinalized
}
// TODO Verify KZG inclusion proof of data column sidecar
// TODO Verify KZG proofs of column sidecar
parentState, err := s.cfg.stateGen.StateByRoot(ctx, [32]byte(ds.SignedBlockHeader.Header.ParentRoot))
if err != nil {
return pubsub.ValidationIgnore, err
}
if err := blocks.VerifyBlockHeaderSignatureUsingCurrentFork(parentState, ds.SignedBlockHeader); err != nil {
return pubsub.ValidationReject, err
}
// In the event the block is more than an epoch ahead from its
// parent state, we have to advance the state forward.
parentRoot := ds.SignedBlockHeader.Header.ParentRoot
parentState, err = transition.ProcessSlotsUsingNextSlotCache(ctx, parentState, parentRoot, ds.SignedBlockHeader.Header.Slot)
if err != nil {
return pubsub.ValidationIgnore, err
}
idx, err := helpers.BeaconProposerIndex(ctx, parentState)
if err != nil {
return pubsub.ValidationIgnore, err
}
if ds.SignedBlockHeader.Header.ProposerIndex != idx {
return pubsub.ValidationReject, errors.New("incorrect proposer index")
}
startTime, err := slots.ToTime(uint64(s.cfg.chain.GenesisTime().Unix()), ds.SignedBlockHeader.Header.Slot)
if err != nil {
return pubsub.ValidationIgnore, err
}
sinceSlotStartTime := receivedTime.Sub(startTime)
validationTime := s.cfg.clock.Now().Sub(receivedTime)
log.WithFields(logrus.Fields{
"sinceSlotStartTime": sinceSlotStartTime,
"validationTime": validationTime,
}).Debug("Received data column sidecar")
msg.ValidatorData = ds
return pubsub.ValidationAccept, nil
}
func computeSubnetForColumnSidecar(colIdx uint64) uint64 {
return colIdx % params.BeaconConfig().DataColumnSidecarSubnetCount
}

View File

@@ -73,6 +73,8 @@ type Flags struct {
PrepareAllPayloads bool // PrepareAllPayloads informs the engine to prepare a block on every slot.
// BlobSaveFsync requires blob saving to block on fsync to ensure blobs are durably persisted before passing DA.
BlobSaveFsync bool
// EnablePeerDAS enables running the node with the experimental data availability sampling scheme.
EnablePeerDAS bool
SaveInvalidBlock bool // SaveInvalidBlock saves invalid block to temp.
SaveInvalidBlob bool // SaveInvalidBlob saves invalid blob to temp.
@@ -254,6 +256,12 @@ func ConfigureBeaconChain(ctx *cli.Context) error {
logEnabled(EnableQUIC)
cfg.EnableQUIC = true
}
// For the p.o.c we enable it by default.
cfg.EnablePeerDAS = true
if ctx.IsSet(EnablePeerDAS.Name) {
logEnabled(EnablePeerDAS)
cfg.EnablePeerDAS = true
}
cfg.AggregateIntervals = [3]time.Duration{aggregateFirstInterval.Value, aggregateSecondInterval.Value, aggregateThirdInterval.Value}
Init(cfg)

View File

@@ -166,6 +166,10 @@ var (
Name: "enable-quic",
Usage: "Enables connection using the QUIC protocol for peers which support it.",
}
EnablePeerDAS = &cli.BoolFlag{
Name: "peer-das",
Usage: "Enables Prysm to run with the experimental peer data availability sampling scheme.",
}
)
// devModeFlags holds list of flags that are set when development mode is on.
@@ -223,6 +227,7 @@ var BeaconChainFlags = append(deprecatedBeaconFlags, append(deprecatedFlags, []c
EnableLightClient,
BlobSaveFsync,
EnableQUIC,
EnablePeerDAS,
}...)...)
// E2EBeaconChainFlags contains a list of the beacon chain feature flags to be tested in E2E.

View File

@@ -253,6 +253,11 @@ type BeaconChainConfig struct {
MaxDepositRequestsPerPayload uint64 `yaml:"MAX_DEPOSIT_REQUESTS_PER_PAYLOAD" spec:"true"` // MaxDepositRequestsPerPayload is the maximum number of execution layer deposits in each payload
UnsetDepositRequestsStartIndex uint64 `yaml:"UNSET_DEPOSIT_REQUESTS_START_INDEX" spec:"true"` // UnsetDepositRequestsStartIndex is used to check the start index for eip6110
// PeerDAS Values
SamplesPerSlot uint64 `yaml:"SAMPLES_PER_SLOT"` // SamplesPerSlot refers to the humber of random samples a node queries per slot.
CustodyRequirement uint64 `yaml:"CUSTODY_REQUIREMENT"` // CustodyRequirement refers to the minimum amount of subnets a peer must custody and serve samples from.
MinEpochsForDataColumnSidecarsRequest primitives.Epoch `yaml:"MIN_EPOCHS_FOR_DATA_COLUMN_SIDECARS_REQUESTS"` // MinEpochsForDataColumnSidecarsRequest is the minimum number of epochs the node will keep the data columns for.
// Networking Specific Parameters
GossipMaxSize uint64 `yaml:"GOSSIP_MAX_SIZE" spec:"true"` // GossipMaxSize is the maximum allowed size of uncompressed gossip messages.
MaxChunkSize uint64 `yaml:"MAX_CHUNK_SIZE" spec:"true"` // MaxChunkSize is the maximum allowed size of uncompressed req/resp chunked responses.

View File

@@ -25,14 +25,13 @@ import (
// IMPORTANT: Use one field per line and sort these alphabetically to reduce conflicts.
var placeholderFields = []string{
"BYTES_PER_LOGS_BLOOM", // Compile time constant on ExecutionPayload.logs_bloom.
"CUSTODY_REQUIREMENT",
"EIP6110_FORK_EPOCH",
"EIP6110_FORK_VERSION",
"EIP7002_FORK_EPOCH",
"EIP7002_FORK_VERSION",
"EIP7594_FORK_EPOCH",
"EIP7594_FORK_VERSION",
"FIELD_ELEMENTS_PER_BLOB", // Compile time constant.
"FIELD_ELEMENTS_PER_BLOB", // Compile time constant.CUSTODY_REQUIREMENT
"KZG_COMMITMENT_INCLUSION_PROOF_DEPTH", // Compile time constant on BlobSidecar.commitment_inclusion_proof.
"MAX_BLOBS_PER_BLOCK",
"MAX_BLOB_COMMITMENTS_PER_BLOCK", // Compile time constant on BeaconBlockBodyDeneb.blob_kzg_commitments.
@@ -40,7 +39,6 @@ var placeholderFields = []string{
"MAX_EXTRA_DATA_BYTES", // Compile time constant on ExecutionPayload.extra_data.
"MAX_TRANSACTIONS_PER_PAYLOAD", // Compile time constant on ExecutionPayload.transactions.
"REORG_HEAD_WEIGHT_THRESHOLD",
"SAMPLES_PER_SLOT",
"TARGET_NUMBER_OF_PEERS",
"UPDATE_TIMEOUT",
"WHISK_EPOCHS_PER_SHUFFLING_PHASE",

View File

@@ -293,6 +293,11 @@ var mainnetBeaconConfig = &BeaconChainConfig{
MaxDepositRequestsPerPayload: 8192, // 2**13 (= 8192)
UnsetDepositRequestsStartIndex: math.MaxUint64,
// PeerDAS
SamplesPerSlot: 8,
CustodyRequirement: 1,
MinEpochsForDataColumnSidecarsRequest: 4096,
// Values related to networking parameters.
GossipMaxSize: 10 * 1 << 20, // 10 MiB
MaxChunkSize: 10 * 1 << 20, // 10 MiB

View File

@@ -80,6 +80,32 @@ func MerkleProofKZGCommitment(body interfaces.ReadOnlyBeaconBlockBody, index int
return proof, nil
}
// MerkleProofKZGCommitments constructs a Merkle proof of inclusion of the KZG
// commitments into the Beacon Block with the given `body`
func MerkleProofKZGCommitments(body interfaces.ReadOnlyBeaconBlockBody) ([][]byte, error) {
bodyVersion := body.Version()
if bodyVersion < version.Deneb {
return nil, errUnsupportedBeaconBlockBody
}
membersRoots, err := topLevelRoots(body)
if err != nil {
return nil, errors.Wrap(err, "top level roots")
}
sparse, err := trie.GenerateTrieFromItems(membersRoots, logBodyLength)
if err != nil {
return nil, errors.Wrap(err, "generate trie from items")
}
proof, err := sparse.MerkleProof(kzgPosition)
if err != nil {
return nil, errors.Wrap(err, "merkle proof")
}
return proof, nil
}
// leavesFromCommitments hashes each commitment to construct a slice of roots
func leavesFromCommitments(commitments [][]byte) [][]byte {
leaves := make([][]byte, len(commitments))

View File

@@ -860,8 +860,8 @@ def prysm_deps():
importpath = "github.com/ethereum/c-kzg-4844",
patch_args = ["-p1"],
patches = ["//third_party:com_github_ethereum_c_kzg_4844.patch"],
sum = "h1:3MS1s4JtA868KpJxroZoepdV0ZKBp3u/O5HcZ7R3nlY=",
version = "v0.4.0",
sum = "h1:ffWmm0RUR2+VqJsCkf94HqgEwZi2fgbm2iq+O/GdJNI=",
version = "v1.0.1-0.20240422190800-13be436f5927",
)
go_repository(
name = "com_github_ethereum_go_ethereum",

2
go.mod
View File

@@ -17,6 +17,7 @@ require (
github.com/dgraph-io/ristretto v0.0.4-0.20210318174700-74754f61e018
github.com/dustin/go-humanize v1.0.0
github.com/emicklei/dot v0.11.0
github.com/ethereum/c-kzg-4844 v1.0.1-0.20240422190800-13be436f5927
github.com/ethereum/go-ethereum v1.13.5
github.com/fjl/memsize v0.0.0-20190710130421-bcb5799ab5e5
github.com/fsnotify/fsnotify v1.6.0
@@ -134,7 +135,6 @@ require (
github.com/docker/go-units v0.5.0 // indirect
github.com/dop251/goja v0.0.0-20230806174421-c933cf95e127 // indirect
github.com/elastic/gosigar v0.14.2 // indirect
github.com/ethereum/c-kzg-4844 v0.4.0 // indirect
github.com/ferranbt/fastssz v0.0.0-20210120143747-11b9eff30ea9 // indirect
github.com/flynn/noise v1.1.0 // indirect
github.com/francoispqt/gojay v1.2.13 // indirect

4
go.sum
View File

@@ -262,8 +262,8 @@ github.com/envoyproxy/go-control-plane v0.9.7/go.mod h1:cwu0lG7PUMfa9snN8LXBig5y
github.com/envoyproxy/go-control-plane v0.9.9-0.20201210154907-fd9021fe5dad/go.mod h1:cXg6YxExXjJnVBQHBLXeUAgxn2UodCpnH306RInaBQk=
github.com/envoyproxy/go-control-plane v0.9.9-0.20210217033140-668b12f5399d/go.mod h1:cXg6YxExXjJnVBQHBLXeUAgxn2UodCpnH306RInaBQk=
github.com/envoyproxy/protoc-gen-validate v0.1.0/go.mod h1:iSmxcyjqTsJpI2R4NaDN7+kN2VEUnK/pcBlmesArF7c=
github.com/ethereum/c-kzg-4844 v0.4.0 h1:3MS1s4JtA868KpJxroZoepdV0ZKBp3u/O5HcZ7R3nlY=
github.com/ethereum/c-kzg-4844 v0.4.0/go.mod h1:VewdlzQmpT5QSrVhbBuGoCdFJkpaJlO1aQputP83wc0=
github.com/ethereum/c-kzg-4844 v1.0.1-0.20240422190800-13be436f5927 h1:ffWmm0RUR2+VqJsCkf94HqgEwZi2fgbm2iq+O/GdJNI=
github.com/ethereum/c-kzg-4844 v1.0.1-0.20240422190800-13be436f5927/go.mod h1:VewdlzQmpT5QSrVhbBuGoCdFJkpaJlO1aQputP83wc0=
github.com/ethereum/go-ethereum v1.13.5 h1:U6TCRciCqZRe4FPXmy1sMGxTfuk8P7u2UoinF3VbaFk=
github.com/ethereum/go-ethereum v1.13.5/go.mod h1:yMTu38GSuyxaYzQMViqNmQ1s3cE84abZexQmTgenWk0=
github.com/evanphx/json-patch v4.9.0+incompatible/go.mod h1:50XU6AFN0ol/bzJsmQLiYLvXMP4fmwYFNcr97nuDLSk=

View File

@@ -0,0 +1,36 @@
// Copyright 2024 Offchain Labs.
//
// Licensed under the Apache License, Version 2.0 (the "License");
// you may not use this file except in compliance with the License.
// You may obtain a copy of the License at
//
// http://www.apache.org/licenses/LICENSE-2.0
//
// Unless required by applicable law or agreed to in writing, software
// distributed under the License is distributed on an "AS IS" BASIS,
// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
// See the License for the specific language governing permissions and
// limitations under the License.
syntax = "proto3";
package ethereum.eth.v1alpha1;
import "proto/eth/ext/options.proto";
import "proto/prysm/v1alpha1/beacon_block.proto";
option csharp_namespace = "Ethereum.Eth.v1alpha1";
option go_package = "github.com/prysmaticlabs/prysm/v5/proto/prysm/v1alpha1;eth";
option java_multiple_files = true;
option java_outer_classname = "DataColumnsProto";
option java_package = "org.ethereum.eth.v1alpha1";
option php_namespace = "Ethereum\\Eth\\v1alpha1";
message DataColumnSidecar {
uint64 column_index = 1;
repeated bytes data_column = 2 [(ethereum.eth.ext.ssz_size) = "?,bytes_per_cell.size", (ethereum.eth.ext.ssz_max) = "max_blob_commitments.size"];
repeated bytes kzg_commitments = 3 [(ethereum.eth.ext.ssz_size) = "?,48", (ethereum.eth.ext.ssz_max) = "max_blob_commitments.size"];
repeated bytes kzg_proof = 4 [(ethereum.eth.ext.ssz_size) = "?,48", (ethereum.eth.ext.ssz_max) = "max_blob_commitments.size"];
SignedBeaconBlockHeader signed_block_header = 5;
repeated bytes kzg_commitments_inclusion_proof = 6 [(ethereum.eth.ext.ssz_size) = "kzg_commitments_inclusion_proof_depth.size,32"];
}

View File

@@ -128,6 +128,7 @@ ssz_deneb_objs = [
"BlobSidecar",
"BlobSidecars",
"BuilderBidDeneb",
"DataColumnSidecar",
"SignedBeaconBlockContentsDeneb",
"SignedBeaconBlockDeneb",
"SignedBlindedBeaconBlockDeneb",
@@ -360,6 +361,7 @@ ssz_proto_files(
"beacon_block.proto",
"beacon_state.proto",
"blobs.proto",
"data_columns.proto",
"sync_committee.proto",
"withdrawals.proto",
],

File diff suppressed because it is too large Load Diff

225
proto/prysm/v1alpha1/data_columns.pb.go generated Executable file
View File

@@ -0,0 +1,225 @@
// Code generated by protoc-gen-go. DO NOT EDIT.
// versions:
// protoc-gen-go v1.31.0
// protoc v4.25.1
// source: proto/prysm/v1alpha1/data_columns.proto
package eth
import (
reflect "reflect"
sync "sync"
_ "github.com/prysmaticlabs/prysm/v5/proto/eth/ext"
protoreflect "google.golang.org/protobuf/reflect/protoreflect"
protoimpl "google.golang.org/protobuf/runtime/protoimpl"
)
const (
// Verify that this generated code is sufficiently up-to-date.
_ = protoimpl.EnforceVersion(20 - protoimpl.MinVersion)
// Verify that runtime/protoimpl is sufficiently up-to-date.
_ = protoimpl.EnforceVersion(protoimpl.MaxVersion - 20)
)
type DataColumnSidecar struct {
state protoimpl.MessageState
sizeCache protoimpl.SizeCache
unknownFields protoimpl.UnknownFields
ColumnIndex uint64 `protobuf:"varint,1,opt,name=column_index,json=columnIndex,proto3" json:"column_index,omitempty"`
DataColumn [][]byte `protobuf:"bytes,2,rep,name=data_column,json=dataColumn,proto3" json:"data_column,omitempty" ssz-max:"4096" ssz-size:"?,2048"`
KzgCommitments [][]byte `protobuf:"bytes,3,rep,name=kzg_commitments,json=kzgCommitments,proto3" json:"kzg_commitments,omitempty" ssz-max:"4096" ssz-size:"?,48"`
KzgProof [][]byte `protobuf:"bytes,4,rep,name=kzg_proof,json=kzgProof,proto3" json:"kzg_proof,omitempty" ssz-max:"4096" ssz-size:"?,48"`
SignedBlockHeader *SignedBeaconBlockHeader `protobuf:"bytes,5,opt,name=signed_block_header,json=signedBlockHeader,proto3" json:"signed_block_header,omitempty"`
KzgCommitmentsInclusionProof [][]byte `protobuf:"bytes,6,rep,name=kzg_commitments_inclusion_proof,json=kzgCommitmentsInclusionProof,proto3" json:"kzg_commitments_inclusion_proof,omitempty" ssz-size:"4,32"`
}
func (x *DataColumnSidecar) Reset() {
*x = DataColumnSidecar{}
if protoimpl.UnsafeEnabled {
mi := &file_proto_prysm_v1alpha1_data_columns_proto_msgTypes[0]
ms := protoimpl.X.MessageStateOf(protoimpl.Pointer(x))
ms.StoreMessageInfo(mi)
}
}
func (x *DataColumnSidecar) String() string {
return protoimpl.X.MessageStringOf(x)
}
func (*DataColumnSidecar) ProtoMessage() {}
func (x *DataColumnSidecar) ProtoReflect() protoreflect.Message {
mi := &file_proto_prysm_v1alpha1_data_columns_proto_msgTypes[0]
if protoimpl.UnsafeEnabled && x != nil {
ms := protoimpl.X.MessageStateOf(protoimpl.Pointer(x))
if ms.LoadMessageInfo() == nil {
ms.StoreMessageInfo(mi)
}
return ms
}
return mi.MessageOf(x)
}
// Deprecated: Use DataColumnSidecar.ProtoReflect.Descriptor instead.
func (*DataColumnSidecar) Descriptor() ([]byte, []int) {
return file_proto_prysm_v1alpha1_data_columns_proto_rawDescGZIP(), []int{0}
}
func (x *DataColumnSidecar) GetColumnIndex() uint64 {
if x != nil {
return x.ColumnIndex
}
return 0
}
func (x *DataColumnSidecar) GetDataColumn() [][]byte {
if x != nil {
return x.DataColumn
}
return nil
}
func (x *DataColumnSidecar) GetKzgCommitments() [][]byte {
if x != nil {
return x.KzgCommitments
}
return nil
}
func (x *DataColumnSidecar) GetKzgProof() [][]byte {
if x != nil {
return x.KzgProof
}
return nil
}
func (x *DataColumnSidecar) GetSignedBlockHeader() *SignedBeaconBlockHeader {
if x != nil {
return x.SignedBlockHeader
}
return nil
}
func (x *DataColumnSidecar) GetKzgCommitmentsInclusionProof() [][]byte {
if x != nil {
return x.KzgCommitmentsInclusionProof
}
return nil
}
var File_proto_prysm_v1alpha1_data_columns_proto protoreflect.FileDescriptor
var file_proto_prysm_v1alpha1_data_columns_proto_rawDesc = []byte{
0x0a, 0x27, 0x70, 0x72, 0x6f, 0x74, 0x6f, 0x2f, 0x70, 0x72, 0x79, 0x73, 0x6d, 0x2f, 0x76, 0x31,
0x61, 0x6c, 0x70, 0x68, 0x61, 0x31, 0x2f, 0x64, 0x61, 0x74, 0x61, 0x5f, 0x63, 0x6f, 0x6c, 0x75,
0x6d, 0x6e, 0x73, 0x2e, 0x70, 0x72, 0x6f, 0x74, 0x6f, 0x12, 0x15, 0x65, 0x74, 0x68, 0x65, 0x72,
0x65, 0x75, 0x6d, 0x2e, 0x65, 0x74, 0x68, 0x2e, 0x76, 0x31, 0x61, 0x6c, 0x70, 0x68, 0x61, 0x31,
0x1a, 0x1b, 0x70, 0x72, 0x6f, 0x74, 0x6f, 0x2f, 0x65, 0x74, 0x68, 0x2f, 0x65, 0x78, 0x74, 0x2f,
0x6f, 0x70, 0x74, 0x69, 0x6f, 0x6e, 0x73, 0x2e, 0x70, 0x72, 0x6f, 0x74, 0x6f, 0x1a, 0x27, 0x70,
0x72, 0x6f, 0x74, 0x6f, 0x2f, 0x70, 0x72, 0x79, 0x73, 0x6d, 0x2f, 0x76, 0x31, 0x61, 0x6c, 0x70,
0x68, 0x61, 0x31, 0x2f, 0x62, 0x65, 0x61, 0x63, 0x6f, 0x6e, 0x5f, 0x62, 0x6c, 0x6f, 0x63, 0x6b,
0x2e, 0x70, 0x72, 0x6f, 0x74, 0x6f, 0x22, 0x86, 0x03, 0x0a, 0x11, 0x44, 0x61, 0x74, 0x61, 0x43,
0x6f, 0x6c, 0x75, 0x6d, 0x6e, 0x53, 0x69, 0x64, 0x65, 0x63, 0x61, 0x72, 0x12, 0x21, 0x0a, 0x0c,
0x63, 0x6f, 0x6c, 0x75, 0x6d, 0x6e, 0x5f, 0x69, 0x6e, 0x64, 0x65, 0x78, 0x18, 0x01, 0x20, 0x01,
0x28, 0x04, 0x52, 0x0b, 0x63, 0x6f, 0x6c, 0x75, 0x6d, 0x6e, 0x49, 0x6e, 0x64, 0x65, 0x78, 0x12,
0x33, 0x0a, 0x0b, 0x64, 0x61, 0x74, 0x61, 0x5f, 0x63, 0x6f, 0x6c, 0x75, 0x6d, 0x6e, 0x18, 0x02,
0x20, 0x03, 0x28, 0x0c, 0x42, 0x12, 0x8a, 0xb5, 0x18, 0x06, 0x3f, 0x2c, 0x32, 0x30, 0x34, 0x38,
0x92, 0xb5, 0x18, 0x04, 0x34, 0x30, 0x39, 0x36, 0x52, 0x0a, 0x64, 0x61, 0x74, 0x61, 0x43, 0x6f,
0x6c, 0x75, 0x6d, 0x6e, 0x12, 0x39, 0x0a, 0x0f, 0x6b, 0x7a, 0x67, 0x5f, 0x63, 0x6f, 0x6d, 0x6d,
0x69, 0x74, 0x6d, 0x65, 0x6e, 0x74, 0x73, 0x18, 0x03, 0x20, 0x03, 0x28, 0x0c, 0x42, 0x10, 0x8a,
0xb5, 0x18, 0x04, 0x3f, 0x2c, 0x34, 0x38, 0x92, 0xb5, 0x18, 0x04, 0x34, 0x30, 0x39, 0x36, 0x52,
0x0e, 0x6b, 0x7a, 0x67, 0x43, 0x6f, 0x6d, 0x6d, 0x69, 0x74, 0x6d, 0x65, 0x6e, 0x74, 0x73, 0x12,
0x2d, 0x0a, 0x09, 0x6b, 0x7a, 0x67, 0x5f, 0x70, 0x72, 0x6f, 0x6f, 0x66, 0x18, 0x04, 0x20, 0x03,
0x28, 0x0c, 0x42, 0x10, 0x8a, 0xb5, 0x18, 0x04, 0x3f, 0x2c, 0x34, 0x38, 0x92, 0xb5, 0x18, 0x04,
0x34, 0x30, 0x39, 0x36, 0x52, 0x08, 0x6b, 0x7a, 0x67, 0x50, 0x72, 0x6f, 0x6f, 0x66, 0x12, 0x5e,
0x0a, 0x13, 0x73, 0x69, 0x67, 0x6e, 0x65, 0x64, 0x5f, 0x62, 0x6c, 0x6f, 0x63, 0x6b, 0x5f, 0x68,
0x65, 0x61, 0x64, 0x65, 0x72, 0x18, 0x05, 0x20, 0x01, 0x28, 0x0b, 0x32, 0x2e, 0x2e, 0x65, 0x74,
0x68, 0x65, 0x72, 0x65, 0x75, 0x6d, 0x2e, 0x65, 0x74, 0x68, 0x2e, 0x76, 0x31, 0x61, 0x6c, 0x70,
0x68, 0x61, 0x31, 0x2e, 0x53, 0x69, 0x67, 0x6e, 0x65, 0x64, 0x42, 0x65, 0x61, 0x63, 0x6f, 0x6e,
0x42, 0x6c, 0x6f, 0x63, 0x6b, 0x48, 0x65, 0x61, 0x64, 0x65, 0x72, 0x52, 0x11, 0x73, 0x69, 0x67,
0x6e, 0x65, 0x64, 0x42, 0x6c, 0x6f, 0x63, 0x6b, 0x48, 0x65, 0x61, 0x64, 0x65, 0x72, 0x12, 0x4f,
0x0a, 0x1f, 0x6b, 0x7a, 0x67, 0x5f, 0x63, 0x6f, 0x6d, 0x6d, 0x69, 0x74, 0x6d, 0x65, 0x6e, 0x74,
0x73, 0x5f, 0x69, 0x6e, 0x63, 0x6c, 0x75, 0x73, 0x69, 0x6f, 0x6e, 0x5f, 0x70, 0x72, 0x6f, 0x6f,
0x66, 0x18, 0x06, 0x20, 0x03, 0x28, 0x0c, 0x42, 0x08, 0x8a, 0xb5, 0x18, 0x04, 0x34, 0x2c, 0x33,
0x32, 0x52, 0x1c, 0x6b, 0x7a, 0x67, 0x43, 0x6f, 0x6d, 0x6d, 0x69, 0x74, 0x6d, 0x65, 0x6e, 0x74,
0x73, 0x49, 0x6e, 0x63, 0x6c, 0x75, 0x73, 0x69, 0x6f, 0x6e, 0x50, 0x72, 0x6f, 0x6f, 0x66, 0x42,
0x9b, 0x01, 0x0a, 0x19, 0x6f, 0x72, 0x67, 0x2e, 0x65, 0x74, 0x68, 0x65, 0x72, 0x65, 0x75, 0x6d,
0x2e, 0x65, 0x74, 0x68, 0x2e, 0x76, 0x31, 0x61, 0x6c, 0x70, 0x68, 0x61, 0x31, 0x42, 0x10, 0x44,
0x61, 0x74, 0x61, 0x43, 0x6f, 0x6c, 0x75, 0x6d, 0x6e, 0x73, 0x50, 0x72, 0x6f, 0x74, 0x6f, 0x50,
0x01, 0x5a, 0x3a, 0x67, 0x69, 0x74, 0x68, 0x75, 0x62, 0x2e, 0x63, 0x6f, 0x6d, 0x2f, 0x70, 0x72,
0x79, 0x73, 0x6d, 0x61, 0x74, 0x69, 0x63, 0x6c, 0x61, 0x62, 0x73, 0x2f, 0x70, 0x72, 0x79, 0x73,
0x6d, 0x2f, 0x76, 0x35, 0x2f, 0x70, 0x72, 0x6f, 0x74, 0x6f, 0x2f, 0x70, 0x72, 0x79, 0x73, 0x6d,
0x2f, 0x76, 0x31, 0x61, 0x6c, 0x70, 0x68, 0x61, 0x31, 0x3b, 0x65, 0x74, 0x68, 0xaa, 0x02, 0x15,
0x45, 0x74, 0x68, 0x65, 0x72, 0x65, 0x75, 0x6d, 0x2e, 0x45, 0x74, 0x68, 0x2e, 0x76, 0x31, 0x61,
0x6c, 0x70, 0x68, 0x61, 0x31, 0xca, 0x02, 0x15, 0x45, 0x74, 0x68, 0x65, 0x72, 0x65, 0x75, 0x6d,
0x5c, 0x45, 0x74, 0x68, 0x5c, 0x76, 0x31, 0x61, 0x6c, 0x70, 0x68, 0x61, 0x31, 0x62, 0x06, 0x70,
0x72, 0x6f, 0x74, 0x6f, 0x33,
}
var (
file_proto_prysm_v1alpha1_data_columns_proto_rawDescOnce sync.Once
file_proto_prysm_v1alpha1_data_columns_proto_rawDescData = file_proto_prysm_v1alpha1_data_columns_proto_rawDesc
)
func file_proto_prysm_v1alpha1_data_columns_proto_rawDescGZIP() []byte {
file_proto_prysm_v1alpha1_data_columns_proto_rawDescOnce.Do(func() {
file_proto_prysm_v1alpha1_data_columns_proto_rawDescData = protoimpl.X.CompressGZIP(file_proto_prysm_v1alpha1_data_columns_proto_rawDescData)
})
return file_proto_prysm_v1alpha1_data_columns_proto_rawDescData
}
var file_proto_prysm_v1alpha1_data_columns_proto_msgTypes = make([]protoimpl.MessageInfo, 1)
var file_proto_prysm_v1alpha1_data_columns_proto_goTypes = []interface{}{
(*DataColumnSidecar)(nil), // 0: ethereum.eth.v1alpha1.DataColumnSidecar
(*SignedBeaconBlockHeader)(nil), // 1: ethereum.eth.v1alpha1.SignedBeaconBlockHeader
}
var file_proto_prysm_v1alpha1_data_columns_proto_depIdxs = []int32{
1, // 0: ethereum.eth.v1alpha1.DataColumnSidecar.signed_block_header:type_name -> ethereum.eth.v1alpha1.SignedBeaconBlockHeader
1, // [1:1] is the sub-list for method output_type
1, // [1:1] is the sub-list for method input_type
1, // [1:1] is the sub-list for extension type_name
1, // [1:1] is the sub-list for extension extendee
0, // [0:1] is the sub-list for field type_name
}
func init() { file_proto_prysm_v1alpha1_data_columns_proto_init() }
func file_proto_prysm_v1alpha1_data_columns_proto_init() {
if File_proto_prysm_v1alpha1_data_columns_proto != nil {
return
}
file_proto_prysm_v1alpha1_beacon_block_proto_init()
if !protoimpl.UnsafeEnabled {
file_proto_prysm_v1alpha1_data_columns_proto_msgTypes[0].Exporter = func(v interface{}, i int) interface{} {
switch v := v.(*DataColumnSidecar); i {
case 0:
return &v.state
case 1:
return &v.sizeCache
case 2:
return &v.unknownFields
default:
return nil
}
}
}
type x struct{}
out := protoimpl.TypeBuilder{
File: protoimpl.DescBuilder{
GoPackagePath: reflect.TypeOf(x{}).PkgPath(),
RawDescriptor: file_proto_prysm_v1alpha1_data_columns_proto_rawDesc,
NumEnums: 0,
NumMessages: 1,
NumExtensions: 0,
NumServices: 0,
},
GoTypes: file_proto_prysm_v1alpha1_data_columns_proto_goTypes,
DependencyIndexes: file_proto_prysm_v1alpha1_data_columns_proto_depIdxs,
MessageInfos: file_proto_prysm_v1alpha1_data_columns_proto_msgTypes,
}.Build()
File_proto_prysm_v1alpha1_data_columns_proto = out.File
file_proto_prysm_v1alpha1_data_columns_proto_rawDesc = nil
file_proto_prysm_v1alpha1_data_columns_proto_goTypes = nil
file_proto_prysm_v1alpha1_data_columns_proto_depIdxs = nil
}

View File

@@ -0,0 +1,4 @@
//go:build ignore
// +build ignore
package ignore

View File

@@ -0,0 +1,36 @@
// Copyright 2024 Offchain Labs.
//
// Licensed under the Apache License, Version 2.0 (the "License");
// you may not use this file except in compliance with the License.
// You may obtain a copy of the License at
//
// http://www.apache.org/licenses/LICENSE-2.0
//
// Unless required by applicable law or agreed to in writing, software
// distributed under the License is distributed on an "AS IS" BASIS,
// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
// See the License for the specific language governing permissions and
// limitations under the License.
syntax = "proto3";
package ethereum.eth.v1alpha1;
import "proto/eth/ext/options.proto";
import "proto/prysm/v1alpha1/beacon_block.proto";
option csharp_namespace = "Ethereum.Eth.v1alpha1";
option go_package = "github.com/prysmaticlabs/prysm/v5/proto/prysm/v1alpha1;eth";
option java_multiple_files = true;
option java_outer_classname = "DataColumnsProto";
option java_package = "org.ethereum.eth.v1alpha1";
option php_namespace = "Ethereum\\Eth\\v1alpha1";
message DataColumnSidecar {
uint64 column_index = 1;
repeated bytes data_column = 2 [(ethereum.eth.ext.ssz_size) = "?,bytes_per_cell.size", (ethereum.eth.ext.ssz_max) = "max_blob_commitments.size"];
repeated bytes kzg_commitments = 3 [(ethereum.eth.ext.ssz_size) = "?,48", (ethereum.eth.ext.ssz_max) = "max_blob_commitments.size"];
repeated bytes kzg_proof = 4 [(ethereum.eth.ext.ssz_size) = "?,48", (ethereum.eth.ext.ssz_max) = "max_blob_commitments.size"];
SignedBeaconBlockHeader signed_block_header = 5;
repeated bytes kzg_commitments_inclusion_proof = 6 [(ethereum.eth.ext.ssz_size) = "kzg_commitments_inclusion_proof_depth.size,32"];
}

View File

@@ -1,5 +1,5 @@
// Code generated by fastssz. DO NOT EDIT.
// Hash: adfca9994daf736f0526568d87329503d997b98308a409e71cd510115380af5a
// Hash: 010cb7c7b894346eed8775f7f0b83250f7f0d9f1e06d0cd0c6929350299dcb60
package eth
import (
@@ -3593,3 +3593,302 @@ func (b *BlobIdentifier) HashTreeRootWith(hh *ssz.Hasher) (err error) {
hh.Merkleize(indx)
return
}
// MarshalSSZ ssz marshals the DataColumnSidecar object
func (d *DataColumnSidecar) MarshalSSZ() ([]byte, error) {
return ssz.MarshalSSZ(d)
}
// MarshalSSZTo ssz marshals the DataColumnSidecar object to a target array
func (d *DataColumnSidecar) MarshalSSZTo(buf []byte) (dst []byte, err error) {
dst = buf
offset := int(356)
// Field (0) 'ColumnIndex'
dst = ssz.MarshalUint64(dst, d.ColumnIndex)
// Offset (1) 'DataColumn'
dst = ssz.WriteOffset(dst, offset)
offset += len(d.DataColumn) * 2048
// Offset (2) 'KzgCommitments'
dst = ssz.WriteOffset(dst, offset)
offset += len(d.KzgCommitments) * 48
// Offset (3) 'KzgProof'
dst = ssz.WriteOffset(dst, offset)
offset += len(d.KzgProof) * 48
// Field (4) 'SignedBlockHeader'
if d.SignedBlockHeader == nil {
d.SignedBlockHeader = new(SignedBeaconBlockHeader)
}
if dst, err = d.SignedBlockHeader.MarshalSSZTo(dst); err != nil {
return
}
// Field (5) 'KzgCommitmentsInclusionProof'
if size := len(d.KzgCommitmentsInclusionProof); size != 4 {
err = ssz.ErrVectorLengthFn("--.KzgCommitmentsInclusionProof", size, 4)
return
}
for ii := 0; ii < 4; ii++ {
if size := len(d.KzgCommitmentsInclusionProof[ii]); size != 32 {
err = ssz.ErrBytesLengthFn("--.KzgCommitmentsInclusionProof[ii]", size, 32)
return
}
dst = append(dst, d.KzgCommitmentsInclusionProof[ii]...)
}
// Field (1) 'DataColumn'
if size := len(d.DataColumn); size > 4096 {
err = ssz.ErrListTooBigFn("--.DataColumn", size, 4096)
return
}
for ii := 0; ii < len(d.DataColumn); ii++ {
if size := len(d.DataColumn[ii]); size != 2048 {
err = ssz.ErrBytesLengthFn("--.DataColumn[ii]", size, 2048)
return
}
dst = append(dst, d.DataColumn[ii]...)
}
// Field (2) 'KzgCommitments'
if size := len(d.KzgCommitments); size > 4096 {
err = ssz.ErrListTooBigFn("--.KzgCommitments", size, 4096)
return
}
for ii := 0; ii < len(d.KzgCommitments); ii++ {
if size := len(d.KzgCommitments[ii]); size != 48 {
err = ssz.ErrBytesLengthFn("--.KzgCommitments[ii]", size, 48)
return
}
dst = append(dst, d.KzgCommitments[ii]...)
}
// Field (3) 'KzgProof'
if size := len(d.KzgProof); size > 4096 {
err = ssz.ErrListTooBigFn("--.KzgProof", size, 4096)
return
}
for ii := 0; ii < len(d.KzgProof); ii++ {
if size := len(d.KzgProof[ii]); size != 48 {
err = ssz.ErrBytesLengthFn("--.KzgProof[ii]", size, 48)
return
}
dst = append(dst, d.KzgProof[ii]...)
}
return
}
// UnmarshalSSZ ssz unmarshals the DataColumnSidecar object
func (d *DataColumnSidecar) UnmarshalSSZ(buf []byte) error {
var err error
size := uint64(len(buf))
if size < 356 {
return ssz.ErrSize
}
tail := buf
var o1, o2, o3 uint64
// Field (0) 'ColumnIndex'
d.ColumnIndex = ssz.UnmarshallUint64(buf[0:8])
// Offset (1) 'DataColumn'
if o1 = ssz.ReadOffset(buf[8:12]); o1 > size {
return ssz.ErrOffset
}
if o1 != 356 {
return ssz.ErrInvalidVariableOffset
}
// Offset (2) 'KzgCommitments'
if o2 = ssz.ReadOffset(buf[12:16]); o2 > size || o1 > o2 {
return ssz.ErrOffset
}
// Offset (3) 'KzgProof'
if o3 = ssz.ReadOffset(buf[16:20]); o3 > size || o2 > o3 {
return ssz.ErrOffset
}
// Field (4) 'SignedBlockHeader'
if d.SignedBlockHeader == nil {
d.SignedBlockHeader = new(SignedBeaconBlockHeader)
}
if err = d.SignedBlockHeader.UnmarshalSSZ(buf[20:228]); err != nil {
return err
}
// Field (5) 'KzgCommitmentsInclusionProof'
d.KzgCommitmentsInclusionProof = make([][]byte, 4)
for ii := 0; ii < 4; ii++ {
if cap(d.KzgCommitmentsInclusionProof[ii]) == 0 {
d.KzgCommitmentsInclusionProof[ii] = make([]byte, 0, len(buf[228:356][ii*32:(ii+1)*32]))
}
d.KzgCommitmentsInclusionProof[ii] = append(d.KzgCommitmentsInclusionProof[ii], buf[228:356][ii*32:(ii+1)*32]...)
}
// Field (1) 'DataColumn'
{
buf = tail[o1:o2]
num, err := ssz.DivideInt2(len(buf), 2048, 4096)
if err != nil {
return err
}
d.DataColumn = make([][]byte, num)
for ii := 0; ii < num; ii++ {
if cap(d.DataColumn[ii]) == 0 {
d.DataColumn[ii] = make([]byte, 0, len(buf[ii*2048:(ii+1)*2048]))
}
d.DataColumn[ii] = append(d.DataColumn[ii], buf[ii*2048:(ii+1)*2048]...)
}
}
// Field (2) 'KzgCommitments'
{
buf = tail[o2:o3]
num, err := ssz.DivideInt2(len(buf), 48, 4096)
if err != nil {
return err
}
d.KzgCommitments = make([][]byte, num)
for ii := 0; ii < num; ii++ {
if cap(d.KzgCommitments[ii]) == 0 {
d.KzgCommitments[ii] = make([]byte, 0, len(buf[ii*48:(ii+1)*48]))
}
d.KzgCommitments[ii] = append(d.KzgCommitments[ii], buf[ii*48:(ii+1)*48]...)
}
}
// Field (3) 'KzgProof'
{
buf = tail[o3:]
num, err := ssz.DivideInt2(len(buf), 48, 4096)
if err != nil {
return err
}
d.KzgProof = make([][]byte, num)
for ii := 0; ii < num; ii++ {
if cap(d.KzgProof[ii]) == 0 {
d.KzgProof[ii] = make([]byte, 0, len(buf[ii*48:(ii+1)*48]))
}
d.KzgProof[ii] = append(d.KzgProof[ii], buf[ii*48:(ii+1)*48]...)
}
}
return err
}
// SizeSSZ returns the ssz encoded size in bytes for the DataColumnSidecar object
func (d *DataColumnSidecar) SizeSSZ() (size int) {
size = 356
// Field (1) 'DataColumn'
size += len(d.DataColumn) * 2048
// Field (2) 'KzgCommitments'
size += len(d.KzgCommitments) * 48
// Field (3) 'KzgProof'
size += len(d.KzgProof) * 48
return
}
// HashTreeRoot ssz hashes the DataColumnSidecar object
func (d *DataColumnSidecar) HashTreeRoot() ([32]byte, error) {
return ssz.HashWithDefaultHasher(d)
}
// HashTreeRootWith ssz hashes the DataColumnSidecar object with a hasher
func (d *DataColumnSidecar) HashTreeRootWith(hh *ssz.Hasher) (err error) {
indx := hh.Index()
// Field (0) 'ColumnIndex'
hh.PutUint64(d.ColumnIndex)
// Field (1) 'DataColumn'
{
if size := len(d.DataColumn); size > 4096 {
err = ssz.ErrListTooBigFn("--.DataColumn", size, 4096)
return
}
subIndx := hh.Index()
for _, i := range d.DataColumn {
if len(i) != 2048 {
err = ssz.ErrBytesLength
return
}
hh.PutBytes(i)
}
numItems := uint64(len(d.DataColumn))
hh.MerkleizeWithMixin(subIndx, numItems, 4096)
}
// Field (2) 'KzgCommitments'
{
if size := len(d.KzgCommitments); size > 4096 {
err = ssz.ErrListTooBigFn("--.KzgCommitments", size, 4096)
return
}
subIndx := hh.Index()
for _, i := range d.KzgCommitments {
if len(i) != 48 {
err = ssz.ErrBytesLength
return
}
hh.PutBytes(i)
}
numItems := uint64(len(d.KzgCommitments))
hh.MerkleizeWithMixin(subIndx, numItems, 4096)
}
// Field (3) 'KzgProof'
{
if size := len(d.KzgProof); size > 4096 {
err = ssz.ErrListTooBigFn("--.KzgProof", size, 4096)
return
}
subIndx := hh.Index()
for _, i := range d.KzgProof {
if len(i) != 48 {
err = ssz.ErrBytesLength
return
}
hh.PutBytes(i)
}
numItems := uint64(len(d.KzgProof))
hh.MerkleizeWithMixin(subIndx, numItems, 4096)
}
// Field (4) 'SignedBlockHeader'
if err = d.SignedBlockHeader.HashTreeRootWith(hh); err != nil {
return
}
// Field (5) 'KzgCommitmentsInclusionProof'
{
if size := len(d.KzgCommitmentsInclusionProof); size != 4 {
err = ssz.ErrVectorLengthFn("--.KzgCommitmentsInclusionProof", size, 4)
return
}
subIndx := hh.Index()
for _, i := range d.KzgCommitmentsInclusionProof {
if len(i) != 32 {
err = ssz.ErrBytesLength
return
}
hh.Append(i)
}
hh.Merkleize(subIndx)
}
hh.Merkleize(indx)
return
}

View File

@@ -1,5 +1,5 @@
// Code generated by fastssz. DO NOT EDIT.
// Hash: 84572d8fa233c45a41477bced891ee355cc1745ae0fad290f110b7f6b5ed12e1
// Hash: 471816576c63e346ccec36bfea9338737f5fa4cd27e211dd68c9bf6455fe66ad
package eth
import (
@@ -30,20 +30,20 @@ func (a *AttestationElectra) MarshalSSZTo(buf []byte) (dst []byte, err error) {
return
}
// Field (2) 'CommitteeBits'
if size := len(a.CommitteeBits); size != 8 {
err = ssz.ErrBytesLengthFn("--.CommitteeBits", size, 8)
return
}
dst = append(dst, a.CommitteeBits...)
// Field (3) 'Signature'
// Field (2) 'Signature'
if size := len(a.Signature); size != 96 {
err = ssz.ErrBytesLengthFn("--.Signature", size, 96)
return
}
dst = append(dst, a.Signature...)
// Field (3) 'CommitteeBits'
if size := len(a.CommitteeBits); size != 8 {
err = ssz.ErrBytesLengthFn("--.CommitteeBits", size, 8)
return
}
dst = append(dst, a.CommitteeBits...)
// Field (0) 'AggregationBits'
if size := len(a.AggregationBits); size > 131072 {
err = ssz.ErrBytesLengthFn("--.AggregationBits", size, 131072)
@@ -82,17 +82,17 @@ func (a *AttestationElectra) UnmarshalSSZ(buf []byte) error {
return err
}
// Field (2) 'CommitteeBits'
if cap(a.CommitteeBits) == 0 {
a.CommitteeBits = make([]byte, 0, len(buf[132:140]))
}
a.CommitteeBits = append(a.CommitteeBits, buf[132:140]...)
// Field (3) 'Signature'
// Field (2) 'Signature'
if cap(a.Signature) == 0 {
a.Signature = make([]byte, 0, len(buf[140:236]))
a.Signature = make([]byte, 0, len(buf[132:228]))
}
a.Signature = append(a.Signature, buf[140:236]...)
a.Signature = append(a.Signature, buf[132:228]...)
// Field (3) 'CommitteeBits'
if cap(a.CommitteeBits) == 0 {
a.CommitteeBits = make([]byte, 0, len(buf[228:236]))
}
a.CommitteeBits = append(a.CommitteeBits, buf[228:236]...)
// Field (0) 'AggregationBits'
{
@@ -139,20 +139,20 @@ func (a *AttestationElectra) HashTreeRootWith(hh *ssz.Hasher) (err error) {
return
}
// Field (2) 'CommitteeBits'
if size := len(a.CommitteeBits); size != 8 {
err = ssz.ErrBytesLengthFn("--.CommitteeBits", size, 8)
return
}
hh.PutBytes(a.CommitteeBits)
// Field (3) 'Signature'
// Field (2) 'Signature'
if size := len(a.Signature); size != 96 {
err = ssz.ErrBytesLengthFn("--.Signature", size, 96)
return
}
hh.PutBytes(a.Signature)
// Field (3) 'CommitteeBits'
if size := len(a.CommitteeBits); size != 8 {
err = ssz.ErrBytesLengthFn("--.CommitteeBits", size, 8)
return
}
hh.PutBytes(a.CommitteeBits)
hh.Merkleize(indx)
return
}

View File

@@ -36,6 +36,11 @@ mainnet = {
"pending_partial_withdrawals_limit": "134217728",
"pending_consolidations_limit": "262144",
"max_consolidation_requests_per_payload.size": "1",
"field_elements_per_cell.size": "64",
"field_elements_per_ext_blob.size": "8192",
"bytes_per_cell.size": "2048", # FIELD_ELEMENTS_PER_CELL * BYTES_PER_FIELD_ELEMENT
"cells_per_blob.size": "128",
"kzg_commitments_inclusion_proof_depth.size": "4",
}
minimal = {
@@ -68,6 +73,11 @@ minimal = {
"pending_partial_withdrawals_limit": "64",
"pending_consolidations_limit": "64",
"max_consolidation_requests_per_payload.size": "1",
"field_elements_per_cell.size": "64",
"field_elements_per_ext_blob.size": "8192",
"bytes_per_cell.size": "2048", # FIELD_ELEMENTS_PER_CELL * BYTES_PER_FIELD_ELEMENT
"cells_per_blob.size": "128",
"kzg_commitments_inclusion_proof_depth.size": "4",
}
###### Rules definitions #######