From e170bd00fc7d40eee60f07e9acc5283a094d1b11 Mon Sep 17 00:00:00 2001 From: taylanisikdemir Date: Wed, 30 Oct 2024 17:59:35 -0700 Subject: [PATCH] Write tests for resource impl (#6452) --- common/persistence/client/bean.go | 2 +- common/pprof.go | 2 + common/pprof_mock.go | 70 +++++ common/resource/params.go | 4 + .../{resourceImpl.go => resource_impl.go} | 123 +++++---- common/resource/resource_impl_test.go | 245 ++++++++++++++++++ common/resource/resource_mock.go | 2 +- common/resource/resource_test.go | 42 --- common/resource/{resource.go => types.go} | 134 +++++----- common/rpc/peer_chooser.go | 2 + common/rpc/peer_chooser_mock.go | 169 ++++++++++++ 11 files changed, 618 insertions(+), 177 deletions(-) create mode 100644 common/pprof_mock.go rename common/resource/{resourceImpl.go => resource_impl.go} (89%) create mode 100644 common/resource/resource_impl_test.go delete mode 100644 common/resource/resource_test.go rename common/resource/{resource.go => types.go} (56%) create mode 100644 common/rpc/peer_chooser_mock.go diff --git a/common/persistence/client/bean.go b/common/persistence/client/bean.go index 2f543aaad15..7344ea4b0f8 100644 --- a/common/persistence/client/bean.go +++ b/common/persistence/client/bean.go @@ -98,7 +98,7 @@ func NewBeanFromFactory( factory Factory, params *Params, serviceConfig *service.Config, -) (*BeanImpl, error) { +) (Bean, error) { metadataMgr, err := factory.NewDomainManager() if err != nil { diff --git a/common/pprof.go b/common/pprof.go index 9cfb03907ba..1ad30e1449f 100644 --- a/common/pprof.go +++ b/common/pprof.go @@ -20,6 +20,8 @@ package common +//go:generate mockgen -package $GOPACKAGE -source $GOFILE -destination pprof_mock.go -package common github.com/uber/cadence/common PProfInitializer + type ( // PProfInitializer initialize the pprof based on config PProfInitializer interface { diff --git a/common/pprof_mock.go b/common/pprof_mock.go new file mode 100644 index 00000000000..96cddeab9df --- /dev/null +++ b/common/pprof_mock.go @@ -0,0 +1,70 @@ +// The MIT License (MIT) + +// Copyright (c) 2017-2020 Uber Technologies Inc. + +// Permission is hereby granted, free of charge, to any person obtaining a copy +// of this software and associated documentation files (the "Software"), to deal +// in the Software without restriction, including without limitation the rights +// to use, copy, modify, merge, publish, distribute, sublicense, and/or sell +// copies of the Software, and to permit persons to whom the Software is +// furnished to do so, subject to the following conditions: +// +// The above copyright notice and this permission notice shall be included in all +// copies or substantial portions of the Software. +// +// THE SOFTWARE IS PROVIDED "AS IS", WITHOUT WARRANTY OF ANY KIND, EXPRESS OR +// IMPLIED, INCLUDING BUT NOT LIMITED TO THE WARRANTIES OF MERCHANTABILITY, +// FITNESS FOR A PARTICULAR PURPOSE AND NONINFRINGEMENT. IN NO EVENT SHALL THE +// AUTHORS OR COPYRIGHT HOLDERS BE LIABLE FOR ANY CLAIM, DAMAGES OR OTHER +// LIABILITY, WHETHER IN AN ACTION OF CONTRACT, TORT OR OTHERWISE, ARISING FROM, +// OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER DEALINGS IN THE +// SOFTWARE. + +// Code generated by MockGen. DO NOT EDIT. +// Source: pprof.go + +// Package common is a generated GoMock package. +package common + +import ( + reflect "reflect" + + gomock "github.com/golang/mock/gomock" +) + +// MockPProfInitializer is a mock of PProfInitializer interface. +type MockPProfInitializer struct { + ctrl *gomock.Controller + recorder *MockPProfInitializerMockRecorder +} + +// MockPProfInitializerMockRecorder is the mock recorder for MockPProfInitializer. +type MockPProfInitializerMockRecorder struct { + mock *MockPProfInitializer +} + +// NewMockPProfInitializer creates a new mock instance. +func NewMockPProfInitializer(ctrl *gomock.Controller) *MockPProfInitializer { + mock := &MockPProfInitializer{ctrl: ctrl} + mock.recorder = &MockPProfInitializerMockRecorder{mock} + return mock +} + +// EXPECT returns an object that allows the caller to indicate expected use. +func (m *MockPProfInitializer) EXPECT() *MockPProfInitializerMockRecorder { + return m.recorder +} + +// Start mocks base method. +func (m *MockPProfInitializer) Start() error { + m.ctrl.T.Helper() + ret := m.ctrl.Call(m, "Start") + ret0, _ := ret[0].(error) + return ret0 +} + +// Start indicates an expected call of Start. +func (mr *MockPProfInitializerMockRecorder) Start() *gomock.Call { + mr.mock.ctrl.T.Helper() + return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Start", reflect.TypeOf((*MockPProfInitializer)(nil).Start)) +} diff --git a/common/resource/params.go b/common/resource/params.go index 7b57a71c827..9389ec27d13 100644 --- a/common/resource/params.go +++ b/common/resource/params.go @@ -43,8 +43,10 @@ import ( "github.com/uber/cadence/common/messaging" "github.com/uber/cadence/common/metrics" "github.com/uber/cadence/common/partition" + persistenceClient "github.com/uber/cadence/common/persistence/client" "github.com/uber/cadence/common/pinot" "github.com/uber/cadence/common/rpc" + "github.com/uber/cadence/common/service" ) type ( @@ -89,5 +91,7 @@ type ( TimeSource clock.TimeSource // HistoryClientFn is used by integration tests to mock a history client HistoryClientFn func() history.Client + // NewPersistenceBeanFn can be used to override the default persistence bean creation in unit tests to avoid DB setup + NewPersistenceBeanFn func(persistenceClient.Factory, *persistenceClient.Params, *service.Config) (persistenceClient.Bean, error) } ) diff --git a/common/resource/resourceImpl.go b/common/resource/resource_impl.go similarity index 89% rename from common/resource/resourceImpl.go rename to common/resource/resource_impl.go index 0fa491b08ab..1eb6ba7dfcb 100644 --- a/common/resource/resourceImpl.go +++ b/common/resource/resource_impl.go @@ -70,89 +70,80 @@ func NewResourceFactory() ResourceFactory { type resourceImplFactory struct{} -func (*resourceImplFactory) NewResource(params *Params, +func (*resourceImplFactory) NewResource( + params *Params, serviceName string, serviceConfig *service.Config, ) (resource Resource, err error) { return New(params, serviceName, serviceConfig) } -type ( +// Impl contains all common resources shared across frontend / matching / history / worker +type Impl struct { + status int32 - // VisibilityManagerInitializer is the function each service should implement - // for visibility manager initialization - VisibilityManagerInitializer func( - persistenceBean persistenceClient.Bean, - logger log.Logger, - ) (persistence.VisibilityManager, error) + // static infos + numShards int + serviceName string + hostInfo membership.HostInfo + metricsScope tally.Scope + clusterMetadata cluster.Metadata - // Impl contains all common resources shared across frontend / matching / history / worker - Impl struct { - status int32 + // other common resources - // static infos - numShards int - serviceName string - hostInfo membership.HostInfo - metricsScope tally.Scope - clusterMetadata cluster.Metadata + domainCache cache.DomainCache + domainMetricsScopeCache cache.DomainMetricsScopeCache + timeSource clock.TimeSource + payloadSerializer persistence.PayloadSerializer + metricsClient metrics.Client + messagingClient messaging.Client + blobstoreClient blobstore.Client + archivalMetadata archiver.ArchivalMetadata + archiverProvider provider.ArchiverProvider + domainReplicationQueue domain.ReplicationQueue - // other common resources + // membership infos - domainCache cache.DomainCache - domainMetricsScopeCache cache.DomainMetricsScopeCache - timeSource clock.TimeSource - payloadSerializer persistence.PayloadSerializer - metricsClient metrics.Client - messagingClient messaging.Client - blobstoreClient blobstore.Client - archivalMetadata archiver.ArchivalMetadata - archiverProvider provider.ArchiverProvider - domainReplicationQueue domain.ReplicationQueue + membershipResolver membership.Resolver - // membership infos + // internal services clients - membershipResolver membership.Resolver + sdkClient workflowserviceclient.Interface + frontendRawClient frontend.Client + frontendClient frontend.Client + matchingRawClient matching.Client + matchingClient matching.Client + historyRawClient history.Client + historyClient history.Client + clientBean client.Bean - // internal services clients + // persistence clients + persistenceBean persistenceClient.Bean - sdkClient workflowserviceclient.Interface - frontendRawClient frontend.Client - frontendClient frontend.Client - matchingRawClient matching.Client - matchingClient matching.Client - historyRawClient history.Client - historyClient history.Client - clientBean client.Bean + // hostName + hostName string - // persistence clients - persistenceBean persistenceClient.Bean + // loggers + logger log.Logger + throttledLogger log.Logger - // hostName - hostName string + // for registering handlers + dispatcher *yarpc.Dispatcher - // loggers - logger log.Logger - throttledLogger log.Logger + // internal vars - // for registering handlers - dispatcher *yarpc.Dispatcher + pprofInitializer common.PProfInitializer + runtimeMetricsReporter *metrics.RuntimeMetricsReporter + rpcFactory rpc.Factory - // internal vars - - pprofInitializer common.PProfInitializer - runtimeMetricsReporter *metrics.RuntimeMetricsReporter - rpcFactory rpc.Factory + isolationGroups isolationgroup.State + isolationGroupConfigStore configstore.Client + partitioner partition.Partitioner - isolationGroups isolationgroup.State - isolationGroupConfigStore configstore.Client - partitioner partition.Partitioner + asyncWorkflowQueueProvider queue.Provider - asyncWorkflowQueueProvider queue.Provider - - ratelimiterAggregatorClient qrpc.Client - } -) + ratelimiterAggregatorClient qrpc.Client +} var _ Resource = (*Impl)(nil) @@ -195,7 +186,11 @@ func New( return nil, err } - persistenceBean, err := persistenceClient.NewBeanFromFactory(persistenceClient.NewFactory( + newPersistenceBeanFn := persistenceClient.NewBeanFromFactory + if params.NewPersistenceBeanFn != nil { + newPersistenceBeanFn = params.NewPersistenceBeanFn + } + persistenceBean, err := newPersistenceBeanFn(persistenceClient.NewFactory( ¶ms.PersistenceConfig, func() float64 { return permember.PerMember( @@ -613,9 +608,7 @@ func (h *Impl) GetHistoryManager() persistence.HistoryManager { } // GetExecutionManager return execution manager for given shard ID -func (h *Impl) GetExecutionManager( - shardID int, -) (persistence.ExecutionManager, error) { +func (h *Impl) GetExecutionManager(shardID int) (persistence.ExecutionManager, error) { return h.persistenceBean.GetExecutionManager(shardID) } diff --git a/common/resource/resource_impl_test.go b/common/resource/resource_impl_test.go new file mode 100644 index 00000000000..93e9ecfc3d6 --- /dev/null +++ b/common/resource/resource_impl_test.go @@ -0,0 +1,245 @@ +// The MIT License (MIT) + +// Copyright (c) 2017-2020 Uber Technologies Inc. + +// Permission is hereby granted, free of charge, to any person obtaining a copy +// of this software and associated documentation files (the "Software"), to deal +// in the Software without restriction, including without limitation the rights +// to use, copy, modify, merge, publish, distribute, sublicense, and/or sell +// copies of the Software, and to permit persons to whom the Software is +// furnished to do so, subject to the following conditions: +// +// The above copyright notice and this permission notice shall be included in all +// copies or substantial portions of the Software. +// +// THE SOFTWARE IS PROVIDED "AS IS", WITHOUT WARRANTY OF ANY KIND, EXPRESS OR +// IMPLIED, INCLUDING BUT NOT LIMITED TO THE WARRANTIES OF MERCHANTABILITY, +// FITNESS FOR A PARTICULAR PURPOSE AND NONINFRINGEMENT. IN NO EVENT SHALL THE +// AUTHORS OR COPYRIGHT HOLDERS BE LIABLE FOR ANY CLAIM, DAMAGES OR OTHER +// LIABILITY, WHETHER IN AN ACTION OF CONTRACT, TORT OR OTHERWISE, ARISING FROM, +// OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER DEALINGS IN THE +// SOFTWARE. + +package resource + +import ( + "runtime/debug" + "testing" + "time" + + "github.com/golang/mock/gomock" + "github.com/stretchr/testify/assert" + "github.com/stretchr/testify/mock" + "github.com/uber-go/tally" + "go.uber.org/goleak" + + "github.com/uber/cadence/common" + "github.com/uber/cadence/common/archiver/provider" + "github.com/uber/cadence/common/asyncworkflow/queue" + "github.com/uber/cadence/common/clock" + "github.com/uber/cadence/common/cluster" + "github.com/uber/cadence/common/config" + "github.com/uber/cadence/common/dynamicconfig" + "github.com/uber/cadence/common/log/testlogger" + "github.com/uber/cadence/common/membership" + "github.com/uber/cadence/common/metrics" + "github.com/uber/cadence/common/persistence" + persistenceClient "github.com/uber/cadence/common/persistence/client" + "github.com/uber/cadence/common/rpc" + "github.com/uber/cadence/common/service" +) + +func TestStartStop(t *testing.T) { + defer func() { + if r := recover(); r != nil { + t.Fatalf("Panic: %v, stacktrace: %s", r, debug.Stack()) + } + }() + + ctrl := gomock.NewController(t) + serviceName := "test-service" + hostName := "test-host" + metricsCl := metrics.NewNoopMetricsClient() + logger := testlogger.New(t) + dc := dynamicconfig.NewInMemoryClient() + + // membership resolver mocks + memberRes := membership.NewMockResolver(ctrl) + memberRes.EXPECT().MemberCount(gomock.Any()).Return(4, nil).AnyTimes() + memberRes.EXPECT().Subscribe(gomock.Any(), gomock.Any(), gomock.Any()).Return(nil).AnyTimes() + memberRes.EXPECT().Unsubscribe(gomock.Any(), gomock.Any()).Return(nil).AnyTimes() + memberRes.EXPECT().Start().AnyTimes() + memberRes.EXPECT().Stop().AnyTimes() + selfHostInfo := membership.NewHostInfo("localhost:0") + memberRes.EXPECT().WhoAmI().Return(selfHostInfo, nil).AnyTimes() + + // pprof mocks + pprof := common.NewMockPProfInitializer(ctrl) + pprof.EXPECT().Start().Return(nil).Times(1) + + // rpc mocks + clusterMetadata := cluster.NewMetadata(1, "primary-cluster", "primary-cluster", map[string]config.ClusterInformation{ + "primary-cluster": {InitialFailoverVersion: 1, Enabled: true, RPCTransport: "tchannel", RPCAddress: "localhost:0"}, + "secondary-cluster": {InitialFailoverVersion: 1, Enabled: true, RPCTransport: "tchannel", RPCAddress: "localhost:0"}, + }, nil, metricsCl, logger) + directOutboundPCF := rpc.NewDirectPeerChooserFactory(serviceName, logger) + directConnRetainFn := func(opts ...dynamicconfig.FilterOption) bool { return false } + pcf := rpc.NewMockPeerChooserFactory(ctrl) + peerChooser := rpc.NewMockPeerChooser(ctrl) + peerChooser.EXPECT().Start().Return(nil).AnyTimes() + peerChooser.EXPECT().Stop().Return(nil).AnyTimes() + pcf.EXPECT().CreatePeerChooser(gomock.Any(), gomock.Any()).Return(peerChooser, nil).AnyTimes() + ob := rpc.CombineOutbounds( + rpc.NewCrossDCOutbounds(clusterMetadata.GetAllClusterInfo(), pcf), + rpc.NewDirectOutboundBuilder(service.History, true, nil, directOutboundPCF, directConnRetainFn), + rpc.NewDirectOutboundBuilder(service.Matching, true, nil, directOutboundPCF, directConnRetainFn), + ) + rpcFac := rpc.NewFactory(logger, rpc.Params{ + ServiceName: serviceName, + TChannelAddress: "localhost:0", + GRPCAddress: "localhost:0", + OutboundsBuilder: ob, + }) + + // persistence mocks + persistenceClientBean := persistenceClient.NewMockBean(ctrl) + domainMgr := persistence.NewMockDomainManager(ctrl) + domainMgr.EXPECT().GetMetadata(gomock.Any()).Return(&persistence.GetMetadataResponse{ + NotificationVersion: 2, + }, nil).AnyTimes() + domain := &persistence.GetDomainResponse{ + Info: &persistence.DomainInfo{ + ID: "test-domain-id", + Name: "test-domain-name", + Data: map[string]string{"k1": "v1"}, + }, + Config: &persistence.DomainConfig{}, + ReplicationConfig: &persistence.DomainReplicationConfig{}, + NotificationVersion: 1, // should be less than notification version for this domain to be loaded by cache. + } + domainMgr.EXPECT().ListDomains(gomock.Any(), gomock.Any()).Return(&persistence.ListDomainsResponse{ + Domains: []*persistence.GetDomainResponse{domain}, + }, nil).AnyTimes() + domainMgr.EXPECT().GetDomain(gomock.Any(), gomock.Any()).Return(domain, nil).AnyTimes() + domainReplMgr := persistence.NewMockQueueManager(ctrl) + historyMgr := persistence.NewMockHistoryManager(ctrl) + taskMgr := persistence.NewMockTaskManager(ctrl) + visMgr := persistence.NewMockVisibilityManager(ctrl) + shardMgr := persistence.NewMockShardManager(ctrl) + execMgr := persistence.NewMockExecutionManager(ctrl) + persistenceClientBean.EXPECT().GetDomainManager().Return(domainMgr).AnyTimes() + persistenceClientBean.EXPECT().GetTaskManager().Return(taskMgr).AnyTimes() + persistenceClientBean.EXPECT().GetVisibilityManager().Return(visMgr).AnyTimes() + persistenceClientBean.EXPECT().GetShardManager().Return(shardMgr).AnyTimes() + persistenceClientBean.EXPECT().GetExecutionManager(gomock.Any()).Return(execMgr, nil).AnyTimes() + persistenceClientBean.EXPECT().GetDomainReplicationQueueManager().Return(domainReplMgr).AnyTimes() + persistenceClientBean.EXPECT().GetHistoryManager().Return(historyMgr).AnyTimes() + persistenceClientBean.EXPECT().Close().Times(1) + + // archiver provider mocks + archiveProvider := &provider.MockArchiverProvider{} + archiveProvider.On("RegisterBootstrapContainer", mock.Anything, mock.Anything, mock.Anything).Return(nil).Times(1) + + // params + params := &Params{ + Name: serviceName, + HostName: hostName, + PersistenceConfig: config.Persistence{ + NumHistoryShards: 1024, + DefaultStore: "nosql-store", + DataStores: map[string]config.DataStore{ + "nosql-store": { + NoSQL: &config.NoSQL{}, + }, + }, + }, + ClusterMetadata: clusterMetadata, + Logger: logger, + MetricScope: tally.NoopScope, + MetricsClient: metricsCl, + RPCFactory: rpcFac, + MembershipResolver: memberRes, + DynamicConfig: dc, + TimeSource: clock.NewRealTimeSource(), + PProfInitializer: pprof, + NewPersistenceBeanFn: func(persistenceClient.Factory, *persistenceClient.Params, *service.Config) (persistenceClient.Bean, error) { + return persistenceClientBean, nil + }, + ArchiverProvider: archiveProvider, + AsyncWorkflowQueueProvider: queue.NewMockProvider(ctrl), + } + + // bare minimum service config + svcCfg := &service.Config{ + ThrottledLoggerMaxRPS: func(opts ...dynamicconfig.FilterOption) int { + return 100 + }, + PersistenceGlobalMaxQPS: func(opts ...dynamicconfig.FilterOption) int { + return 100 + }, + PersistenceMaxQPS: func(opts ...dynamicconfig.FilterOption) int { + return 100 + }, + } + + i, err := New(params, serviceName, svcCfg) + if err != nil { + t.Fatal(err) + } + + i.Start() + i.Start() // should be no-op + defer func() { + i.Stop() + i.Stop() // should be no-op + goleak.VerifyNone(t) + }() + + time.Sleep(100 * time.Millisecond) // wait for go routines to start + + // validations + assert.Equal(t, serviceName, i.GetServiceName()) + assert.Equal(t, selfHostInfo, i.GetHostInfo()) + assert.Equal(t, params.ClusterMetadata, i.GetClusterMetadata()) + gotDomain, err := i.GetDomainCache().GetDomainByID("test-domain-id") + assert.NoError(t, err) + assert.Equal(t, domain.Info, gotDomain.GetInfo()) + assert.NotNil(t, i.GetDomainMetricsScopeCache()) + assert.NotNil(t, i.GetTimeSource()) + assert.NotNil(t, i.GetPayloadSerializer()) + assert.Equal(t, metricsCl, i.GetMetricsClient()) + assert.Equal(t, params.MessagingClient, i.GetMessagingClient()) + assert.Equal(t, params.BlobstoreClient, i.GetBlobstoreClient()) + assert.Equal(t, params.ArchivalMetadata, i.GetArchivalMetadata()) + assert.Equal(t, archiveProvider, i.GetArchiverProvider()) + assert.NotNil(t, i.GetDomainReplicationQueue()) + assert.Equal(t, memberRes, i.GetMembershipResolver()) + assert.Equal(t, params.PublicClient, i.GetSDKClient()) + assert.NotNil(t, i.GetFrontendRawClient()) + assert.NotNil(t, i.GetFrontendClient()) + assert.NotNil(t, i.GetMatchingRawClient()) + assert.NotNil(t, i.GetMatchingClient()) + assert.NotNil(t, i.GetHistoryRawClient()) + assert.NotNil(t, i.GetHistoryClient()) + assert.NotNil(t, i.GetRatelimiterAggregatorsClient()) + assert.NotNil(t, i.GetRemoteAdminClient("secondary-cluster")) + assert.NotNil(t, i.GetRemoteFrontendClient("secondary-cluster")) + assert.NotNil(t, i.GetClientBean()) + assert.Equal(t, domainMgr, i.GetDomainManager()) + assert.Equal(t, taskMgr, i.GetTaskManager()) + assert.Equal(t, visMgr, i.GetVisibilityManager()) + assert.Equal(t, shardMgr, i.GetShardManager()) + assert.Equal(t, historyMgr, i.GetHistoryManager()) + em, err := i.GetExecutionManager(3) + assert.NoError(t, err) + assert.Equal(t, execMgr, em) + assert.Equal(t, persistenceClientBean, i.GetPersistenceBean()) + assert.Equal(t, hostName, i.GetHostName()) + assert.NotNil(t, i.GetLogger()) + assert.NotNil(t, i.GetThrottledLogger()) + assert.NotNil(t, i.GetDispatcher()) + assert.NotNil(t, i.GetIsolationGroupState()) + assert.Nil(t, i.GetIsolationGroupStore()) + assert.NotNil(t, i.GetPartitioner()) + assert.Equal(t, params.AsyncWorkflowQueueProvider, i.GetAsyncWorkflowQueueProvider()) +} diff --git a/common/resource/resource_mock.go b/common/resource/resource_mock.go index 3e6c1a8d40d..52e12dcdba2 100644 --- a/common/resource/resource_mock.go +++ b/common/resource/resource_mock.go @@ -21,7 +21,7 @@ // SOFTWARE. // Code generated by MockGen. DO NOT EDIT. -// Source: resource.go +// Source: types.go // Package resource is a generated GoMock package. package resource diff --git a/common/resource/resource_test.go b/common/resource/resource_test.go deleted file mode 100644 index de41a02a4d0..00000000000 --- a/common/resource/resource_test.go +++ /dev/null @@ -1,42 +0,0 @@ -// The MIT License (MIT) - -// Copyright (c) 2017-2020 Uber Technologies Inc. - -// Permission is hereby granted, free of charge, to any person obtaining a copy -// of this software and associated documentation files (the "Software"), to deal -// in the Software without restriction, including without limitation the rights -// to use, copy, modify, merge, publish, distribute, sublicense, and/or sell -// copies of the Software, and to permit persons to whom the Software is -// furnished to do so, subject to the following conditions: -// -// The above copyright notice and this permission notice shall be included in all -// copies or substantial portions of the Software. -// -// THE SOFTWARE IS PROVIDED "AS IS", WITHOUT WARRANTY OF ANY KIND, EXPRESS OR -// IMPLIED, INCLUDING BUT NOT LIMITED TO THE WARRANTIES OF MERCHANTABILITY, -// FITNESS FOR A PARTICULAR PURPOSE AND NONINFRINGEMENT. IN NO EVENT SHALL THE -// AUTHORS OR COPYRIGHT HOLDERS BE LIABLE FOR ANY CLAIM, DAMAGES OR OTHER -// LIABILITY, WHETHER IN AN ACTION OF CONTRACT, TORT OR OTHERWISE, ARISING FROM, -// OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER DEALINGS IN THE -// SOFTWARE. - -package resource - -import ( - "testing" - - "github.com/stretchr/testify/assert" -) - -func TestShutdown(t *testing.T) { - i := Impl{} - assert.NotPanics(t, func() { - i.Stop() - }) -} - -func TestNewResource(t *testing.T) { - assert.NotPanics(t, func() { - ensureGetAllIsolationGroupsFnIsSet(&Params{}) - }) -} diff --git a/common/resource/resource.go b/common/resource/types.go similarity index 56% rename from common/resource/resource.go rename to common/resource/types.go index 4f5aa81c6e8..12e2fdf565a 100644 --- a/common/resource/resource.go +++ b/common/resource/types.go @@ -60,71 +60,69 @@ type ResourceFactory interface { ) (resource Resource, err error) } -type ( - // Resource is the interface which expose common resources - Resource interface { - common.Daemon - - // static infos - - GetServiceName() string - GetHostInfo() membership.HostInfo - GetArchivalMetadata() archiver.ArchivalMetadata - GetClusterMetadata() cluster.Metadata - - // other common resources - - GetDomainCache() cache.DomainCache - GetDomainMetricsScopeCache() cache.DomainMetricsScopeCache - GetTimeSource() clock.TimeSource - GetPayloadSerializer() persistence.PayloadSerializer - GetMetricsClient() metrics.Client - GetArchiverProvider() provider.ArchiverProvider - GetMessagingClient() messaging.Client - GetBlobstoreClient() blobstore.Client - GetDomainReplicationQueue() domain.ReplicationQueue - - // membership infos - GetMembershipResolver() membership.Resolver - - // internal services clients - - GetSDKClient() workflowserviceclient.Interface - GetFrontendRawClient() frontend.Client - GetFrontendClient() frontend.Client - GetMatchingRawClient() matching.Client - GetMatchingClient() matching.Client - GetHistoryRawClient() history.Client - GetHistoryClient() history.Client - GetRatelimiterAggregatorsClient() qrpc.Client - GetRemoteAdminClient(cluster string) admin.Client - GetRemoteFrontendClient(cluster string) frontend.Client - GetClientBean() client.Bean - - // persistence clients - GetDomainManager() persistence.DomainManager - GetTaskManager() persistence.TaskManager - GetVisibilityManager() persistence.VisibilityManager - GetShardManager() persistence.ShardManager - GetHistoryManager() persistence.HistoryManager - GetExecutionManager(int) (persistence.ExecutionManager, error) - GetPersistenceBean() persistenceClient.Bean - - // GetHostName get host name - GetHostName() string - - // loggers - GetLogger() log.Logger - GetThrottledLogger() log.Logger - - // for registering handlers - GetDispatcher() *yarpc.Dispatcher - - // GetIsolationGroupState returns the isolationGroupState - GetIsolationGroupState() isolationgroup.State - GetPartitioner() partition.Partitioner - GetIsolationGroupStore() configstore.Client - - GetAsyncWorkflowQueueProvider() queue.Provider - } -) +// Resource is the interface which expose common resources +type Resource interface { + common.Daemon + + // static infos + + GetServiceName() string + GetHostInfo() membership.HostInfo + GetArchivalMetadata() archiver.ArchivalMetadata + GetClusterMetadata() cluster.Metadata + + // other common resources + + GetDomainCache() cache.DomainCache + GetDomainMetricsScopeCache() cache.DomainMetricsScopeCache + GetTimeSource() clock.TimeSource + GetPayloadSerializer() persistence.PayloadSerializer + GetMetricsClient() metrics.Client + GetArchiverProvider() provider.ArchiverProvider + GetMessagingClient() messaging.Client + GetBlobstoreClient() blobstore.Client + GetDomainReplicationQueue() domain.ReplicationQueue + + // membership infos + GetMembershipResolver() membership.Resolver + + // internal services clients + + GetSDKClient() workflowserviceclient.Interface + GetFrontendRawClient() frontend.Client + GetFrontendClient() frontend.Client + GetMatchingRawClient() matching.Client + GetMatchingClient() matching.Client + GetHistoryRawClient() history.Client + GetHistoryClient() history.Client + GetRatelimiterAggregatorsClient() qrpc.Client + GetRemoteAdminClient(cluster string) admin.Client + GetRemoteFrontendClient(cluster string) frontend.Client + GetClientBean() client.Bean + + // persistence clients + GetDomainManager() persistence.DomainManager + GetTaskManager() persistence.TaskManager + GetVisibilityManager() persistence.VisibilityManager + GetShardManager() persistence.ShardManager + GetHistoryManager() persistence.HistoryManager + GetExecutionManager(int) (persistence.ExecutionManager, error) + GetPersistenceBean() persistenceClient.Bean + + // GetHostName get host name + GetHostName() string + + // loggers + GetLogger() log.Logger + GetThrottledLogger() log.Logger + + // for registering handlers + GetDispatcher() *yarpc.Dispatcher + + // GetIsolationGroupState returns the isolationGroupState + GetIsolationGroupState() isolationgroup.State + GetPartitioner() partition.Partitioner + GetIsolationGroupStore() configstore.Client + + GetAsyncWorkflowQueueProvider() queue.Provider +} diff --git a/common/rpc/peer_chooser.go b/common/rpc/peer_chooser.go index d2c16344101..e1da84033f9 100644 --- a/common/rpc/peer_chooser.go +++ b/common/rpc/peer_chooser.go @@ -18,6 +18,8 @@ // OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER DEALINGS IN // THE SOFTWARE. +//go:generate mockgen -package $GOPACKAGE -source $GOFILE -destination peer_chooser_mock.go -self_package github.com/uber/cadence/common/rpc + package rpc import ( diff --git a/common/rpc/peer_chooser_mock.go b/common/rpc/peer_chooser_mock.go new file mode 100644 index 00000000000..935debc3980 --- /dev/null +++ b/common/rpc/peer_chooser_mock.go @@ -0,0 +1,169 @@ +// The MIT License (MIT) + +// Copyright (c) 2017-2020 Uber Technologies Inc. + +// Permission is hereby granted, free of charge, to any person obtaining a copy +// of this software and associated documentation files (the "Software"), to deal +// in the Software without restriction, including without limitation the rights +// to use, copy, modify, merge, publish, distribute, sublicense, and/or sell +// copies of the Software, and to permit persons to whom the Software is +// furnished to do so, subject to the following conditions: +// +// The above copyright notice and this permission notice shall be included in all +// copies or substantial portions of the Software. +// +// THE SOFTWARE IS PROVIDED "AS IS", WITHOUT WARRANTY OF ANY KIND, EXPRESS OR +// IMPLIED, INCLUDING BUT NOT LIMITED TO THE WARRANTIES OF MERCHANTABILITY, +// FITNESS FOR A PARTICULAR PURPOSE AND NONINFRINGEMENT. IN NO EVENT SHALL THE +// AUTHORS OR COPYRIGHT HOLDERS BE LIABLE FOR ANY CLAIM, DAMAGES OR OTHER +// LIABILITY, WHETHER IN AN ACTION OF CONTRACT, TORT OR OTHERWISE, ARISING FROM, +// OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER DEALINGS IN THE +// SOFTWARE. + +// Code generated by MockGen. DO NOT EDIT. +// Source: peer_chooser.go + +// Package rpc is a generated GoMock package. +package rpc + +import ( + context "context" + reflect "reflect" + + gomock "github.com/golang/mock/gomock" + peer "go.uber.org/yarpc/api/peer" + transport "go.uber.org/yarpc/api/transport" + + membership "github.com/uber/cadence/common/membership" +) + +// MockPeerChooserFactory is a mock of PeerChooserFactory interface. +type MockPeerChooserFactory struct { + ctrl *gomock.Controller + recorder *MockPeerChooserFactoryMockRecorder +} + +// MockPeerChooserFactoryMockRecorder is the mock recorder for MockPeerChooserFactory. +type MockPeerChooserFactoryMockRecorder struct { + mock *MockPeerChooserFactory +} + +// NewMockPeerChooserFactory creates a new mock instance. +func NewMockPeerChooserFactory(ctrl *gomock.Controller) *MockPeerChooserFactory { + mock := &MockPeerChooserFactory{ctrl: ctrl} + mock.recorder = &MockPeerChooserFactoryMockRecorder{mock} + return mock +} + +// EXPECT returns an object that allows the caller to indicate expected use. +func (m *MockPeerChooserFactory) EXPECT() *MockPeerChooserFactoryMockRecorder { + return m.recorder +} + +// CreatePeerChooser mocks base method. +func (m *MockPeerChooserFactory) CreatePeerChooser(transport peer.Transport, opts PeerChooserOptions) (PeerChooser, error) { + m.ctrl.T.Helper() + ret := m.ctrl.Call(m, "CreatePeerChooser", transport, opts) + ret0, _ := ret[0].(PeerChooser) + ret1, _ := ret[1].(error) + return ret0, ret1 +} + +// CreatePeerChooser indicates an expected call of CreatePeerChooser. +func (mr *MockPeerChooserFactoryMockRecorder) CreatePeerChooser(transport, opts interface{}) *gomock.Call { + mr.mock.ctrl.T.Helper() + return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "CreatePeerChooser", reflect.TypeOf((*MockPeerChooserFactory)(nil).CreatePeerChooser), transport, opts) +} + +// MockPeerChooser is a mock of PeerChooser interface. +type MockPeerChooser struct { + ctrl *gomock.Controller + recorder *MockPeerChooserMockRecorder +} + +// MockPeerChooserMockRecorder is the mock recorder for MockPeerChooser. +type MockPeerChooserMockRecorder struct { + mock *MockPeerChooser +} + +// NewMockPeerChooser creates a new mock instance. +func NewMockPeerChooser(ctrl *gomock.Controller) *MockPeerChooser { + mock := &MockPeerChooser{ctrl: ctrl} + mock.recorder = &MockPeerChooserMockRecorder{mock} + return mock +} + +// EXPECT returns an object that allows the caller to indicate expected use. +func (m *MockPeerChooser) EXPECT() *MockPeerChooserMockRecorder { + return m.recorder +} + +// Choose mocks base method. +func (m *MockPeerChooser) Choose(arg0 context.Context, arg1 *transport.Request) (peer.Peer, func(error), error) { + m.ctrl.T.Helper() + ret := m.ctrl.Call(m, "Choose", arg0, arg1) + ret0, _ := ret[0].(peer.Peer) + ret1, _ := ret[1].(func(error)) + ret2, _ := ret[2].(error) + return ret0, ret1, ret2 +} + +// Choose indicates an expected call of Choose. +func (mr *MockPeerChooserMockRecorder) Choose(arg0, arg1 interface{}) *gomock.Call { + mr.mock.ctrl.T.Helper() + return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Choose", reflect.TypeOf((*MockPeerChooser)(nil).Choose), arg0, arg1) +} + +// IsRunning mocks base method. +func (m *MockPeerChooser) IsRunning() bool { + m.ctrl.T.Helper() + ret := m.ctrl.Call(m, "IsRunning") + ret0, _ := ret[0].(bool) + return ret0 +} + +// IsRunning indicates an expected call of IsRunning. +func (mr *MockPeerChooserMockRecorder) IsRunning() *gomock.Call { + mr.mock.ctrl.T.Helper() + return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "IsRunning", reflect.TypeOf((*MockPeerChooser)(nil).IsRunning)) +} + +// Start mocks base method. +func (m *MockPeerChooser) Start() error { + m.ctrl.T.Helper() + ret := m.ctrl.Call(m, "Start") + ret0, _ := ret[0].(error) + return ret0 +} + +// Start indicates an expected call of Start. +func (mr *MockPeerChooserMockRecorder) Start() *gomock.Call { + mr.mock.ctrl.T.Helper() + return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Start", reflect.TypeOf((*MockPeerChooser)(nil).Start)) +} + +// Stop mocks base method. +func (m *MockPeerChooser) Stop() error { + m.ctrl.T.Helper() + ret := m.ctrl.Call(m, "Stop") + ret0, _ := ret[0].(error) + return ret0 +} + +// Stop indicates an expected call of Stop. +func (mr *MockPeerChooserMockRecorder) Stop() *gomock.Call { + mr.mock.ctrl.T.Helper() + return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "Stop", reflect.TypeOf((*MockPeerChooser)(nil).Stop)) +} + +// UpdatePeers mocks base method. +func (m *MockPeerChooser) UpdatePeers(arg0 []membership.HostInfo) { + m.ctrl.T.Helper() + m.ctrl.Call(m, "UpdatePeers", arg0) +} + +// UpdatePeers indicates an expected call of UpdatePeers. +func (mr *MockPeerChooserMockRecorder) UpdatePeers(arg0 interface{}) *gomock.Call { + mr.mock.ctrl.T.Helper() + return mr.mock.ctrl.RecordCallWithMethodType(mr.mock, "UpdatePeers", reflect.TypeOf((*MockPeerChooser)(nil).UpdatePeers), arg0) +}