Hook to slot stream instead of block stream on the VC (#13327)

* Hook to slot stream instead of block stream on the VC

* Implement StreamSlots in the BN

* mock update

* fix tests

* don't return from stream

* Terence's review

* deepsource second complain

---------

Co-authored-by: rkapka <rkapka@wp.pl>
This commit is contained in:
Potuz 2023-12-13 20:13:56 -03:00 committed by GitHub
parent c47c52152b
commit 96df81d5c5
No known key found for this signature in database
GPG Key ID: 4AEE18F83AFDEB23
29 changed files with 2803 additions and 2139 deletions

View File

@ -7,6 +7,7 @@ import (
"github.com/prysmaticlabs/prysm/v4/beacon-chain/core/feed"
blockfeed "github.com/prysmaticlabs/prysm/v4/beacon-chain/core/feed/block"
statefeed "github.com/prysmaticlabs/prysm/v4/beacon-chain/core/feed/state"
"github.com/prysmaticlabs/prysm/v4/consensus-types/primitives"
ethpb "github.com/prysmaticlabs/prysm/v4/proto/prysm/v1alpha1"
"github.com/prysmaticlabs/prysm/v4/runtime/version"
"google.golang.org/grpc/codes"
@ -46,6 +47,53 @@ func (vs *Server) StreamBlocksAltair(req *ethpb.StreamBlocksRequest, stream ethp
}
}
// StreamSlots sends a block's slot to clients every single time a block is received by the beacon node.
func (vs *Server) StreamSlots(req *ethpb.StreamSlotsRequest, stream ethpb.BeaconNodeValidator_StreamSlotsServer) error {
ch := make(chan *feed.Event, 1)
var sub event.Subscription
if req.VerifiedOnly {
sub = vs.StateNotifier.StateFeed().Subscribe(ch)
} else {
sub = vs.BlockNotifier.BlockFeed().Subscribe(ch)
}
defer sub.Unsubscribe()
for {
select {
case ev := <-ch:
var s primitives.Slot
if req.VerifiedOnly {
if ev.Type != statefeed.BlockProcessed {
continue
}
data, ok := ev.Data.(*statefeed.BlockProcessedData)
if !ok || data == nil {
continue
}
s = data.Slot
} else {
if ev.Type != blockfeed.ReceivedBlock {
continue
}
data, ok := ev.Data.(*blockfeed.ReceivedBlockData)
if !ok || data == nil {
continue
}
s = data.SignedBlock.Block().Slot()
}
if err := stream.Send(&ethpb.StreamSlotsResponse{Slot: s}); err != nil {
return status.Errorf(codes.Unavailable, "Could not send over stream: %v", err)
}
case <-sub.Err():
return status.Error(codes.Aborted, "Subscriber closed, exiting goroutine")
case <-vs.Ctx.Done():
return status.Error(codes.Canceled, "Context canceled")
case <-stream.Context().Done():
return status.Error(codes.Canceled, "Context canceled")
}
}
}
func sendVerifiedBlocks(stream ethpb.BeaconNodeValidator_StreamBlocksAltairServer, blockEvent *feed.Event) error {
if blockEvent.Type != statefeed.BlockProcessed {
return nil

View File

@ -239,3 +239,121 @@ func TestServer_StreamCapellaBlocksVerified_OnHeadUpdated(t *testing.T) {
}
<-exitRoutine
}
func TestServer_StreamSlotsVerified_ContextCanceled(t *testing.T) {
ctx := context.Background()
chainService := &chainMock.ChainService{}
ctx, cancel := context.WithCancel(ctx)
server := &Server{
Ctx: ctx,
StateNotifier: chainService.StateNotifier(),
HeadFetcher: chainService,
}
exitRoutine := make(chan bool)
ctrl := gomock.NewController(t)
defer ctrl.Finish()
mockStream := mock.NewMockBeaconNodeValidator_StreamSlotsServer(ctrl)
mockStream.EXPECT().Context().Return(ctx)
go func(tt *testing.T) {
assert.ErrorContains(tt, "Context canceled", server.StreamSlots(&ethpb.StreamSlotsRequest{
VerifiedOnly: true,
}, mockStream))
<-exitRoutine
}(t)
cancel()
exitRoutine <- true
}
func TestServer_StreamSlots_ContextCanceled(t *testing.T) {
ctx := context.Background()
chainService := &chainMock.ChainService{}
ctx, cancel := context.WithCancel(ctx)
server := &Server{
Ctx: ctx,
BlockNotifier: chainService.BlockNotifier(),
HeadFetcher: chainService,
}
exitRoutine := make(chan bool)
ctrl := gomock.NewController(t)
defer ctrl.Finish()
mockStream := mock.NewMockBeaconNodeValidator_StreamSlotsServer(ctrl)
mockStream.EXPECT().Context().Return(ctx)
go func(tt *testing.T) {
assert.ErrorContains(tt, "Context canceled", server.StreamSlots(&ethpb.StreamSlotsRequest{}, mockStream))
<-exitRoutine
}(t)
cancel()
exitRoutine <- true
}
func TestServer_StreamSlots_OnHeadUpdated(t *testing.T) {
params.SetupTestConfigCleanup(t)
params.OverrideBeaconConfig(params.BeaconConfig())
ctx := context.Background()
chainService := &chainMock.ChainService{}
server := &Server{
Ctx: ctx,
BlockNotifier: chainService.BlockNotifier(),
}
exitRoutine := make(chan bool)
ctrl := gomock.NewController(t)
defer ctrl.Finish()
mockStream := mock.NewMockBeaconNodeValidator_StreamSlotsServer(ctrl)
mockStream.EXPECT().Send(&ethpb.StreamSlotsResponse{Slot: 123}).Do(func(arg0 interface{}) {
exitRoutine <- true
})
mockStream.EXPECT().Context().Return(ctx).AnyTimes()
go func(tt *testing.T) {
assert.NoError(tt, server.StreamSlots(&ethpb.StreamSlotsRequest{}, mockStream), "Could not call RPC method")
}(t)
wrappedBlk, err := blocks.NewSignedBeaconBlock(&ethpb.SignedBeaconBlock{Block: &ethpb.BeaconBlock{Slot: 123, Body: &ethpb.BeaconBlockBody{}}})
require.NoError(t, err)
// Send in a loop to ensure it is delivered (busy wait for the service to subscribe to the state feed).
for sent := 0; sent == 0; {
sent = server.BlockNotifier.BlockFeed().Send(&feed.Event{
Type: blockfeed.ReceivedBlock,
Data: &blockfeed.ReceivedBlockData{SignedBlock: wrappedBlk},
})
}
<-exitRoutine
}
func TestServer_StreamSlotsVerified_OnHeadUpdated(t *testing.T) {
ctx := context.Background()
chainService := &chainMock.ChainService{}
server := &Server{
Ctx: ctx,
StateNotifier: chainService.StateNotifier(),
}
exitRoutine := make(chan bool)
ctrl := gomock.NewController(t)
defer ctrl.Finish()
mockStream := mock.NewMockBeaconNodeValidator_StreamSlotsServer(ctrl)
mockStream.EXPECT().Send(&ethpb.StreamSlotsResponse{Slot: 123}).Do(func(arg0 interface{}) {
exitRoutine <- true
})
mockStream.EXPECT().Context().Return(ctx).AnyTimes()
go func(tt *testing.T) {
assert.NoError(tt, server.StreamSlots(&ethpb.StreamSlotsRequest{
VerifiedOnly: true,
}, mockStream), "Could not call RPC method")
}(t)
wrappedBlk, err := blocks.NewSignedBeaconBlock(&ethpb.SignedBeaconBlock{Block: &ethpb.BeaconBlock{Slot: 123, Body: &ethpb.BeaconBlockBody{}}})
require.NoError(t, err)
// Send in a loop to ensure it is delivered (busy wait for the service to subscribe to the state feed).
for sent := 0; sent == 0; {
sent = server.StateNotifier.StateFeed().Send(&feed.Event{
Type: statefeed.BlockProcessed,
Data: &statefeed.BlockProcessedData{Slot: 123, BlockRoot: [32]byte{}, SignedBlock: wrappedBlk},
})
}
<-exitRoutine
}

View File

@ -11,9 +11,8 @@ iface_mock_path="testing/validator-mock"
proto_mocks_v1alpha1=(
"$mock_path/beacon_service_mock.go BeaconChainClient,BeaconChain_StreamChainHeadClient,BeaconChain_StreamAttestationsClient,BeaconChain_StreamBlocksClient,BeaconChain_StreamValidatorsInfoClient,BeaconChain_StreamIndexedAttestationsClient"
"$mock_path/beacon_chain_service_mock.go BeaconChain_StreamChainHeadServer,BeaconChain_StreamAttestationsServer,BeaconChain_StreamBlocksServer,BeaconChain_StreamValidatorsInfoServer,BeaconChain_StreamIndexedAttestationsServer"
"$mock_path/beacon_validator_server_mock.go BeaconNodeValidatorServer,BeaconNodeValidator_WaitForActivationServer,BeaconNodeValidator_WaitForChainStartServer,BeaconNodeValidator_StreamDutiesServer"
"$mock_path/beacon_validator_client_mock.go BeaconNodeValidatorClient,BeaconNodeValidator_WaitForChainStartClient,BeaconNodeValidator_WaitForActivationClient,BeaconNodeValidator_StreamDutiesClient"
"$mock_path/slasher_client_mock.go SlasherClient"
"$mock_path/beacon_validator_server_mock.go BeaconNodeValidatorServer,BeaconNodeValidator_WaitForActivationServer,BeaconNodeValidator_WaitForChainStartServer,BeaconNodeValidator_StreamDutiesServer,BeaconNodeValidator_StreamSlotsServer"
"$mock_path/beacon_validator_client_mock.go BeaconNodeValidatorClient,BeaconNodeValidator_WaitForChainStartClient,BeaconNodeValidator_WaitForActivationClient,BeaconNodeValidator_StreamDutiesClient,BeaconNodeValidator_StreamSlotsClient"
"$mock_path/node_service_mock.go NodeClient"
)
@ -25,41 +24,12 @@ for ((i = 0; i < ${#proto_mocks_v1alpha1[@]}; i++)); do
GO11MODULE=on mockgen -package=mock -destination="$file" github.com/prysmaticlabs/prysm/v4/proto/prysm/v1alpha1 "$interfaces"
done
# github.com/prysmaticlabs/prysm/v4/proto/eth/service
# ---------------------------------------------------
proto_mocks_service=(
"$mock_path/event_service_mock.go EventsClient,Events_StreamEventsClient,Events_StreamEventsServer"
)
for ((i = 0; i < ${#proto_mocks_service[@]}; i++)); do
file=${proto_mocks_service[i]% *};
interfaces=${proto_mocks_service[i]#* };
echo "generating $file for interfaces: $interfaces";
echo
GO11MODULE=on mockgen -package=mock -destination="$file" github.com/prysmaticlabs/prysm/v4/proto/eth/service "$interfaces"
done
# github.com/prysmaticlabs/prysm/proto/v4/prysm/v1alpha1/validator-client
# -----------------------------------------------------------------------
proto_mocks_v1alpha1_validator_clients=(
"$mock_path/keymanager_mock.go RemoteSignerClient"
)
for ((i = 0; i < ${#proto_mocks_v1alpha1_validator_clients[@]}; i++)); do
file=${proto_mocks_v1alpha1_validator_clients[i]% *};
interfaces=${proto_mocks_v1alpha1_validator_clients[i]#* };
echo "generating $file for interfaces: $interfaces";
echo
GO11MODULE=on mockgen -package=mock -destination="$file" github.com/prysmaticlabs/prysm/v4/proto/prysm/v1alpha1/validator-client "$interfaces"
done
# github.com/prysmaticlabs/prysm/v4/validator/client/iface
# --------------------------------------------------------
iface_mocks=(
"$iface_mock_path/beacon_chain_client_mock.go BeaconChainClient"
"$iface_mock_path/prysm_beacon_chain_client_mock.go PrysmBeaconChainClient"
"$iface_mock_path/node_client_mock.go NodeClient"
"$iface_mock_path/slasher_client_mock.go SlasherClient"
"$iface_mock_path/validator_client_mock.go ValidatorClient"
)

File diff suppressed because it is too large Load Diff

View File

@ -824,6 +824,34 @@ func local_request_BeaconNodeValidator_SubmitSignedContributionAndProof_0(ctx co
}
var (
filter_BeaconNodeValidator_StreamSlots_0 = &utilities.DoubleArray{Encoding: map[string]int{}, Base: []int(nil), Check: []int(nil)}
)
func request_BeaconNodeValidator_StreamSlots_0(ctx context.Context, marshaler runtime.Marshaler, client BeaconNodeValidatorClient, req *http.Request, pathParams map[string]string) (BeaconNodeValidator_StreamSlotsClient, runtime.ServerMetadata, error) {
var protoReq StreamSlotsRequest
var metadata runtime.ServerMetadata
if err := req.ParseForm(); err != nil {
return nil, metadata, status.Errorf(codes.InvalidArgument, "%v", err)
}
if err := runtime.PopulateQueryParameters(&protoReq, req.Form, filter_BeaconNodeValidator_StreamSlots_0); err != nil {
return nil, metadata, status.Errorf(codes.InvalidArgument, "%v", err)
}
stream, err := client.StreamSlots(ctx, &protoReq)
if err != nil {
return nil, metadata, err
}
header, err := stream.Header()
if err != nil {
return nil, metadata, err
}
metadata.HeaderMD = header
return stream, metadata, nil
}
var (
filter_BeaconNodeValidator_StreamBlocksAltair_0 = &utilities.DoubleArray{Encoding: map[string]int{}, Base: []int(nil), Check: []int(nil)}
)
@ -1466,6 +1494,13 @@ func RegisterBeaconNodeValidatorHandlerServer(ctx context.Context, mux *runtime.
})
mux.Handle("GET", pattern_BeaconNodeValidator_StreamSlots_0, func(w http.ResponseWriter, req *http.Request, pathParams map[string]string) {
err := status.Error(codes.Unimplemented, "streaming calls are not yet supported in the in-process transport")
_, outboundMarshaler := runtime.MarshalerForRequest(mux, req)
runtime.HTTPError(ctx, mux, outboundMarshaler, w, req, err)
return
})
mux.Handle("GET", pattern_BeaconNodeValidator_StreamBlocksAltair_0, func(w http.ResponseWriter, req *http.Request, pathParams map[string]string) {
err := status.Error(codes.Unimplemented, "streaming calls are not yet supported in the in-process transport")
_, outboundMarshaler := runtime.MarshalerForRequest(mux, req)
@ -2063,6 +2098,26 @@ func RegisterBeaconNodeValidatorHandlerClient(ctx context.Context, mux *runtime.
})
mux.Handle("GET", pattern_BeaconNodeValidator_StreamSlots_0, func(w http.ResponseWriter, req *http.Request, pathParams map[string]string) {
ctx, cancel := context.WithCancel(req.Context())
defer cancel()
inboundMarshaler, outboundMarshaler := runtime.MarshalerForRequest(mux, req)
rctx, err := runtime.AnnotateContext(ctx, mux, req, "/ethereum.eth.v1alpha1.BeaconNodeValidator/StreamSlots")
if err != nil {
runtime.HTTPError(ctx, mux, outboundMarshaler, w, req, err)
return
}
resp, md, err := request_BeaconNodeValidator_StreamSlots_0(rctx, inboundMarshaler, client, req, pathParams)
ctx = runtime.NewServerMetadataContext(ctx, md)
if err != nil {
runtime.HTTPError(ctx, mux, outboundMarshaler, w, req, err)
return
}
forward_BeaconNodeValidator_StreamSlots_0(ctx, mux, outboundMarshaler, w, req, func() (proto.Message, error) { return resp.Recv() }, mux.GetForwardResponseOptions()...)
})
mux.Handle("GET", pattern_BeaconNodeValidator_StreamBlocksAltair_0, func(w http.ResponseWriter, req *http.Request, pathParams map[string]string) {
ctx, cancel := context.WithCancel(req.Context())
defer cancel()
@ -2195,6 +2250,8 @@ var (
pattern_BeaconNodeValidator_SubmitSignedContributionAndProof_0 = runtime.MustPattern(runtime.NewPattern(1, []int{2, 0, 2, 1, 2, 2, 2, 3}, []string{"eth", "v1alpha1", "validator", "signed_contribution_and_proof"}, ""))
pattern_BeaconNodeValidator_StreamSlots_0 = runtime.MustPattern(runtime.NewPattern(1, []int{2, 0, 2, 1, 2, 2, 2, 3, 2, 4}, []string{"eth", "v1alpha1", "validator", "blocks", "stream_slots"}, ""))
pattern_BeaconNodeValidator_StreamBlocksAltair_0 = runtime.MustPattern(runtime.NewPattern(1, []int{2, 0, 2, 1, 2, 2, 2, 3, 2, 4}, []string{"eth", "v1alpha1", "validator", "blocks", "stream"}, ""))
pattern_BeaconNodeValidator_SubmitValidatorRegistrations_0 = runtime.MustPattern(runtime.NewPattern(1, []int{2, 0, 2, 1, 2, 2, 2, 3}, []string{"eth", "v1alpha1", "validator", "registration"}, ""))
@ -2253,6 +2310,8 @@ var (
forward_BeaconNodeValidator_SubmitSignedContributionAndProof_0 = runtime.ForwardResponseMessage
forward_BeaconNodeValidator_StreamSlots_0 = runtime.ForwardResponseStream
forward_BeaconNodeValidator_StreamBlocksAltair_0 = runtime.ForwardResponseStream
forward_BeaconNodeValidator_SubmitValidatorRegistrations_0 = runtime.ForwardResponseMessage

View File

@ -294,6 +294,16 @@ service BeaconNodeValidator {
};
}
// Server-side stream of all slots of valid blocks as they are received by
// the beacon chain node.
// DEPRECATED: This endpoint is superseded by the /eth/v1/events Beacon API endpoint
rpc StreamSlots(StreamSlotsRequest) returns (stream StreamSlotsResponse) {
option (google.api.http) = {
get: "/eth/v1alpha1/validator/blocks/stream_slots"
};
option deprecated = true;
}
// Server-side stream of all signed blocks as they are received by
// the beacon chain node.
// DEPRECATED: This endpoint is superseded by the /eth/v1/events Beacon API endpoint
@ -356,6 +366,13 @@ message SyncSubcommitteeIndexResponse {
repeated uint64 indices = 1 [(ethereum.eth.ext.cast_type) = "github.com/prysmaticlabs/prysm/v4/consensus-types/primitives.CommitteeIndex"];
}
// DEPRECATED: This endpoint StreamSlots is superseded by the /eth/v1/events Beacon API endpoint
message StreamSlotsResponse {
option deprecated = true;
uint64 slot = 1 [(ethereum.eth.ext.cast_type) = "github.com/prysmaticlabs/prysm/v4/consensus-types/primitives.Slot"];
}
// DEPRECATED: This endpoint StreamBlocks is superseded by the /eth/v1/events Beacon API endpoint
message StreamBlocksResponse {
option deprecated = true;
@ -740,8 +757,16 @@ message DoppelGangerResponse {
repeated ValidatorResponse responses = 1;
}
// Request to only return slots with blocks that where verified by the beacon node.
// DEPRECATED: This endpoint StreamSlots is superseded by the /eth/v1/events Beacon API endpoint.
message StreamSlotsRequest {
option deprecated = true;
bool verified_only = 1;
}
// Request to only return blocks that is verified by the beacon node.
// DEPRECATED: This endpoint StreamBlocks is superseded by the /eth/v1/events Beacon API endpoint, there is an internal GRPC endpoint /internal/eth/v1/events, this message will no longer be needed.
// DEPRECATED: This endpoint StreamBlocks is superseded by the /eth/v1/events Beacon API endpoint
message StreamBlocksRequest {
option deprecated = true;
bool verified_only = 1;

View File

@ -13,31 +13,31 @@ import (
metadata "google.golang.org/grpc/metadata"
)
// BeaconChain_StreamChainHeadServer is a mock of BeaconChain_StreamChainHeadServer interface.
type BeaconChain_StreamChainHeadServer struct {
// MockBeaconChain_StreamChainHeadServer is a mock of BeaconChain_StreamChainHeadServer interface.
type MockBeaconChain_StreamChainHeadServer struct {
ctrl *gomock.Controller
recorder *BeaconChain_StreamChainHeadServerMockRecorder
recorder *MockBeaconChain_StreamChainHeadServerMockRecorder
}
// BeaconChain_StreamChainHeadServerMockRecorder is the mock recorder for MockBeaconChain_StreamChainHeadServer.
type BeaconChain_StreamChainHeadServerMockRecorder struct {
mock *BeaconChain_StreamChainHeadServer
// MockBeaconChain_StreamChainHeadServerMockRecorder is the mock recorder for MockBeaconChain_StreamChainHeadServer.
type MockBeaconChain_StreamChainHeadServerMockRecorder struct {
mock *MockBeaconChain_StreamChainHeadServer
}
// NewMockBeaconChain_StreamChainHeadServer creates a new mock instance.
func NewMockBeaconChain_StreamChainHeadServer(ctrl *gomock.Controller) *BeaconChain_StreamChainHeadServer {
mock := &BeaconChain_StreamChainHeadServer{ctrl: ctrl}
mock.recorder = &BeaconChain_StreamChainHeadServerMockRecorder{mock}
func NewMockBeaconChain_StreamChainHeadServer(ctrl *gomock.Controller) *MockBeaconChain_StreamChainHeadServer {
mock := &MockBeaconChain_StreamChainHeadServer{ctrl: ctrl}
mock.recorder = &MockBeaconChain_StreamChainHeadServerMockRecorder{mock}
return mock
}
// EXPECT returns an object that allows the caller to indicate expected use.
func (m *BeaconChain_StreamChainHeadServer) EXPECT() *BeaconChain_StreamChainHeadServerMockRecorder {
func (m *MockBeaconChain_StreamChainHeadServer) EXPECT() *MockBeaconChain_StreamChainHeadServerMockRecorder {
return m.recorder
}
// Context mocks base method.
func (m *BeaconChain_StreamChainHeadServer) Context() context.Context {
func (m *MockBeaconChain_StreamChainHeadServer) Context() context.Context {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "Context")
ret0, _ := ret[0].(context.Context)
@ -45,13 +45,13 @@ func (m *BeaconChain_StreamChainHeadServer) Context() context.Context {
}
// Context indicates an expected call of Context.
func (mr *BeaconChain_StreamChainHeadServerMockRecorder) Context() *gomock.Call {
func (mr *MockBeaconChain_StreamChainHeadServerMockRecorder) Context() *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Context", reflect.TypeOf((*BeaconChain_StreamChainHeadServer)(nil).Context))
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Context", reflect.TypeOf((*MockBeaconChain_StreamChainHeadServer)(nil).Context))
}
// RecvMsg mocks base method.
func (m *BeaconChain_StreamChainHeadServer) RecvMsg(arg0 interface{}) error {
func (m *MockBeaconChain_StreamChainHeadServer) RecvMsg(arg0 interface{}) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "RecvMsg", arg0)
ret0, _ := ret[0].(error)
@ -59,13 +59,13 @@ func (m *BeaconChain_StreamChainHeadServer) RecvMsg(arg0 interface{}) error {
}
// RecvMsg indicates an expected call of RecvMsg.
func (mr *BeaconChain_StreamChainHeadServerMockRecorder) RecvMsg(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamChainHeadServerMockRecorder) RecvMsg(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "RecvMsg", reflect.TypeOf((*BeaconChain_StreamChainHeadServer)(nil).RecvMsg), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "RecvMsg", reflect.TypeOf((*MockBeaconChain_StreamChainHeadServer)(nil).RecvMsg), arg0)
}
// Send mocks base method.
func (m *BeaconChain_StreamChainHeadServer) Send(arg0 *eth.ChainHead) error {
func (m *MockBeaconChain_StreamChainHeadServer) Send(arg0 *eth.ChainHead) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "Send", arg0)
ret0, _ := ret[0].(error)
@ -73,13 +73,13 @@ func (m *BeaconChain_StreamChainHeadServer) Send(arg0 *eth.ChainHead) error {
}
// Send indicates an expected call of Send.
func (mr *BeaconChain_StreamChainHeadServerMockRecorder) Send(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamChainHeadServerMockRecorder) Send(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Send", reflect.TypeOf((*BeaconChain_StreamChainHeadServer)(nil).Send), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Send", reflect.TypeOf((*MockBeaconChain_StreamChainHeadServer)(nil).Send), arg0)
}
// SendHeader mocks base method.
func (m *BeaconChain_StreamChainHeadServer) SendHeader(arg0 metadata.MD) error {
func (m *MockBeaconChain_StreamChainHeadServer) SendHeader(arg0 metadata.MD) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "SendHeader", arg0)
ret0, _ := ret[0].(error)
@ -87,13 +87,13 @@ func (m *BeaconChain_StreamChainHeadServer) SendHeader(arg0 metadata.MD) error {
}
// SendHeader indicates an expected call of SendHeader.
func (mr *BeaconChain_StreamChainHeadServerMockRecorder) SendHeader(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamChainHeadServerMockRecorder) SendHeader(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SendHeader", reflect.TypeOf((*BeaconChain_StreamChainHeadServer)(nil).SendHeader), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SendHeader", reflect.TypeOf((*MockBeaconChain_StreamChainHeadServer)(nil).SendHeader), arg0)
}
// SendMsg mocks base method.
func (m *BeaconChain_StreamChainHeadServer) SendMsg(arg0 interface{}) error {
func (m *MockBeaconChain_StreamChainHeadServer) SendMsg(arg0 interface{}) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "SendMsg", arg0)
ret0, _ := ret[0].(error)
@ -101,13 +101,13 @@ func (m *BeaconChain_StreamChainHeadServer) SendMsg(arg0 interface{}) error {
}
// SendMsg indicates an expected call of SendMsg.
func (mr *BeaconChain_StreamChainHeadServerMockRecorder) SendMsg(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamChainHeadServerMockRecorder) SendMsg(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SendMsg", reflect.TypeOf((*BeaconChain_StreamChainHeadServer)(nil).SendMsg), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SendMsg", reflect.TypeOf((*MockBeaconChain_StreamChainHeadServer)(nil).SendMsg), arg0)
}
// SetHeader mocks base method.
func (m *BeaconChain_StreamChainHeadServer) SetHeader(arg0 metadata.MD) error {
func (m *MockBeaconChain_StreamChainHeadServer) SetHeader(arg0 metadata.MD) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "SetHeader", arg0)
ret0, _ := ret[0].(error)
@ -115,48 +115,48 @@ func (m *BeaconChain_StreamChainHeadServer) SetHeader(arg0 metadata.MD) error {
}
// SetHeader indicates an expected call of SetHeader.
func (mr *BeaconChain_StreamChainHeadServerMockRecorder) SetHeader(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamChainHeadServerMockRecorder) SetHeader(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SetHeader", reflect.TypeOf((*BeaconChain_StreamChainHeadServer)(nil).SetHeader), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SetHeader", reflect.TypeOf((*MockBeaconChain_StreamChainHeadServer)(nil).SetHeader), arg0)
}
// SetTrailer mocks base method.
func (m *BeaconChain_StreamChainHeadServer) SetTrailer(arg0 metadata.MD) {
func (m *MockBeaconChain_StreamChainHeadServer) SetTrailer(arg0 metadata.MD) {
m.ctrl.T.Helper()
m.ctrl.Call(m, "SetTrailer", arg0)
}
// SetTrailer indicates an expected call of SetTrailer.
func (mr *BeaconChain_StreamChainHeadServerMockRecorder) SetTrailer(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamChainHeadServerMockRecorder) SetTrailer(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SetTrailer", reflect.TypeOf((*BeaconChain_StreamChainHeadServer)(nil).SetTrailer), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SetTrailer", reflect.TypeOf((*MockBeaconChain_StreamChainHeadServer)(nil).SetTrailer), arg0)
}
// BeaconChain_StreamAttestationsServer is a mock of BeaconChain_StreamAttestationsServer interface.
type BeaconChain_StreamAttestationsServer struct {
// MockBeaconChain_StreamAttestationsServer is a mock of BeaconChain_StreamAttestationsServer interface.
type MockBeaconChain_StreamAttestationsServer struct {
ctrl *gomock.Controller
recorder *BeaconChain_StreamAttestationsServerMockRecorder
recorder *MockBeaconChain_StreamAttestationsServerMockRecorder
}
// BeaconChain_StreamAttestationsServerMockRecorder is the mock recorder for MockBeaconChain_StreamAttestationsServer.
type BeaconChain_StreamAttestationsServerMockRecorder struct {
mock *BeaconChain_StreamAttestationsServer
// MockBeaconChain_StreamAttestationsServerMockRecorder is the mock recorder for MockBeaconChain_StreamAttestationsServer.
type MockBeaconChain_StreamAttestationsServerMockRecorder struct {
mock *MockBeaconChain_StreamAttestationsServer
}
// NewMockBeaconChain_StreamAttestationsServer creates a new mock instance.
func NewMockBeaconChain_StreamAttestationsServer(ctrl *gomock.Controller) *BeaconChain_StreamAttestationsServer {
mock := &BeaconChain_StreamAttestationsServer{ctrl: ctrl}
mock.recorder = &BeaconChain_StreamAttestationsServerMockRecorder{mock}
func NewMockBeaconChain_StreamAttestationsServer(ctrl *gomock.Controller) *MockBeaconChain_StreamAttestationsServer {
mock := &MockBeaconChain_StreamAttestationsServer{ctrl: ctrl}
mock.recorder = &MockBeaconChain_StreamAttestationsServerMockRecorder{mock}
return mock
}
// EXPECT returns an object that allows the caller to indicate expected use.
func (m *BeaconChain_StreamAttestationsServer) EXPECT() *BeaconChain_StreamAttestationsServerMockRecorder {
func (m *MockBeaconChain_StreamAttestationsServer) EXPECT() *MockBeaconChain_StreamAttestationsServerMockRecorder {
return m.recorder
}
// Context mocks base method.
func (m *BeaconChain_StreamAttestationsServer) Context() context.Context {
func (m *MockBeaconChain_StreamAttestationsServer) Context() context.Context {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "Context")
ret0, _ := ret[0].(context.Context)
@ -164,13 +164,13 @@ func (m *BeaconChain_StreamAttestationsServer) Context() context.Context {
}
// Context indicates an expected call of Context.
func (mr *BeaconChain_StreamAttestationsServerMockRecorder) Context() *gomock.Call {
func (mr *MockBeaconChain_StreamAttestationsServerMockRecorder) Context() *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Context", reflect.TypeOf((*BeaconChain_StreamAttestationsServer)(nil).Context))
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Context", reflect.TypeOf((*MockBeaconChain_StreamAttestationsServer)(nil).Context))
}
// RecvMsg mocks base method.
func (m *BeaconChain_StreamAttestationsServer) RecvMsg(arg0 interface{}) error {
func (m *MockBeaconChain_StreamAttestationsServer) RecvMsg(arg0 interface{}) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "RecvMsg", arg0)
ret0, _ := ret[0].(error)
@ -178,13 +178,13 @@ func (m *BeaconChain_StreamAttestationsServer) RecvMsg(arg0 interface{}) error {
}
// RecvMsg indicates an expected call of RecvMsg.
func (mr *BeaconChain_StreamAttestationsServerMockRecorder) RecvMsg(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamAttestationsServerMockRecorder) RecvMsg(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "RecvMsg", reflect.TypeOf((*BeaconChain_StreamAttestationsServer)(nil).RecvMsg), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "RecvMsg", reflect.TypeOf((*MockBeaconChain_StreamAttestationsServer)(nil).RecvMsg), arg0)
}
// Send mocks base method.
func (m *BeaconChain_StreamAttestationsServer) Send(arg0 *eth.Attestation) error {
func (m *MockBeaconChain_StreamAttestationsServer) Send(arg0 *eth.Attestation) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "Send", arg0)
ret0, _ := ret[0].(error)
@ -192,13 +192,13 @@ func (m *BeaconChain_StreamAttestationsServer) Send(arg0 *eth.Attestation) error
}
// Send indicates an expected call of Send.
func (mr *BeaconChain_StreamAttestationsServerMockRecorder) Send(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamAttestationsServerMockRecorder) Send(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Send", reflect.TypeOf((*BeaconChain_StreamAttestationsServer)(nil).Send), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Send", reflect.TypeOf((*MockBeaconChain_StreamAttestationsServer)(nil).Send), arg0)
}
// SendHeader mocks base method.
func (m *BeaconChain_StreamAttestationsServer) SendHeader(arg0 metadata.MD) error {
func (m *MockBeaconChain_StreamAttestationsServer) SendHeader(arg0 metadata.MD) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "SendHeader", arg0)
ret0, _ := ret[0].(error)
@ -206,13 +206,13 @@ func (m *BeaconChain_StreamAttestationsServer) SendHeader(arg0 metadata.MD) erro
}
// SendHeader indicates an expected call of SendHeader.
func (mr *BeaconChain_StreamAttestationsServerMockRecorder) SendHeader(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamAttestationsServerMockRecorder) SendHeader(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SendHeader", reflect.TypeOf((*BeaconChain_StreamAttestationsServer)(nil).SendHeader), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SendHeader", reflect.TypeOf((*MockBeaconChain_StreamAttestationsServer)(nil).SendHeader), arg0)
}
// SendMsg mocks base method.
func (m *BeaconChain_StreamAttestationsServer) SendMsg(arg0 interface{}) error {
func (m *MockBeaconChain_StreamAttestationsServer) SendMsg(arg0 interface{}) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "SendMsg", arg0)
ret0, _ := ret[0].(error)
@ -220,13 +220,13 @@ func (m *BeaconChain_StreamAttestationsServer) SendMsg(arg0 interface{}) error {
}
// SendMsg indicates an expected call of SendMsg.
func (mr *BeaconChain_StreamAttestationsServerMockRecorder) SendMsg(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamAttestationsServerMockRecorder) SendMsg(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SendMsg", reflect.TypeOf((*BeaconChain_StreamAttestationsServer)(nil).SendMsg), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SendMsg", reflect.TypeOf((*MockBeaconChain_StreamAttestationsServer)(nil).SendMsg), arg0)
}
// SetHeader mocks base method.
func (m *BeaconChain_StreamAttestationsServer) SetHeader(arg0 metadata.MD) error {
func (m *MockBeaconChain_StreamAttestationsServer) SetHeader(arg0 metadata.MD) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "SetHeader", arg0)
ret0, _ := ret[0].(error)
@ -234,48 +234,48 @@ func (m *BeaconChain_StreamAttestationsServer) SetHeader(arg0 metadata.MD) error
}
// SetHeader indicates an expected call of SetHeader.
func (mr *BeaconChain_StreamAttestationsServerMockRecorder) SetHeader(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamAttestationsServerMockRecorder) SetHeader(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SetHeader", reflect.TypeOf((*BeaconChain_StreamAttestationsServer)(nil).SetHeader), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SetHeader", reflect.TypeOf((*MockBeaconChain_StreamAttestationsServer)(nil).SetHeader), arg0)
}
// SetTrailer mocks base method.
func (m *BeaconChain_StreamAttestationsServer) SetTrailer(arg0 metadata.MD) {
func (m *MockBeaconChain_StreamAttestationsServer) SetTrailer(arg0 metadata.MD) {
m.ctrl.T.Helper()
m.ctrl.Call(m, "SetTrailer", arg0)
}
// SetTrailer indicates an expected call of SetTrailer.
func (mr *BeaconChain_StreamAttestationsServerMockRecorder) SetTrailer(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamAttestationsServerMockRecorder) SetTrailer(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SetTrailer", reflect.TypeOf((*BeaconChain_StreamAttestationsServer)(nil).SetTrailer), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SetTrailer", reflect.TypeOf((*MockBeaconChain_StreamAttestationsServer)(nil).SetTrailer), arg0)
}
// BeaconChain_StreamBlocksServer is a mock of BeaconChain_StreamBlocksServer interface.
type BeaconChain_StreamBlocksServer struct {
// MockBeaconChain_StreamBlocksServer is a mock of BeaconChain_StreamBlocksServer interface.
type MockBeaconChain_StreamBlocksServer struct {
ctrl *gomock.Controller
recorder *BeaconChain_StreamBlocksServerMockRecorder
recorder *MockBeaconChain_StreamBlocksServerMockRecorder
}
// BeaconChain_StreamBlocksServerMockRecorder is the mock recorder for MockBeaconChain_StreamBlocksServer.
type BeaconChain_StreamBlocksServerMockRecorder struct {
mock *BeaconChain_StreamBlocksServer
// MockBeaconChain_StreamBlocksServerMockRecorder is the mock recorder for MockBeaconChain_StreamBlocksServer.
type MockBeaconChain_StreamBlocksServerMockRecorder struct {
mock *MockBeaconChain_StreamBlocksServer
}
// NewMockBeaconChain_StreamBlocksServer creates a new mock instance.
func NewMockBeaconChain_StreamBlocksServer(ctrl *gomock.Controller) *BeaconChain_StreamBlocksServer {
mock := &BeaconChain_StreamBlocksServer{ctrl: ctrl}
mock.recorder = &BeaconChain_StreamBlocksServerMockRecorder{mock}
func NewMockBeaconChain_StreamBlocksServer(ctrl *gomock.Controller) *MockBeaconChain_StreamBlocksServer {
mock := &MockBeaconChain_StreamBlocksServer{ctrl: ctrl}
mock.recorder = &MockBeaconChain_StreamBlocksServerMockRecorder{mock}
return mock
}
// EXPECT returns an object that allows the caller to indicate expected use.
func (m *BeaconChain_StreamBlocksServer) EXPECT() *BeaconChain_StreamBlocksServerMockRecorder {
func (m *MockBeaconChain_StreamBlocksServer) EXPECT() *MockBeaconChain_StreamBlocksServerMockRecorder {
return m.recorder
}
// Context mocks base method.
func (m *BeaconChain_StreamBlocksServer) Context() context.Context {
func (m *MockBeaconChain_StreamBlocksServer) Context() context.Context {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "Context")
ret0, _ := ret[0].(context.Context)
@ -283,13 +283,13 @@ func (m *BeaconChain_StreamBlocksServer) Context() context.Context {
}
// Context indicates an expected call of Context.
func (mr *BeaconChain_StreamBlocksServerMockRecorder) Context() *gomock.Call {
func (mr *MockBeaconChain_StreamBlocksServerMockRecorder) Context() *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Context", reflect.TypeOf((*BeaconChain_StreamBlocksServer)(nil).Context))
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Context", reflect.TypeOf((*MockBeaconChain_StreamBlocksServer)(nil).Context))
}
// RecvMsg mocks base method.
func (m *BeaconChain_StreamBlocksServer) RecvMsg(arg0 interface{}) error {
func (m *MockBeaconChain_StreamBlocksServer) RecvMsg(arg0 interface{}) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "RecvMsg", arg0)
ret0, _ := ret[0].(error)
@ -297,13 +297,13 @@ func (m *BeaconChain_StreamBlocksServer) RecvMsg(arg0 interface{}) error {
}
// RecvMsg indicates an expected call of RecvMsg.
func (mr *BeaconChain_StreamBlocksServerMockRecorder) RecvMsg(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamBlocksServerMockRecorder) RecvMsg(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "RecvMsg", reflect.TypeOf((*BeaconChain_StreamBlocksServer)(nil).RecvMsg), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "RecvMsg", reflect.TypeOf((*MockBeaconChain_StreamBlocksServer)(nil).RecvMsg), arg0)
}
// Send mocks base method.
func (m *BeaconChain_StreamBlocksServer) Send(arg0 *eth.SignedBeaconBlock) error {
func (m *MockBeaconChain_StreamBlocksServer) Send(arg0 *eth.SignedBeaconBlock) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "Send", arg0)
ret0, _ := ret[0].(error)
@ -311,13 +311,13 @@ func (m *BeaconChain_StreamBlocksServer) Send(arg0 *eth.SignedBeaconBlock) error
}
// Send indicates an expected call of Send.
func (mr *BeaconChain_StreamBlocksServerMockRecorder) Send(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamBlocksServerMockRecorder) Send(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Send", reflect.TypeOf((*BeaconChain_StreamBlocksServer)(nil).Send), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Send", reflect.TypeOf((*MockBeaconChain_StreamBlocksServer)(nil).Send), arg0)
}
// SendHeader mocks base method.
func (m *BeaconChain_StreamBlocksServer) SendHeader(arg0 metadata.MD) error {
func (m *MockBeaconChain_StreamBlocksServer) SendHeader(arg0 metadata.MD) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "SendHeader", arg0)
ret0, _ := ret[0].(error)
@ -325,13 +325,13 @@ func (m *BeaconChain_StreamBlocksServer) SendHeader(arg0 metadata.MD) error {
}
// SendHeader indicates an expected call of SendHeader.
func (mr *BeaconChain_StreamBlocksServerMockRecorder) SendHeader(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamBlocksServerMockRecorder) SendHeader(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SendHeader", reflect.TypeOf((*BeaconChain_StreamBlocksServer)(nil).SendHeader), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SendHeader", reflect.TypeOf((*MockBeaconChain_StreamBlocksServer)(nil).SendHeader), arg0)
}
// SendMsg mocks base method.
func (m *BeaconChain_StreamBlocksServer) SendMsg(arg0 interface{}) error {
func (m *MockBeaconChain_StreamBlocksServer) SendMsg(arg0 interface{}) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "SendMsg", arg0)
ret0, _ := ret[0].(error)
@ -339,13 +339,13 @@ func (m *BeaconChain_StreamBlocksServer) SendMsg(arg0 interface{}) error {
}
// SendMsg indicates an expected call of SendMsg.
func (mr *BeaconChain_StreamBlocksServerMockRecorder) SendMsg(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamBlocksServerMockRecorder) SendMsg(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SendMsg", reflect.TypeOf((*BeaconChain_StreamBlocksServer)(nil).SendMsg), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SendMsg", reflect.TypeOf((*MockBeaconChain_StreamBlocksServer)(nil).SendMsg), arg0)
}
// SetHeader mocks base method.
func (m *BeaconChain_StreamBlocksServer) SetHeader(arg0 metadata.MD) error {
func (m *MockBeaconChain_StreamBlocksServer) SetHeader(arg0 metadata.MD) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "SetHeader", arg0)
ret0, _ := ret[0].(error)
@ -353,48 +353,48 @@ func (m *BeaconChain_StreamBlocksServer) SetHeader(arg0 metadata.MD) error {
}
// SetHeader indicates an expected call of SetHeader.
func (mr *BeaconChain_StreamBlocksServerMockRecorder) SetHeader(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamBlocksServerMockRecorder) SetHeader(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SetHeader", reflect.TypeOf((*BeaconChain_StreamBlocksServer)(nil).SetHeader), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SetHeader", reflect.TypeOf((*MockBeaconChain_StreamBlocksServer)(nil).SetHeader), arg0)
}
// SetTrailer mocks base method.
func (m *BeaconChain_StreamBlocksServer) SetTrailer(arg0 metadata.MD) {
func (m *MockBeaconChain_StreamBlocksServer) SetTrailer(arg0 metadata.MD) {
m.ctrl.T.Helper()
m.ctrl.Call(m, "SetTrailer", arg0)
}
// SetTrailer indicates an expected call of SetTrailer.
func (mr *BeaconChain_StreamBlocksServerMockRecorder) SetTrailer(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamBlocksServerMockRecorder) SetTrailer(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SetTrailer", reflect.TypeOf((*BeaconChain_StreamBlocksServer)(nil).SetTrailer), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SetTrailer", reflect.TypeOf((*MockBeaconChain_StreamBlocksServer)(nil).SetTrailer), arg0)
}
// BeaconChain_StreamValidatorsInfoServer is a mock of BeaconChain_StreamValidatorsInfoServer interface.
type BeaconChain_StreamValidatorsInfoServer struct {
// MockBeaconChain_StreamValidatorsInfoServer is a mock of BeaconChain_StreamValidatorsInfoServer interface.
type MockBeaconChain_StreamValidatorsInfoServer struct {
ctrl *gomock.Controller
recorder *BeaconChain_StreamValidatorsInfoServerMockRecorder
recorder *MockBeaconChain_StreamValidatorsInfoServerMockRecorder
}
// BeaconChain_StreamValidatorsInfoServerMockRecorder is the mock recorder for MockBeaconChain_StreamValidatorsInfoServer.
type BeaconChain_StreamValidatorsInfoServerMockRecorder struct {
mock *BeaconChain_StreamValidatorsInfoServer
// MockBeaconChain_StreamValidatorsInfoServerMockRecorder is the mock recorder for MockBeaconChain_StreamValidatorsInfoServer.
type MockBeaconChain_StreamValidatorsInfoServerMockRecorder struct {
mock *MockBeaconChain_StreamValidatorsInfoServer
}
// NewMockBeaconChain_StreamValidatorsInfoServer creates a new mock instance.
func NewMockBeaconChain_StreamValidatorsInfoServer(ctrl *gomock.Controller) *BeaconChain_StreamValidatorsInfoServer {
mock := &BeaconChain_StreamValidatorsInfoServer{ctrl: ctrl}
mock.recorder = &BeaconChain_StreamValidatorsInfoServerMockRecorder{mock}
func NewMockBeaconChain_StreamValidatorsInfoServer(ctrl *gomock.Controller) *MockBeaconChain_StreamValidatorsInfoServer {
mock := &MockBeaconChain_StreamValidatorsInfoServer{ctrl: ctrl}
mock.recorder = &MockBeaconChain_StreamValidatorsInfoServerMockRecorder{mock}
return mock
}
// EXPECT returns an object that allows the caller to indicate expected use.
func (m *BeaconChain_StreamValidatorsInfoServer) EXPECT() *BeaconChain_StreamValidatorsInfoServerMockRecorder {
func (m *MockBeaconChain_StreamValidatorsInfoServer) EXPECT() *MockBeaconChain_StreamValidatorsInfoServerMockRecorder {
return m.recorder
}
// Context mocks base method.
func (m *BeaconChain_StreamValidatorsInfoServer) Context() context.Context {
func (m *MockBeaconChain_StreamValidatorsInfoServer) Context() context.Context {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "Context")
ret0, _ := ret[0].(context.Context)
@ -402,13 +402,13 @@ func (m *BeaconChain_StreamValidatorsInfoServer) Context() context.Context {
}
// Context indicates an expected call of Context.
func (mr *BeaconChain_StreamValidatorsInfoServerMockRecorder) Context() *gomock.Call {
func (mr *MockBeaconChain_StreamValidatorsInfoServerMockRecorder) Context() *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Context", reflect.TypeOf((*BeaconChain_StreamValidatorsInfoServer)(nil).Context))
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Context", reflect.TypeOf((*MockBeaconChain_StreamValidatorsInfoServer)(nil).Context))
}
// Recv mocks base method.
func (m *BeaconChain_StreamValidatorsInfoServer) Recv() (*eth.ValidatorChangeSet, error) {
func (m *MockBeaconChain_StreamValidatorsInfoServer) Recv() (*eth.ValidatorChangeSet, error) {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "Recv")
ret0, _ := ret[0].(*eth.ValidatorChangeSet)
@ -417,13 +417,13 @@ func (m *BeaconChain_StreamValidatorsInfoServer) Recv() (*eth.ValidatorChangeSet
}
// Recv indicates an expected call of Recv.
func (mr *BeaconChain_StreamValidatorsInfoServerMockRecorder) Recv() *gomock.Call {
func (mr *MockBeaconChain_StreamValidatorsInfoServerMockRecorder) Recv() *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Recv", reflect.TypeOf((*BeaconChain_StreamValidatorsInfoServer)(nil).Recv))
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Recv", reflect.TypeOf((*MockBeaconChain_StreamValidatorsInfoServer)(nil).Recv))
}
// RecvMsg mocks base method.
func (m *BeaconChain_StreamValidatorsInfoServer) RecvMsg(arg0 interface{}) error {
func (m *MockBeaconChain_StreamValidatorsInfoServer) RecvMsg(arg0 interface{}) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "RecvMsg", arg0)
ret0, _ := ret[0].(error)
@ -431,13 +431,13 @@ func (m *BeaconChain_StreamValidatorsInfoServer) RecvMsg(arg0 interface{}) error
}
// RecvMsg indicates an expected call of RecvMsg.
func (mr *BeaconChain_StreamValidatorsInfoServerMockRecorder) RecvMsg(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamValidatorsInfoServerMockRecorder) RecvMsg(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "RecvMsg", reflect.TypeOf((*BeaconChain_StreamValidatorsInfoServer)(nil).RecvMsg), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "RecvMsg", reflect.TypeOf((*MockBeaconChain_StreamValidatorsInfoServer)(nil).RecvMsg), arg0)
}
// Send mocks base method.
func (m *BeaconChain_StreamValidatorsInfoServer) Send(arg0 *eth.ValidatorInfo) error {
func (m *MockBeaconChain_StreamValidatorsInfoServer) Send(arg0 *eth.ValidatorInfo) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "Send", arg0)
ret0, _ := ret[0].(error)
@ -445,13 +445,13 @@ func (m *BeaconChain_StreamValidatorsInfoServer) Send(arg0 *eth.ValidatorInfo) e
}
// Send indicates an expected call of Send.
func (mr *BeaconChain_StreamValidatorsInfoServerMockRecorder) Send(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamValidatorsInfoServerMockRecorder) Send(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Send", reflect.TypeOf((*BeaconChain_StreamValidatorsInfoServer)(nil).Send), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Send", reflect.TypeOf((*MockBeaconChain_StreamValidatorsInfoServer)(nil).Send), arg0)
}
// SendHeader mocks base method.
func (m *BeaconChain_StreamValidatorsInfoServer) SendHeader(arg0 metadata.MD) error {
func (m *MockBeaconChain_StreamValidatorsInfoServer) SendHeader(arg0 metadata.MD) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "SendHeader", arg0)
ret0, _ := ret[0].(error)
@ -459,13 +459,13 @@ func (m *BeaconChain_StreamValidatorsInfoServer) SendHeader(arg0 metadata.MD) er
}
// SendHeader indicates an expected call of SendHeader.
func (mr *BeaconChain_StreamValidatorsInfoServerMockRecorder) SendHeader(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamValidatorsInfoServerMockRecorder) SendHeader(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SendHeader", reflect.TypeOf((*BeaconChain_StreamValidatorsInfoServer)(nil).SendHeader), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SendHeader", reflect.TypeOf((*MockBeaconChain_StreamValidatorsInfoServer)(nil).SendHeader), arg0)
}
// SendMsg mocks base method.
func (m *BeaconChain_StreamValidatorsInfoServer) SendMsg(arg0 interface{}) error {
func (m *MockBeaconChain_StreamValidatorsInfoServer) SendMsg(arg0 interface{}) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "SendMsg", arg0)
ret0, _ := ret[0].(error)
@ -473,13 +473,13 @@ func (m *BeaconChain_StreamValidatorsInfoServer) SendMsg(arg0 interface{}) error
}
// SendMsg indicates an expected call of SendMsg.
func (mr *BeaconChain_StreamValidatorsInfoServerMockRecorder) SendMsg(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamValidatorsInfoServerMockRecorder) SendMsg(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SendMsg", reflect.TypeOf((*BeaconChain_StreamValidatorsInfoServer)(nil).SendMsg), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SendMsg", reflect.TypeOf((*MockBeaconChain_StreamValidatorsInfoServer)(nil).SendMsg), arg0)
}
// SetHeader mocks base method.
func (m *BeaconChain_StreamValidatorsInfoServer) SetHeader(arg0 metadata.MD) error {
func (m *MockBeaconChain_StreamValidatorsInfoServer) SetHeader(arg0 metadata.MD) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "SetHeader", arg0)
ret0, _ := ret[0].(error)
@ -487,48 +487,48 @@ func (m *BeaconChain_StreamValidatorsInfoServer) SetHeader(arg0 metadata.MD) err
}
// SetHeader indicates an expected call of SetHeader.
func (mr *BeaconChain_StreamValidatorsInfoServerMockRecorder) SetHeader(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamValidatorsInfoServerMockRecorder) SetHeader(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SetHeader", reflect.TypeOf((*BeaconChain_StreamValidatorsInfoServer)(nil).SetHeader), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SetHeader", reflect.TypeOf((*MockBeaconChain_StreamValidatorsInfoServer)(nil).SetHeader), arg0)
}
// SetTrailer mocks base method.
func (m *BeaconChain_StreamValidatorsInfoServer) SetTrailer(arg0 metadata.MD) {
func (m *MockBeaconChain_StreamValidatorsInfoServer) SetTrailer(arg0 metadata.MD) {
m.ctrl.T.Helper()
m.ctrl.Call(m, "SetTrailer", arg0)
}
// SetTrailer indicates an expected call of SetTrailer.
func (mr *BeaconChain_StreamValidatorsInfoServerMockRecorder) SetTrailer(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamValidatorsInfoServerMockRecorder) SetTrailer(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SetTrailer", reflect.TypeOf((*BeaconChain_StreamValidatorsInfoServer)(nil).SetTrailer), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SetTrailer", reflect.TypeOf((*MockBeaconChain_StreamValidatorsInfoServer)(nil).SetTrailer), arg0)
}
// BeaconChain_StreamIndexedAttestationsServer is a mock of BeaconChain_StreamIndexedAttestationsServer interface.
type BeaconChain_StreamIndexedAttestationsServer struct {
// MockBeaconChain_StreamIndexedAttestationsServer is a mock of BeaconChain_StreamIndexedAttestationsServer interface.
type MockBeaconChain_StreamIndexedAttestationsServer struct {
ctrl *gomock.Controller
recorder *BeaconChain_StreamIndexedAttestationsServerMockRecorder
recorder *MockBeaconChain_StreamIndexedAttestationsServerMockRecorder
}
// BeaconChain_StreamIndexedAttestationsServerMockRecorder is the mock recorder for MockBeaconChain_StreamIndexedAttestationsServer.
type BeaconChain_StreamIndexedAttestationsServerMockRecorder struct {
mock *BeaconChain_StreamIndexedAttestationsServer
// MockBeaconChain_StreamIndexedAttestationsServerMockRecorder is the mock recorder for MockBeaconChain_StreamIndexedAttestationsServer.
type MockBeaconChain_StreamIndexedAttestationsServerMockRecorder struct {
mock *MockBeaconChain_StreamIndexedAttestationsServer
}
// NewMockBeaconChain_StreamIndexedAttestationsServer creates a new mock instance.
func NewMockBeaconChain_StreamIndexedAttestationsServer(ctrl *gomock.Controller) *BeaconChain_StreamIndexedAttestationsServer {
mock := &BeaconChain_StreamIndexedAttestationsServer{ctrl: ctrl}
mock.recorder = &BeaconChain_StreamIndexedAttestationsServerMockRecorder{mock}
func NewMockBeaconChain_StreamIndexedAttestationsServer(ctrl *gomock.Controller) *MockBeaconChain_StreamIndexedAttestationsServer {
mock := &MockBeaconChain_StreamIndexedAttestationsServer{ctrl: ctrl}
mock.recorder = &MockBeaconChain_StreamIndexedAttestationsServerMockRecorder{mock}
return mock
}
// EXPECT returns an object that allows the caller to indicate expected use.
func (m *BeaconChain_StreamIndexedAttestationsServer) EXPECT() *BeaconChain_StreamIndexedAttestationsServerMockRecorder {
func (m *MockBeaconChain_StreamIndexedAttestationsServer) EXPECT() *MockBeaconChain_StreamIndexedAttestationsServerMockRecorder {
return m.recorder
}
// Context mocks base method.
func (m *BeaconChain_StreamIndexedAttestationsServer) Context() context.Context {
func (m *MockBeaconChain_StreamIndexedAttestationsServer) Context() context.Context {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "Context")
ret0, _ := ret[0].(context.Context)
@ -536,13 +536,13 @@ func (m *BeaconChain_StreamIndexedAttestationsServer) Context() context.Context
}
// Context indicates an expected call of Context.
func (mr *BeaconChain_StreamIndexedAttestationsServerMockRecorder) Context() *gomock.Call {
func (mr *MockBeaconChain_StreamIndexedAttestationsServerMockRecorder) Context() *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Context", reflect.TypeOf((*BeaconChain_StreamIndexedAttestationsServer)(nil).Context))
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Context", reflect.TypeOf((*MockBeaconChain_StreamIndexedAttestationsServer)(nil).Context))
}
// RecvMsg mocks base method.
func (m *BeaconChain_StreamIndexedAttestationsServer) RecvMsg(arg0 interface{}) error {
func (m *MockBeaconChain_StreamIndexedAttestationsServer) RecvMsg(arg0 interface{}) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "RecvMsg", arg0)
ret0, _ := ret[0].(error)
@ -550,13 +550,13 @@ func (m *BeaconChain_StreamIndexedAttestationsServer) RecvMsg(arg0 interface{})
}
// RecvMsg indicates an expected call of RecvMsg.
func (mr *BeaconChain_StreamIndexedAttestationsServerMockRecorder) RecvMsg(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamIndexedAttestationsServerMockRecorder) RecvMsg(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "RecvMsg", reflect.TypeOf((*BeaconChain_StreamIndexedAttestationsServer)(nil).RecvMsg), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "RecvMsg", reflect.TypeOf((*MockBeaconChain_StreamIndexedAttestationsServer)(nil).RecvMsg), arg0)
}
// Send mocks base method.
func (m *BeaconChain_StreamIndexedAttestationsServer) Send(arg0 *eth.IndexedAttestation) error {
func (m *MockBeaconChain_StreamIndexedAttestationsServer) Send(arg0 *eth.IndexedAttestation) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "Send", arg0)
ret0, _ := ret[0].(error)
@ -564,13 +564,13 @@ func (m *BeaconChain_StreamIndexedAttestationsServer) Send(arg0 *eth.IndexedAtte
}
// Send indicates an expected call of Send.
func (mr *BeaconChain_StreamIndexedAttestationsServerMockRecorder) Send(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamIndexedAttestationsServerMockRecorder) Send(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Send", reflect.TypeOf((*BeaconChain_StreamIndexedAttestationsServer)(nil).Send), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Send", reflect.TypeOf((*MockBeaconChain_StreamIndexedAttestationsServer)(nil).Send), arg0)
}
// SendHeader mocks base method.
func (m *BeaconChain_StreamIndexedAttestationsServer) SendHeader(arg0 metadata.MD) error {
func (m *MockBeaconChain_StreamIndexedAttestationsServer) SendHeader(arg0 metadata.MD) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "SendHeader", arg0)
ret0, _ := ret[0].(error)
@ -578,13 +578,13 @@ func (m *BeaconChain_StreamIndexedAttestationsServer) SendHeader(arg0 metadata.M
}
// SendHeader indicates an expected call of SendHeader.
func (mr *BeaconChain_StreamIndexedAttestationsServerMockRecorder) SendHeader(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamIndexedAttestationsServerMockRecorder) SendHeader(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SendHeader", reflect.TypeOf((*BeaconChain_StreamIndexedAttestationsServer)(nil).SendHeader), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SendHeader", reflect.TypeOf((*MockBeaconChain_StreamIndexedAttestationsServer)(nil).SendHeader), arg0)
}
// SendMsg mocks base method.
func (m *BeaconChain_StreamIndexedAttestationsServer) SendMsg(arg0 interface{}) error {
func (m *MockBeaconChain_StreamIndexedAttestationsServer) SendMsg(arg0 interface{}) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "SendMsg", arg0)
ret0, _ := ret[0].(error)
@ -592,13 +592,13 @@ func (m *BeaconChain_StreamIndexedAttestationsServer) SendMsg(arg0 interface{})
}
// SendMsg indicates an expected call of SendMsg.
func (mr *BeaconChain_StreamIndexedAttestationsServerMockRecorder) SendMsg(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamIndexedAttestationsServerMockRecorder) SendMsg(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SendMsg", reflect.TypeOf((*BeaconChain_StreamIndexedAttestationsServer)(nil).SendMsg), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SendMsg", reflect.TypeOf((*MockBeaconChain_StreamIndexedAttestationsServer)(nil).SendMsg), arg0)
}
// SetHeader mocks base method.
func (m *BeaconChain_StreamIndexedAttestationsServer) SetHeader(arg0 metadata.MD) error {
func (m *MockBeaconChain_StreamIndexedAttestationsServer) SetHeader(arg0 metadata.MD) error {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "SetHeader", arg0)
ret0, _ := ret[0].(error)
@ -606,19 +606,19 @@ func (m *BeaconChain_StreamIndexedAttestationsServer) SetHeader(arg0 metadata.MD
}
// SetHeader indicates an expected call of SetHeader.
func (mr *BeaconChain_StreamIndexedAttestationsServerMockRecorder) SetHeader(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamIndexedAttestationsServerMockRecorder) SetHeader(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SetHeader", reflect.TypeOf((*BeaconChain_StreamIndexedAttestationsServer)(nil).SetHeader), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SetHeader", reflect.TypeOf((*MockBeaconChain_StreamIndexedAttestationsServer)(nil).SetHeader), arg0)
}
// SetTrailer mocks base method.
func (m *BeaconChain_StreamIndexedAttestationsServer) SetTrailer(arg0 metadata.MD) {
func (m *MockBeaconChain_StreamIndexedAttestationsServer) SetTrailer(arg0 metadata.MD) {
m.ctrl.T.Helper()
m.ctrl.Call(m, "SetTrailer", arg0)
}
// SetTrailer indicates an expected call of SetTrailer.
func (mr *BeaconChain_StreamIndexedAttestationsServerMockRecorder) SetTrailer(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconChain_StreamIndexedAttestationsServerMockRecorder) SetTrailer(arg0 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SetTrailer", reflect.TypeOf((*BeaconChain_StreamIndexedAttestationsServer)(nil).SetTrailer), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "SetTrailer", reflect.TypeOf((*MockBeaconChain_StreamIndexedAttestationsServer)(nil).SetTrailer), arg0)
}

File diff suppressed because it is too large Load Diff

File diff suppressed because it is too large Load Diff

File diff suppressed because it is too large Load Diff

View File

@ -14,31 +14,31 @@ import (
emptypb "google.golang.org/protobuf/types/known/emptypb"
)
// NodeClient is a mock of NodeClient interface.
type NodeClient struct {
// MockNodeClient is a mock of NodeClient interface.
type MockNodeClient struct {
ctrl *gomock.Controller
recorder *NodeClientMockRecorder
recorder *MockNodeClientMockRecorder
}
// NodeClientMockRecorder is the mock recorder for MockNodeClient.
type NodeClientMockRecorder struct {
mock *NodeClient
// MockNodeClientMockRecorder is the mock recorder for MockNodeClient.
type MockNodeClientMockRecorder struct {
mock *MockNodeClient
}
// NewMockNodeClient creates a new mock instance.
func NewMockNodeClient(ctrl *gomock.Controller) *NodeClient {
mock := &NodeClient{ctrl: ctrl}
mock.recorder = &NodeClientMockRecorder{mock}
func NewMockNodeClient(ctrl *gomock.Controller) *MockNodeClient {
mock := &MockNodeClient{ctrl: ctrl}
mock.recorder = &MockNodeClientMockRecorder{mock}
return mock
}
// EXPECT returns an object that allows the caller to indicate expected use.
func (m *NodeClient) EXPECT() *NodeClientMockRecorder {
func (m *MockNodeClient) EXPECT() *MockNodeClientMockRecorder {
return m.recorder
}
// GetETH1ConnectionStatus mocks base method.
func (m *NodeClient) GetETH1ConnectionStatus(arg0 context.Context, arg1 *emptypb.Empty, arg2 ...grpc.CallOption) (*eth.ETH1ConnectionStatus, error) {
func (m *MockNodeClient) GetETH1ConnectionStatus(arg0 context.Context, arg1 *emptypb.Empty, arg2 ...grpc.CallOption) (*eth.ETH1ConnectionStatus, error) {
m.ctrl.T.Helper()
varargs := []interface{}{arg0, arg1}
for _, a := range arg2 {
@ -51,14 +51,14 @@ func (m *NodeClient) GetETH1ConnectionStatus(arg0 context.Context, arg1 *emptypb
}
// GetETH1ConnectionStatus indicates an expected call of GetETH1ConnectionStatus.
func (mr *NodeClientMockRecorder) GetETH1ConnectionStatus(arg0, arg1 interface{}, arg2 ...interface{}) *gomock.Call {
func (mr *MockNodeClientMockRecorder) GetETH1ConnectionStatus(arg0, arg1 interface{}, arg2 ...interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
varargs := append([]interface{}{arg0, arg1}, arg2...)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "GetETH1ConnectionStatus", reflect.TypeOf((*NodeClient)(nil).GetETH1ConnectionStatus), varargs...)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "GetETH1ConnectionStatus", reflect.TypeOf((*MockNodeClient)(nil).GetETH1ConnectionStatus), varargs...)
}
// GetGenesis mocks base method.
func (m *NodeClient) GetGenesis(arg0 context.Context, arg1 *emptypb.Empty, arg2 ...grpc.CallOption) (*eth.Genesis, error) {
func (m *MockNodeClient) GetGenesis(arg0 context.Context, arg1 *emptypb.Empty, arg2 ...grpc.CallOption) (*eth.Genesis, error) {
m.ctrl.T.Helper()
varargs := []interface{}{arg0, arg1}
for _, a := range arg2 {
@ -71,14 +71,14 @@ func (m *NodeClient) GetGenesis(arg0 context.Context, arg1 *emptypb.Empty, arg2
}
// GetGenesis indicates an expected call of GetGenesis.
func (mr *NodeClientMockRecorder) GetGenesis(arg0, arg1 interface{}, arg2 ...interface{}) *gomock.Call {
func (mr *MockNodeClientMockRecorder) GetGenesis(arg0, arg1 interface{}, arg2 ...interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
varargs := append([]interface{}{arg0, arg1}, arg2...)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "GetGenesis", reflect.TypeOf((*NodeClient)(nil).GetGenesis), varargs...)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "GetGenesis", reflect.TypeOf((*MockNodeClient)(nil).GetGenesis), varargs...)
}
// GetHost mocks base method.
func (m *NodeClient) GetHost(arg0 context.Context, arg1 *emptypb.Empty, arg2 ...grpc.CallOption) (*eth.HostData, error) {
func (m *MockNodeClient) GetHost(arg0 context.Context, arg1 *emptypb.Empty, arg2 ...grpc.CallOption) (*eth.HostData, error) {
m.ctrl.T.Helper()
varargs := []interface{}{arg0, arg1}
for _, a := range arg2 {
@ -91,14 +91,14 @@ func (m *NodeClient) GetHost(arg0 context.Context, arg1 *emptypb.Empty, arg2 ...
}
// GetHost indicates an expected call of GetHost.
func (mr *NodeClientMockRecorder) GetHost(arg0, arg1 interface{}, arg2 ...interface{}) *gomock.Call {
func (mr *MockNodeClientMockRecorder) GetHost(arg0, arg1 interface{}, arg2 ...interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
varargs := append([]interface{}{arg0, arg1}, arg2...)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "GetHost", reflect.TypeOf((*NodeClient)(nil).GetHost), varargs...)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "GetHost", reflect.TypeOf((*MockNodeClient)(nil).GetHost), varargs...)
}
// GetPeer mocks base method.
func (m *NodeClient) GetPeer(arg0 context.Context, arg1 *eth.PeerRequest, arg2 ...grpc.CallOption) (*eth.Peer, error) {
func (m *MockNodeClient) GetPeer(arg0 context.Context, arg1 *eth.PeerRequest, arg2 ...grpc.CallOption) (*eth.Peer, error) {
m.ctrl.T.Helper()
varargs := []interface{}{arg0, arg1}
for _, a := range arg2 {
@ -111,14 +111,14 @@ func (m *NodeClient) GetPeer(arg0 context.Context, arg1 *eth.PeerRequest, arg2 .
}
// GetPeer indicates an expected call of GetPeer.
func (mr *NodeClientMockRecorder) GetPeer(arg0, arg1 interface{}, arg2 ...interface{}) *gomock.Call {
func (mr *MockNodeClientMockRecorder) GetPeer(arg0, arg1 interface{}, arg2 ...interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
varargs := append([]interface{}{arg0, arg1}, arg2...)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "GetPeer", reflect.TypeOf((*NodeClient)(nil).GetPeer), varargs...)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "GetPeer", reflect.TypeOf((*MockNodeClient)(nil).GetPeer), varargs...)
}
// GetSyncStatus mocks base method.
func (m *NodeClient) GetSyncStatus(arg0 context.Context, arg1 *emptypb.Empty, arg2 ...grpc.CallOption) (*eth.SyncStatus, error) {
func (m *MockNodeClient) GetSyncStatus(arg0 context.Context, arg1 *emptypb.Empty, arg2 ...grpc.CallOption) (*eth.SyncStatus, error) {
m.ctrl.T.Helper()
varargs := []interface{}{arg0, arg1}
for _, a := range arg2 {
@ -131,14 +131,14 @@ func (m *NodeClient) GetSyncStatus(arg0 context.Context, arg1 *emptypb.Empty, ar
}
// GetSyncStatus indicates an expected call of GetSyncStatus.
func (mr *NodeClientMockRecorder) GetSyncStatus(arg0, arg1 interface{}, arg2 ...interface{}) *gomock.Call {
func (mr *MockNodeClientMockRecorder) GetSyncStatus(arg0, arg1 interface{}, arg2 ...interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
varargs := append([]interface{}{arg0, arg1}, arg2...)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "GetSyncStatus", reflect.TypeOf((*NodeClient)(nil).GetSyncStatus), varargs...)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "GetSyncStatus", reflect.TypeOf((*MockNodeClient)(nil).GetSyncStatus), varargs...)
}
// GetVersion mocks base method.
func (m *NodeClient) GetVersion(arg0 context.Context, arg1 *emptypb.Empty, arg2 ...grpc.CallOption) (*eth.Version, error) {
func (m *MockNodeClient) GetVersion(arg0 context.Context, arg1 *emptypb.Empty, arg2 ...grpc.CallOption) (*eth.Version, error) {
m.ctrl.T.Helper()
varargs := []interface{}{arg0, arg1}
for _, a := range arg2 {
@ -151,14 +151,14 @@ func (m *NodeClient) GetVersion(arg0 context.Context, arg1 *emptypb.Empty, arg2
}
// GetVersion indicates an expected call of GetVersion.
func (mr *NodeClientMockRecorder) GetVersion(arg0, arg1 interface{}, arg2 ...interface{}) *gomock.Call {
func (mr *MockNodeClientMockRecorder) GetVersion(arg0, arg1 interface{}, arg2 ...interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
varargs := append([]interface{}{arg0, arg1}, arg2...)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "GetVersion", reflect.TypeOf((*NodeClient)(nil).GetVersion), varargs...)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "GetVersion", reflect.TypeOf((*MockNodeClient)(nil).GetVersion), varargs...)
}
// ListImplementedServices mocks base method.
func (m *NodeClient) ListImplementedServices(arg0 context.Context, arg1 *emptypb.Empty, arg2 ...grpc.CallOption) (*eth.ImplementedServices, error) {
func (m *MockNodeClient) ListImplementedServices(arg0 context.Context, arg1 *emptypb.Empty, arg2 ...grpc.CallOption) (*eth.ImplementedServices, error) {
m.ctrl.T.Helper()
varargs := []interface{}{arg0, arg1}
for _, a := range arg2 {
@ -171,14 +171,14 @@ func (m *NodeClient) ListImplementedServices(arg0 context.Context, arg1 *emptypb
}
// ListImplementedServices indicates an expected call of ListImplementedServices.
func (mr *NodeClientMockRecorder) ListImplementedServices(arg0, arg1 interface{}, arg2 ...interface{}) *gomock.Call {
func (mr *MockNodeClientMockRecorder) ListImplementedServices(arg0, arg1 interface{}, arg2 ...interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
varargs := append([]interface{}{arg0, arg1}, arg2...)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "ListImplementedServices", reflect.TypeOf((*NodeClient)(nil).ListImplementedServices), varargs...)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "ListImplementedServices", reflect.TypeOf((*MockNodeClient)(nil).ListImplementedServices), varargs...)
}
// ListPeers mocks base method.
func (m *NodeClient) ListPeers(arg0 context.Context, arg1 *emptypb.Empty, arg2 ...grpc.CallOption) (*eth.Peers, error) {
func (m *MockNodeClient) ListPeers(arg0 context.Context, arg1 *emptypb.Empty, arg2 ...grpc.CallOption) (*eth.Peers, error) {
m.ctrl.T.Helper()
varargs := []interface{}{arg0, arg1}
for _, a := range arg2 {
@ -191,8 +191,8 @@ func (m *NodeClient) ListPeers(arg0 context.Context, arg1 *emptypb.Empty, arg2 .
}
// ListPeers indicates an expected call of ListPeers.
func (mr *NodeClientMockRecorder) ListPeers(arg0, arg1 interface{}, arg2 ...interface{}) *gomock.Call {
func (mr *MockNodeClientMockRecorder) ListPeers(arg0, arg1 interface{}, arg2 ...interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
varargs := append([]interface{}{arg0, arg1}, arg2...)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "ListPeers", reflect.TypeOf((*NodeClient)(nil).ListPeers), varargs...)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "ListPeers", reflect.TypeOf((*MockNodeClient)(nil).ListPeers), varargs...)
}

View File

@ -247,19 +247,19 @@ func (mr *MockValidatorClientMockRecorder) ProposeExit(arg0, arg1 interface{}) *
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "ProposeExit", reflect.TypeOf((*MockValidatorClient)(nil).ProposeExit), arg0, arg1)
}
// StreamBlocksAltair mocks base method.
func (m *MockValidatorClient) StreamBlocksAltair(arg0 context.Context, arg1 *eth.StreamBlocksRequest) (eth.BeaconNodeValidator_StreamBlocksAltairClient, error) {
// StreamSlots mocks base method.
func (m *MockValidatorClient) StreamSlots(arg0 context.Context, arg1 *eth.StreamSlotsRequest) (eth.BeaconNodeValidator_StreamSlotsClient, error) {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "StreamBlocksAltair", arg0, arg1)
ret0, _ := ret[0].(eth.BeaconNodeValidator_StreamBlocksAltairClient)
ret := m.ctrl.Call(m, "StreamSlots", arg0, arg1)
ret0, _ := ret[0].(eth.BeaconNodeValidator_StreamSlotsClient)
ret1, _ := ret[1].(error)
return ret0, ret1
}
// StreamBlocksAltair indicates an expected call of StreamBlocksAltair.
func (mr *MockValidatorClientMockRecorder) StreamBlocksAltair(arg0, arg1 interface{}) *gomock.Call {
// StreamSlots indicates an expected call of StreamSlots.
func (mr *MockValidatorClientMockRecorder) StreamSlots(arg0, arg1 interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "StreamBlocksAltair", reflect.TypeOf((*MockValidatorClient)(nil).StreamBlocksAltair), arg0, arg1)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "StreamSlots", reflect.TypeOf((*MockValidatorClient)(nil).StreamSlots), arg0, arg1)
}
// SubmitAggregateSelectionProof mocks base method.

View File

@ -170,7 +170,7 @@ func (m *Validator) Keymanager() (keymanager.IKeymanager, error) {
return m.Km, nil
}
func (_ *Validator) ReceiveBlocks(_ context.Context, _ chan<- error) {
func (_ *Validator) ReceiveSlots(_ context.Context, _ chan<- error) {
panic("implement me")
}

View File

@ -14,7 +14,6 @@ import (
"github.com/prysmaticlabs/prysm/v4/config/features"
fieldparams "github.com/prysmaticlabs/prysm/v4/config/fieldparams"
"github.com/prysmaticlabs/prysm/v4/config/params"
"github.com/prysmaticlabs/prysm/v4/consensus-types/interfaces"
"github.com/prysmaticlabs/prysm/v4/consensus-types/primitives"
"github.com/prysmaticlabs/prysm/v4/crypto/hash"
"github.com/prysmaticlabs/prysm/v4/encoding/bytesutil"
@ -248,6 +247,23 @@ func (v *validator) saveAttesterIndexToData(data *ethpb.AttestationData, index p
return nil
}
// highestSlot returns the highest slot with a valid block seen by the validator
func (v *validator) highestSlot() primitives.Slot {
v.highestValidSlotLock.Lock()
defer v.highestValidSlotLock.Unlock()
return v.highestValidSlot
}
// setHighestSlot sets the highest slot with a valid block seen by the validator
func (v *validator) setHighestSlot(slot primitives.Slot) {
v.highestValidSlotLock.Lock()
defer v.highestValidSlotLock.Unlock()
if slot > v.highestValidSlot {
v.highestValidSlot = slot
v.slotFeed.Send(slot)
}
}
// waitOneThirdOrValidBlock waits until (a) or (b) whichever comes first:
//
// (a) the validator has received a valid block that is the same slot as input slot
@ -257,12 +273,9 @@ func (v *validator) waitOneThirdOrValidBlock(ctx context.Context, slot primitive
defer span.End()
// Don't need to wait if requested slot is the same as highest valid slot.
v.highestValidSlotLock.Lock()
if slot <= v.highestValidSlot {
v.highestValidSlotLock.Unlock()
if slot <= v.highestSlot() {
return
}
v.highestValidSlotLock.Unlock()
delay := slots.DivideSlotBy(3 /* a third of the slot duration */)
startTime := slots.StartTime(v.genesisTime, slot)
@ -274,15 +287,15 @@ func (v *validator) waitOneThirdOrValidBlock(ctx context.Context, slot primitive
t := time.NewTimer(wait)
defer t.Stop()
bChannel := make(chan interfaces.ReadOnlySignedBeaconBlock, 1)
sub := v.blockFeed.Subscribe(bChannel)
ch := make(chan primitives.Slot, 1)
sub := v.slotFeed.Subscribe(ch)
defer sub.Unsubscribe()
for {
select {
case b := <-bChannel:
case s := <-ch:
if features.Get().AttestTimely {
if slot <= b.Block().Slot() {
if slot <= s {
return
}
}

View File

@ -16,7 +16,6 @@ import (
"github.com/prysmaticlabs/prysm/v4/config/features"
fieldparams "github.com/prysmaticlabs/prysm/v4/config/fieldparams"
"github.com/prysmaticlabs/prysm/v4/config/params"
"github.com/prysmaticlabs/prysm/v4/consensus-types/blocks"
"github.com/prysmaticlabs/prysm/v4/consensus-types/primitives"
"github.com/prysmaticlabs/prysm/v4/encoding/bytesutil"
ethpb "github.com/prysmaticlabs/prysm/v4/proto/prysm/v1alpha1"
@ -482,7 +481,7 @@ func TestServer_WaitToSlotOneThird_CanWait(t *testing.T) {
v := &validator{
genesisTime: genesisTime,
blockFeed: new(event.Feed),
slotFeed: new(event.Feed),
}
timeToSleep := params.BeaconConfig().SecondsPerSlot / 3
@ -501,7 +500,7 @@ func TestServer_WaitToSlotOneThird_SameReqSlot(t *testing.T) {
v := &validator{
genesisTime: genesisTime,
blockFeed: new(event.Feed),
slotFeed: new(event.Feed),
highestValidSlot: currentSlot,
}
@ -522,19 +521,14 @@ func TestServer_WaitToSlotOneThird_ReceiveBlockSlot(t *testing.T) {
v := &validator{
genesisTime: genesisTime,
blockFeed: new(event.Feed),
slotFeed: new(event.Feed),
}
wg := &sync.WaitGroup{}
wg.Add(1)
go func() {
time.Sleep(100 * time.Millisecond)
wsb, err := blocks.NewSignedBeaconBlock(
&ethpb.SignedBeaconBlock{
Block: &ethpb.BeaconBlock{Slot: currentSlot, Body: &ethpb.BeaconBlockBody{}},
})
require.NoError(t, err)
v.blockFeed.Send(wsb)
v.slotFeed.Send(currentSlot)
wg.Done()
}()

View File

@ -107,6 +107,10 @@ func (c *beaconApiValidatorClient) ProposeExit(ctx context.Context, in *ethpb.Si
return c.proposeExit(ctx, in)
}
func (c *beaconApiValidatorClient) StreamSlots(ctx context.Context, in *ethpb.StreamSlotsRequest) (ethpb.BeaconNodeValidator_StreamSlotsClient, error) {
return c.streamSlots(ctx, in, time.Second), nil
}
func (c *beaconApiValidatorClient) StreamBlocksAltair(ctx context.Context, in *ethpb.StreamBlocksRequest) (ethpb.BeaconNodeValidator_StreamBlocksAltairClient, error) {
return c.streamBlocks(ctx, in, time.Second), nil
}

View File

@ -1,5 +1,5 @@
// Code generated by MockGen. DO NOT EDIT.
// Source: github.com/prysmaticlabs/prysm/v4/validator/client/beacon-api (interfaces: BeaconBlockConverter)
// Source: validator/client/beacon-api/beacon_block_converter.go
// Package mock is a generated GoMock package.
package mock
@ -36,61 +36,61 @@ func (m *MockBeaconBlockConverter) EXPECT() *MockBeaconBlockConverterMockRecorde
}
// ConvertRESTAltairBlockToProto mocks base method.
func (m *MockBeaconBlockConverter) ConvertRESTAltairBlockToProto(arg0 *shared.BeaconBlockAltair) (*eth.BeaconBlockAltair, error) {
func (m *MockBeaconBlockConverter) ConvertRESTAltairBlockToProto(block *shared.BeaconBlockAltair) (*eth.BeaconBlockAltair, error) {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "ConvertRESTAltairBlockToProto", arg0)
ret := m.ctrl.Call(m, "ConvertRESTAltairBlockToProto", block)
ret0, _ := ret[0].(*eth.BeaconBlockAltair)
ret1, _ := ret[1].(error)
return ret0, ret1
}
// ConvertRESTAltairBlockToProto indicates an expected call of ConvertRESTAltairBlockToProto.
func (mr *MockBeaconBlockConverterMockRecorder) ConvertRESTAltairBlockToProto(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconBlockConverterMockRecorder) ConvertRESTAltairBlockToProto(block interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "ConvertRESTAltairBlockToProto", reflect.TypeOf((*MockBeaconBlockConverter)(nil).ConvertRESTAltairBlockToProto), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "ConvertRESTAltairBlockToProto", reflect.TypeOf((*MockBeaconBlockConverter)(nil).ConvertRESTAltairBlockToProto), block)
}
// ConvertRESTBellatrixBlockToProto mocks base method.
func (m *MockBeaconBlockConverter) ConvertRESTBellatrixBlockToProto(arg0 *shared.BeaconBlockBellatrix) (*eth.BeaconBlockBellatrix, error) {
func (m *MockBeaconBlockConverter) ConvertRESTBellatrixBlockToProto(block *shared.BeaconBlockBellatrix) (*eth.BeaconBlockBellatrix, error) {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "ConvertRESTBellatrixBlockToProto", arg0)
ret := m.ctrl.Call(m, "ConvertRESTBellatrixBlockToProto", block)
ret0, _ := ret[0].(*eth.BeaconBlockBellatrix)
ret1, _ := ret[1].(error)
return ret0, ret1
}
// ConvertRESTBellatrixBlockToProto indicates an expected call of ConvertRESTBellatrixBlockToProto.
func (mr *MockBeaconBlockConverterMockRecorder) ConvertRESTBellatrixBlockToProto(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconBlockConverterMockRecorder) ConvertRESTBellatrixBlockToProto(block interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "ConvertRESTBellatrixBlockToProto", reflect.TypeOf((*MockBeaconBlockConverter)(nil).ConvertRESTBellatrixBlockToProto), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "ConvertRESTBellatrixBlockToProto", reflect.TypeOf((*MockBeaconBlockConverter)(nil).ConvertRESTBellatrixBlockToProto), block)
}
// ConvertRESTCapellaBlockToProto mocks base method.
func (m *MockBeaconBlockConverter) ConvertRESTCapellaBlockToProto(arg0 *shared.BeaconBlockCapella) (*eth.BeaconBlockCapella, error) {
func (m *MockBeaconBlockConverter) ConvertRESTCapellaBlockToProto(block *shared.BeaconBlockCapella) (*eth.BeaconBlockCapella, error) {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "ConvertRESTCapellaBlockToProto", arg0)
ret := m.ctrl.Call(m, "ConvertRESTCapellaBlockToProto", block)
ret0, _ := ret[0].(*eth.BeaconBlockCapella)
ret1, _ := ret[1].(error)
return ret0, ret1
}
// ConvertRESTCapellaBlockToProto indicates an expected call of ConvertRESTCapellaBlockToProto.
func (mr *MockBeaconBlockConverterMockRecorder) ConvertRESTCapellaBlockToProto(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconBlockConverterMockRecorder) ConvertRESTCapellaBlockToProto(block interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "ConvertRESTCapellaBlockToProto", reflect.TypeOf((*MockBeaconBlockConverter)(nil).ConvertRESTCapellaBlockToProto), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "ConvertRESTCapellaBlockToProto", reflect.TypeOf((*MockBeaconBlockConverter)(nil).ConvertRESTCapellaBlockToProto), block)
}
// ConvertRESTPhase0BlockToProto mocks base method.
func (m *MockBeaconBlockConverter) ConvertRESTPhase0BlockToProto(arg0 *shared.BeaconBlock) (*eth.BeaconBlock, error) {
func (m *MockBeaconBlockConverter) ConvertRESTPhase0BlockToProto(block *shared.BeaconBlock) (*eth.BeaconBlock, error) {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "ConvertRESTPhase0BlockToProto", arg0)
ret := m.ctrl.Call(m, "ConvertRESTPhase0BlockToProto", block)
ret0, _ := ret[0].(*eth.BeaconBlock)
ret1, _ := ret[1].(error)
return ret0, ret1
}
// ConvertRESTPhase0BlockToProto indicates an expected call of ConvertRESTPhase0BlockToProto.
func (mr *MockBeaconBlockConverterMockRecorder) ConvertRESTPhase0BlockToProto(arg0 interface{}) *gomock.Call {
func (mr *MockBeaconBlockConverterMockRecorder) ConvertRESTPhase0BlockToProto(block interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "ConvertRESTPhase0BlockToProto", reflect.TypeOf((*MockBeaconBlockConverter)(nil).ConvertRESTPhase0BlockToProto), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "ConvertRESTPhase0BlockToProto", reflect.TypeOf((*MockBeaconBlockConverter)(nil).ConvertRESTPhase0BlockToProto), block)
}

View File

@ -1,5 +1,5 @@
// Code generated by MockGen. DO NOT EDIT.
// Source: github.com/prysmaticlabs/prysm/v4/validator/client/beacon-api (interfaces: GenesisProvider)
// Source: validator/client/beacon-api/genesis.go
// Package mock is a generated GoMock package.
package mock
@ -37,9 +37,9 @@ func (m *MockGenesisProvider) EXPECT() *MockGenesisProviderMockRecorder {
}
// GetGenesis mocks base method.
func (m *MockGenesisProvider) GetGenesis(arg0 context.Context) (*beacon.Genesis, *httputil.DefaultErrorJson, error) {
func (m *MockGenesisProvider) GetGenesis(ctx context.Context) (*beacon.Genesis, *httputil.DefaultErrorJson, error) {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "GetGenesis", arg0)
ret := m.ctrl.Call(m, "GetGenesis", ctx)
ret0, _ := ret[0].(*beacon.Genesis)
ret1, _ := ret[1].(*httputil.DefaultErrorJson)
ret2, _ := ret[2].(error)
@ -47,7 +47,7 @@ func (m *MockGenesisProvider) GetGenesis(arg0 context.Context) (*beacon.Genesis,
}
// GetGenesis indicates an expected call of GetGenesis.
func (mr *MockGenesisProviderMockRecorder) GetGenesis(arg0 interface{}) *gomock.Call {
func (mr *MockGenesisProviderMockRecorder) GetGenesis(ctx interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "GetGenesis", reflect.TypeOf((*MockGenesisProvider)(nil).GetGenesis), arg0)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "GetGenesis", reflect.TypeOf((*MockGenesisProvider)(nil).GetGenesis), ctx)
}

View File

@ -1,5 +1,5 @@
// Code generated by MockGen. DO NOT EDIT.
// Source: github.com/prysmaticlabs/prysm/v4/validator/client/beacon-api (interfaces: JsonRestHandler)
// Source: validator/client/beacon-api/json_rest_handler.go
// Package mock is a generated GoMock package.
package mock
@ -10,7 +10,7 @@ import (
reflect "reflect"
gomock "github.com/golang/mock/gomock"
http "github.com/prysmaticlabs/prysm/v4/network/httputil"
httputil "github.com/prysmaticlabs/prysm/v4/network/httputil"
)
// MockJsonRestHandler is a mock of JsonRestHandler interface.
@ -37,31 +37,31 @@ func (m *MockJsonRestHandler) EXPECT() *MockJsonRestHandlerMockRecorder {
}
// Get mocks base method.
func (m *MockJsonRestHandler) Get(arg0 context.Context, arg1 string, arg2 interface{}) (*http.DefaultErrorJson, error) {
func (m *MockJsonRestHandler) Get(ctx context.Context, query string, resp interface{}) (*httputil.DefaultErrorJson, error) {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "Get", arg0, arg1, arg2)
ret0, _ := ret[0].(*http.DefaultErrorJson)
ret := m.ctrl.Call(m, "Get", ctx, query, resp)
ret0, _ := ret[0].(*httputil.DefaultErrorJson)
ret1, _ := ret[1].(error)
return ret0, ret1
}
// Get indicates an expected call of Get.
func (mr *MockJsonRestHandlerMockRecorder) Get(arg0, arg1, arg2 interface{}) *gomock.Call {
func (mr *MockJsonRestHandlerMockRecorder) Get(ctx, query, resp interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Get", reflect.TypeOf((*MockJsonRestHandler)(nil).Get), arg0, arg1, arg2)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Get", reflect.TypeOf((*MockJsonRestHandler)(nil).Get), ctx, query, resp)
}
// Post mocks base method.
func (m *MockJsonRestHandler) Post(arg0 context.Context, arg1 string, arg2 map[string]string, arg3 *bytes.Buffer, arg4 interface{}) (*http.DefaultErrorJson, error) {
func (m *MockJsonRestHandler) Post(ctx context.Context, endpoint string, headers map[string]string, data *bytes.Buffer, resp interface{}) (*httputil.DefaultErrorJson, error) {
m.ctrl.T.Helper()
ret := m.ctrl.Call(m, "Post", arg0, arg1, arg2, arg3, arg4)
ret0, _ := ret[0].(*http.DefaultErrorJson)
ret := m.ctrl.Call(m, "Post", ctx, endpoint, headers, data, resp)
ret0, _ := ret[0].(*httputil.DefaultErrorJson)
ret1, _ := ret[1].(error)
return ret0, ret1
}
// Post indicates an expected call of Post.
func (mr *MockJsonRestHandlerMockRecorder) Post(arg0, arg1, arg2, arg3, arg4 interface{}) *gomock.Call {
func (mr *MockJsonRestHandlerMockRecorder) Post(ctx, endpoint, headers, data, resp interface{}) *gomock.Call {
mr.mock.ctrl.T.Helper()
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Post", reflect.TypeOf((*MockJsonRestHandler)(nil).Post), arg0, arg1, arg2, arg3, arg4)
return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Post", reflect.TypeOf((*MockJsonRestHandler)(nil).Post), ctx, endpoint, headers, data, resp)
}

View File

@ -1,5 +1,5 @@
// Code generated by MockGen. DO NOT EDIT.
// Source: github.com/prysmaticlabs/prysm/v4/validator/client/beacon-api (interfaces: StateValidatorsProvider)
// Source: validator/client/beacon-api/state_validators.go
// Package mock is a generated GoMock package.
package mock

View File

@ -21,6 +21,15 @@ type abstractSignedBlockResponseJson struct {
Data json.RawMessage `json:"data"`
}
type streamSlotsClient struct {
grpc.ClientStream
ctx context.Context
beaconApiClient beaconApiValidatorClient
streamSlotsRequest *ethpb.StreamSlotsRequest
prevBlockSlot primitives.Slot
pingDelay time.Duration
}
type streamBlocksAltairClient struct {
grpc.ClientStream
ctx context.Context
@ -36,6 +45,15 @@ type headSignedBeaconBlockResult struct {
slot primitives.Slot
}
func (c beaconApiValidatorClient) streamSlots(ctx context.Context, in *ethpb.StreamSlotsRequest, pingDelay time.Duration) ethpb.BeaconNodeValidator_StreamSlotsClient {
return &streamSlotsClient{
ctx: ctx,
beaconApiClient: c,
streamSlotsRequest: in,
pingDelay: pingDelay,
}
}
func (c beaconApiValidatorClient) streamBlocks(ctx context.Context, in *ethpb.StreamBlocksRequest, pingDelay time.Duration) ethpb.BeaconNodeValidator_StreamBlocksAltairClient {
return &streamBlocksAltairClient{
ctx: ctx,
@ -45,6 +63,31 @@ func (c beaconApiValidatorClient) streamBlocks(ctx context.Context, in *ethpb.St
}
}
func (c *streamSlotsClient) Recv() (*ethpb.StreamSlotsResponse, error) {
result, err := c.beaconApiClient.getHeadSignedBeaconBlock(c.ctx)
if err != nil {
return nil, errors.Wrap(err, "failed to get latest signed block")
}
// We keep querying the beacon chain for the latest block until we receive a new slot
for (c.streamSlotsRequest.VerifiedOnly && result.executionOptimistic) || c.prevBlockSlot == result.slot {
select {
case <-time.After(c.pingDelay):
result, err = c.beaconApiClient.getHeadSignedBeaconBlock(c.ctx)
if err != nil {
return nil, errors.Wrap(err, "failed to get latest signed block")
}
case <-c.ctx.Done():
return nil, errors.New("context canceled")
}
}
c.prevBlockSlot = result.slot
return &ethpb.StreamSlotsResponse{
Slot: result.slot,
}, nil
}
func (c *streamBlocksAltairClient) Recv() (*ethpb.StreamBlocksResponse, error) {
result, err := c.beaconApiClient.getHeadSignedBeaconBlock(c.ctx)
if err != nil {

View File

@ -71,6 +71,10 @@ func (c *grpcValidatorClient) ProposeExit(ctx context.Context, in *ethpb.SignedV
return c.beaconNodeValidatorClient.ProposeExit(ctx, in)
}
func (c *grpcValidatorClient) StreamSlots(ctx context.Context, in *ethpb.StreamSlotsRequest) (ethpb.BeaconNodeValidator_StreamSlotsClient, error) {
return c.beaconNodeValidatorClient.StreamSlots(ctx, in)
}
func (c *grpcValidatorClient) StreamBlocksAltair(ctx context.Context, in *ethpb.StreamBlocksRequest) (ethpb.BeaconNodeValidator_StreamBlocksAltairClient, error) {
return c.beaconNodeValidatorClient.StreamBlocksAltair(ctx, in)
}

View File

@ -57,7 +57,7 @@ type Validator interface {
UpdateDomainDataCaches(ctx context.Context, slot primitives.Slot)
WaitForKeymanagerInitialization(ctx context.Context) error
Keymanager() (keymanager.IKeymanager, error)
ReceiveBlocks(ctx context.Context, connectionErrorChannel chan<- error)
ReceiveSlots(ctx context.Context, connectionErrorChannel chan<- error)
HandleKeyReload(ctx context.Context, currentKeys [][fieldparams.BLSPubkeyLength]byte) (bool, error)
CheckDoppelGanger(ctx context.Context) error
PushProposerSettings(ctx context.Context, km keymanager.IKeymanager, slot primitives.Slot, deadline time.Time) error

View File

@ -32,6 +32,6 @@ type ValidatorClient interface {
GetSyncSubcommitteeIndex(ctx context.Context, in *ethpb.SyncSubcommitteeIndexRequest) (*ethpb.SyncSubcommitteeIndexResponse, error)
GetSyncCommitteeContribution(ctx context.Context, in *ethpb.SyncCommitteeContributionRequest) (*ethpb.SyncCommitteeContribution, error)
SubmitSignedContributionAndProof(ctx context.Context, in *ethpb.SignedContributionAndProof) (*empty.Empty, error)
StreamBlocksAltair(ctx context.Context, in *ethpb.StreamBlocksRequest) (ethpb.BeaconNodeValidator_StreamBlocksAltairClient, error)
StreamSlots(ctx context.Context, in *ethpb.StreamSlotsRequest) (ethpb.BeaconNodeValidator_StreamSlotsClient, error)
SubmitValidatorRegistrations(ctx context.Context, in *ethpb.SignedValidatorRegistrationsV1) (*empty.Empty, error)
}

View File

@ -42,7 +42,7 @@ func run(ctx context.Context, v iface.Validator) {
}
connectionErrorChannel := make(chan error, 1)
go v.ReceiveBlocks(ctx, connectionErrorChannel)
go v.ReceiveSlots(ctx, connectionErrorChannel)
if err := v.UpdateDuties(ctx, headSlot); err != nil {
handleAssignmentError(err, headSlot)
}
@ -81,10 +81,10 @@ func run(ctx context.Context, v iface.Validator) {
sub.Unsubscribe()
close(accountsChangedChan)
return // Exit if context is canceled.
case blocksError := <-connectionErrorChannel:
if blocksError != nil {
log.WithError(blocksError).Warn("block stream interrupted")
go v.ReceiveBlocks(ctx, connectionErrorChannel)
case slotsError := <-connectionErrorChannel:
if slotsError != nil {
log.WithError(slotsError).Warn("slots stream interrupted")
go v.ReceiveSlots(ctx, connectionErrorChannel)
continue
}
case currentKeys := <-accountsChangedChan:

View File

@ -17,7 +17,6 @@ import (
fieldparams "github.com/prysmaticlabs/prysm/v4/config/fieldparams"
"github.com/prysmaticlabs/prysm/v4/config/params"
validatorserviceconfig "github.com/prysmaticlabs/prysm/v4/config/validator/service"
"github.com/prysmaticlabs/prysm/v4/consensus-types/interfaces"
"github.com/prysmaticlabs/prysm/v4/consensus-types/primitives"
ethpb "github.com/prysmaticlabs/prysm/v4/proto/prysm/v1alpha1"
"github.com/prysmaticlabs/prysm/v4/validator/accounts/wallet"
@ -215,7 +214,7 @@ func (v *ValidatorService) Start() {
interopKeysConfig: v.interopKeysConfig,
wallet: v.wallet,
walletInitializedFeed: v.walletInitializedFeed,
blockFeed: new(event.Feed),
slotFeed: new(event.Feed),
graffitiStruct: v.graffitiStruct,
graffitiOrderedIndex: graffitiOrderedIndex,
eipImportBlacklistedPublicKeys: slashablePublicKeys,
@ -226,16 +225,6 @@ func (v *ValidatorService) Start() {
validatorRegBatchSize: v.validatorRegBatchSize,
}
// To resolve a race condition at startup due to the interface
// nature of the abstracted block type. We initialize
// the inner type of the feed before hand. So that
// during future accesses, there will be no panics here
// from type incompatibility.
tempChan := make(chan interfaces.ReadOnlySignedBeaconBlock)
sub := valStruct.blockFeed.Subscribe(tempChan)
sub.Unsubscribe()
close(tempChan)
v.validator = valStruct
go run(v.ctx, v.validator)
}

View File

@ -217,8 +217,8 @@ func (_ *FakeValidator) CheckDoppelGanger(_ context.Context) error {
return nil
}
// ReceiveBlocks for mocking
func (fv *FakeValidator) ReceiveBlocks(_ context.Context, connectionErrorChannel chan<- error) {
// ReceiveSlots for mocking
func (fv *FakeValidator) ReceiveSlots(_ context.Context, connectionErrorChannel chan<- error) {
fv.ReceiveBlocksCalled++
if fv.RetryTillSuccess > fv.ReceiveBlocksCalled {
connectionErrorChannel <- iface.ErrConnectionIssue

View File

@ -26,8 +26,6 @@ import (
fieldparams "github.com/prysmaticlabs/prysm/v4/config/fieldparams"
"github.com/prysmaticlabs/prysm/v4/config/params"
validatorserviceconfig "github.com/prysmaticlabs/prysm/v4/config/validator/service"
"github.com/prysmaticlabs/prysm/v4/consensus-types/blocks"
"github.com/prysmaticlabs/prysm/v4/consensus-types/interfaces"
"github.com/prysmaticlabs/prysm/v4/consensus-types/primitives"
"github.com/prysmaticlabs/prysm/v4/crypto/hash"
"github.com/prysmaticlabs/prysm/v4/encoding/bytesutil"
@ -89,7 +87,7 @@ type validator struct {
domainDataCache *ristretto.Cache
highestValidSlot primitives.Slot
genesisTime uint64
blockFeed *event.Feed
slotFeed *event.Feed
interopKeysConfig *local.InteropKeymanagerConfig
wallet *wallet.Wallet
graffitiStruct *graffiti.Graffiti
@ -317,58 +315,32 @@ func (v *validator) WaitForSync(ctx context.Context) error {
}
}
// ReceiveBlocks starts a gRPC client stream listener to obtain
// blocks from the beacon node. Upon receiving a block, the service
// ReceiveSlots starts a gRPC client stream listener to obtain
// slots from the beacon node when it imports a block. Upon receiving a slot, the service
// broadcasts it to a feed for other usages to subscribe to.
func (v *validator) ReceiveBlocks(ctx context.Context, connectionErrorChannel chan<- error) {
stream, err := v.validatorClient.StreamBlocksAltair(ctx, &ethpb.StreamBlocksRequest{VerifiedOnly: true})
func (v *validator) ReceiveSlots(ctx context.Context, connectionErrorChannel chan<- error) {
stream, err := v.validatorClient.StreamSlots(ctx, &ethpb.StreamSlotsRequest{VerifiedOnly: true})
if err != nil {
log.WithError(err).Error("Failed to retrieve blocks stream, " + iface.ErrConnectionIssue.Error())
log.WithError(err).Error("Failed to retrieve slots stream, " + iface.ErrConnectionIssue.Error())
connectionErrorChannel <- errors.Wrap(iface.ErrConnectionIssue, err.Error())
return
}
for {
if ctx.Err() == context.Canceled {
log.WithError(ctx.Err()).Error("Context canceled - shutting down blocks receiver")
log.WithError(ctx.Err()).Error("Context canceled - shutting down slots receiver")
return
}
res, err := stream.Recv()
if err != nil {
log.WithError(err).Error("Could not receive blocks from beacon node, " + iface.ErrConnectionIssue.Error())
log.WithError(err).Error("Could not receive slots from beacon node, " + iface.ErrConnectionIssue.Error())
connectionErrorChannel <- errors.Wrap(iface.ErrConnectionIssue, err.Error())
return
}
if res == nil || res.Block == nil {
if res == nil {
continue
}
var blk interfaces.ReadOnlySignedBeaconBlock
switch b := res.Block.(type) {
case *ethpb.StreamBlocksResponse_Phase0Block:
blk, err = blocks.NewSignedBeaconBlock(b.Phase0Block)
case *ethpb.StreamBlocksResponse_AltairBlock:
blk, err = blocks.NewSignedBeaconBlock(b.AltairBlock)
case *ethpb.StreamBlocksResponse_BellatrixBlock:
blk, err = blocks.NewSignedBeaconBlock(b.BellatrixBlock)
case *ethpb.StreamBlocksResponse_CapellaBlock:
blk, err = blocks.NewSignedBeaconBlock(b.CapellaBlock)
case *ethpb.StreamBlocksResponse_DenebBlock:
blk, err = blocks.NewSignedBeaconBlock(b.DenebBlock)
}
if err != nil {
log.WithError(err).Error("Failed to wrap signed block")
continue
}
if blk == nil || blk.IsNil() {
log.Error("Received nil block")
continue
}
v.highestValidSlotLock.Lock()
if blk.Block().Slot() > v.highestValidSlot {
v.highestValidSlot = blk.Block().Slot()
}
v.highestValidSlotLock.Unlock()
v.blockFeed.Send(blk)
v.setHighestSlot(res.Slot)
}
}

View File

@ -901,93 +901,31 @@ func TestCheckAndLogValidatorStatus_OK(t *testing.T) {
}
}
func TestService_ReceiveBlocks_NilBlock(t *testing.T) {
ctrl := gomock.NewController(t)
defer ctrl.Finish()
valClient := validatormock.NewMockValidatorClient(ctrl)
v := validator{
blockFeed: new(event.Feed),
validatorClient: valClient,
}
stream := mock2.NewMockBeaconNodeValidatorAltair_StreamBlocksClient(ctrl)
ctx, cancel := context.WithCancel(context.Background())
valClient.EXPECT().StreamBlocksAltair(
gomock.Any(),
&ethpb.StreamBlocksRequest{VerifiedOnly: true},
).Return(stream, nil)
stream.EXPECT().Context().Return(ctx).AnyTimes()
stream.EXPECT().Recv().Return(
&ethpb.StreamBlocksResponse{Block: &ethpb.StreamBlocksResponse_Phase0Block{
Phase0Block: &ethpb.SignedBeaconBlock{}}},
nil,
).Do(func() {
cancel()
})
connectionErrorChannel := make(chan error)
v.ReceiveBlocks(ctx, connectionErrorChannel)
require.Equal(t, primitives.Slot(0), v.highestValidSlot)
}
func TestService_ReceiveBlocks_SetHighest(t *testing.T) {
func TestService_ReceiveSlots_SetHighest(t *testing.T) {
ctrl := gomock.NewController(t)
defer ctrl.Finish()
client := validatormock.NewMockValidatorClient(ctrl)
v := validator{
validatorClient: client,
blockFeed: new(event.Feed),
slotFeed: new(event.Feed),
}
stream := mock2.NewMockBeaconNodeValidatorAltair_StreamBlocksClient(ctrl)
stream := mock2.NewMockBeaconNodeValidator_StreamSlotsClient(ctrl)
ctx, cancel := context.WithCancel(context.Background())
client.EXPECT().StreamBlocksAltair(
client.EXPECT().StreamSlots(
gomock.Any(),
&ethpb.StreamBlocksRequest{VerifiedOnly: true},
&ethpb.StreamSlotsRequest{VerifiedOnly: true},
).Return(stream, nil)
stream.EXPECT().Context().Return(ctx).AnyTimes()
slot := primitives.Slot(100)
stream.EXPECT().Recv().Return(
&ethpb.StreamBlocksResponse{
Block: &ethpb.StreamBlocksResponse_Phase0Block{
Phase0Block: &ethpb.SignedBeaconBlock{Block: &ethpb.BeaconBlock{Slot: slot, Body: &ethpb.BeaconBlockBody{}}}},
},
&ethpb.StreamSlotsResponse{Slot: 123},
nil,
).Do(func() {
cancel()
})
connectionErrorChannel := make(chan error)
v.ReceiveBlocks(ctx, connectionErrorChannel)
require.Equal(t, slot, v.highestValidSlot)
}
func TestService_ReceiveBlocks_SetHighestDeneb(t *testing.T) {
ctrl := gomock.NewController(t)
defer ctrl.Finish()
client := validatormock.NewMockValidatorClient(ctrl)
v := validator{
validatorClient: client,
blockFeed: new(event.Feed),
}
stream := mock2.NewMockBeaconNodeValidatorAltair_StreamBlocksClient(ctrl)
ctx, cancel := context.WithCancel(context.Background())
client.EXPECT().StreamBlocksAltair(
gomock.Any(),
&ethpb.StreamBlocksRequest{VerifiedOnly: true},
).Return(stream, nil)
stream.EXPECT().Context().Return(ctx).AnyTimes()
slot := primitives.Slot(100)
stream.EXPECT().Recv().Return(
&ethpb.StreamBlocksResponse{
Block: &ethpb.StreamBlocksResponse_DenebBlock{
DenebBlock: &ethpb.SignedBeaconBlockDeneb{Block: &ethpb.BeaconBlockDeneb{Slot: slot, Body: &ethpb.BeaconBlockBodyDeneb{}}}},
},
nil,
).Do(func() {
cancel()
})
connectionErrorChannel := make(chan error)
v.ReceiveBlocks(ctx, connectionErrorChannel)
require.Equal(t, slot, v.highestValidSlot)
v.ReceiveSlots(ctx, connectionErrorChannel)
require.Equal(t, primitives.Slot(123), v.highestValidSlot)
}
type doppelGangerRequestMatcher struct {