From 5db8286a50df2a262c215273d3a2f58ef8869439 Mon Sep 17 00:00:00 2001 From: Patrick Date: Mon, 8 Jul 2024 11:44:45 -0400 Subject: [PATCH] Using a provider proxied commit store implementation in commit (#1080) ## Motivation Continues the work done in feature/ccip-1250 and adds a provider based implementation of a CommitStore ## Solution ![Provider Proxy Commit Store Approach](https://github.com/smartcontractkit/ccip/assets/16763321/b9bcab73-9289-4867-88bc-6a6019d86c1f) --------- Co-authored-by: ilija42 <57732589+ilija42@users.noreply.github.com> Co-authored-by: Aaron Lu <50029043+aalu1418@users.noreply.github.com> --- core/services/ocr2/delegate.go | 31 +-- .../ocr2/plugins/ccip/ccipcommit/factory.go | 1 - .../plugins/ccip/ccipcommit/initializers.go | 16 +- .../ocr2/plugins/ccip/exportinternal.go | 16 ++ .../internal/ccipdata/v1_2_0/commit_store.go | 6 +- .../plugins/ccip/prices/da_price_estimator.go | 4 +- .../ccip/prices/gas_price_estimator.go | 18 +- .../ocr2/plugins/ccip/proxycommitstore.go | 135 ++++++++++++ core/services/relay/evm/ccip.go | 205 ++++++++++++++++++ core/services/relay/evm/commit_provider.go | 29 ++- core/services/relay/evm/evm.go | 2 + 11 files changed, 397 insertions(+), 66 deletions(-) create mode 100644 core/services/ocr2/plugins/ccip/proxycommitstore.go create mode 100644 core/services/relay/evm/ccip.go diff --git a/core/services/ocr2/delegate.go b/core/services/ocr2/delegate.go index 429a086fe0..32b91c3d93 100644 --- a/core/services/ocr2/delegate.go +++ b/core/services/ocr2/delegate.go @@ -9,7 +9,7 @@ import ( "strconv" "time" - chainselectors "github.com/smartcontractkit/chain-selectors" + "gopkg.in/guregu/null.v4" cciptypes "github.com/smartcontractkit/chainlink-common/pkg/types/ccip" "github.com/smartcontractkit/chainlink-common/pkg/types/core" @@ -18,12 +18,12 @@ import ( "github.com/ethereum/go-ethereum/common" "github.com/pkg/errors" "github.com/prometheus/client_golang/prometheus" + chainselectors "github.com/smartcontractkit/chain-selectors" "github.com/smartcontractkit/libocr/commontypes" libocr2 "github.com/smartcontractkit/libocr/offchainreporting2plus" "github.com/smartcontractkit/libocr/offchainreporting2plus/ocr3types" ocrtypes "github.com/smartcontractkit/libocr/offchainreporting2plus/types" "google.golang.org/grpc" - "gopkg.in/guregu/null.v4" ocr2keepers20 "github.com/smartcontractkit/chainlink-automation/pkg/v2" ocr2keepers20config "github.com/smartcontractkit/chainlink-automation/pkg/v2/config" @@ -32,7 +32,6 @@ import ( ocr2keepers20runner "github.com/smartcontractkit/chainlink-automation/pkg/v2/runner" ocr2keepers21config "github.com/smartcontractkit/chainlink-automation/pkg/v3/config" ocr2keepers21 "github.com/smartcontractkit/chainlink-automation/pkg/v3/plugin" - commonlogger "github.com/smartcontractkit/chainlink-common/pkg/logger" "github.com/smartcontractkit/chainlink-common/pkg/loop" "github.com/smartcontractkit/chainlink-common/pkg/loop/reportingplugins" @@ -41,7 +40,6 @@ import ( "github.com/smartcontractkit/chainlink-common/pkg/types" llotypes "github.com/smartcontractkit/chainlink-common/pkg/types/llo" "github.com/smartcontractkit/chainlink-common/pkg/utils/mailbox" - "github.com/smartcontractkit/chainlink-vrf/altbn_128" dkgpkg "github.com/smartcontractkit/chainlink-vrf/dkg" "github.com/smartcontractkit/chainlink-vrf/ocr2vrf" @@ -279,12 +277,12 @@ func (d *Delegate) JobType() job.Type { return job.OffchainReporting2 } -func (d *Delegate) BeforeJobCreated(spec job.Job) { +func (d *Delegate) BeforeJobCreated(_ job.Job) { // This is only called first time the job is created d.isNewlyCreatedJob = true } -func (d *Delegate) AfterJobCreated(spec job.Job) {} -func (d *Delegate) BeforeJobDeleted(spec job.Job) {} +func (d *Delegate) AfterJobCreated(_ job.Job) {} +func (d *Delegate) BeforeJobDeleted(_ job.Job) {} func (d *Delegate) OnDeleteJob(ctx context.Context, jb job.Job) error { // If the job spec is malformed in any way, we report the error but return nil so that // the job deletion itself isn't blocked. @@ -1287,10 +1285,10 @@ func (d *Delegate) newServicesOCR2VRF( lggr.ErrorIf(d.jobORM.RecordError(ctx, jb.ID, msg), "unable to record error") }) dkgReportingPluginFactoryDecorator := func(wrapped ocrtypes.ReportingPluginFactory) ocrtypes.ReportingPluginFactory { - return promwrapper.NewPromFactory(wrapped, "DKG", string(relay.NetworkEVM), chain.ID()) + return promwrapper.NewPromFactory(wrapped, "DKG", relay.NetworkEVM, chain.ID()) } vrfReportingPluginFactoryDecorator := func(wrapped ocrtypes.ReportingPluginFactory) ocrtypes.ReportingPluginFactory { - return promwrapper.NewPromFactory(wrapped, "OCR2VRF", string(relay.NetworkEVM), chain.ID()) + return promwrapper.NewPromFactory(wrapped, "OCR2VRF", relay.NetworkEVM, chain.ID()) } noopMonitoringEndpoint := telemetry.NoopAgent{} oracles, err2 := ocr2vrf.NewOCR2VRF(ocr2vrf.DKGVRFArgs{ @@ -1815,11 +1813,6 @@ func (d *Delegate) newServicesCCIPCommit(ctx context.Context, lggr logger.Sugare return nil, ErrJobSpecNoRelayer{Err: err, PluginName: string(spec.PluginType)} } - dstChain, err := d.legacyChains.Get(dstRid.ChainID) - if err != nil { - return nil, fmt.Errorf("ccip services; failed to get chain %s: %w", dstRid.ChainID, err) - } - logError := func(msg string) { lggr.ErrorIf(d.jobORM.RecordError(context.Background(), jb.ID, msg), "unable to record error") } @@ -1846,12 +1839,6 @@ func (d *Delegate) newServicesCCIPCommit(ctx context.Context, lggr logger.Sugare return nil, err } - srcChainIDstr := strconv.FormatUint(srcChainID, 10) - srcChain, err := d.legacyChains.Get(srcChainIDstr) - if err != nil { - return nil, fmt.Errorf("open source chain: %w", err) - } - oracleArgsNoPlugin := libocr2.OCR2OracleArgs{ BinaryNetworkEndpointFactory: d.peerWrapper.Peer2, V2Bootstrappers: bootstrapPeers, @@ -1871,7 +1858,7 @@ func (d *Delegate) newServicesCCIPCommit(ctx context.Context, lggr logger.Sugare MetricsRegisterer: prometheus.WrapRegistererWith(map[string]string{"job_name": jb.Name.ValueOrZero()}, prometheus.DefaultRegisterer), } - return ccipcommit.NewCommitServices(ctx, d.ds, srcProvider, dstProvider, srcChain, dstChain, d.legacyChains, jb, lggr, d.pipelineRunner, oracleArgsNoPlugin, d.isNewlyCreatedJob, int64(srcChainID), dstChainID, logError) + return ccipcommit.NewCommitServices(ctx, d.ds, srcProvider, dstProvider, d.legacyChains, jb, lggr, d.pipelineRunner, oracleArgsNoPlugin, d.isNewlyCreatedJob, int64(srcChainID), dstChainID, logError) } func newCCIPCommitPluginBytes(isSourceProvider bool, sourceStartBlock uint64, destStartBlock uint64) config.CommitPluginConfig { @@ -1985,7 +1972,7 @@ func (d *Delegate) ccipCommitGetSrcProvider(ctx context.Context, jb job.Job, plu func (d *Delegate) newServicesCCIPExecution(ctx context.Context, lggr logger.SugaredLogger, jb job.Job, bootstrapPeers []commontypes.BootstrapperLocator, kb ocr2key.KeyBundle, ocrDB *db, lc ocrtypes.LocalConfig, transmitterID string) ([]job.ServiceCtx, error) { spec := jb.OCR2OracleSpec if spec.Relay != relay.NetworkEVM { - return nil, fmt.Errorf("Non evm chains are not supported for CCIP execution") + return nil, fmt.Errorf("non evm chains are not supported for CCIP execution") } dstRid, err := spec.RelayID() diff --git a/core/services/ocr2/plugins/ccip/ccipcommit/factory.go b/core/services/ocr2/plugins/ccip/ccipcommit/factory.go index d4f5e03221..95d6398f56 100644 --- a/core/services/ocr2/plugins/ccip/ccipcommit/factory.go +++ b/core/services/ocr2/plugins/ccip/ccipcommit/factory.go @@ -27,7 +27,6 @@ type CommitReportingPluginFactory struct { } // NewCommitReportingPluginFactory return a new CommitReportingPluginFactory. -// TODO: wire through a GasEstimatorProvider here to solve two relayer one provider problem for CommitStoreReader func NewCommitReportingPluginFactory(config CommitPluginStaticConfig) *CommitReportingPluginFactory { return &CommitReportingPluginFactory{ config: config, diff --git a/core/services/ocr2/plugins/ccip/ccipcommit/initializers.go b/core/services/ocr2/plugins/ccip/ccipcommit/initializers.go index 0ba086bac6..0d8a5414dc 100644 --- a/core/services/ocr2/plugins/ccip/ccipcommit/initializers.go +++ b/core/services/ocr2/plugins/ccip/ccipcommit/initializers.go @@ -43,7 +43,7 @@ import ( "github.com/smartcontractkit/chainlink/v2/core/services/pipeline" ) -func NewCommitServices(ctx context.Context, ds sqlutil.DataSource, srcProvider commontypes.CCIPCommitProvider, dstProvider commontypes.CCIPCommitProvider, srcChain legacyevm.Chain, dstChain legacyevm.Chain, chainSet legacyevm.LegacyChainContainer, jb job.Job, lggr logger.Logger, pr pipeline.Runner, argsNoPlugin libocr2.OCR2OracleArgs, new bool, sourceChainID int64, destChainID int64, logError func(string)) ([]job.ServiceCtx, error) { +func NewCommitServices(ctx context.Context, ds sqlutil.DataSource, srcProvider commontypes.CCIPCommitProvider, dstProvider commontypes.CCIPCommitProvider, chainSet legacyevm.LegacyChainContainer, jb job.Job, lggr logger.Logger, pr pipeline.Runner, argsNoPlugin libocr2.OCR2OracleArgs, new bool, sourceChainID int64, destChainID int64, logError func(string)) ([]job.ServiceCtx, error) { spec := jb.OCR2OracleSpec var pluginConfig ccipconfig.CommitPluginJobSpecConfig @@ -52,25 +52,19 @@ func NewCommitServices(ctx context.Context, ds sqlutil.DataSource, srcProvider c return nil, err } - // TODO CCIP-2493 EVM family specific behavior leaked for CommitStore, which requires access to two relayers - versionFinder := factory.NewEvmVersionFinder() commitStoreAddress := common.HexToAddress(spec.ContractID) - sourceMaxGasPrice := srcChain.Config().EVM().GasEstimator().PriceMax().ToInt() - commitStoreReader, err := ccip.NewCommitStoreReader(lggr, versionFinder, ccipcalc.EvmAddrToGeneric(commitStoreAddress), dstChain.Client(), dstChain.LogPoller()) - if err != nil { - return nil, err - } - - err = commitStoreReader.SetGasEstimator(ctx, srcChain.GasEstimator()) + srcCommitStore, err := srcProvider.NewCommitStoreReader(ctx, ccipcalc.EvmAddrToGeneric(commitStoreAddress)) if err != nil { return nil, err } - err = commitStoreReader.SetSourceMaxGasPrice(ctx, sourceMaxGasPrice) + dstCommitStore, err := dstProvider.NewCommitStoreReader(ctx, ccipcalc.EvmAddrToGeneric(commitStoreAddress)) if err != nil { return nil, err } + var commitStoreReader ccipdata.CommitStoreReader + commitStoreReader = ccip.NewProviderProxyCommitStoreReader(srcCommitStore, dstCommitStore) commitLggr := lggr.Named("CCIPCommit").With("sourceChain", sourceChainID, "destChain", destChainID) var priceGetter pricegetter.PriceGetter diff --git a/core/services/ocr2/plugins/ccip/exportinternal.go b/core/services/ocr2/plugins/ccip/exportinternal.go index 26922abe47..035aa6f68a 100644 --- a/core/services/ocr2/plugins/ccip/exportinternal.go +++ b/core/services/ocr2/plugins/ccip/exportinternal.go @@ -3,6 +3,7 @@ package ccip import ( "context" "math/big" + "time" "github.com/ethereum/go-ethereum/common" @@ -17,6 +18,7 @@ import ( "github.com/smartcontractkit/chainlink/v2/core/services/ocr2/plugins/ccip/internal/ccipdata/batchreader" "github.com/smartcontractkit/chainlink/v2/core/services/ocr2/plugins/ccip/internal/ccipdata/ccipdataprovider" "github.com/smartcontractkit/chainlink/v2/core/services/ocr2/plugins/ccip/internal/ccipdata/factory" + "github.com/smartcontractkit/chainlink/v2/core/services/ocr2/plugins/ccip/internal/ccipdata/v1_2_0" "github.com/smartcontractkit/chainlink/v2/core/services/ocr2/plugins/ccip/internal/pricegetter" "github.com/smartcontractkit/chainlink/v2/core/services/ocr2/plugins/ccip/internal/rpclib" ) @@ -101,3 +103,17 @@ func (c *ChainAgnosticPriceRegistry) NewPriceRegistryReader(ctx context.Context, func NewChainAgnosticPriceRegistry(provider ChainAgnosticPriceRegistryFactory) *ChainAgnosticPriceRegistry { return &ChainAgnosticPriceRegistry{provider} } + +type JSONCommitOffchainConfigV1_2_0 = v1_2_0.JSONCommitOffchainConfig +type CommitOnchainConfig = ccipdata.CommitOnchainConfig + +func NewCommitOffchainConfig( + gasPriceDeviationPPB uint32, + gasPriceHeartBeat time.Duration, + tokenPriceDeviationPPB uint32, + tokenPriceHeartBeat time.Duration, + inflightCacheExpiry time.Duration, + priceReportingDisabled bool, +) ccip.CommitOffchainConfig { + return ccipdata.NewCommitOffchainConfig(gasPriceDeviationPPB, gasPriceHeartBeat, tokenPriceDeviationPPB, tokenPriceHeartBeat, inflightCacheExpiry, priceReportingDisabled) +} diff --git a/core/services/ocr2/plugins/ccip/internal/ccipdata/v1_2_0/commit_store.go b/core/services/ocr2/plugins/ccip/internal/ccipdata/v1_2_0/commit_store.go index f94c8e1a7b..a21781e84c 100644 --- a/core/services/ocr2/plugins/ccip/internal/ccipdata/v1_2_0/commit_store.go +++ b/core/services/ocr2/plugins/ccip/internal/ccipdata/v1_2_0/commit_store.go @@ -49,7 +49,7 @@ type CommitStore struct { // Dynamic config configMu sync.RWMutex - gasPriceEstimator prices.DAGasPriceEstimator + gasPriceEstimator *prices.DAGasPriceEstimator offchainConfig cciptypes.CommitOffchainConfig } @@ -228,7 +228,6 @@ func (c JSONCommitOffchainConfig) Validate() error { return nil } -// TODO: Pass a Gas Estimator through to the plugin directly func (c *CommitStore) ChangeConfig(_ context.Context, onchainConfig []byte, offchainConfig []byte) (cciptypes.Address, error) { onchainConfigParsed, err := abihelpers.DecodeAbiStruct[ccipdata.CommitOnchainConfig](onchainConfig) if err != nil { @@ -250,7 +249,6 @@ func (c *CommitStore) ChangeConfig(_ context.Context, onchainConfig []byte, offc return "", fmt.Errorf("this CommitStore sourceMaxGasPrice is nil. SetSourceMaxGasPrice should be called before ChangeConfig") } - // TODO: do this in the factory c.gasPriceEstimator = prices.NewDAGasPriceEstimator( *c.estimator, c.sourceMaxGasPrice, @@ -450,6 +448,6 @@ func NewCommitStore(lggr logger.Logger, addr common.Address, ec client.Client, l // The fields below are initially empty and set on ChangeConfig method offchainConfig: cciptypes.CommitOffchainConfig{}, - gasPriceEstimator: prices.DAGasPriceEstimator{}, + gasPriceEstimator: nil, }, nil } diff --git a/core/services/ocr2/plugins/ccip/prices/da_price_estimator.go b/core/services/ocr2/plugins/ccip/prices/da_price_estimator.go index 83869dc0ef..7c75b9bdd9 100644 --- a/core/services/ocr2/plugins/ccip/prices/da_price_estimator.go +++ b/core/services/ocr2/plugins/ccip/prices/da_price_estimator.go @@ -26,8 +26,8 @@ func NewDAGasPriceEstimator( maxGasPrice *big.Int, deviationPPB int64, daDeviationPPB int64, -) DAGasPriceEstimator { - return DAGasPriceEstimator{ +) *DAGasPriceEstimator { + return &DAGasPriceEstimator{ execEstimator: NewExecGasPriceEstimator(estimator, maxGasPrice, deviationPPB), l1Oracle: estimator.L1Oracle(), priceEncodingLength: daGasPriceEncodingLength, diff --git a/core/services/ocr2/plugins/ccip/prices/gas_price_estimator.go b/core/services/ocr2/plugins/ccip/prices/gas_price_estimator.go index 610dc3b3da..c5cf9e1b86 100644 --- a/core/services/ocr2/plugins/ccip/prices/gas_price_estimator.go +++ b/core/services/ocr2/plugins/ccip/prices/gas_price_estimator.go @@ -54,25 +54,11 @@ func NewGasPriceEstimatorForCommitPlugin( daDeviationPPB int64, execDeviationPPB int64, ) (GasPriceEstimatorCommit, error) { - execEstimator := ExecGasPriceEstimator{ - estimator: estimator, - maxGasPrice: maxExecGasPrice, - deviationPPB: execDeviationPPB, - } - switch commitStoreVersion.String() { case "1.0.0", "1.1.0": - return execEstimator, nil + return NewExecGasPriceEstimator(estimator, maxExecGasPrice, execDeviationPPB), nil case "1.2.0": - return DAGasPriceEstimator{ - execEstimator: execEstimator, - l1Oracle: estimator.L1Oracle(), - priceEncodingLength: daGasPriceEncodingLength, - daDeviationPPB: daDeviationPPB, - daOverheadGas: 0, - gasPerDAByte: 0, - daMultiplier: 0, - }, nil + return NewDAGasPriceEstimator(estimator, maxExecGasPrice, execDeviationPPB, daDeviationPPB), nil default: return nil, errors.Errorf("Invalid commitStore version: %s", commitStoreVersion) } diff --git a/core/services/ocr2/plugins/ccip/proxycommitstore.go b/core/services/ocr2/plugins/ccip/proxycommitstore.go new file mode 100644 index 0000000000..b06f957bd5 --- /dev/null +++ b/core/services/ocr2/plugins/ccip/proxycommitstore.go @@ -0,0 +1,135 @@ +package ccip + +import ( + "context" + "fmt" + "io" + "math/big" + "time" + + "go.uber.org/multierr" + + "github.com/smartcontractkit/chainlink/v2/core/chains/evm/gas" + + cciptypes "github.com/smartcontractkit/chainlink-common/pkg/types/ccip" +) + +// The disjunct methods in IncompleteSourceCommitStoreReader and IncompleteDestCommitStoreReader satisfy the full +// CommitStoreReader iface in Union +var _ cciptypes.CommitStoreReader = (*ProviderProxyCommitStoreReader)(nil) + +// ProviderProxyCommitStoreReader is a CommitStoreReader that proxies to two custom provider grpc backed implementations +// of a CommitStoreReader. +// [ProviderProxyCommitStoreReader] lives in the memory space of the reporting plugin factory and reporting plugin, and should have no chain-specific details. +// Why? Historical implementations of a commit store consumed in reporting plugins mixed usage of a gas estimator from +// the source relayer and contract read and write abilities to a dest relayer. This is not valid in LOOP world. +type ProviderProxyCommitStoreReader struct { + srcCommitStoreReader IncompleteSourceCommitStoreReader + dstCommitStoreReader IncompleteDestCommitStoreReader +} + +// IncompleteSourceCommitStoreReader contains only the methods of CommitStoreReader that are serviced by the source chain/relayer. +type IncompleteSourceCommitStoreReader interface { + ChangeConfig(ctx context.Context, onchainConfig []byte, offchainConfig []byte) (cciptypes.Address, error) + GasPriceEstimator(ctx context.Context) (cciptypes.GasPriceEstimatorCommit, error) + OffchainConfig(ctx context.Context) (cciptypes.CommitOffchainConfig, error) + io.Closer +} + +// IncompleteDestCommitStoreReader contains only the methods of CommitStoreReader that are serviced by the dest chain/relayer. +type IncompleteDestCommitStoreReader interface { + DecodeCommitReport(ctx context.Context, report []byte) (cciptypes.CommitStoreReport, error) + EncodeCommitReport(ctx context.Context, report cciptypes.CommitStoreReport) ([]byte, error) + GetAcceptedCommitReportsGteTimestamp(ctx context.Context, ts time.Time, confirmations int) ([]cciptypes.CommitStoreReportWithTxMeta, error) + GetCommitReportMatchingSeqNum(ctx context.Context, seqNum uint64, confirmations int) ([]cciptypes.CommitStoreReportWithTxMeta, error) + GetCommitStoreStaticConfig(ctx context.Context) (cciptypes.CommitStoreStaticConfig, error) + GetExpectedNextSequenceNumber(ctx context.Context) (uint64, error) + GetLatestPriceEpochAndRound(ctx context.Context) (uint64, error) + IsBlessed(ctx context.Context, root [32]byte) (bool, error) + IsDestChainHealthy(ctx context.Context) (bool, error) + IsDown(ctx context.Context) (bool, error) + VerifyExecutionReport(ctx context.Context, report cciptypes.ExecReport) (bool, error) + io.Closer +} + +func NewProviderProxyCommitStoreReader(srcReader cciptypes.CommitStoreReader, dstReader cciptypes.CommitStoreReader) *ProviderProxyCommitStoreReader { + return &ProviderProxyCommitStoreReader{ + srcCommitStoreReader: srcReader, + dstCommitStoreReader: dstReader, + } +} + +// ChangeConfig updates the offchainConfig values for the source relayer gas estimator by calling ChangeConfig +// on the source relayer. Once this is called, GasPriceEstimator and OffchainConfig can be called. +func (p *ProviderProxyCommitStoreReader) ChangeConfig(ctx context.Context, onchainConfig []byte, offchainConfig []byte) (cciptypes.Address, error) { + return p.srcCommitStoreReader.ChangeConfig(ctx, onchainConfig, offchainConfig) +} + +func (p *ProviderProxyCommitStoreReader) DecodeCommitReport(ctx context.Context, report []byte) (cciptypes.CommitStoreReport, error) { + return p.dstCommitStoreReader.DecodeCommitReport(ctx, report) +} + +func (p *ProviderProxyCommitStoreReader) EncodeCommitReport(ctx context.Context, report cciptypes.CommitStoreReport) ([]byte, error) { + return p.dstCommitStoreReader.EncodeCommitReport(ctx, report) +} + +// GasPriceEstimator constructs a gas price estimator on the source relayer +func (p *ProviderProxyCommitStoreReader) GasPriceEstimator(ctx context.Context) (cciptypes.GasPriceEstimatorCommit, error) { + return p.srcCommitStoreReader.GasPriceEstimator(ctx) +} + +func (p *ProviderProxyCommitStoreReader) GetAcceptedCommitReportsGteTimestamp(ctx context.Context, ts time.Time, confirmations int) ([]cciptypes.CommitStoreReportWithTxMeta, error) { + return p.dstCommitStoreReader.GetAcceptedCommitReportsGteTimestamp(ctx, ts, confirmations) +} + +func (p *ProviderProxyCommitStoreReader) GetCommitReportMatchingSeqNum(ctx context.Context, seqNum uint64, confirmations int) ([]cciptypes.CommitStoreReportWithTxMeta, error) { + return p.dstCommitStoreReader.GetCommitReportMatchingSeqNum(ctx, seqNum, confirmations) +} + +func (p *ProviderProxyCommitStoreReader) GetCommitStoreStaticConfig(ctx context.Context) (cciptypes.CommitStoreStaticConfig, error) { + return p.dstCommitStoreReader.GetCommitStoreStaticConfig(ctx) +} + +func (p *ProviderProxyCommitStoreReader) GetExpectedNextSequenceNumber(ctx context.Context) (uint64, error) { + return p.dstCommitStoreReader.GetExpectedNextSequenceNumber(ctx) +} + +func (p *ProviderProxyCommitStoreReader) GetLatestPriceEpochAndRound(ctx context.Context) (uint64, error) { + return p.dstCommitStoreReader.GetLatestPriceEpochAndRound(ctx) +} + +func (p *ProviderProxyCommitStoreReader) IsBlessed(ctx context.Context, root [32]byte) (bool, error) { + return p.dstCommitStoreReader.IsBlessed(ctx, root) +} + +func (p *ProviderProxyCommitStoreReader) IsDestChainHealthy(ctx context.Context) (bool, error) { + return p.dstCommitStoreReader.IsDestChainHealthy(ctx) +} + +func (p *ProviderProxyCommitStoreReader) IsDown(ctx context.Context) (bool, error) { + return p.dstCommitStoreReader.IsDown(ctx) +} + +func (p *ProviderProxyCommitStoreReader) OffchainConfig(ctx context.Context) (cciptypes.CommitOffchainConfig, error) { + return p.srcCommitStoreReader.OffchainConfig(ctx) +} + +func (p *ProviderProxyCommitStoreReader) VerifyExecutionReport(ctx context.Context, report cciptypes.ExecReport) (bool, error) { + return p.dstCommitStoreReader.VerifyExecutionReport(ctx, report) +} + +// SetGasEstimator is invalid on ProviderProxyCommitStoreReader. The provider based impl's do not have SetGasEstimator +// defined, so this serves no purpose other than satisfying an interface. +func (p *ProviderProxyCommitStoreReader) SetGasEstimator(ctx context.Context, gpe gas.EvmFeeEstimator) error { + return fmt.Errorf("invalid usage of ProviderProxyCommitStoreReader") +} + +// SetSourceMaxGasPrice is invalid on ProviderProxyCommitStoreReader. The provider based impl's do not have SetSourceMaxGasPrice +// defined, so this serves no purpose other than satisfying an interface. +func (p *ProviderProxyCommitStoreReader) SetSourceMaxGasPrice(ctx context.Context, sourceMaxGasPrice *big.Int) error { + return fmt.Errorf("invalid usage of ProviderProxyCommitStoreReader") +} + +func (p *ProviderProxyCommitStoreReader) Close() error { + return multierr.Append(p.srcCommitStoreReader.Close(), p.dstCommitStoreReader.Close()) +} diff --git a/core/services/relay/evm/ccip.go b/core/services/relay/evm/ccip.go new file mode 100644 index 0000000000..34a732e145 --- /dev/null +++ b/core/services/relay/evm/ccip.go @@ -0,0 +1,205 @@ +package evm + +import ( + "context" + "fmt" + "math/big" + "time" + + "github.com/smartcontractkit/chainlink/v2/core/services/ocr2/plugins/ccip/abihelpers" + + "github.com/smartcontractkit/chainlink/v2/core/chains/evm/client" + "github.com/smartcontractkit/chainlink/v2/core/chains/evm/logpoller" + "github.com/smartcontractkit/chainlink/v2/core/logger" + + "github.com/smartcontractkit/chainlink/v2/core/chains/evm/gas" + "github.com/smartcontractkit/chainlink/v2/core/services/ocr2/plugins/ccip" + ccipconfig "github.com/smartcontractkit/chainlink/v2/core/services/ocr2/plugins/ccip/config" + "github.com/smartcontractkit/chainlink/v2/core/services/ocr2/plugins/ccip/prices" + + cciptypes "github.com/smartcontractkit/chainlink-common/pkg/types/ccip" +) + +var _ cciptypes.CommitStoreReader = (*IncompleteSourceCommitStoreReader)(nil) +var _ cciptypes.CommitStoreReader = (*IncompleteDestCommitStoreReader)(nil) + +// IncompleteSourceCommitStoreReader is an implementation of CommitStoreReader with the only valid methods being +// GasPriceEstimator, ChangeConfig, and OffchainConfig +type IncompleteSourceCommitStoreReader struct { + estimator gas.EvmFeeEstimator + gasPriceEstimator *prices.DAGasPriceEstimator + sourceMaxGasPrice *big.Int + offchainConfig cciptypes.CommitOffchainConfig +} + +func NewIncompleteSourceCommitStoreReader(estimator gas.EvmFeeEstimator, sourceMaxGasPrice *big.Int) *IncompleteSourceCommitStoreReader { + return &IncompleteSourceCommitStoreReader{ + estimator: estimator, + sourceMaxGasPrice: sourceMaxGasPrice, + } +} + +func (i *IncompleteSourceCommitStoreReader) ChangeConfig(ctx context.Context, onchainConfig []byte, offchainConfig []byte) (cciptypes.Address, error) { + onchainConfigParsed, err := abihelpers.DecodeAbiStruct[ccip.CommitOnchainConfig](onchainConfig) + if err != nil { + return "", err + } + + offchainConfigParsed, err := ccipconfig.DecodeOffchainConfig[ccip.JSONCommitOffchainConfigV1_2_0](offchainConfig) + if err != nil { + return "", err + } + + i.gasPriceEstimator = prices.NewDAGasPriceEstimator( + i.estimator, + i.sourceMaxGasPrice, + int64(offchainConfigParsed.ExecGasPriceDeviationPPB), + int64(offchainConfigParsed.DAGasPriceDeviationPPB), + ) + i.offchainConfig = ccip.NewCommitOffchainConfig( + offchainConfigParsed.ExecGasPriceDeviationPPB, + offchainConfigParsed.GasPriceHeartBeat.Duration(), + offchainConfigParsed.TokenPriceDeviationPPB, + offchainConfigParsed.TokenPriceHeartBeat.Duration(), + offchainConfigParsed.InflightCacheExpiry.Duration(), + offchainConfigParsed.PriceReportingDisabled, + ) + + return cciptypes.Address(onchainConfigParsed.PriceRegistry.String()), nil +} + +func (i *IncompleteSourceCommitStoreReader) DecodeCommitReport(ctx context.Context, report []byte) (cciptypes.CommitStoreReport, error) { + return cciptypes.CommitStoreReport{}, fmt.Errorf("invalid usage of IncompleteSourceCommitStoreReader") +} + +func (i *IncompleteSourceCommitStoreReader) EncodeCommitReport(ctx context.Context, report cciptypes.CommitStoreReport) ([]byte, error) { + return []byte{}, fmt.Errorf("invalid usage of IncompleteSourceCommitStoreReader") +} + +// GasPriceEstimator returns an ExecGasPriceEstimator to satisfy the GasPriceEstimatorCommit interface, +// with deviationPPB values hardcoded to 0 when this implementation is first constructed. +// When ChangeConfig is called, another call to this method must be made to fetch a GasPriceEstimator with updated values +func (i *IncompleteSourceCommitStoreReader) GasPriceEstimator(ctx context.Context) (cciptypes.GasPriceEstimatorCommit, error) { + return i.gasPriceEstimator, nil +} + +func (i *IncompleteSourceCommitStoreReader) GetAcceptedCommitReportsGteTimestamp(ctx context.Context, ts time.Time, confirmations int) ([]cciptypes.CommitStoreReportWithTxMeta, error) { + return nil, fmt.Errorf("invalid usage of IncompleteSourceCommitStoreReader") +} + +func (i *IncompleteSourceCommitStoreReader) GetCommitReportMatchingSeqNum(ctx context.Context, seqNum uint64, confirmations int) ([]cciptypes.CommitStoreReportWithTxMeta, error) { + return nil, fmt.Errorf("invalid usage of IncompleteSourceCommitStoreReader") +} + +func (i *IncompleteSourceCommitStoreReader) GetCommitStoreStaticConfig(ctx context.Context) (cciptypes.CommitStoreStaticConfig, error) { + return cciptypes.CommitStoreStaticConfig{}, fmt.Errorf("invalid usage of IncompleteSourceCommitStoreReader") +} + +func (i *IncompleteSourceCommitStoreReader) GetExpectedNextSequenceNumber(ctx context.Context) (uint64, error) { + return 0, fmt.Errorf("invalid usage of IncompleteSourceCommitStoreReader") +} + +func (i *IncompleteSourceCommitStoreReader) GetLatestPriceEpochAndRound(ctx context.Context) (uint64, error) { + return 0, fmt.Errorf("invalid usage of IncompleteSourceCommitStoreReader") +} + +func (i *IncompleteSourceCommitStoreReader) IsBlessed(ctx context.Context, root [32]byte) (bool, error) { + return false, fmt.Errorf("invalid usage of IncompleteSourceCommitStoreReader") +} + +func (i *IncompleteSourceCommitStoreReader) IsDestChainHealthy(ctx context.Context) (bool, error) { + return false, fmt.Errorf("invalid usage of IncompleteSourceCommitStoreReader") +} + +func (i *IncompleteSourceCommitStoreReader) IsDown(ctx context.Context) (bool, error) { + return false, fmt.Errorf("invalid usage of IncompleteSourceCommitStoreReader") +} + +func (i *IncompleteSourceCommitStoreReader) OffchainConfig(ctx context.Context) (cciptypes.CommitOffchainConfig, error) { + return i.offchainConfig, nil +} + +func (i *IncompleteSourceCommitStoreReader) VerifyExecutionReport(ctx context.Context, report cciptypes.ExecReport) (bool, error) { + return false, fmt.Errorf("invalid usage of IncompleteSourceCommitStoreReader") +} + +func (i *IncompleteSourceCommitStoreReader) Close() error { + return fmt.Errorf("invalid usage of IncompleteSourceCommitStoreReader") +} + +// IncompleteDestCommitStoreReader is an implementation of CommitStoreReader with all valid methods except +// GasPriceEstimator, ChangeConfig, and OffchainConfig. +type IncompleteDestCommitStoreReader struct { + cs cciptypes.CommitStoreReader +} + +func NewIncompleteDestCommitStoreReader(lggr logger.Logger, versionFinder ccip.VersionFinder, address cciptypes.Address, ec client.Client, lp logpoller.LogPoller) (*IncompleteDestCommitStoreReader, error) { + cs, err := ccip.NewCommitStoreReader(lggr, versionFinder, address, ec, lp) + if err != nil { + return nil, err + } + + return &IncompleteDestCommitStoreReader{ + cs: cs, + }, nil +} + +func (i *IncompleteDestCommitStoreReader) ChangeConfig(ctx context.Context, onchainConfig []byte, offchainConfig []byte) (cciptypes.Address, error) { + return "", fmt.Errorf("invalid usage of IncompleteDestCommitStoreReader") +} + +func (i *IncompleteDestCommitStoreReader) DecodeCommitReport(ctx context.Context, report []byte) (cciptypes.CommitStoreReport, error) { + return i.cs.DecodeCommitReport(ctx, report) +} + +func (i *IncompleteDestCommitStoreReader) EncodeCommitReport(ctx context.Context, report cciptypes.CommitStoreReport) ([]byte, error) { + return i.cs.EncodeCommitReport(ctx, report) +} + +func (i *IncompleteDestCommitStoreReader) GasPriceEstimator(ctx context.Context) (cciptypes.GasPriceEstimatorCommit, error) { + return nil, fmt.Errorf("invalid usage of IncompleteDestCommitStoreReader") +} + +func (i *IncompleteDestCommitStoreReader) GetAcceptedCommitReportsGteTimestamp(ctx context.Context, ts time.Time, confirmations int) ([]cciptypes.CommitStoreReportWithTxMeta, error) { + return i.cs.GetAcceptedCommitReportsGteTimestamp(ctx, ts, confirmations) +} + +func (i *IncompleteDestCommitStoreReader) GetCommitReportMatchingSeqNum(ctx context.Context, seqNum uint64, confirmations int) ([]cciptypes.CommitStoreReportWithTxMeta, error) { + return i.cs.GetCommitReportMatchingSeqNum(ctx, seqNum, confirmations) +} + +func (i *IncompleteDestCommitStoreReader) GetCommitStoreStaticConfig(ctx context.Context) (cciptypes.CommitStoreStaticConfig, error) { + return i.cs.GetCommitStoreStaticConfig(ctx) +} + +func (i *IncompleteDestCommitStoreReader) GetExpectedNextSequenceNumber(ctx context.Context) (uint64, error) { + return i.cs.GetExpectedNextSequenceNumber(ctx) +} + +func (i *IncompleteDestCommitStoreReader) GetLatestPriceEpochAndRound(ctx context.Context) (uint64, error) { + return i.cs.GetLatestPriceEpochAndRound(ctx) +} + +func (i *IncompleteDestCommitStoreReader) IsBlessed(ctx context.Context, root [32]byte) (bool, error) { + return i.cs.IsBlessed(ctx, root) +} + +func (i *IncompleteDestCommitStoreReader) IsDestChainHealthy(ctx context.Context) (bool, error) { + return i.cs.IsDestChainHealthy(ctx) +} + +func (i *IncompleteDestCommitStoreReader) IsDown(ctx context.Context) (bool, error) { + return i.cs.IsDown(ctx) +} + +func (i *IncompleteDestCommitStoreReader) OffchainConfig(ctx context.Context) (cciptypes.CommitOffchainConfig, error) { + return cciptypes.CommitOffchainConfig{}, fmt.Errorf("invalid usage of IncompleteDestCommitStoreReader") +} + +func (i *IncompleteDestCommitStoreReader) VerifyExecutionReport(ctx context.Context, report cciptypes.ExecReport) (bool, error) { + return i.cs.VerifyExecutionReport(ctx, report) +} + +func (i *IncompleteDestCommitStoreReader) Close() error { + return i.cs.Close() +} diff --git a/core/services/relay/evm/commit_provider.go b/core/services/relay/evm/commit_provider.go index cccf6b0276..f2a25842f0 100644 --- a/core/services/relay/evm/commit_provider.go +++ b/core/services/relay/evm/commit_provider.go @@ -22,10 +22,12 @@ var _ commontypes.CCIPCommitProvider = (*SrcCommitProvider)(nil) var _ commontypes.CCIPCommitProvider = (*DstCommitProvider)(nil) type SrcCommitProvider struct { - lggr logger.Logger - startBlock uint64 - client client.Client - lp logpoller.LogPoller + lggr logger.Logger + startBlock uint64 + client client.Client + lp logpoller.LogPoller + estimator gas.EvmFeeEstimator + maxGasPrice *big.Int } func NewSrcCommitProvider( @@ -33,12 +35,16 @@ func NewSrcCommitProvider( startBlock uint64, client client.Client, lp logpoller.LogPoller, + srcEstimator gas.EvmFeeEstimator, + maxGasPrice *big.Int, ) commontypes.CCIPCommitProvider { return &SrcCommitProvider{ - lggr: lggr, - startBlock: startBlock, - client: client, - lp: lp, + lggr: lggr, + startBlock: startBlock, + client: client, + lp: lp, + estimator: srcEstimator, + maxGasPrice: maxGasPrice, } } @@ -181,11 +187,14 @@ func (P DstCommitProvider) NewPriceGetter(ctx context.Context) (priceGetter ccip } func (P SrcCommitProvider) NewCommitStoreReader(ctx context.Context, commitStoreAddress cciptypes.Address) (commitStoreReader cciptypes.CommitStoreReader, err error) { - return nil, fmt.Errorf("can't construct a commit store reader from one relayer") + commitStoreReader = NewIncompleteSourceCommitStoreReader(P.estimator, P.maxGasPrice) + return } func (P DstCommitProvider) NewCommitStoreReader(ctx context.Context, commitStoreAddress cciptypes.Address) (commitStoreReader cciptypes.CommitStoreReader, err error) { - return nil, fmt.Errorf("can't construct a commit store reader from one relayer") + versionFinder := ccip.NewEvmVersionFinder() + commitStoreReader, err = NewIncompleteDestCommitStoreReader(P.lggr, versionFinder, commitStoreAddress, P.client, P.lp) + return } func (P SrcCommitProvider) NewOnRampReader(ctx context.Context, onRampAddress cciptypes.Address, sourceChainSelector uint64, destChainSelector uint64) (onRampReader cciptypes.OnRampReader, err error) { diff --git a/core/services/relay/evm/evm.go b/core/services/relay/evm/evm.go index 7203c142f2..cd2f4010b4 100644 --- a/core/services/relay/evm/evm.go +++ b/core/services/relay/evm/evm.go @@ -412,6 +412,8 @@ func (r *Relayer) NewCCIPCommitProvider(rargs commontypes.RelayArgs, pargs commo sourceStartBlock, r.chain.Client(), r.chain.LogPoller(), + r.chain.GasEstimator(), + r.chain.Config().EVM().GasEstimator().PriceMax().ToInt(), ), nil }