Commit 978355d8 authored by protolambda's avatar protolambda Committed by GitHub

op-e2e: cleanup endpoints and dialing (#11594)

* op-e2e: cleanup endpoints and dialing

* op-e2e: fix accidental wrong dial, fix endpoint-test, adjust proposer poll interval

* op-e2e: fix test deadlock, fix L1 RPC no-HTTP exception

* op-e2e: any RPC for test, HTTP mode only applied to nodes

* op-e2e: fix lint
parent 99d677fb
......@@ -13,6 +13,7 @@ import (
)
func TestArtifacts(t *testing.T) {
InitParallel(t)
logger := testlog.Logger(t, log.LevelWarn) // lower this log level to get verbose test dump of all artifacts
af := foundry.OpenArtifactsDir("../packages/contracts-bedrock/forge-artifacts")
artifacts, err := af.ListArtifacts()
......
......@@ -29,13 +29,12 @@ func TestERC20BridgeDeposits(t *testing.T) {
sys, err := cfg.Start(t)
require.Nil(t, err, "Error starting up system")
defer sys.Close()
log := testlog.Logger(t, log.LevelInfo)
log.Info("genesis", "l2", sys.RollupConfig.Genesis.L2, "l1", sys.RollupConfig.Genesis.L1, "l2_time", sys.RollupConfig.Genesis.L2Time)
l1Client := sys.Clients["l1"]
l2Client := sys.Clients["sequencer"]
l1Client := sys.NodeClient("l1")
l2Client := sys.NodeClient("sequencer")
opts, err := bind.NewKeyedTransactorWithChainID(sys.Cfg.Secrets.Alice, cfg.L1ChainIDBig())
require.Nil(t, err)
......
......@@ -9,23 +9,19 @@ import (
"github.com/stretchr/testify/require"
batcherFlags "github.com/ethereum-optimism/optimism/op-batcher/flags"
"github.com/ethereum-optimism/optimism/op-e2e/e2eutils/wait"
"github.com/ethereum-optimism/optimism/op-service/testlog"
"github.com/ethereum/go-ethereum/accounts/abi/bind"
"github.com/ethereum/go-ethereum/common"
"github.com/ethereum/go-ethereum/common/hexutil"
"github.com/ethereum/go-ethereum/core"
"github.com/ethereum/go-ethereum/log"
"github.com/ethereum/go-ethereum/rpc"
batcherFlags "github.com/ethereum-optimism/optimism/op-batcher/flags"
"github.com/ethereum-optimism/optimism/op-e2e/e2eutils/wait"
"github.com/ethereum-optimism/optimism/op-service/client"
"github.com/ethereum-optimism/optimism/op-service/sources"
"github.com/ethereum-optimism/optimism/op-service/testlog"
)
func setupAliceAccount(t *testing.T, cfg SystemConfig, sys *System, ethPrivKey *ecdsa.PrivateKey) {
l1Client := sys.Clients["l1"]
l2Verif := sys.Clients["verifier"]
l1Client := sys.NodeClient("l1")
l2Verif := sys.NodeClient("verifier")
// Send Transaction & wait for success
fromAddr := cfg.Secrets.Addresses().Alice
......@@ -68,13 +64,12 @@ func TestBrotliBatcherFjord(t *testing.T) {
// set up batcher to use brotli
sys, err := cfg.Start(t, SystemConfigOption{"compressionAlgo", "brotli", nil})
require.Nil(t, err, "Error starting up system")
defer sys.Close()
log := testlog.Logger(t, log.LevelInfo)
log.Info("genesis", "l2", sys.RollupConfig.Genesis.L2, "l1", sys.RollupConfig.Genesis.L1, "l2_time", sys.RollupConfig.Genesis.L2Time)
l2Seq := sys.Clients["sequencer"]
l2Verif := sys.Clients["verifier"]
l2Seq := sys.NodeClient("sequencer")
l2Verif := sys.NodeClient("verifier")
// Transactor Account and set up the account
ethPrivKey := cfg.Secrets.Alice
......@@ -101,9 +96,7 @@ func TestBrotliBatcherFjord(t *testing.T) {
require.Equal(t, verifBlock.ParentHash(), seqBlock.ParentHash(), "Verifier and sequencer blocks parent hashes not the same after including a batch tx")
require.Equal(t, verifBlock.Hash(), seqBlock.Hash(), "Verifier and sequencer blocks not the same after including a batch tx")
rollupRPCClient, err := rpc.DialContext(context.Background(), sys.RollupNodes["sequencer"].HTTPEndpoint())
require.NoError(t, err)
rollupClient := sources.NewRollupClient(client.NewBaseRPCClient(rollupRPCClient))
rollupClient := sys.RollupClient("sequencer")
// basic check that sync status works
seqStatus, err := rollupClient.SyncStatus(context.Background())
require.NoError(t, err)
......
......@@ -52,11 +52,10 @@ func TestCheckFjordScript(t *testing.T) {
sys, err := cfg.Start(t)
require.NoError(t, err, "Error starting up system")
defer sys.Close()
checkFjordConfig := &fjordChecks.CheckFjordConfig{
Log: log,
L2: sys.Clients["sequencer"],
L2: sys.NodeClient("sequencer"),
Key: sys.Cfg.Secrets.Alice,
Addr: sys.Cfg.Secrets.Addresses().Alice,
}
......
......@@ -35,9 +35,9 @@ func TestCustomGasToken(t *testing.T) {
sys, err := cfg.Start(t)
require.NoError(t, err, "Error starting up system")
defer sys.Close()
l1Client := sys.Clients["l1"]
l2Client := sys.Clients["sequencer"]
l1Client := sys.NodeClient("l1")
l2Client := sys.NodeClient("sequencer")
aliceOpts, err := bind.NewKeyedTransactorWithChainID(cfg.Secrets.Alice, cfg.L1ChainIDBig())
require.NoError(t, err)
......@@ -134,7 +134,7 @@ func TestCustomGasToken(t *testing.T) {
// and assert token balance is increased on L1.
checkWithdrawal := func(t *testing.T) {
l2Seq := l2Client
l2Verif := sys.Clients["verifier"]
l2Verif := sys.NodeClient("verifier")
fromAddr := aliceOpts.From
ethPrivKey := cfg.Secrets.Alice
......@@ -463,7 +463,7 @@ func callViaSafe(opts *bind.TransactOpts, client *ethclient.Client, safeAddress
// It reads existing parameters from the SystemConfig contract, inserts the supplied cgtAddress and reinitializes that contract.
// To do this it uses the ProxyAdmin and StorageSetter from the supplied cfg.
func setCustomGasToken(t *testing.T, cfg SystemConfig, sys *System, cgtAddress common.Address) {
l1Client := sys.Clients["l1"]
l1Client := sys.NodeClient("l1")
deployerOpts, err := bind.NewKeyedTransactorWithChainID(cfg.Secrets.Deployer, cfg.L1ChainIDBig())
require.NoError(t, err)
......@@ -581,7 +581,7 @@ func setCustomGasToken(t *testing.T, cfg SystemConfig, sys *System, cgtAddress c
depositTx, err := derive.UnmarshalDepositLogEvent(&depositEvent.Raw)
require.NoError(t, err)
l2Client := sys.Clients["sequencer"]
l2Client := sys.NodeClient("sequencer")
receipt, err = wait.ForReceiptOK(context.Background(), l2Client, types.NewTx(depositTx).Hash())
require.NoError(t, err)
......
......@@ -21,10 +21,9 @@ func TestMintOnRevertedDeposit(t *testing.T) {
delete(cfg.Nodes, "verifier")
sys, err := cfg.Start(t)
require.NoError(t, err, "Error starting up system")
defer sys.Close()
l1Client := sys.Clients["l1"]
l2Verif := sys.Clients["sequencer"]
l1Client := sys.NodeClient("l1")
l2Verif := sys.NodeClient("sequencer")
// create signer
aliceKey := cfg.Secrets.Alice
......@@ -82,10 +81,9 @@ func TestDepositTxCreateContract(t *testing.T) {
sys, err := cfg.Start(t)
require.NoError(t, err, "Error starting up system")
defer sys.Close()
l1Client := sys.Clients["l1"]
l2Client := sys.Clients["sequencer"]
l1Client := sys.NodeClient("l1")
l2Client := sys.NodeClient("sequencer")
opts, err := bind.NewKeyedTransactorWithChainID(cfg.Secrets.Alice, cfg.L1ChainIDBig())
require.NoError(t, err)
......
......@@ -11,7 +11,6 @@ import (
"testing"
"time"
"github.com/ethereum-optimism/optimism/op-service/metrics"
"github.com/ethereum/go-ethereum/ethclient"
"github.com/stretchr/testify/require"
......@@ -26,13 +25,15 @@ import (
"github.com/ethereum-optimism/optimism/op-e2e/e2eutils/wait"
"github.com/ethereum-optimism/optimism/op-node/rollup"
"github.com/ethereum-optimism/optimism/op-service/cliapp"
"github.com/ethereum-optimism/optimism/op-service/endpoint"
"github.com/ethereum-optimism/optimism/op-service/metrics"
"github.com/ethereum-optimism/optimism/op-service/testlog"
)
type EndpointProvider interface {
NodeEndpoint(name string) string
RollupEndpoint(name string) string
L1BeaconEndpoint() string
NodeEndpoint(name string) endpoint.RPC
RollupEndpoint(name string) endpoint.RPC
L1BeaconEndpoint() endpoint.RestHTTP
}
type Helper struct {
......@@ -155,9 +156,9 @@ func NewChallenger(t *testing.T, ctx context.Context, sys EndpointProvider, name
func NewChallengerConfig(t *testing.T, sys EndpointProvider, l2NodeName string, options ...Option) *config.Config {
// Use the NewConfig method to ensure we pick up any defaults that are set.
l1Endpoint := sys.NodeEndpoint("l1")
l1Beacon := sys.L1BeaconEndpoint()
cfg := config.NewConfig(common.Address{}, l1Endpoint, l1Beacon, sys.RollupEndpoint(l2NodeName), sys.NodeEndpoint(l2NodeName), t.TempDir())
l1Endpoint := sys.NodeEndpoint("l1").RPC()
l1Beacon := sys.L1BeaconEndpoint().RestHTTP()
cfg := config.NewConfig(common.Address{}, l1Endpoint, l1Beacon, sys.RollupEndpoint(l2NodeName).RPC(), sys.NodeEndpoint(l2NodeName).RPC(), t.TempDir())
// The devnet can't set the absolute prestate output root because the contracts are deployed in L1 genesis
// before the L2 genesis is known.
cfg.AllowInvalidPrestate = true
......
......@@ -19,6 +19,7 @@ import (
"github.com/ethereum-optimism/optimism/op-e2e/e2eutils/transactions"
"github.com/ethereum-optimism/optimism/op-e2e/e2eutils/wait"
"github.com/ethereum-optimism/optimism/op-node/rollup"
"github.com/ethereum-optimism/optimism/op-service/endpoint"
"github.com/ethereum-optimism/optimism/op-service/eth"
"github.com/ethereum-optimism/optimism/op-service/sources"
"github.com/ethereum-optimism/optimism/op-service/sources/batching"
......@@ -70,10 +71,10 @@ func WithFutureProposal() GameOpt {
}
type DisputeSystem interface {
L1BeaconEndpoint() string
NodeEndpoint(name string) string
L1BeaconEndpoint() endpoint.RestHTTP
NodeEndpoint(name string) endpoint.RPC
NodeClient(name string) *ethclient.Client
RollupEndpoint(name string) string
RollupEndpoint(name string) endpoint.RPC
RollupClient(name string) *sources.RollupClient
L1Deployments() *genesis.L1Deployments
......
......@@ -2,10 +2,8 @@ package geth
import (
"fmt"
"math/big"
"time"
"github.com/ethereum-optimism/optimism/op-service/clock"
"github.com/ethereum/go-ethereum/cmd/utils"
"github.com/ethereum/go-ethereum/common"
"github.com/ethereum/go-ethereum/core"
......@@ -22,11 +20,13 @@ import (
// Force-load the tracer engines to trigger registration
_ "github.com/ethereum/go-ethereum/eth/tracers/js"
_ "github.com/ethereum/go-ethereum/eth/tracers/native"
"github.com/ethereum-optimism/optimism/op-service/clock"
)
func InitL1(chainID uint64, blockTime uint64, finalizedDistance uint64, genesis *core.Genesis, c clock.Clock, blobPoolDir string, beaconSrv Beacon, opts ...GethOption) (*node.Node, *eth.Ethereum, error) {
func InitL1(blockTime uint64, finalizedDistance uint64, genesis *core.Genesis, c clock.Clock, blobPoolDir string, beaconSrv Beacon, opts ...GethOption) (*GethInstance, error) {
ethConfig := &ethconfig.Config{
NetworkId: chainID,
NetworkId: genesis.Config.ChainID.Uint64(),
Genesis: genesis,
BlobPool: blobpool.Config{
Datadir: blobPoolDir,
......@@ -53,24 +53,24 @@ func InitL1(chainID uint64, blockTime uint64, finalizedDistance uint64, genesis
HTTPModules: []string{"debug", "admin", "eth", "txpool", "net", "rpc", "web3", "personal", "engine"},
}
l1Node, l1Eth, err := createGethNode(false, nodeConfig, ethConfig, opts...)
gethInstance, err := createGethNode(false, nodeConfig, ethConfig, opts...)
if err != nil {
return nil, nil, err
return nil, err
}
// Instead of running a whole beacon node, we run this fake-proof-of-stake sidecar that sequences L1 blocks using the Engine API.
l1Node.RegisterLifecycle(&fakePoS{
gethInstance.Node.RegisterLifecycle(&fakePoS{
clock: c,
eth: l1Eth,
eth: gethInstance.Backend,
log: log.Root(), // geth logger is global anyway. Would be nice to replace with a local logger though.
blockTime: blockTime,
finalizedDistance: finalizedDistance,
safeDistance: 4,
engineAPI: catalyst.NewConsensusAPI(l1Eth),
engineAPI: catalyst.NewConsensusAPI(gethInstance.Backend),
beacon: beaconSrv,
})
return l1Node, l1Eth, nil
return gethInstance, nil
}
func defaultNodeConfig(name string, jwtPath string) *node.Config {
......@@ -91,9 +91,9 @@ func defaultNodeConfig(name string, jwtPath string) *node.Config {
type GethOption func(ethCfg *ethconfig.Config, nodeCfg *node.Config) error
// InitL2 inits a L2 geth node.
func InitL2(name string, l2ChainID *big.Int, genesis *core.Genesis, jwtPath string, opts ...GethOption) (*node.Node, *eth.Ethereum, error) {
func InitL2(name string, genesis *core.Genesis, jwtPath string, opts ...GethOption) (*GethInstance, error) {
ethConfig := &ethconfig.Config{
NetworkId: l2ChainID.Uint64(),
NetworkId: genesis.Config.ChainID.Uint64(),
Genesis: genesis,
StateScheme: rawdb.HashScheme,
Miner: miner.Config{
......@@ -113,10 +113,10 @@ func InitL2(name string, l2ChainID *big.Int, genesis *core.Genesis, jwtPath stri
// The private keys are added to the keystore and are unlocked.
// If the node is l2, catalyst is enabled.
// The node should be started and then closed when done.
func createGethNode(l2 bool, nodeCfg *node.Config, ethCfg *ethconfig.Config, opts ...GethOption) (*node.Node, *eth.Ethereum, error) {
func createGethNode(l2 bool, nodeCfg *node.Config, ethCfg *ethconfig.Config, opts ...GethOption) (*GethInstance, error) {
for i, opt := range opts {
if err := opt(ethCfg, nodeCfg); err != nil {
return nil, nil, fmt.Errorf("failed to apply geth option %d: %w", i, err)
return nil, fmt.Errorf("failed to apply geth option %d: %w", i, err)
}
}
ethCfg.StateScheme = rawdb.HashScheme
......@@ -124,13 +124,13 @@ func createGethNode(l2 bool, nodeCfg *node.Config, ethCfg *ethconfig.Config, opt
n, err := node.New(nodeCfg)
if err != nil {
n.Close()
return nil, nil, err
return nil, err
}
backend, err := eth.New(n, ethCfg)
if err != nil {
n.Close()
return nil, nil, err
return nil, err
}
......@@ -144,8 +144,11 @@ func createGethNode(l2 bool, nodeCfg *node.Config, ethCfg *ethconfig.Config, opt
if l2 {
if err := catalyst.Register(n, backend); err != nil {
n.Close()
return nil, nil, err
return nil, err
}
}
return n, backend, nil
return &GethInstance{
Backend: backend,
Node: n,
}, nil
}
package geth
import (
"github.com/ethereum/go-ethereum/eth"
"github.com/ethereum/go-ethereum/node"
"github.com/ethereum-optimism/optimism/op-e2e/e2eutils/services"
"github.com/ethereum-optimism/optimism/op-service/endpoint"
)
type GethInstance struct {
Backend *eth.Ethereum
Node *node.Node
}
var _ services.EthInstance = (*GethInstance)(nil)
func (gi *GethInstance) UserRPC() endpoint.RPC {
fallback := endpoint.WsOrHttpRPC{
WsURL: gi.Node.WSEndpoint(),
HttpURL: gi.Node.HTTPEndpoint(),
}
srv, err := gi.Node.RPCHandler()
if err != nil {
return fallback
}
return &endpoint.ServerRPC{
Fallback: fallback,
Server: srv,
}
}
func (gi *GethInstance) AuthRPC() endpoint.RPC {
// TODO: can we rely on the in-process RPC server to support the auth namespaces?
return endpoint.WsOrHttpRPC{
WsURL: gi.Node.WSAuthEndpoint(),
HttpURL: gi.Node.HTTPAuthEndpoint(),
}
}
func (gi *GethInstance) Close() error {
return gi.Node.Close()
}
package opnode
import (
"context"
"github.com/ethereum/go-ethereum/log"
"github.com/ethereum-optimism/optimism/op-e2e/e2eutils/services"
"github.com/ethereum-optimism/optimism/op-node/metrics"
rollupNode "github.com/ethereum-optimism/optimism/op-node/node"
"github.com/ethereum-optimism/optimism/op-node/p2p"
"github.com/ethereum-optimism/optimism/op-service/cliapp"
"github.com/ethereum-optimism/optimism/op-service/endpoint"
)
type Opnode struct {
node *rollupNode.OpNode
}
func (o *Opnode) UserRPC() endpoint.RPC {
return endpoint.HttpURL(o.node.HTTPEndpoint())
}
func (o *Opnode) Stop(ctx context.Context) error {
return o.node.Stop(ctx)
}
func (o *Opnode) Stopped() bool {
return o.node.Stopped()
}
func (o *Opnode) RuntimeConfig() rollupNode.ReadonlyRuntimeConfig {
return o.node.RuntimeConfig()
}
func (o *Opnode) P2P() p2p.Node {
return o.node.P2P()
}
var _ services.RollupNode = (*Opnode)(nil)
func NewOpnode(l log.Logger, c *rollupNode.Config, errFn func(error)) (*Opnode, error) {
var cycle cliapp.Lifecycle
c.Cancel = func(errCause error) {
l.Warn("node requested early shutdown!", "err", errCause)
go func() {
postCtx, postCancel := context.WithCancel(context.Background())
postCancel() // don't allow the stopping to continue for longer than needed
if err := cycle.Stop(postCtx); err != nil {
errFn(err)
}
l.Warn("closed op-node!")
}()
}
node, err := rollupNode.New(context.Background(), c, l, "", metrics.NewMetrics(""))
if err != nil {
return nil, err
}
cycle = node
err = node.Start(context.Background())
if err != nil {
return nil, err
}
return &Opnode{node: node}, nil
}
package services
import (
"context"
rollupNode "github.com/ethereum-optimism/optimism/op-node/node"
"github.com/ethereum-optimism/optimism/op-node/p2p"
"github.com/ethereum-optimism/optimism/op-service/endpoint"
)
// EthInstance is either an in process Geth or external process exposing its
// endpoints over the network
type EthInstance interface {
UserRPC() endpoint.RPC
AuthRPC() endpoint.RPC
Close() error
}
type RollupNode interface {
UserRPC() endpoint.RPC
Stop(ctx context.Context) error
Stopped() bool
RuntimeConfig() rollupNode.ReadonlyRuntimeConfig
P2P() p2p.Node
}
......@@ -10,26 +10,22 @@ import (
"github.com/stretchr/testify/assert"
"github.com/stretchr/testify/require"
"github.com/ethereum/go-ethereum/accounts/abi/bind"
"github.com/ethereum/go-ethereum/common"
"github.com/ethereum/go-ethereum/common/hexutil"
"github.com/ethereum/go-ethereum/core"
"github.com/ethereum/go-ethereum/core/types"
"github.com/ethereum/go-ethereum/log"
"github.com/ethereum/go-ethereum/params"
"github.com/ethereum/go-ethereum/rpc"
batcherFlags "github.com/ethereum-optimism/optimism/op-batcher/flags"
"github.com/ethereum-optimism/optimism/op-e2e/bindings"
gethutils "github.com/ethereum-optimism/optimism/op-e2e/e2eutils/geth"
"github.com/ethereum-optimism/optimism/op-e2e/e2eutils/transactions"
"github.com/ethereum-optimism/optimism/op-e2e/e2eutils/wait"
"github.com/ethereum-optimism/optimism/op-node/rollup/derive"
"github.com/ethereum-optimism/optimism/op-service/client"
"github.com/ethereum-optimism/optimism/op-service/eth"
"github.com/ethereum-optimism/optimism/op-service/sources"
"github.com/ethereum-optimism/optimism/op-service/testlog"
"github.com/ethereum-optimism/optimism/op-service/testutils"
"github.com/ethereum/go-ethereum/accounts/abi/bind"
"github.com/ethereum/go-ethereum/common"
"github.com/ethereum/go-ethereum/common/hexutil"
"github.com/ethereum/go-ethereum/core"
"github.com/ethereum/go-ethereum/core/types"
"github.com/ethereum/go-ethereum/log"
"github.com/ethereum/go-ethereum/params"
)
// TestSystem4844E2E runs the SystemE2E test with 4844 enabled on L1, and active on the rollup in
......@@ -64,7 +60,7 @@ func testSystem4844E2E(t *testing.T, multiBlob bool, daType batcherFlags.DataAva
// front. This lets us test the ability for the batcher to clear out the incompatible
// transaction. The hook used here makes sure we make the jamming call before batch submission
// is started, as is required by the function.
jamChan := make(chan error)
var jamChan chan error
jamCtx, jamCancel := context.WithTimeout(context.Background(), 20*time.Second)
action := SystemConfigOption{
key: "beforeBatcherStart",
......@@ -72,26 +68,28 @@ func testSystem4844E2E(t *testing.T, multiBlob bool, daType batcherFlags.DataAva
driver := s.BatchSubmitter.TestDriver()
err := driver.JamTxPool(jamCtx)
require.NoError(t, err)
jamChan = make(chan error)
go func() {
jamChan <- driver.WaitOnJammingTx(jamCtx)
}()
},
}
defer func() {
jamCancel()
require.NoError(t, <-jamChan, "jam tx error")
if jamChan != nil { // only check if we actually got to a successful batcher start
jamCancel()
require.NoError(t, <-jamChan, "jam tx error")
}
}()
sys, err := cfg.Start(t, action)
require.NoError(t, err, "Error starting up system")
defer sys.Close()
log := testlog.Logger(t, log.LevelInfo)
log.Info("genesis", "l2", sys.RollupConfig.Genesis.L2, "l1", sys.RollupConfig.Genesis.L1, "l2_time", sys.RollupConfig.Genesis.L2Time)
l1Client := sys.Clients["l1"]
l2Seq := sys.Clients["sequencer"]
l2Verif := sys.Clients["verifier"]
l1Client := sys.NodeClient("l1")
l2Seq := sys.NodeClient("sequencer")
l2Verif := sys.NodeClient("verifier")
// Transactor Account
ethPrivKey := cfg.Secrets.Alice
......@@ -144,9 +142,7 @@ func testSystem4844E2E(t *testing.T, multiBlob bool, daType batcherFlags.DataAva
require.Equal(t, verifBlock.ParentHash(), seqBlock.ParentHash(), "Verifier and sequencer blocks parent hashes not the same after including a batch tx")
require.Equal(t, verifBlock.Hash(), seqBlock.Hash(), "Verifier and sequencer blocks not the same after including a batch tx")
rollupRPCClient, err := rpc.DialContext(context.Background(), sys.RollupNodes["sequencer"].HTTPEndpoint())
require.NoError(t, err)
rollupClient := sources.NewRollupClient(client.NewBaseRPCClient(rollupRPCClient))
rollupClient := sys.RollupClient("sequencer")
// basic check that sync status works
seqStatus, err := rollupClient.SyncStatus(context.Background())
require.NoError(t, err)
......@@ -258,12 +254,11 @@ func TestBatcherAutoDA(t *testing.T) {
sys, err := cfg.Start(t)
require.NoError(t, err, "Error starting up system")
defer sys.Close()
log := testlog.Logger(t, log.LevelInfo)
log.Info("genesis", "l2", sys.RollupConfig.Genesis.L2, "l1", sys.RollupConfig.Genesis.L1, "l2_time", sys.RollupConfig.Genesis.L2Time)
l1Client := sys.Clients["l1"]
l1Client := sys.NodeClient("l1")
ctx, cancel := context.WithTimeout(context.Background(), 30*time.Second)
defer cancel()
......
......@@ -12,6 +12,7 @@ import (
"github.com/ethereum-optimism/optimism/op-e2e/config"
"github.com/ethereum-optimism/optimism/op-e2e/external"
"github.com/ethereum-optimism/optimism/op-service/endpoint"
"github.com/ethereum/go-ethereum/common"
"github.com/ethereum/go-ethereum/core"
"github.com/ethereum/go-ethereum/core/types"
......@@ -34,20 +35,18 @@ type ExternalEthClient struct {
Endpoints external.Endpoints
}
func (eec *ExternalEthClient) HTTPEndpoint() string {
return eec.Endpoints.HTTPEndpoint
}
func (eec *ExternalEthClient) WSEndpoint() string {
return eec.Endpoints.WSEndpoint
}
func (eec *ExternalEthClient) HTTPAuthEndpoint() string {
return eec.Endpoints.HTTPAuthEndpoint
func (eec *ExternalEthClient) UserRPC() endpoint.RPC {
return endpoint.WsOrHttpRPC{
WsURL: eec.Endpoints.WSEndpoint,
HttpURL: eec.Endpoints.HTTPEndpoint,
}
}
func (eec *ExternalEthClient) WSAuthEndpoint() string {
return eec.Endpoints.WSAuthEndpoint
func (eec *ExternalEthClient) AuthRPC() endpoint.RPC {
return endpoint.WsOrHttpRPC{
WsURL: eec.Endpoints.WSAuthEndpoint,
HttpURL: eec.Endpoints.HTTPAuthEndpoint,
}
}
func (eec *ExternalEthClient) Close() error {
......
......@@ -9,9 +9,11 @@ import (
"testing"
"time"
"github.com/stretchr/testify/require"
e2e "github.com/ethereum-optimism/optimism/op-e2e"
"github.com/ethereum-optimism/optimism/op-e2e/config"
"github.com/stretchr/testify/require"
"github.com/ethereum-optimism/optimism/op-service/endpoint"
)
func TestShim(t *testing.T) {
......@@ -41,13 +43,13 @@ func TestShim(t *testing.T) {
}).Run(t)
t.Cleanup(func() { _ = ec.Close() })
for _, endpoint := range []string{
ec.HTTPEndpoint(),
ec.HTTPAuthEndpoint(),
ec.WSEndpoint(),
ec.WSAuthEndpoint(),
for _, rpcEndpoint := range []string{
ec.UserRPC().(endpoint.HttpRPC).HttpRPC(),
ec.AuthRPC().(endpoint.HttpRPC).HttpRPC(),
ec.UserRPC().(endpoint.WsRPC).WsRPC(),
ec.AuthRPC().(endpoint.WsRPC).WsRPC(),
} {
plainURL, err := url.ParseRequestURI(endpoint)
plainURL, err := url.ParseRequestURI(rpcEndpoint)
require.NoError(t, err)
_, err = net.DialTimeout("tcp", plainURL.Host, time.Second)
require.NoError(t, err, "could not connect to HTTP port")
......
......@@ -18,7 +18,6 @@ import (
"github.com/ethereum/go-ethereum/crypto"
"github.com/ethereum/go-ethereum/ethclient"
"github.com/ethereum/go-ethereum/log"
"github.com/ethereum/go-ethereum/rpc"
"github.com/pkg/errors"
"github.com/stretchr/testify/require"
......@@ -27,9 +26,7 @@ import (
op_e2e "github.com/ethereum-optimism/optimism/op-e2e"
"github.com/ethereum-optimism/optimism/op-e2e/bindings"
"github.com/ethereum-optimism/optimism/op-e2e/e2eutils/wait"
"github.com/ethereum-optimism/optimism/op-service/client"
"github.com/ethereum-optimism/optimism/op-service/predeploys"
"github.com/ethereum-optimism/optimism/op-service/sources"
"github.com/ethereum-optimism/optimism/op-service/testlog"
)
......@@ -50,16 +47,13 @@ func TestBenchmarkCannon_FPP(t *testing.T) {
sys, err := cfg.Start(t)
require.Nil(t, err, "Error starting up system")
defer sys.Close()
log := testlog.Logger(t, log.LevelInfo)
log.Info("genesis", "l2", sys.RollupConfig.Genesis.L2, "l1", sys.RollupConfig.Genesis.L1, "l2_time", sys.RollupConfig.Genesis.L2Time)
l1Client := sys.Clients["l1"]
l2Seq := sys.Clients["sequencer"]
rollupRPCClient, err := rpc.DialContext(context.Background(), sys.RollupNodes["sequencer"].HTTPEndpoint())
require.Nil(t, err)
rollupClient := sources.NewRollupClient(client.NewBaseRPCClient(rollupRPCClient))
l1Client := sys.NodeClient("l1")
l2Seq := sys.NodeClient("sequencer")
rollupClient := sys.RollupClient("sequencer")
require.NoError(t, wait.ForUnsafeBlock(ctx, rollupClient, 1))
// Agreed state: 200 Big Contracts deployed at max size - total codesize is 5.90 MB
......
......@@ -14,7 +14,6 @@ import (
"github.com/ethereum/go-ethereum/core/types"
"github.com/ethereum/go-ethereum/log"
"github.com/ethereum/go-ethereum/params"
"github.com/ethereum/go-ethereum/rpc"
"github.com/stretchr/testify/require"
"github.com/ethereum-optimism/optimism/cannon/mipsevm/singlethreaded"
......@@ -26,9 +25,7 @@ import (
"github.com/ethereum-optimism/optimism/op-e2e/e2eutils/challenger"
"github.com/ethereum-optimism/optimism/op-e2e/e2eutils/disputegame"
"github.com/ethereum-optimism/optimism/op-e2e/e2eutils/wait"
"github.com/ethereum-optimism/optimism/op-service/client"
"github.com/ethereum-optimism/optimism/op-service/ioutil"
"github.com/ethereum-optimism/optimism/op-service/sources"
"github.com/ethereum-optimism/optimism/op-service/testlog"
)
......@@ -90,17 +87,13 @@ func TestPrecompiles(t *testing.T) {
sys, err := cfg.Start(t)
require.Nil(t, err, "Error starting up system")
defer sys.Close()
log := testlog.Logger(t, log.LevelInfo)
log.Info("genesis", "l2", sys.RollupConfig.Genesis.L2, "l1", sys.RollupConfig.Genesis.L1, "l2_time", sys.RollupConfig.Genesis.L2Time)
l1Client := sys.Clients["l1"]
l2Seq := sys.Clients["sequencer"]
rollupRPCClient, err := rpc.DialContext(context.Background(), sys.RollupNodes["sequencer"].HTTPEndpoint())
require.Nil(t, err)
rollupClient := sources.NewRollupClient(client.NewBaseRPCClient(rollupRPCClient))
l1Client := sys.NodeClient("l1")
l2Seq := sys.NodeClient("sequencer")
rollupClient := sys.RollupClient("sequencer")
aliceKey := cfg.Secrets.Alice
t.Log("Capture current L2 head as agreed starting point")
......@@ -147,9 +140,8 @@ func TestPrecompiles(t *testing.T) {
}
ctx := context.Background()
sys, _ := StartFaultDisputeSystem(t, WithBlobBatches())
defer sys.Close()
l2Seq := sys.Clients["sequencer"]
l2Seq := sys.NodeClient("sequencer")
aliceKey := sys.Cfg.Secrets.Alice
receipt := op_e2e.SendL2Tx(t, sys.Cfg, l2Seq, aliceKey, func(opts *op_e2e.TxOpts) {
opts.Gas = 1_000_000
......@@ -194,17 +186,13 @@ func TestGranitePrecompiles(t *testing.T) {
sys, err := cfg.Start(t)
require.Nil(t, err, "Error starting up system")
defer sys.Close()
log := testlog.Logger(t, log.LevelInfo)
log.Info("genesis", "l2", sys.RollupConfig.Genesis.L2, "l1", sys.RollupConfig.Genesis.L1, "l2_time", sys.RollupConfig.Genesis.L2Time)
l1Client := sys.Clients["l1"]
l2Seq := sys.Clients["sequencer"]
rollupRPCClient, err := rpc.DialContext(context.Background(), sys.RollupNodes["sequencer"].HTTPEndpoint())
require.Nil(t, err)
rollupClient := sources.NewRollupClient(client.NewBaseRPCClient(rollupRPCClient))
l1Client := sys.NodeClient("l1")
l2Seq := sys.NodeClient("sequencer")
rollupClient := sys.RollupClient("sequencer")
aliceKey := cfg.Secrets.Alice
t.Log("Capture current L2 head as agreed starting point")
......@@ -258,10 +246,10 @@ func TestGranitePrecompiles(t *testing.T) {
}
func runCannon(t *testing.T, ctx context.Context, sys *op_e2e.System, inputs utils.LocalGameInputs, l2Node string, extraVmArgs ...string) {
l1Endpoint := sys.NodeEndpoint("l1")
l1Beacon := sys.L1BeaconEndpoint()
rollupEndpoint := sys.RollupEndpoint("sequencer")
l2Endpoint := sys.NodeEndpoint("sequencer")
l1Endpoint := sys.NodeEndpoint("l1").RPC()
l1Beacon := sys.L1BeaconEndpoint().RestHTTP()
rollupEndpoint := sys.RollupEndpoint("sequencer").RPC()
l2Endpoint := sys.NodeEndpoint("sequencer").RPC()
cannonOpts := challenger.WithCannon(t, sys.RollupCfg(), sys.L2Genesis())
dir := t.TempDir()
proofsDir := filepath.Join(dir, "cannon-proofs")
......
......@@ -61,11 +61,11 @@ func StartFaultDisputeSystem(t *testing.T, opts ...faultDisputeConfigOpts) (*op_
}
sys, err := cfg.Start(t)
require.Nil(t, err, "Error starting up system")
return sys, sys.Clients["l1"]
return sys, sys.NodeClient("l1")
}
func SendKZGPointEvaluationTx(t *testing.T, sys *op_e2e.System, l2Node string, privateKey *ecdsa.PrivateKey) *types.Receipt {
return op_e2e.SendL2Tx(t, sys.Cfg, sys.Clients[l2Node], privateKey, func(opts *op_e2e.TxOpts) {
return op_e2e.SendL2Tx(t, sys.Cfg, sys.NodeClient(l2Node), privateKey, func(opts *op_e2e.TxOpts) {
precompile := common.BytesToAddress([]byte{0x0a})
opts.Gas = 100_000
opts.ToAddr = &precompile
......
......@@ -21,8 +21,8 @@ func TestTxGossip(t *testing.T) {
sys, err := cfg.Start(t)
require.NoError(t, err, "Start system")
seqClient := sys.Clients["sequencer"]
verifClient := sys.Clients["verifier"]
seqClient := sys.NodeClient("sequencer")
verifClient := sys.NodeClient("verifier")
geth.ConnectP2P(t, seqClient, verifClient)
// This prevents the below tx-sending from flaking in CI
......
......@@ -4,7 +4,6 @@ import (
"context"
"errors"
"fmt"
"math/big"
"reflect"
"testing"
......@@ -12,6 +11,7 @@ import (
"github.com/ethereum-optimism/optimism/op-e2e/config"
"github.com/ethereum-optimism/optimism/op-e2e/e2eutils"
"github.com/ethereum-optimism/optimism/op-e2e/e2eutils/geth"
"github.com/ethereum-optimism/optimism/op-e2e/e2eutils/services"
"github.com/ethereum-optimism/optimism/op-node/rollup"
"github.com/ethereum-optimism/optimism/op-node/rollup/derive"
"github.com/ethereum-optimism/optimism/op-service/client"
......@@ -39,7 +39,7 @@ var (
// OpGeth is an actor that functions as a l2 op-geth node
// It provides useful functions for advancing and querying the chain
type OpGeth struct {
node EthInstance
node services.EthInstance
l2Engine *sources.EngineClient
L2Client *ethclient.Client
SystemConfig eth.SystemConfig
......@@ -85,11 +85,11 @@ func NewOpGeth(t testing.TB, ctx context.Context, cfg *SystemConfig) (*OpGeth, e
SystemConfig: e2eutils.SystemConfigFromDeployConfig(cfg.DeployConfig),
}
var node EthInstance
var node services.EthInstance
if cfg.ExternalL2Shim == "" {
gethNode, _, err := geth.InitL2("l2", big.NewInt(int64(cfg.DeployConfig.L2ChainID)), l2Genesis, cfg.JWTFilePath)
gethNode, err := geth.InitL2("l2", l2Genesis, cfg.JWTFilePath)
require.NoError(t, err)
require.NoError(t, gethNode.Start())
require.NoError(t, gethNode.Node.Start())
node = gethNode
} else {
externalNode := (&ExternalRunner{
......@@ -102,7 +102,7 @@ func NewOpGeth(t testing.TB, ctx context.Context, cfg *SystemConfig) (*OpGeth, e
}
auth := rpc.WithHTTPAuth(gn.NewJWTAuth(cfg.JWTSecret))
l2Node, err := client.NewRPC(ctx, logger, node.WSAuthEndpoint(), client.WithGethRPCOptions(auth))
l2Node, err := client.NewRPC(ctx, logger, node.AuthRPC().RPC(), client.WithGethRPCOptions(auth))
require.NoError(t, err)
// Finally create the engine client
......@@ -117,7 +117,7 @@ func NewOpGeth(t testing.TB, ctx context.Context, cfg *SystemConfig) (*OpGeth, e
)
require.NoError(t, err)
l2Client, err := ethclient.Dial(selectEndpoint(node))
l2Client, err := ethclient.Dial(node.UserRPC().RPC())
require.NoError(t, err)
genesisPayload, err := eth.BlockAsPayload(l2GenesisBlock, cfg.DeployConfig.CanyonTime(l2GenesisBlock.Time()))
......
......@@ -59,7 +59,6 @@ func TestTxGasSameAsBlockGasLimit(t *testing.T) {
cfg := DefaultSystemConfig(t)
sys, err := cfg.Start(t)
require.Nil(t, err, "Error starting up system")
defer sys.Close()
ethPrivKey := sys.Cfg.Secrets.Alice
tx := types.MustSignNewTx(ethPrivKey, types.LatestSignerForChainID(cfg.L2ChainIDBig()), &types.DynamicFeeTx{
......@@ -68,7 +67,7 @@ func TestTxGasSameAsBlockGasLimit(t *testing.T) {
})
ctx, cancel := context.WithTimeout(context.Background(), 30*time.Second)
defer cancel()
l2Seq := sys.Clients["sequencer"]
l2Seq := sys.NodeClient("sequencer")
err = l2Seq.SendTransaction(ctx, tx)
require.ErrorContains(t, err, txpool.ErrGasLimit.Error())
}
......
......@@ -82,7 +82,7 @@ func setupSequencerFailoverTest(t *testing.T) (*System, map[string]*conductor, f
// start sequencing on leader
lid, _ := findLeader(t, conductors)
unsafeHead, err := sys.Clients[lid].BlockByNumber(ctx, nil)
unsafeHead, err := sys.NodeClient(lid).BlockByNumber(ctx, nil)
require.NoError(t, err)
require.Equal(t, uint64(0), unsafeHead.NumberU64())
require.NoError(t, sys.RollupClient(lid).StartSequencer(ctx, unsafeHead.Hash()))
......@@ -93,9 +93,9 @@ func setupSequencerFailoverTest(t *testing.T) (*System, map[string]*conductor, f
// weirdly, batcher does not submit a batch until unsafe block 9.
// It became normal after that and submits a batch every L1 block (2s) per configuration.
// Since our health monitor checks on safe head progression, wait for batcher to become normal before proceeding.
_, err = wait.ForNextSafeBlock(ctx, sys.Clients[Sequencer1Name])
_, err = wait.ForNextSafeBlock(ctx, sys.NodeClient(Sequencer1Name))
require.NoError(t, err)
_, err = wait.ForNextSafeBlock(ctx, sys.Clients[Sequencer1Name])
_, err = wait.ForNextSafeBlock(ctx, sys.NodeClient(Sequencer1Name))
require.NoError(t, err)
// make sure conductor reports all sequencers as healthy, this means they're syncing correctly.
......@@ -189,8 +189,8 @@ func setupHAInfra(t *testing.T, ctx context.Context) (*System, map[string]*condu
}
for _, cfg := range conductorCfgs {
cfg := cfg
nodePRC := sys.RollupNodes[cfg.name].HTTPEndpoint()
engineRPC := sys.EthInstances[cfg.name].HTTPEndpoint()
nodePRC := sys.RollupNodes[cfg.name].UserRPC().RPC()
engineRPC := sys.EthInstances[cfg.name].UserRPC().RPC()
if conductors[cfg.name], err = setupConductor(t, cfg.name, t.TempDir(), nodePRC, engineRPC, cfg.port, cfg.bootstrap, *sys.RollupConfig); err != nil {
return nil, nil, err
}
......@@ -252,6 +252,7 @@ func setupConductor(
if err != nil {
return nil, err
}
t.Cleanup(rawClient.Close)
client := conrpc.NewAPIClient(rawClient)
return &conductor{
......@@ -277,7 +278,7 @@ func setupBatcher(t *testing.T, sys *System, conductors map[string]*conductor) {
conductors[Sequencer3Name].RPCEndpoint(),
}, ",")
batcherCLIConfig := &bss.CLIConfig{
L1EthRpc: sys.EthInstances["l1"].WSEndpoint(),
L1EthRpc: sys.EthInstances["l1"].UserRPC().RPC(),
L2EthRpc: l2EthRpc,
RollupRpc: rollupRpc,
MaxPendingTransactions: 0,
......@@ -287,7 +288,7 @@ func setupBatcher(t *testing.T, sys *System, conductors map[string]*conductor) {
ApproxComprRatio: 0.4,
SubSafetyMargin: 4,
PollInterval: 1 * time.Second,
TxMgrConfig: newTxMgrConfig(sys.EthInstances["l1"].WSEndpoint(), sys.Cfg.Secrets.Batcher),
TxMgrConfig: newTxMgrConfig(sys.EthInstances["l1"].UserRPC().RPC(), sys.Cfg.Secrets.Batcher),
LogConfig: oplog.CLIConfig{
Level: log.LevelDebug,
Format: oplog.FormatText,
......
......@@ -100,8 +100,8 @@ func TestSequencerFailover_ConductorRPC(t *testing.T) {
nonvoter, err := retry.Do[*conductor](ctx, maxSetupRetries, retryStrategy, func() (*conductor, error) {
return setupConductor(
t, VerifierName, t.TempDir(),
sys.RollupEndpoint(Sequencer3Name),
sys.NodeEndpoint(Sequencer3Name),
sys.RollupEndpoint(Sequencer3Name).RPC(),
sys.NodeEndpoint(Sequencer3Name).RPC(),
findAvailablePort(t),
false,
*sys.RollupConfig,
......
This diff is collapsed.
......@@ -23,14 +23,10 @@ func TestStopStartSequencer(t *testing.T) {
cfg := DefaultSystemConfig(t)
sys, err := cfg.Start(t)
require.Nil(t, err, "Error starting up system")
defer sys.Close()
l2Seq := sys.Clients["sequencer"]
rollupNode := sys.RollupNodes["sequencer"]
l2Seq := sys.NodeClient("sequencer")
nodeRPC, err := rpc.DialContext(context.Background(), rollupNode.HTTPEndpoint())
require.Nil(t, err, "Error dialing node")
rollupClient := sources.NewRollupClient(client.NewBaseRPCClient(nodeRPC))
rollupClient := sys.RollupClient("sequencer")
ctx, cancel := context.WithTimeout(context.Background(), 5*time.Second)
defer cancel()
......@@ -91,11 +87,10 @@ func TestPersistSequencerStateWhenChanged(t *testing.T) {
sys, err := cfg.Start(t)
require.NoError(t, err)
defer sys.Close()
assertPersistedSequencerState(t, stateFile, node.StateStarted)
rollupRPCClient, err := rpc.DialContext(ctx, sys.RollupNodes["sequencer"].HTTPEndpoint())
rollupRPCClient, err := rpc.DialContext(ctx, sys.RollupNodes["sequencer"].UserRPC().RPC())
require.Nil(t, err)
rollupClient := sources.NewRollupClient(client.NewBaseRPCClient(rollupRPCClient))
......@@ -126,9 +121,8 @@ func TestLoadSequencerStateOnStarted_Stopped(t *testing.T) {
sys, err := cfg.Start(t)
require.NoError(t, err)
defer sys.Close()
rollupRPCClient, err := rpc.DialContext(ctx, sys.RollupNodes["sequencer"].HTTPEndpoint())
rollupRPCClient, err := rpc.DialContext(ctx, sys.RollupNodes["sequencer"].UserRPC().RPC())
require.Nil(t, err)
rollupClient := sources.NewRollupClient(client.NewBaseRPCClient(rollupRPCClient))
......@@ -160,9 +154,8 @@ func TestLoadSequencerStateOnStarted_Started(t *testing.T) {
sys, err := cfg.Start(t)
require.NoError(t, err)
defer sys.Close()
rollupRPCClient, err := rpc.DialContext(ctx, sys.RollupNodes["sequencer"].HTTPEndpoint())
rollupRPCClient, err := rpc.DialContext(ctx, sys.RollupNodes["sequencer"].UserRPC().RPC())
require.Nil(t, err)
rollupClient := sources.NewRollupClient(client.NewBaseRPCClient(rollupRPCClient))
......@@ -185,10 +178,9 @@ func TestPostUnsafePayload(t *testing.T) {
sys, err := cfg.Start(t)
require.NoError(t, err)
defer sys.Close()
l2Seq := sys.Clients["sequencer"]
l2Ver := sys.Clients["verifier"]
l2Seq := sys.NodeClient("sequencer")
l2Ver := sys.NodeClient("verifier")
rollupClient := sys.RollupClient("verifier")
require.NoError(t, wait.ForBlock(ctx, l2Seq, 2), "Chain did not advance after starting sequencer")
......
......@@ -20,8 +20,6 @@ import (
"github.com/ethereum-optimism/optimism/op-program/client/claim"
opp "github.com/ethereum-optimism/optimism/op-program/host"
oppconf "github.com/ethereum-optimism/optimism/op-program/host/config"
"github.com/ethereum-optimism/optimism/op-service/client"
"github.com/ethereum-optimism/optimism/op-service/sources"
"github.com/ethereum-optimism/optimism/op-service/testlog"
)
......@@ -101,16 +99,13 @@ func testVerifyL2OutputRootEmptyBlock(t *testing.T, detached bool, spanBatchActi
sys, err := cfg.Start(t)
require.Nil(t, err, "Error starting up system")
defer sys.Close()
log := testlog.Logger(t, log.LevelInfo)
log.Info("genesis", "l2", sys.RollupConfig.Genesis.L2, "l1", sys.RollupConfig.Genesis.L1, "l2_time", sys.RollupConfig.Genesis.L2Time)
l1Client := sys.Clients["l1"]
l2Seq := sys.Clients["sequencer"]
rollupRPCClient, err := rpc.DialContext(context.Background(), sys.RollupNodes["sequencer"].HTTPEndpoint())
require.Nil(t, err)
rollupClient := sources.NewRollupClient(client.NewBaseRPCClient(rollupRPCClient))
l1Client := sys.NodeClient("l1")
l2Seq := sys.NodeClient("sequencer")
rollupClient := sys.RollupClient("sequencer")
// Avoids flaky test by avoiding reorgs at epoch 0
t.Log("Wait for safe head to advance once for setup")
......@@ -201,16 +196,14 @@ func testVerifyL2OutputRoot(t *testing.T, detached bool, spanBatchActivated bool
sys, err := cfg.Start(t)
require.Nil(t, err, "Error starting up system")
defer sys.Close()
log := testlog.Logger(t, log.LevelInfo)
log.Info("genesis", "l2", sys.RollupConfig.Genesis.L2, "l1", sys.RollupConfig.Genesis.L1, "l2_time", sys.RollupConfig.Genesis.L2Time)
l1Client := sys.Clients["l1"]
l2Seq := sys.Clients["sequencer"]
rollupRPCClient, err := rpc.DialContext(context.Background(), sys.RollupNodes["sequencer"].HTTPEndpoint())
require.Nil(t, err)
rollupClient := sources.NewRollupClient(client.NewBaseRPCClient(rollupRPCClient))
l1Client := sys.NodeClient("l1")
l2Seq := sys.NodeClient("sequencer")
rollupClient := sys.RollupClient("sequencer")
t.Log("Sending transactions to setup existing state, prior to challenged period")
aliceKey := cfg.Secrets.Alice
......@@ -286,8 +279,8 @@ type FaultProofProgramTestScenario struct {
func testFaultProofProgramScenario(t *testing.T, ctx context.Context, sys *System, s *FaultProofProgramTestScenario) {
preimageDir := t.TempDir()
fppConfig := oppconf.NewConfig(sys.RollupConfig, sys.L2GenesisCfg.Config, s.L1Head, s.L2Head, s.L2OutputRoot, common.Hash(s.L2Claim), s.L2ClaimBlockNumber)
fppConfig.L1URL = sys.NodeEndpoint("l1")
fppConfig.L2URL = sys.NodeEndpoint("sequencer")
fppConfig.L1URL = sys.NodeEndpoint("l1").RPC()
fppConfig.L2URL = sys.NodeEndpoint("sequencer").RPC()
fppConfig.DataDir = preimageDir
if s.Detached {
// When running in detached mode we need to compile the client executable since it will be called directly.
......
This diff is collapsed.
......@@ -52,12 +52,11 @@ func TestGasPriceOracleFeeUpdates(t *testing.T) {
cfg := DefaultSystemConfig(t)
sys, err := cfg.Start(t)
require.NoError(t, err, "Error starting up system")
defer sys.Close()
// Obtain our sequencer, verifier, and transactor keypair.
l1Client := sys.Clients["l1"]
l2Seq := sys.Clients["sequencer"]
// l2Verif := sys.Clients["verifier"]
l1Client := sys.NodeClient("l1")
l2Seq := sys.NodeClient("sequencer")
// l2Verif := sys.NodeClient("verifier")
ethPrivKey := cfg.Secrets.SysCfgOwner
// Bind to the SystemConfig & GasPriceOracle contracts
......@@ -128,11 +127,10 @@ func TestL2SequencerRPCDepositTx(t *testing.T) {
cfg := DefaultSystemConfig(t)
sys, err := cfg.Start(t)
require.Nil(t, err, "Error starting up system")
defer sys.Close()
// Obtain our sequencer, verifier, and transactor keypair.
l2Seq := sys.Clients["sequencer"]
l2Verif := sys.Clients["verifier"]
l2Seq := sys.NodeClient("sequencer")
l2Verif := sys.NodeClient("verifier")
txSigningKey := sys.Cfg.Secrets.Alice
require.Nil(t, err)
......@@ -237,12 +235,11 @@ func TestMixedDepositValidity(t *testing.T) {
cfg := DefaultSystemConfig(t)
sys, testAccounts, err := startConfigWithTestAccounts(t, &cfg, accountUsedToDeposit)
require.Nil(t, err, "Error starting up system")
defer sys.Close()
// Obtain our sequencer, verifier, and transactor keypair.
l1Client := sys.Clients["l1"]
l2Seq := sys.Clients["sequencer"]
l2Verif := sys.Clients["verifier"]
l1Client := sys.NodeClient("l1")
l2Seq := sys.NodeClient("sequencer")
l2Verif := sys.NodeClient("verifier")
require.NoError(t, err)
// Define our L1 transaction timeout duration.
......@@ -408,12 +405,11 @@ func TestMixedWithdrawalValidity(t *testing.T) {
require.Equal(t, cfg.DeployConfig.FundDevAccounts, true)
sys, err := cfg.Start(t)
require.NoError(t, err, "error starting up system")
defer sys.Close()
// Obtain our sequencer, verifier, and transactor keypair.
l1Client := sys.Clients["l1"]
l2Seq := sys.Clients["sequencer"]
l2Verif := sys.Clients["verifier"]
l1Client := sys.NodeClient("l1")
l2Seq := sys.NodeClient("sequencer")
l2Verif := sys.NodeClient("verifier")
require.NoError(t, err)
systemConfig, err := legacybindings.NewSystemConfigCaller(cfg.L1Deployments.SystemConfigProxy, l1Client)
......@@ -554,7 +550,7 @@ func TestMixedWithdrawalValidity(t *testing.T) {
header, err = l2Verif.HeaderByNumber(ctx, new(big.Int).SetUint64(blockNumber))
require.Nil(t, err)
rpcClient, err := rpc.Dial(sys.EthInstances["verifier"].WSEndpoint())
rpcClient, err := rpc.Dial(sys.EthInstances["verifier"].UserRPC().RPC())
require.Nil(t, err)
proofCl := gethclient.New(rpcClient)
receiptCl := ethclient.NewClient(rpcClient)
......
package endpoint
// RestHTTP is an interface for an endpoint to provide flexibility.
// By default the RestHTTP just returns an REST-ful HTTP endpoint string.
// But the RestHTTP can implement one or more extension interfaces,
// to provide alternative ways of establishing a connection,
// or even a fully initialized client binding.
type RestHTTP interface {
RestHTTP() string
}
// RestHTTPURL is an HTTP endpoint URL string
type RestHTTPURL string
func (url RestHTTPURL) RestHTTP() string {
return string(url)
}
package endpoint
import (
"github.com/ethereum/go-ethereum/rpc"
)
// RPC is an interface for an endpoint to provide flexibility.
// By default the RPC just returns an RPC endpoint string.
// But the RPC can implement one or more extension interfaces,
// to provide alternative ways of establishing a connection,
// or even a fully initialized client binding.
type RPC interface {
RPC() string
}
// WsRPC is an RPC extension interface,
// to explicitly provide the Websocket RPC option.
type WsRPC interface {
RPC
WsRPC() string
}
// HttpRPC is an RPC extension interface,
// to explicitly provide the HTTP RPC option.
type HttpRPC interface {
RPC
HttpRPC() string
}
// ClientRPC is an RPC extension interface,
// providing the option to attach in-process to a client,
// rather than dialing an endpoint.
type ClientRPC interface {
RPC
ClientRPC() *rpc.Client
}
// HttpURL is an HTTP endpoint URL
type HttpURL string
func (url HttpURL) RPC() string {
return string(url)
}
func (url HttpURL) HttpRPC() string {
return string(url)
}
// WsURL is a websocket endpoint URL
type WsURL string
func (url WsURL) RPC() string {
return string(url)
}
func (url WsURL) WsRPC() string {
return string(url)
}
// WsOrHttpRPC provides optionality between
// a websocket RPC endpoint and a HTTP RPC endpoint.
// The default is the websocket endpoint.
type WsOrHttpRPC struct {
WsURL string
HttpURL string
}
func (r WsOrHttpRPC) RPC() string {
return r.WsURL
}
func (r WsOrHttpRPC) WsRPC() string {
return r.WsURL
}
func (r WsOrHttpRPC) HttpRPC() string {
return r.HttpURL
}
// ServerRPC is a very flexible RPC: it can attach in-process to a server,
// or select one of the fallback RPC methods.
type ServerRPC struct {
Fallback WsOrHttpRPC
Server *rpc.Server
}
func (e *ServerRPC) RPC() string {
return e.Fallback.RPC()
}
func (e *ServerRPC) WsRPC() string {
return e.Fallback.WsRPC()
}
func (e *ServerRPC) HttpRPC() string {
return e.Fallback.HttpRPC()
}
func (e *ServerRPC) ClientRPC() *rpc.Client {
return rpc.DialInProc(e.Server)
}
type Dialer func(v string) *rpc.Client
type RPCPreference int
const (
PreferAnyRPC RPCPreference = iota
PreferHttpRPC
PreferWSRPC
)
// DialRPC navigates the RPC interface,
// to find the optimal version of the PRC to dial or attach to.
func DialRPC(preference RPCPreference, rpc RPC, dialer Dialer) *rpc.Client {
if v, ok := rpc.(HttpRPC); preference == PreferHttpRPC && ok {
return dialer(v.HttpRPC())
}
if v, ok := rpc.(WsRPC); preference == PreferWSRPC && ok {
return dialer(v.WsRPC())
}
if v, ok := rpc.(ClientRPC); ok {
return v.ClientRPC()
}
return dialer(rpc.RPC())
}
// SelectRPC selects an endpoint URL, based on preference.
// For more optimal dialing, use DialRPC.
func SelectRPC(preference RPCPreference, rpc RPC) string {
if v, ok := rpc.(HttpRPC); preference == PreferHttpRPC && ok {
return v.HttpRPC()
}
if v, ok := rpc.(WsRPC); preference == PreferWSRPC && ok {
return v.WsRPC()
}
return rpc.RPC()
}
Markdown is supported
0% or
You are about to add 0 people to the discussion. Proceed with caution.
Finish editing this message first!
Please register or to comment