network/services: unify service lifecycle management

Run with Start, Stop with Shutdown, make behavior uniform.
This commit is contained in:
Roman Khimov 2022-01-12 04:11:21 +03:00
parent c942402957
commit 5dd4db2c02
11 changed files with 65 additions and 60 deletions

View file

@ -129,10 +129,6 @@ func NewService(cfg Config) (Service, error) {
finished: make(chan struct{}), finished: make(chan struct{}),
} }
if cfg.Wallet == nil {
return srv, nil
}
var err error var err error
if srv.wallet, err = wallet.NewWalletFromFile(cfg.Wallet.Path); err != nil { if srv.wallet, err = wallet.NewWalletFromFile(cfg.Wallet.Path); err != nil {

View file

@ -16,8 +16,8 @@ type Oracle interface {
UpdateOracleNodes(keys.PublicKeys) UpdateOracleNodes(keys.PublicKeys)
// UpdateNativeContract updates oracle contract native script and hash. // UpdateNativeContract updates oracle contract native script and hash.
UpdateNativeContract([]byte, []byte, util.Uint160, int) UpdateNativeContract([]byte, []byte, util.Uint160, int)
// Run runs oracle module. Must be invoked in a separate goroutine. // Start runs oracle module.
Run() Start()
// Shutdown shutdowns oracle module. // Shutdown shutdowns oracle module.
Shutdown() Shutdown()
} }

View file

@ -140,9 +140,9 @@ func TestNotary(t *testing.T) {
}) })
mp1.RunSubscriptions() mp1.RunSubscriptions()
go ntr1.Run() ntr1.Start()
t.Cleanup(func() { t.Cleanup(func() {
ntr1.Stop() ntr1.Shutdown()
mp1.StopSubscriptions() mp1.StopSubscriptions()
}) })

View file

@ -305,7 +305,7 @@ func TestOracleFull(t *testing.T) {
require.NoError(t, bc.contracts.Management.PutContractState(bc.dao, cs)) require.NoError(t, bc.contracts.Management.PutContractState(bc.dao, cs))
go bc.Run() go bc.Run()
go orc.Run() orc.Start()
t.Cleanup(orc.Shutdown) t.Cleanup(orc.Shutdown)
bc.setNodesByRole(t, true, noderoles.Oracle, keys.PublicKeys{acc.PrivateKey().PublicKey()}) bc.setNodesByRole(t, true, noderoles.Oracle, keys.PublicKeys{acc.PrivateKey().PublicKey()})
@ -351,7 +351,7 @@ func TestNotYetRunningOracle(t *testing.T) {
ids = []uint64{3} ids = []uint64{3}
orc.RemoveRequests(ids) // 3 removed from pending -> 2, 4 in pending. orc.RemoveRequests(ids) // 3 removed from pending -> 2, 4 in pending.
go orc.Run() orc.Start()
t.Cleanup(orc.Shutdown) t.Cleanup(orc.Shutdown)
require.Eventually(t, func() bool { return mp.Count() == 2 }, require.Eventually(t, func() bool { return mp.Count() == 2 },

View file

@ -199,7 +199,7 @@ func TestStateRootFull(t *testing.T) {
lastValidated.Store(ep) lastValidated.Store(ep)
}) })
require.NoError(t, err) require.NoError(t, err)
srv.Run() srv.Start()
t.Cleanup(srv.Shutdown) t.Cleanup(srv.Shutdown)
bc.setNodesByRole(t, true, noderoles.StateValidator, pubs) bc.setNodesByRole(t, true, noderoles.StateValidator, pubs)

View file

@ -52,6 +52,12 @@ var (
) )
type ( type (
// Service is a service abstraction (oracle, state root, consensus, etc).
Service interface {
Start()
Shutdown()
}
// Server represents the local Node in the network. Its transport could // Server represents the local Node in the network. Its transport could
// be of any kind. // be of any kind.
Server struct { Server struct {
@ -77,6 +83,7 @@ type (
extensiblePool *extpool.Pool extensiblePool *extpool.Pool
notaryFeer NotaryFeer notaryFeer NotaryFeer
notaryModule *notary.Notary notaryModule *notary.Notary
services []Service
txInLock sync.Mutex txInLock sync.Mutex
txInMap map[util.Uint256]struct{} txInMap map[util.Uint256]struct{}
@ -179,6 +186,7 @@ func newServerFromConstructors(config ServerConfig, chain blockchainer.Blockchai
return nil, fmt.Errorf("failed to create Notary module: %w", err) return nil, fmt.Errorf("failed to create Notary module: %w", err)
} }
s.notaryModule = n s.notaryModule = n
s.services = append(s.services, n)
chain.SetNotary(n) chain.SetNotary(n)
} }
} else if config.P2PNotaryCfg.Enabled { } else if config.P2PNotaryCfg.Enabled {
@ -197,6 +205,7 @@ func newServerFromConstructors(config ServerConfig, chain blockchainer.Blockchai
return nil, fmt.Errorf("can't initialize StateRoot service: %w", err) return nil, fmt.Errorf("can't initialize StateRoot service: %w", err)
} }
s.stateRoot = sr s.stateRoot = sr
s.services = append(s.services, sr)
sSync := chain.GetStateSyncModule() sSync := chain.GetStateSyncModule()
s.stateSync = sSync s.stateSync = sSync
@ -221,25 +230,29 @@ func newServerFromConstructors(config ServerConfig, chain blockchainer.Blockchai
} }
}) })
s.oracle = orc s.oracle = orc
s.services = append(s.services, orc)
chain.SetOracle(orc) chain.SetOracle(orc)
} }
srv, err := newConsensus(consensus.Config{ if config.Wallet != nil {
Logger: log, srv, err := newConsensus(consensus.Config{
Broadcast: s.handleNewPayload, Logger: log,
Chain: chain, Broadcast: s.handleNewPayload,
ProtocolConfiguration: chain.GetConfig(), Chain: chain,
RequestTx: s.requestTx, ProtocolConfiguration: chain.GetConfig(),
Wallet: config.Wallet, RequestTx: s.requestTx,
Wallet: config.Wallet,
TimePerBlock: config.TimePerBlock, TimePerBlock: config.TimePerBlock,
}) })
if err != nil { if err != nil {
return nil, err return nil, err
}
s.consensus = srv
s.services = append(s.services, srv)
} }
s.consensus = srv
if s.MinPeers < 0 { if s.MinPeers < 0 {
s.log.Info("bad MinPeers configured, using the default value", s.log.Info("bad MinPeers configured, using the default value",
zap.Int("configured", s.MinPeers), zap.Int("configured", s.MinPeers),
@ -299,20 +312,13 @@ func (s *Server) Shutdown() {
s.log.Info("shutting down server", zap.Int("peers", s.PeerCount())) s.log.Info("shutting down server", zap.Int("peers", s.PeerCount()))
s.transport.Close() s.transport.Close()
s.discovery.Close() s.discovery.Close()
s.consensus.Shutdown()
for _, p := range s.getPeers(nil) { for _, p := range s.getPeers(nil) {
p.Disconnect(errServerShutdown) p.Disconnect(errServerShutdown)
} }
s.bQueue.discard() s.bQueue.discard()
s.bSyncQueue.discard() s.bSyncQueue.discard()
if s.StateRootCfg.Enabled { for _, svc := range s.services {
s.stateRoot.Shutdown() svc.Shutdown()
}
if s.oracle != nil {
s.oracle.Shutdown()
}
if s.notaryModule != nil {
s.notaryModule.Stop()
} }
if s.chain.P2PSigExtensionsEnabled() { if s.chain.P2PSigExtensionsEnabled() {
s.notaryRequestPool.StopSubscriptions() s.notaryRequestPool.StopSubscriptions()
@ -460,20 +466,11 @@ func (s *Server) tryStartServices() {
if s.IsInSync() && s.syncReached.CAS(false, true) { if s.IsInSync() && s.syncReached.CAS(false, true) {
s.log.Info("node reached synchronized state, starting services") s.log.Info("node reached synchronized state, starting services")
if s.Wallet != nil {
s.consensus.Start()
}
if s.StateRootCfg.Enabled {
s.stateRoot.Run()
}
if s.oracle != nil {
go s.oracle.Run()
}
if s.chain.P2PSigExtensionsEnabled() { if s.chain.P2PSigExtensionsEnabled() {
s.notaryRequestPool.RunSubscriptions() // WSClient is also a subscriber. s.notaryRequestPool.RunSubscriptions() // WSClient is also a subscriber.
} }
if s.notaryModule != nil { for _, svc := range s.services {
go s.notaryModule.Run() svc.Start()
} }
} }
} }
@ -976,7 +973,9 @@ func (s *Server) handleExtensibleCmd(e *payload.Extensible) error {
} }
switch e.Category { switch e.Category {
case consensus.Category: case consensus.Category:
s.consensus.OnPayload(e) if s.consensus != nil {
s.consensus.OnPayload(e)
}
case stateroot.Category: case stateroot.Category:
err := s.stateRoot.OnPayload(e) err := s.stateRoot.OnPayload(e)
if err != nil { if err != nil {
@ -1009,7 +1008,9 @@ func (s *Server) handleTxCmd(tx *transaction.Transaction) error {
s.txInMap[tx.Hash()] = struct{}{} s.txInMap[tx.Hash()] = struct{}{}
s.txInLock.Unlock() s.txInLock.Unlock()
if s.verifyAndPoolTX(tx) == nil { if s.verifyAndPoolTX(tx) == nil {
s.consensus.OnTransaction(tx) if s.consensus != nil {
s.consensus.OnTransaction(tx)
}
s.broadcastTX(tx, nil) s.broadcastTX(tx, nil)
} }
s.txInLock.Lock() s.txInLock.Lock()

View file

@ -78,7 +78,7 @@ func TestNewServer(t *testing.T) {
}) })
t.Run("consensus error is not dropped", func(t *testing.T) { t.Run("consensus error is not dropped", func(t *testing.T) {
errConsensus := errors.New("can't create consensus") errConsensus := errors.New("can't create consensus")
_, err = newServerFromConstructors(ServerConfig{MinPeers: -1}, bc, zaptest.NewLogger(t), newFakeTransp, _, err = newServerFromConstructors(ServerConfig{Wallet: new(config.Wallet), MinPeers: -1}, bc, zaptest.NewLogger(t), newFakeTransp,
func(consensus.Config) (consensus.Service, error) { return nil, errConsensus }, func(consensus.Config) (consensus.Service, error) { return nil, errConsensus },
newTestDiscovery) newTestDiscovery)
require.True(t, errors.Is(err, errConsensus), "got: %#v", err) require.True(t, errors.Is(err, errConsensus), "got: %#v", err)
@ -104,13 +104,12 @@ func TestServerStartAndShutdown(t *testing.T) {
require.Eventually(t, func() bool { return 1 == s.PeerCount() }, time.Second, time.Millisecond*10) require.Eventually(t, func() bool { return 1 == s.PeerCount() }, time.Second, time.Millisecond*10)
assert.True(t, s.transport.(*fakeTransp).started.Load()) assert.True(t, s.transport.(*fakeTransp).started.Load())
assert.False(t, s.consensus.(*fakeConsensus).started.Load()) assert.Nil(t, s.consensus)
s.Shutdown() s.Shutdown()
<-ch <-ch
require.True(t, s.transport.(*fakeTransp).closed.Load()) require.True(t, s.transport.(*fakeTransp).closed.Load())
require.True(t, s.consensus.(*fakeConsensus).stopped.Load())
err, ok := p.droppedWith.Load().(error) err, ok := p.droppedWith.Load().(error)
require.True(t, ok) require.True(t, ok)
require.True(t, errors.Is(err, errServerShutdown)) require.True(t, errors.Is(err, errServerShutdown))
@ -416,7 +415,8 @@ func TestBlock(t *testing.T) {
} }
func TestConsensus(t *testing.T) { func TestConsensus(t *testing.T) {
s := startTestServer(t) s := newTestServer(t, ServerConfig{Wallet: new(config.Wallet)})
startWithCleanup(t, s)
atomic2.StoreUint32(&s.chain.(*fakechain.FakeChain).Blockheight, 4) atomic2.StoreUint32(&s.chain.(*fakechain.FakeChain).Blockheight, 4)
p := newLocalPeer(t, s) p := newLocalPeer(t, s)
@ -465,7 +465,8 @@ func TestConsensus(t *testing.T) {
} }
func TestTransaction(t *testing.T) { func TestTransaction(t *testing.T) {
s := startTestServer(t) s := newTestServer(t, ServerConfig{Wallet: new(config.Wallet)})
startWithCleanup(t, s)
t.Run("good", func(t *testing.T) { t.Run("good", func(t *testing.T) {
tx := newDummyTx() tx := newDummyTx()

View file

@ -143,12 +143,16 @@ func NewNotary(cfg Config, net netmode.Magic, mp *mempool.Pool, onTransaction fu
}, nil }, nil
} }
// Run runs Notary module and should be called in a separate goroutine. // Start runs Notary module in a separate goroutine.
func (n *Notary) Run() { func (n *Notary) Start() {
n.Config.Log.Info("starting notary service") n.Config.Log.Info("starting notary service")
n.Config.Chain.SubscribeForBlocks(n.blocksCh) n.Config.Chain.SubscribeForBlocks(n.blocksCh)
n.mp.SubscribeForTransactions(n.reqCh) n.mp.SubscribeForTransactions(n.reqCh)
go n.newTxCallbackLoop() go n.newTxCallbackLoop()
go n.mainLoop()
}
func (n *Notary) mainLoop() {
for { for {
select { select {
case <-n.stopCh: case <-n.stopCh:
@ -171,8 +175,8 @@ func (n *Notary) Run() {
} }
} }
// Stop shutdowns Notary module. // Shutdown stops Notary module.
func (n *Notary) Stop() { func (n *Notary) Shutdown() {
close(n.stopCh) close(n.stopCh)
} }

View file

@ -170,15 +170,18 @@ func (o *Oracle) Shutdown() {
o.getBroadcaster().Shutdown() o.getBroadcaster().Shutdown()
} }
// Run runs must be executed in a separate goroutine. // Start runs the oracle service in a separate goroutine.
func (o *Oracle) Run() { func (o *Oracle) Start() {
o.respMtx.Lock() o.respMtx.Lock()
if o.running { if o.running {
o.respMtx.Unlock() o.respMtx.Unlock()
return return
} }
o.Log.Info("starting oracle service") o.Log.Info("starting oracle service")
go o.start()
}
func (o *Oracle) start() {
o.requestMap <- o.pending // Guaranteed to not block, only AddRequests sends to it. o.requestMap <- o.pending // Guaranteed to not block, only AddRequests sends to it.
o.pending = nil o.pending = nil
o.running = true o.running = true

View file

@ -25,7 +25,7 @@ type (
OnPayload(p *payload.Extensible) error OnPayload(p *payload.Extensible) error
AddSignature(height uint32, validatorIndex int32, sig []byte) error AddSignature(height uint32, validatorIndex int32, sig []byte) error
GetConfig() config.StateRoot GetConfig() config.StateRoot
Run() Start()
Shutdown() Shutdown()
} }

View file

@ -17,8 +17,8 @@ const (
firstVoteResendDelay = 3 * time.Second firstVoteResendDelay = 3 * time.Second
) )
// Run runs service instance in a separate goroutine. // Start runs service instance in a separate goroutine.
func (s *service) Run() { func (s *service) Start() {
s.log.Info("starting state validation service") s.log.Info("starting state validation service")
s.chain.SubscribeForBlocks(s.blockCh) s.chain.SubscribeForBlocks(s.blockCh)
go s.run() go s.run()