mirror of https://github.com/milvus-io/milvus.git
Add log to startup process (#5594)
* Add log to startup process Signed-off-by: zhenshan.cao <zhenshan.cao@zilliz.com> * Fixbug: avoid ctx exceed deadline Signed-off-by: zhenshan.cao <zhenshan.cao@zilliz.com> * Fmt code Signed-off-by: zhenshan.cao <zhenshan.cao@zilliz.com> * Fix bug: wrong time unit Signed-off-by: zhenshan.cao <zhenshan.cao@zilliz.com> * Fixbug: datanservice client reconnect use time ctx Signed-off-by: zhenshan.cao <zhenshan.cao@zilliz.com>pull/5604/head^2
parent
2ba93deb5e
commit
03e3b4beb4
|
@ -205,7 +205,6 @@ func (node *DataNode) Init() error {
|
|||
node.dataSyncService.init()
|
||||
node.metaService = newMetaService(node.ctx, replica, node.masterService)
|
||||
node.replica = replica
|
||||
|
||||
return nil
|
||||
}
|
||||
|
||||
|
|
|
@ -88,13 +88,14 @@ func CreateServer(ctx context.Context, factory msgstream.Factory) (*Server, erro
|
|||
}
|
||||
s.insertChannels = s.getInsertChannels()
|
||||
s.createDataNodeClient = func(addr string) (types.DataNode, error) {
|
||||
node, err := grpcdatanodeclient.NewClient(addr, 10)
|
||||
node, err := grpcdatanodeclient.NewClient(addr, 10*time.Second)
|
||||
if err != nil {
|
||||
return nil, err
|
||||
}
|
||||
return node, nil
|
||||
}
|
||||
s.UpdateStateCode(internalpb.StateCode_Abnormal)
|
||||
log.Debug("DataService", zap.Any("State", s.state.Load()))
|
||||
return s, nil
|
||||
}
|
||||
|
||||
|
@ -135,6 +136,7 @@ func (s *Server) Start() error {
|
|||
}
|
||||
|
||||
if err := s.initMeta(); err != nil {
|
||||
log.Debug("DataService initMeta failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
|
||||
|
@ -143,14 +145,17 @@ func (s *Server) Start() error {
|
|||
s.startSegmentAllocator()
|
||||
s.statsHandler = newStatsHandler(s.meta)
|
||||
if err = s.loadMetaFromMaster(); err != nil {
|
||||
log.Debug("DataService loadMetaFromMaster failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
if err = s.initMsgProducer(); err != nil {
|
||||
log.Debug("DataService initMsgProducer failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
s.startServerLoop()
|
||||
s.UpdateStateCode(internalpb.StateCode_Healthy)
|
||||
log.Debug("start success")
|
||||
log.Debug("DataService", zap.Any("State", s.state.Load()))
|
||||
return nil
|
||||
}
|
||||
|
||||
|
@ -163,7 +168,7 @@ func (s *Server) startSegmentAllocator() {
|
|||
func (s *Server) initSegmentInfoChannel() msgstream.MsgStream {
|
||||
segmentInfoStream, _ := s.msFactory.NewMsgStream(s.ctx)
|
||||
segmentInfoStream.AsProducer([]string{Params.SegmentInfoChannelName})
|
||||
log.Debug("dataservice AsProducer: " + Params.SegmentInfoChannelName)
|
||||
log.Debug("DataService AsProducer: " + Params.SegmentInfoChannelName)
|
||||
segmentInfoStream.Start()
|
||||
return segmentInfoStream
|
||||
}
|
||||
|
@ -343,14 +348,14 @@ func (s *Server) startStatsChannel(ctx context.Context) {
|
|||
defer s.serverLoopWg.Done()
|
||||
statsStream, _ := s.msFactory.NewMsgStream(ctx)
|
||||
statsStream.AsConsumer([]string{Params.StatisticsChannelName}, Params.DataServiceSubscriptionName)
|
||||
log.Debug("dataservice AsConsumer: " + Params.StatisticsChannelName + " : " + Params.DataServiceSubscriptionName)
|
||||
log.Debug("DataService AsConsumer: " + Params.StatisticsChannelName + " : " + Params.DataServiceSubscriptionName)
|
||||
// try to restore last processed pos
|
||||
pos, err := s.loadStreamLastPos(streamTypeStats)
|
||||
if err == nil {
|
||||
err = statsStream.Seek([]*internalpb.MsgPosition{pos})
|
||||
if err != nil {
|
||||
log.Error("Failed to seek to last pos for statsStream",
|
||||
zap.String("StatisChanName", Params.StatisticsChannelName),
|
||||
zap.String("StatisticsChanName", Params.StatisticsChannelName),
|
||||
zap.String("DataServiceSubscriptionName", Params.DataServiceSubscriptionName),
|
||||
zap.Error(err))
|
||||
}
|
||||
|
@ -398,7 +403,7 @@ func (s *Server) startSegmentFlushChannel(ctx context.Context) {
|
|||
defer s.serverLoopWg.Done()
|
||||
flushStream, _ := s.msFactory.NewMsgStream(ctx)
|
||||
flushStream.AsConsumer([]string{Params.SegmentInfoChannelName}, Params.DataServiceSubscriptionName)
|
||||
log.Debug("dataservice AsConsumer: " + Params.SegmentInfoChannelName + " : " + Params.DataServiceSubscriptionName)
|
||||
log.Debug("DataService AsConsumer: " + Params.SegmentInfoChannelName + " : " + Params.DataServiceSubscriptionName)
|
||||
|
||||
// try to restore last processed pos
|
||||
pos, err := s.loadStreamLastPos(streamTypeFlush)
|
||||
|
@ -432,7 +437,7 @@ func (s *Server) startSegmentFlushChannel(ctx context.Context) {
|
|||
}
|
||||
fcMsg := msg.(*msgstream.FlushCompletedMsg)
|
||||
err := s.meta.FlushSegment(fcMsg.SegmentID, fcMsg.BeginTimestamp)
|
||||
log.Debug("dataservice flushed segment", zap.Any("segmentID", fcMsg.SegmentID), zap.Error(err))
|
||||
log.Debug("DataService flushed segment", zap.Any("segmentID", fcMsg.SegmentID), zap.Error(err))
|
||||
if err != nil {
|
||||
log.Error("get segment from meta error", zap.Int64("segmentID", fcMsg.SegmentID), zap.Error(err))
|
||||
continue
|
||||
|
@ -517,10 +522,12 @@ func (s *Server) RegisterNode(ctx context.Context, req *datapb.RegisterNodeReque
|
|||
ErrorCode: commonpb.ErrorCode_UnexpectedError,
|
||||
},
|
||||
}
|
||||
log.Debug("DataService: RegisterNode:", zap.String("IP", req.Address.Ip), zap.Int64("Port", req.Address.Port))
|
||||
log.Debug("DataService: RegisterNode:", zap.String("IP", req.Address.Ip), zap.Int64("Port", req.Address.Port),
|
||||
zap.Any("NodeID", req.Base.SourceID))
|
||||
node, err := s.newDataNode(req.Address.Ip, req.Address.Port, req.Base.SourceID)
|
||||
if err != nil {
|
||||
ret.Status.Reason = err.Error()
|
||||
log.Debug("DataService newDataNode failed", zap.Error(err))
|
||||
return ret, nil
|
||||
}
|
||||
|
||||
|
|
|
@ -63,7 +63,9 @@ func (c *Client) Init() error {
|
|||
tracer := opentracing.GlobalTracer()
|
||||
connectGrpcFunc := func() error {
|
||||
log.Debug("DataNodeClient try connect ", zap.String("address", c.address))
|
||||
conn, err := grpc.DialContext(c.ctx, c.address, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
ctx, cancelFunc := context.WithTimeout(c.ctx, c.timeout)
|
||||
defer cancelFunc()
|
||||
conn, err := grpc.DialContext(ctx, c.address, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
grpc.WithUnaryInterceptor(
|
||||
otgrpc.OpenTracingClientInterceptor(tracer)),
|
||||
grpc.WithStreamInterceptor(
|
||||
|
|
|
@ -211,7 +211,7 @@ func (s *Server) init() error {
|
|||
if s.newDataServiceClient != nil {
|
||||
log.Debug("Data service address", zap.String("address", Params.DataServiceAddress))
|
||||
log.Debug("DataNode Init data service client ...")
|
||||
dataServiceClient := s.newDataServiceClient(Params.DataServiceAddress, dn.Params.MetaRootPath, dn.Params.EtcdAddress, 10)
|
||||
dataServiceClient := s.newDataServiceClient(Params.DataServiceAddress, dn.Params.MetaRootPath, dn.Params.EtcdAddress, 3*time.Second)
|
||||
if err = dataServiceClient.Init(); err != nil {
|
||||
log.Debug("DataNode newDataServiceClient failed", zap.Error(err))
|
||||
panic(err)
|
||||
|
|
|
@ -75,8 +75,10 @@ func (c *Client) Init() error {
|
|||
log.Debug("DataServiceClient", zap.Any("c.addr", c.addr))
|
||||
if c.addr != "" {
|
||||
connectGrpcFunc := func() error {
|
||||
ctx, cancelFunc := context.WithTimeout(c.ctx, c.timeout)
|
||||
defer cancelFunc()
|
||||
log.Debug("DataServiceClient try connect ", zap.String("address", c.addr))
|
||||
conn, err := grpc.DialContext(c.ctx, c.addr, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
conn, err := grpc.DialContext(ctx, c.addr, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
grpc.WithUnaryInterceptor(
|
||||
otgrpc.OpenTracingClientInterceptor(tracer)),
|
||||
grpc.WithStreamInterceptor(
|
||||
|
@ -93,10 +95,10 @@ func (c *Client) Init() error {
|
|||
log.Debug("DataServiceClient connect failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
log.Debug("DataServiceClient connect success")
|
||||
} else {
|
||||
return c.reconnect()
|
||||
}
|
||||
log.Debug("DataServiceClient connect success")
|
||||
c.grpcClient = datapb.NewDataServiceClient(c.conn)
|
||||
|
||||
return nil
|
||||
|
@ -118,8 +120,10 @@ func (c *Client) reconnect() error {
|
|||
return err
|
||||
}
|
||||
connectGrpcFunc := func() error {
|
||||
ctx, cancelFunc := context.WithTimeout(c.ctx, c.timeout)
|
||||
defer cancelFunc()
|
||||
log.Debug("DataServiceClient try reconnect ", zap.String("address", c.addr))
|
||||
conn, err := grpc.DialContext(c.ctx, c.addr, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
conn, err := grpc.DialContext(ctx, c.addr, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
grpc.WithUnaryInterceptor(
|
||||
otgrpc.OpenTracingClientInterceptor(tracer)),
|
||||
grpc.WithStreamInterceptor(
|
||||
|
|
|
@ -93,33 +93,42 @@ func (s *Server) init() error {
|
|||
|
||||
err := s.dataService.Register()
|
||||
if err != nil {
|
||||
log.Debug("DataService Register etcd failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
log.Debug("DataService Register etcd success")
|
||||
|
||||
err = s.startGrpc()
|
||||
if err != nil {
|
||||
log.Debug("DataService startGrpc failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
|
||||
s.dataService.UpdateStateCode(internalpb.StateCode_Initializing)
|
||||
log.Debug("DataService", zap.Any("State", internalpb.StateCode_Initializing))
|
||||
|
||||
if s.newMasterServiceClient != nil {
|
||||
log.Debug("master service", zap.String("address", Params.MasterAddress))
|
||||
log.Debug("DataService try to new master service client", zap.String("address", Params.MasterAddress))
|
||||
masterServiceClient, err := s.newMasterServiceClient(Params.MasterAddress)
|
||||
if err != nil {
|
||||
log.Debug("DataService new master service client failed", zap.Error(err))
|
||||
panic(err)
|
||||
}
|
||||
log.Debug("master service client created")
|
||||
|
||||
if err = masterServiceClient.Init(); err != nil {
|
||||
log.Debug("DataService masterServiceClient Init failed", zap.Error(err))
|
||||
panic(err)
|
||||
}
|
||||
if err = masterServiceClient.Start(); err != nil {
|
||||
log.Debug("DataService masterServiceClient Start failed", zap.Error(err))
|
||||
panic(err)
|
||||
}
|
||||
log.Debug("DataService start to wait for MasterService ready")
|
||||
if err = funcutil.WaitForComponentInitOrHealthy(ctx, masterServiceClient, "MasterService", 1000000, 200*time.Millisecond); err != nil {
|
||||
log.Debug("DataService wait for MasterService Ready failed", zap.Error(err))
|
||||
panic(err)
|
||||
}
|
||||
log.Debug("DataService report MasterService is ready")
|
||||
s.dataService.SetMasterClient(masterServiceClient)
|
||||
}
|
||||
|
||||
|
@ -200,7 +209,7 @@ func (s *Server) Run() error {
|
|||
if err := s.init(); err != nil {
|
||||
return err
|
||||
}
|
||||
log.Debug("dataservice init done ...")
|
||||
log.Debug("DataService init done ...")
|
||||
|
||||
if err := s.start(); err != nil {
|
||||
return err
|
||||
|
|
|
@ -57,8 +57,10 @@ func NewClient(address string, timeout time.Duration) (*Client, error) {
|
|||
func (c *Client) Init() error {
|
||||
tracer := opentracing.GlobalTracer()
|
||||
connectGrpcFunc := func() error {
|
||||
log.Debug("indexnode connect ", zap.String("address", c.address))
|
||||
conn, err := grpc.DialContext(c.ctx, c.address, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
ctx, cancelFunc := context.WithTimeout(c.ctx, c.timeout)
|
||||
defer cancelFunc()
|
||||
log.Debug("IndexNodeClient try connect ", zap.String("address", c.address))
|
||||
conn, err := grpc.DialContext(ctx, c.address, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
grpc.WithUnaryInterceptor(
|
||||
otgrpc.OpenTracingClientInterceptor(tracer)),
|
||||
grpc.WithStreamInterceptor(
|
||||
|
@ -71,8 +73,10 @@ func (c *Client) Init() error {
|
|||
}
|
||||
err := retry.Retry(100000, time.Millisecond*200, connectGrpcFunc)
|
||||
if err != nil {
|
||||
log.Debug("IndexNodeClient try connect failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
log.Debug("IndexNodeClient try connect success", zap.String("address", c.address))
|
||||
c.grpcClient = indexpb.NewIndexNodeClient(c.conn)
|
||||
return nil
|
||||
}
|
||||
|
@ -81,8 +85,10 @@ func (c *Client) reconnect() error {
|
|||
tracer := opentracing.GlobalTracer()
|
||||
var err error
|
||||
connectGrpcFunc := func() error {
|
||||
log.Debug("indexnode connect ", zap.String("address", c.address))
|
||||
conn, err := grpc.DialContext(c.ctx, c.address, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
ctx, cancelFunc := context.WithTimeout(c.ctx, c.timeout)
|
||||
defer cancelFunc()
|
||||
log.Debug("IndexNodeClient try reconnect ", zap.String("address", c.address))
|
||||
conn, err := grpc.DialContext(ctx, c.address, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
grpc.WithUnaryInterceptor(
|
||||
otgrpc.OpenTracingClientInterceptor(tracer)),
|
||||
grpc.WithStreamInterceptor(
|
||||
|
@ -96,8 +102,10 @@ func (c *Client) reconnect() error {
|
|||
|
||||
err = retry.Retry(c.reconnTry, 500*time.Millisecond, connectGrpcFunc)
|
||||
if err != nil {
|
||||
log.Debug("IndexNodeClient try reconnect failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
log.Debug("IndexNodeClient try reconnect success", zap.String("address", c.address))
|
||||
c.grpcClient = indexpb.NewIndexNodeClient(c.conn)
|
||||
return nil
|
||||
}
|
||||
|
|
|
@ -19,6 +19,7 @@ import (
|
|||
"net"
|
||||
"strconv"
|
||||
"sync"
|
||||
"time"
|
||||
|
||||
"go.uber.org/zap"
|
||||
|
||||
|
@ -67,10 +68,10 @@ func (s *Server) startGrpcLoop(grpcPort int) {
|
|||
|
||||
defer s.loopWg.Done()
|
||||
|
||||
log.Debug("indexnode", zap.Int("network port: ", grpcPort))
|
||||
log.Debug("IndexNode", zap.Int("network port: ", grpcPort))
|
||||
lis, err := net.Listen("tcp", ":"+strconv.Itoa(grpcPort))
|
||||
if err != nil {
|
||||
log.Warn("indexnode", zap.String("GrpcServer:failed to listen", err.Error()))
|
||||
log.Warn("IndexNode", zap.String("GrpcServer:failed to listen", err.Error()))
|
||||
s.grpcErrChan <- err
|
||||
return
|
||||
}
|
||||
|
@ -118,15 +119,17 @@ func (s *Server) init() error {
|
|||
if err != nil {
|
||||
err = s.Stop()
|
||||
if err != nil {
|
||||
log.Debug("Init failed, and Stop failed")
|
||||
log.Debug("IndexNode Init failed, and Stop failed")
|
||||
}
|
||||
}
|
||||
}()
|
||||
|
||||
err = s.indexnode.Register()
|
||||
if err != nil {
|
||||
log.Debug("IndexNode Register etcd failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
log.Debug("IndexNode Register etcd success")
|
||||
|
||||
s.loopWg.Add(1)
|
||||
go s.startGrpcLoop(Params.Port)
|
||||
|
@ -137,17 +140,19 @@ func (s *Server) init() error {
|
|||
}
|
||||
|
||||
indexServiceAddr := Params.IndexServerAddress
|
||||
s.indexServiceClient = grpcindexserviceclient.NewClient(indexServiceAddr, indexnode.Params.MetaRootPath, []string{indexnode.Params.EtcdAddress}, 10)
|
||||
s.indexServiceClient = grpcindexserviceclient.NewClient(indexServiceAddr, indexnode.Params.MetaRootPath, []string{indexnode.Params.EtcdAddress}, 10*time.Second)
|
||||
err = s.indexServiceClient.Init()
|
||||
if err != nil {
|
||||
log.Debug("IndexNode indexSerticeClient init failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
s.indexnode.SetIndexServiceClient(s.indexServiceClient)
|
||||
|
||||
s.indexnode.UpdateStateCode(internalpb.StateCode_Initializing)
|
||||
|
||||
log.Debug("IndexNode", zap.Any("State", internalpb.StateCode_Initializing))
|
||||
err = s.indexnode.Init()
|
||||
if err != nil {
|
||||
log.Debug("IndexNode Init failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
return nil
|
||||
|
|
|
@ -50,10 +50,12 @@ func getIndexServiceAddress(sess *sessionutil.Session) (string, error) {
|
|||
key := typeutil.IndexServiceRole
|
||||
msess, _, err := sess.GetSessions(key)
|
||||
if err != nil {
|
||||
log.Debug("IndexServiceClient GetSessions failed", zap.Any("key", key), zap.Error(err))
|
||||
return "", err
|
||||
}
|
||||
ms, ok := msess[key]
|
||||
if !ok {
|
||||
log.Debug("IndexServiceClient msess key not existed", zap.Any("key", key), zap.Any("len of msess", len(msess)))
|
||||
return "", fmt.Errorf("number of master service is incorrect, %d", len(msess))
|
||||
}
|
||||
return ms.Address, nil
|
||||
|
@ -73,10 +75,13 @@ func NewClient(address, metaRoot string, etcdAddr []string, timeout time.Duratio
|
|||
|
||||
func (c *Client) Init() error {
|
||||
tracer := opentracing.GlobalTracer()
|
||||
log.Debug("IndexServiceClient Init", zap.Any("c.address", c.address))
|
||||
if c.address != "" {
|
||||
connectGrpcFunc := func() error {
|
||||
log.Debug("indexservice connect ", zap.String("address", c.address))
|
||||
conn, err := grpc.DialContext(c.ctx, c.address, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
ctx, cancelFunc := context.WithTimeout(c.ctx, c.timeout)
|
||||
defer cancelFunc()
|
||||
log.Debug("IndexServiceClient try connect ", zap.String("address", c.address))
|
||||
conn, err := grpc.DialContext(ctx, c.address, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
grpc.WithUnaryInterceptor(
|
||||
otgrpc.OpenTracingClientInterceptor(tracer)),
|
||||
grpc.WithStreamInterceptor(
|
||||
|
@ -91,9 +96,11 @@ func (c *Client) Init() error {
|
|||
if err != nil {
|
||||
return err
|
||||
}
|
||||
log.Debug("IndexServiceClient try connect failed", zap.Error(err))
|
||||
} else {
|
||||
return c.reconnect()
|
||||
}
|
||||
log.Debug("IndexServiceClient try connect success")
|
||||
c.grpcClient = indexpb.NewIndexServiceClient(c.conn)
|
||||
return nil
|
||||
}
|
||||
|
@ -110,11 +117,15 @@ func (c *Client) reconnect() error {
|
|||
}
|
||||
err = retry.Retry(c.reconnTry, 3*time.Second, getIndexServiceAddressFn)
|
||||
if err != nil {
|
||||
log.Debug("IndexServiceClient getIndexServiceAddress failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
log.Debug("IndexServiceClient getIndexServiceAddress success")
|
||||
connectGrpcFunc := func() error {
|
||||
log.Debug("IndexService connect ", zap.String("address", c.address))
|
||||
conn, err := grpc.DialContext(c.ctx, c.address, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
ctx, cancelFunc := context.WithTimeout(c.ctx, c.timeout)
|
||||
defer cancelFunc()
|
||||
log.Debug("IndexServiceClient try connect ", zap.String("address", c.address))
|
||||
conn, err := grpc.DialContext(ctx, c.address, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
grpc.WithUnaryInterceptor(
|
||||
otgrpc.OpenTracingClientInterceptor(tracer)),
|
||||
grpc.WithStreamInterceptor(
|
||||
|
@ -128,8 +139,10 @@ func (c *Client) reconnect() error {
|
|||
|
||||
err = retry.Retry(c.reconnTry, 500*time.Millisecond, connectGrpcFunc)
|
||||
if err != nil {
|
||||
log.Debug("IndexServiceClient try connect failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
log.Debug("IndexServiceClient connect success")
|
||||
c.grpcClient = indexpb.NewIndexServiceClient(c.conn)
|
||||
return nil
|
||||
}
|
||||
|
|
|
@ -78,7 +78,7 @@ func (s *Server) init() error {
|
|||
|
||||
s.loopWg.Add(1)
|
||||
go s.startGrpcLoop(Params.ServicePort)
|
||||
// wait for grpc indexservice loop start
|
||||
// wait for grpc IndexService loop start
|
||||
if err := <-s.grpcErrChan; err != nil {
|
||||
return err
|
||||
}
|
||||
|
@ -153,10 +153,10 @@ func (s *Server) startGrpcLoop(grpcPort int) {
|
|||
|
||||
defer s.loopWg.Done()
|
||||
|
||||
log.Debug("indexservice", zap.Int("network port", grpcPort))
|
||||
log.Debug("IndexService", zap.Int("network port", grpcPort))
|
||||
lis, err := net.Listen("tcp", ":"+strconv.Itoa(grpcPort))
|
||||
if err != nil {
|
||||
log.Warn("indexservice", zap.String("GrpcServer:failed to listen", err.Error()))
|
||||
log.Warn("IndexService", zap.String("GrpcServer:failed to listen", err.Error()))
|
||||
s.grpcErrChan <- err
|
||||
return
|
||||
}
|
||||
|
|
|
@ -16,6 +16,9 @@ import (
|
|||
"fmt"
|
||||
"time"
|
||||
|
||||
"go.uber.org/zap"
|
||||
|
||||
"github.com/milvus-io/milvus/internal/log"
|
||||
"github.com/milvus-io/milvus/internal/proto/commonpb"
|
||||
"github.com/milvus-io/milvus/internal/proto/internalpb"
|
||||
"github.com/milvus-io/milvus/internal/proto/masterpb"
|
||||
|
@ -27,10 +30,11 @@ import (
|
|||
"google.golang.org/grpc"
|
||||
)
|
||||
|
||||
// grpc client
|
||||
// GrpcClient grpc client
|
||||
type GrpcClient struct {
|
||||
grpcClient masterpb.MasterServiceClient
|
||||
conn *grpc.ClientConn
|
||||
ctx context.Context
|
||||
|
||||
//inner member
|
||||
addr string
|
||||
|
@ -45,10 +49,13 @@ func getMasterServiceAddr(sess *sessionutil.Session) (string, error) {
|
|||
key := typeutil.MasterServiceRole
|
||||
msess, _, err := sess.GetSessions(key)
|
||||
if err != nil {
|
||||
log.Debug("MasterServiceClient GetSessions failed", zap.Any("key", key))
|
||||
return "", err
|
||||
}
|
||||
log.Debug("MasterServiceClient GetSessions success")
|
||||
ms, ok := msess[key]
|
||||
if !ok {
|
||||
log.Debug("MasterServiceClient mess key not exist", zap.Any("key", key))
|
||||
return "", fmt.Errorf("number of master service is incorrect, %d", len(msess))
|
||||
}
|
||||
return ms.Address, nil
|
||||
|
@ -57,12 +64,15 @@ func getMasterServiceAddr(sess *sessionutil.Session) (string, error) {
|
|||
func NewClient(addr string, metaRoot string, etcdAddr []string, timeout time.Duration) (*GrpcClient, error) {
|
||||
sess := sessionutil.NewSession(context.Background(), metaRoot, etcdAddr)
|
||||
if sess == nil {
|
||||
return nil, fmt.Errorf("new session error, maybe can not connect to etcd")
|
||||
err := fmt.Errorf("new session error, maybe can not connect to etcd")
|
||||
log.Debug("MasterServiceClient NewClient failed", zap.Error(err))
|
||||
return nil, err
|
||||
}
|
||||
|
||||
return &GrpcClient{
|
||||
grpcClient: nil,
|
||||
conn: nil,
|
||||
ctx: context.Background(),
|
||||
addr: addr,
|
||||
timeout: timeout,
|
||||
reconnTry: 300,
|
||||
|
@ -74,48 +84,57 @@ func NewClient(addr string, metaRoot string, etcdAddr []string, timeout time.Dur
|
|||
func (c *GrpcClient) reconnect() error {
|
||||
addr, err := getMasterServiceAddr(c.sess)
|
||||
if err != nil {
|
||||
log.Debug("MasterServiceClient getMasterServiceAddr failed", zap.Error(err))
|
||||
return nil
|
||||
}
|
||||
log.Debug("MasterServiceClient getMasterServiceAddr success")
|
||||
tracer := opentracing.GlobalTracer()
|
||||
ctx, cancel := context.WithTimeout(context.Background(), c.timeout)
|
||||
defer cancel()
|
||||
for i := 0; i < c.reconnTry; i++ {
|
||||
ctx, cancelFunc := context.WithTimeout(c.ctx, c.timeout)
|
||||
if c.conn, err = grpc.DialContext(ctx, addr, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
grpc.WithUnaryInterceptor(
|
||||
otgrpc.OpenTracingClientInterceptor(tracer)),
|
||||
grpc.WithStreamInterceptor(
|
||||
otgrpc.OpenTracingStreamClientInterceptor(tracer))); err == nil {
|
||||
cancelFunc()
|
||||
break
|
||||
}
|
||||
cancelFunc()
|
||||
}
|
||||
if err != nil {
|
||||
log.Debug("MasterServiceClient try reconnect failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
log.Debug("MasterServiceClient try reconnect success")
|
||||
c.grpcClient = masterpb.NewMasterServiceClient(c.conn)
|
||||
return nil
|
||||
}
|
||||
|
||||
func (c *GrpcClient) Init() error {
|
||||
tracer := opentracing.GlobalTracer()
|
||||
ctx, cancel := context.WithTimeout(context.Background(), c.timeout)
|
||||
defer cancel()
|
||||
var err error
|
||||
log.Debug("MasterServiceClient Init", zap.Any("c.addr", c.addr))
|
||||
if c.addr != "" {
|
||||
for i := 0; i < c.reconnTry; i++ {
|
||||
for i := 0; i < 10000; i++ {
|
||||
ctx, cancelFunc := context.WithTimeout(c.ctx, c.timeout)
|
||||
if c.conn, err = grpc.DialContext(ctx, c.addr, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
grpc.WithUnaryInterceptor(
|
||||
otgrpc.OpenTracingClientInterceptor(tracer)),
|
||||
grpc.WithStreamInterceptor(
|
||||
otgrpc.OpenTracingStreamClientInterceptor(tracer))); err == nil {
|
||||
cancelFunc()
|
||||
break
|
||||
}
|
||||
cancelFunc()
|
||||
}
|
||||
if err != nil {
|
||||
log.Debug("MasterServiceClient connect to master failed", zap.Error(err))
|
||||
return fmt.Errorf("connect to specific address gprc error")
|
||||
}
|
||||
} else {
|
||||
return c.reconnect()
|
||||
}
|
||||
log.Debug("MasterServiceClient connect to master success")
|
||||
c.grpcClient = masterpb.NewMasterServiceClient(c.conn)
|
||||
return nil
|
||||
}
|
||||
|
@ -149,7 +168,7 @@ func (c *GrpcClient) recall(caller func() (interface{}, error)) (interface{}, er
|
|||
return ret, err
|
||||
}
|
||||
|
||||
// TODO: timeout need to be propagated through ctx
|
||||
// GetComponentStates TODO: timeout need to be propagated through ctx
|
||||
func (c *GrpcClient) GetComponentStates(ctx context.Context) (*internalpb.ComponentStates, error) {
|
||||
ret, err := c.recall(func() (interface{}, error) {
|
||||
return c.grpcClient.GetComponentStates(ctx, &internalpb.GetComponentStatesRequest{})
|
||||
|
@ -163,7 +182,7 @@ func (c *GrpcClient) GetTimeTickChannel(ctx context.Context) (*milvuspb.StringRe
|
|||
return ret.(*milvuspb.StringResponse), err
|
||||
}
|
||||
|
||||
//just define a channel, not used currently
|
||||
// GetStatisticsChannel just define a channel, not used currently
|
||||
func (c *GrpcClient) GetStatisticsChannel(ctx context.Context) (*milvuspb.StringResponse, error) {
|
||||
ret, err := c.recall(func() (interface{}, error) {
|
||||
return c.grpcClient.GetStatisticsChannel(ctx, &internalpb.GetStatisticsChannelRequest{})
|
||||
|
@ -171,7 +190,7 @@ func (c *GrpcClient) GetStatisticsChannel(ctx context.Context) (*milvuspb.String
|
|||
return ret.(*milvuspb.StringResponse), err
|
||||
}
|
||||
|
||||
//receive ddl from rpc and time tick from proxy service, and put them into this channel
|
||||
// GetDdChannel receive ddl from rpc and time tick from proxy service, and put them into this channel
|
||||
func (c *GrpcClient) GetDdChannel(ctx context.Context) (*milvuspb.StringResponse, error) {
|
||||
ret, err := c.recall(func() (interface{}, error) {
|
||||
return c.grpcClient.GetDdChannel(ctx, &internalpb.GetDdChannelRequest{})
|
||||
|
@ -179,7 +198,7 @@ func (c *GrpcClient) GetDdChannel(ctx context.Context) (*milvuspb.StringResponse
|
|||
return ret.(*milvuspb.StringResponse), err
|
||||
}
|
||||
|
||||
//DDL request
|
||||
// CreateCollection DDL request
|
||||
func (c *GrpcClient) CreateCollection(ctx context.Context, in *milvuspb.CreateCollectionRequest) (*commonpb.Status, error) {
|
||||
ret, err := c.recall(func() (interface{}, error) {
|
||||
return c.grpcClient.CreateCollection(ctx, in)
|
||||
|
@ -241,7 +260,7 @@ func (c *GrpcClient) ShowPartitions(ctx context.Context, in *milvuspb.ShowPartit
|
|||
return ret.(*milvuspb.ShowPartitionsResponse), err
|
||||
}
|
||||
|
||||
//index builder service
|
||||
// CreateIndex index builder service
|
||||
func (c *GrpcClient) CreateIndex(ctx context.Context, in *milvuspb.CreateIndexRequest) (*commonpb.Status, error) {
|
||||
ret, err := c.recall(func() (interface{}, error) {
|
||||
return c.grpcClient.CreateIndex(ctx, in)
|
||||
|
@ -263,7 +282,7 @@ func (c *GrpcClient) DescribeIndex(ctx context.Context, in *milvuspb.DescribeInd
|
|||
return ret.(*milvuspb.DescribeIndexResponse), err
|
||||
}
|
||||
|
||||
//global timestamp allocator
|
||||
// AllocTimestamp global timestamp allocator
|
||||
func (c *GrpcClient) AllocTimestamp(ctx context.Context, in *masterpb.AllocTimestampRequest) (*masterpb.AllocTimestampResponse, error) {
|
||||
ret, err := c.recall(func() (interface{}, error) {
|
||||
return c.grpcClient.AllocTimestamp(ctx, in)
|
||||
|
@ -286,7 +305,7 @@ func (c *GrpcClient) UpdateChannelTimeTick(ctx context.Context, in *internalpb.C
|
|||
return ret.(*commonpb.Status), err
|
||||
}
|
||||
|
||||
//receiver time tick from proxy service, and put it into this channel
|
||||
// DescribeSegment receiver time tick from proxy service, and put it into this channel
|
||||
func (c *GrpcClient) DescribeSegment(ctx context.Context, in *milvuspb.DescribeSegmentRequest) (*milvuspb.DescribeSegmentResponse, error) {
|
||||
ret, err := c.recall(func() (interface{}, error) {
|
||||
return c.grpcClient.DescribeSegment(ctx, in)
|
||||
|
|
|
@ -44,7 +44,7 @@ import (
|
|||
"github.com/milvus-io/milvus/internal/util/sessionutil"
|
||||
)
|
||||
|
||||
// grpc wrapper
|
||||
// Server grpc wrapper
|
||||
type Server struct {
|
||||
masterService types.MasterComponent
|
||||
grpcServer *grpc.Server
|
||||
|
@ -171,10 +171,10 @@ func (s *Server) init() error {
|
|||
}
|
||||
|
||||
s.masterService.UpdateStateCode(internalpb.StateCode_Initializing)
|
||||
|
||||
log.Debug("MasterService", zap.Any("State", internalpb.StateCode_Initializing))
|
||||
s.masterService.SetNewProxyClient(
|
||||
func(s *sessionutil.Session) (types.ProxyNode, error) {
|
||||
cli := pnc.NewClient(ctx, s.Address, 10)
|
||||
cli := pnc.NewClient(ctx, s.Address, 10*time.Second)
|
||||
if err := cli.Init(); err != nil {
|
||||
return nil, err
|
||||
}
|
||||
|
@ -186,7 +186,7 @@ func (s *Server) init() error {
|
|||
)
|
||||
|
||||
if s.newProxyServiceClient != nil {
|
||||
log.Debug("proxy service", zap.String("address", Params.ProxyServiceAddress))
|
||||
log.Debug("MasterService start to create ProxyService client", zap.String("address", Params.ProxyServiceAddress))
|
||||
proxyService := s.newProxyServiceClient(Params.ProxyServiceAddress)
|
||||
if err := s.masterService.SetProxyService(ctx, proxyService); err != nil {
|
||||
panic(err)
|
||||
|
@ -194,23 +194,23 @@ func (s *Server) init() error {
|
|||
s.proxyService = proxyService
|
||||
}
|
||||
if s.newDataServiceClient != nil {
|
||||
log.Debug("data service", zap.String("address", Params.DataServiceAddress))
|
||||
dataService := s.newDataServiceClient(Params.DataServiceAddress, cms.Params.MetaRootPath, cms.Params.EtcdAddress, 10)
|
||||
log.Debug("MasterService start to create DataService client", zap.String("address", Params.DataServiceAddress))
|
||||
dataService := s.newDataServiceClient(Params.DataServiceAddress, cms.Params.MetaRootPath, cms.Params.EtcdAddress, 10*time.Second)
|
||||
if err := s.masterService.SetDataService(ctx, dataService); err != nil {
|
||||
panic(err)
|
||||
}
|
||||
s.dataService = dataService
|
||||
}
|
||||
if s.newIndexServiceClient != nil {
|
||||
log.Debug("index service", zap.String("address", Params.IndexServiceAddress))
|
||||
indexService := s.newIndexServiceClient(Params.IndexServiceAddress, cms.Params.MetaRootPath, cms.Params.EtcdAddress, 10)
|
||||
log.Debug("MasterService start to create IndexService client", zap.String("address", Params.IndexServiceAddress))
|
||||
indexService := s.newIndexServiceClient(Params.IndexServiceAddress, cms.Params.MetaRootPath, cms.Params.EtcdAddress, 10*time.Second)
|
||||
if err := s.masterService.SetIndexService(indexService); err != nil {
|
||||
panic(err)
|
||||
}
|
||||
s.indexService = indexService
|
||||
}
|
||||
if s.newQueryServiceClient != nil {
|
||||
log.Debug("query service", zap.String("address", Params.QueryServiceAddress))
|
||||
log.Debug("MasterService start to create QueryService client", zap.String("address", Params.QueryServiceAddress))
|
||||
queryService, _ := s.newQueryServiceClient(Params.QueryServiceAddress, cms.Params.MetaRootPath, cms.Params.EtcdAddress)
|
||||
if err := s.masterService.SetQueryService(queryService); err != nil {
|
||||
panic(err)
|
||||
|
@ -312,22 +312,22 @@ func (s *Server) GetComponentStates(ctx context.Context, req *internalpb.GetComp
|
|||
return s.masterService.GetComponentStates(ctx)
|
||||
}
|
||||
|
||||
//receiver time tick from proxy service, and put it into this channel
|
||||
// GetTimeTickChannel receiver time tick from proxy service, and put it into this channel
|
||||
func (s *Server) GetTimeTickChannel(ctx context.Context, req *internalpb.GetTimeTickChannelRequest) (*milvuspb.StringResponse, error) {
|
||||
return s.masterService.GetTimeTickChannel(ctx)
|
||||
}
|
||||
|
||||
//just define a channel, not used currently
|
||||
// GetStatisticsChannel just define a channel, not used currently
|
||||
func (s *Server) GetStatisticsChannel(ctx context.Context, req *internalpb.GetStatisticsChannelRequest) (*milvuspb.StringResponse, error) {
|
||||
return s.masterService.GetStatisticsChannel(ctx)
|
||||
}
|
||||
|
||||
//receive ddl from rpc and time tick from proxy service, and put them into this channel
|
||||
// GetDdChannel receive ddl from rpc and time tick from proxy service, and put them into this channel
|
||||
func (s *Server) GetDdChannel(ctx context.Context, req *internalpb.GetDdChannelRequest) (*milvuspb.StringResponse, error) {
|
||||
return s.masterService.GetDdChannel(ctx)
|
||||
}
|
||||
|
||||
//DDL request
|
||||
// CreateCollection DDL request
|
||||
func (s *Server) CreateCollection(ctx context.Context, in *milvuspb.CreateCollectionRequest) (*commonpb.Status, error) {
|
||||
return s.masterService.CreateCollection(ctx, in)
|
||||
}
|
||||
|
@ -364,7 +364,7 @@ func (s *Server) ShowPartitions(ctx context.Context, in *milvuspb.ShowPartitions
|
|||
return s.masterService.ShowPartitions(ctx, in)
|
||||
}
|
||||
|
||||
//index builder service
|
||||
// CreateIndex index builder service
|
||||
func (s *Server) CreateIndex(ctx context.Context, in *milvuspb.CreateIndexRequest) (*commonpb.Status, error) {
|
||||
return s.masterService.CreateIndex(ctx, in)
|
||||
}
|
||||
|
@ -377,7 +377,7 @@ func (s *Server) DescribeIndex(ctx context.Context, in *milvuspb.DescribeIndexRe
|
|||
return s.masterService.DescribeIndex(ctx, in)
|
||||
}
|
||||
|
||||
//global timestamp allocator
|
||||
// AllocTimestamp global timestamp allocator
|
||||
func (s *Server) AllocTimestamp(ctx context.Context, in *masterpb.AllocTimestampRequest) (*masterpb.AllocTimestampResponse, error) {
|
||||
return s.masterService.AllocTimestamp(ctx, in)
|
||||
}
|
||||
|
|
|
@ -51,8 +51,10 @@ func NewClient(ctx context.Context, address string, timeout time.Duration) *Clie
|
|||
func (c *Client) Init() error {
|
||||
tracer := opentracing.GlobalTracer()
|
||||
connectGrpcFunc := func() error {
|
||||
ctx, cancelFunc := context.WithTimeout(c.ctx, c.timeout)
|
||||
defer cancelFunc()
|
||||
log.Debug("ProxyNodeClient try connect ", zap.String("address", c.address))
|
||||
conn, err := grpc.DialContext(c.ctx, c.address, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
conn, err := grpc.DialContext(ctx, c.address, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
grpc.WithUnaryInterceptor(
|
||||
otgrpc.OpenTracingClientInterceptor(tracer)),
|
||||
grpc.WithStreamInterceptor(
|
||||
|
@ -76,8 +78,10 @@ func (c *Client) Init() error {
|
|||
func (c *Client) reconnect() error {
|
||||
tracer := opentracing.GlobalTracer()
|
||||
connectGrpcFunc := func() error {
|
||||
ctx, cancelFunc := context.WithTimeout(c.ctx, c.timeout)
|
||||
defer cancelFunc()
|
||||
log.Debug("ProxyNodeClient try reconnect ", zap.String("address", c.address))
|
||||
conn, err := grpc.DialContext(c.ctx, c.address, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
conn, err := grpc.DialContext(ctx, c.address, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
grpc.WithUnaryInterceptor(
|
||||
otgrpc.OpenTracingClientInterceptor(tracer)),
|
||||
grpc.WithStreamInterceptor(
|
||||
|
|
|
@ -155,15 +155,6 @@ func (s *Server) init() error {
|
|||
log.Debug("proxynode", zap.Int("proxy port", Params.Port))
|
||||
log.Debug("proxynode", zap.String("proxy address", Params.Address))
|
||||
|
||||
defer func() {
|
||||
if err != nil {
|
||||
err2 := s.Stop()
|
||||
if err2 != nil {
|
||||
log.Debug("Init failed, and Stop failed")
|
||||
}
|
||||
}
|
||||
}()
|
||||
|
||||
err = s.proxynode.Register()
|
||||
if err != nil {
|
||||
log.Debug("ProxyNode Register etcd failed ", zap.Error(err))
|
||||
|
@ -212,7 +203,7 @@ func (s *Server) init() error {
|
|||
|
||||
dataServiceAddr := Params.DataServiceAddress
|
||||
log.Debug("ProxyNode", zap.String("data service address", dataServiceAddr))
|
||||
s.dataServiceClient = grpcdataserviceclient.NewClient(dataServiceAddr, proxynode.Params.MetaRootPath, []string{proxynode.Params.EtcdAddress}, 10)
|
||||
s.dataServiceClient = grpcdataserviceclient.NewClient(dataServiceAddr, proxynode.Params.MetaRootPath, []string{proxynode.Params.EtcdAddress}, 10*time.Second)
|
||||
err = s.dataServiceClient.Init()
|
||||
if err != nil {
|
||||
log.Debug("ProxyNode dataServiceClient init failed ", zap.Error(err))
|
||||
|
@ -223,7 +214,7 @@ func (s *Server) init() error {
|
|||
|
||||
indexServiceAddr := Params.IndexServerAddress
|
||||
log.Debug("ProxyNode", zap.String("index server address", indexServiceAddr))
|
||||
s.indexServiceClient = grpcindexserviceclient.NewClient(indexServiceAddr, proxynode.Params.MetaRootPath, []string{proxynode.Params.EtcdAddress}, 10)
|
||||
s.indexServiceClient = grpcindexserviceclient.NewClient(indexServiceAddr, proxynode.Params.MetaRootPath, []string{proxynode.Params.EtcdAddress}, 10*time.Second)
|
||||
err = s.indexServiceClient.Init()
|
||||
if err != nil {
|
||||
log.Debug("ProxyNode indexServiceClient init failed ", zap.Error(err))
|
||||
|
|
|
@ -42,9 +42,11 @@ func NewClient(address string) *Client {
|
|||
|
||||
func (c *Client) Init() error {
|
||||
tracer := opentracing.GlobalTracer()
|
||||
log.Debug("proxyservice connect ", zap.String("address", c.address))
|
||||
log.Debug("ProxyServiceClient try connect ", zap.String("address", c.address))
|
||||
connectGrpcFunc := func() error {
|
||||
conn, err := grpc.DialContext(c.ctx, c.address, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
ctx, cancelFunc := context.WithTimeout(c.ctx, time.Second*3)
|
||||
defer cancelFunc()
|
||||
conn, err := grpc.DialContext(ctx, c.address, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
grpc.WithUnaryInterceptor(
|
||||
otgrpc.OpenTracingClientInterceptor(tracer)),
|
||||
grpc.WithStreamInterceptor(
|
||||
|
@ -57,8 +59,10 @@ func (c *Client) Init() error {
|
|||
}
|
||||
err := retry.Retry(100000, time.Millisecond*200, connectGrpcFunc)
|
||||
if err != nil {
|
||||
log.Debug("ProxyServiceClient try connect failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
log.Debug("ProxyServiceClient try connect success")
|
||||
return nil
|
||||
}
|
||||
|
||||
|
|
|
@ -106,10 +106,10 @@ func (s *Server) startGrpcLoop(grpcPort int) {
|
|||
|
||||
defer s.wg.Done()
|
||||
|
||||
log.Debug("proxyservice", zap.Int("network port", grpcPort))
|
||||
log.Debug("ProxyService", zap.Int("network port", grpcPort))
|
||||
lis, err := net.Listen("tcp", ":"+strconv.Itoa(grpcPort))
|
||||
if err != nil {
|
||||
log.Warn("proxyservice", zap.String("GrpcServer:failed to listen", err.Error()))
|
||||
log.Warn("ProxyService", zap.String("GrpcServer:failed to listen", err.Error()))
|
||||
s.grpcErrChan <- err
|
||||
return
|
||||
}
|
||||
|
@ -136,7 +136,7 @@ func (s *Server) startGrpcLoop(grpcPort int) {
|
|||
}
|
||||
|
||||
func (s *Server) start() error {
|
||||
log.Debug("proxy ProxyService start ...")
|
||||
log.Debug("ProxyService start ...")
|
||||
if err := s.proxyservice.Start(); err != nil {
|
||||
return err
|
||||
}
|
||||
|
|
|
@ -29,11 +29,6 @@ import (
|
|||
"go.uber.org/zap"
|
||||
)
|
||||
|
||||
const (
|
||||
RPCConnectionTimeout = 30 * time.Second
|
||||
Retry = 3
|
||||
)
|
||||
|
||||
type Client struct {
|
||||
ctx context.Context
|
||||
grpcClient querypb.QueryNodeClient
|
||||
|
@ -50,16 +45,19 @@ func NewClient(address string) (*Client, error) {
|
|||
return nil, fmt.Errorf("address is empty")
|
||||
}
|
||||
return &Client{
|
||||
ctx: context.Background(),
|
||||
addr: address,
|
||||
ctx: context.Background(),
|
||||
addr: address,
|
||||
timeout: 3 * time.Second,
|
||||
}, nil
|
||||
}
|
||||
|
||||
func (c *Client) Init() error {
|
||||
tracer := opentracing.GlobalTracer()
|
||||
connectGrpcFunc := func() error {
|
||||
log.Debug("querynode connect", zap.String("address", c.addr))
|
||||
conn, err := grpc.DialContext(c.ctx, c.addr, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
ctx, cancelFunc := context.WithTimeout(c.ctx, c.timeout)
|
||||
defer cancelFunc()
|
||||
log.Debug("QueryNodeClient try connect", zap.String("address", c.addr))
|
||||
conn, err := grpc.DialContext(ctx, c.addr, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
grpc.WithUnaryInterceptor(
|
||||
otgrpc.OpenTracingClientInterceptor(tracer)),
|
||||
grpc.WithStreamInterceptor(
|
||||
|
@ -72,8 +70,10 @@ func (c *Client) Init() error {
|
|||
}
|
||||
err := retry.Retry(c.reconnTry, time.Millisecond*200, connectGrpcFunc)
|
||||
if err != nil {
|
||||
log.Debug("QueryNodeClient try connect failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
log.Debug("QueryNodeClient try connect success")
|
||||
c.grpcClient = querypb.NewQueryNodeClient(c.conn)
|
||||
return nil
|
||||
}
|
||||
|
@ -82,8 +82,10 @@ func (c *Client) reconnect() error {
|
|||
tracer := opentracing.GlobalTracer()
|
||||
var err error
|
||||
connectGrpcFunc := func() error {
|
||||
log.Debug("querynode connect ", zap.String("address", c.addr))
|
||||
conn, err := grpc.DialContext(c.ctx, c.addr, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
ctx, cancelFunc := context.WithTimeout(c.ctx, c.timeout)
|
||||
defer cancelFunc()
|
||||
log.Debug("QueryNodeClient try reconnect ", zap.String("address", c.addr))
|
||||
conn, err := grpc.DialContext(ctx, c.addr, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
grpc.WithUnaryInterceptor(
|
||||
otgrpc.OpenTracingClientInterceptor(tracer)),
|
||||
grpc.WithStreamInterceptor(
|
||||
|
@ -97,8 +99,10 @@ func (c *Client) reconnect() error {
|
|||
|
||||
err = retry.Retry(c.reconnTry, 500*time.Millisecond, connectGrpcFunc)
|
||||
if err != nil {
|
||||
log.Debug("QueryNodeClient try reconnect failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
log.Debug("QueryNodeClient try reconnect success")
|
||||
c.grpcClient = querypb.NewQueryNodeClient(c.conn)
|
||||
return nil
|
||||
}
|
||||
|
|
|
@ -104,25 +104,30 @@ func (s *Server) init() error {
|
|||
return err
|
||||
}
|
||||
// --- QueryService ---
|
||||
log.Debug("QueryService", zap.String("address", Params.QueryServiceAddress))
|
||||
log.Debug("Init Query service client ...")
|
||||
log.Debug("QueryNode start to new QueryServiceClient", zap.Any("QueryServiceAddress", Params.QueryServiceAddress))
|
||||
queryService, err := qsc.NewClient(ctx, Params.QueryServiceAddress, qn.Params.MetaRootPath, []string{qn.Params.EtcdAddress}, 20*time.Second)
|
||||
if err != nil {
|
||||
log.Debug("QueryNode new QueryServiceClient failed", zap.Error(err))
|
||||
panic(err)
|
||||
}
|
||||
|
||||
if err = queryService.Init(); err != nil {
|
||||
log.Debug("QueryNode QueryServiceClient Init failed", zap.Error(err))
|
||||
panic(err)
|
||||
}
|
||||
|
||||
if err = queryService.Start(); err != nil {
|
||||
log.Debug("QueryNode QueryServiceClient Start failed", zap.Error(err))
|
||||
panic(err)
|
||||
}
|
||||
|
||||
log.Debug("QueryNode start to wait for QueryService ready")
|
||||
err = funcutil.WaitForComponentInitOrHealthy(ctx, queryService, "QueryService", 1000000, time.Millisecond*200)
|
||||
if err != nil {
|
||||
log.Debug("QueryNode wait for QueryService ready failed", zap.Error(err))
|
||||
panic(err)
|
||||
}
|
||||
log.Debug("QueryNode report QueryService is ready")
|
||||
|
||||
if err := s.SetQueryService(queryService); err != nil {
|
||||
panic(err)
|
||||
|
@ -131,76 +136,88 @@ func (s *Server) init() error {
|
|||
// --- Master Server Client ---
|
||||
//ms.Params.Init()
|
||||
addr := Params.MasterAddress
|
||||
log.Debug("Master service", zap.String("address", addr))
|
||||
log.Debug("Init master service client ...")
|
||||
|
||||
log.Debug("QueryNode start to new MasterServiceClient", zap.Any("QueryServiceAddress", addr))
|
||||
masterService, err := msc.NewClient(addr, qn.Params.MetaRootPath, []string{qn.Params.EtcdAddress}, 20*time.Second)
|
||||
if err != nil {
|
||||
log.Debug("QueryNode new MasterServiceClient failed", zap.Error(err))
|
||||
panic(err)
|
||||
}
|
||||
|
||||
if err = masterService.Init(); err != nil {
|
||||
log.Debug("QueryNode MasterServiceClient Init failed", zap.Error(err))
|
||||
panic(err)
|
||||
}
|
||||
|
||||
if err = masterService.Start(); err != nil {
|
||||
log.Debug("QueryNode MasterServiceClient Start failed", zap.Error(err))
|
||||
panic(err)
|
||||
}
|
||||
|
||||
log.Debug("QueryNode start to wait for MasterService ready")
|
||||
err = funcutil.WaitForComponentHealthy(ctx, masterService, "MasterService", 1000000, time.Millisecond*200)
|
||||
if err != nil {
|
||||
log.Debug("QueryNode wait for MasterService ready failed", zap.Error(err))
|
||||
panic(err)
|
||||
}
|
||||
log.Debug("QueryNode report MasterService is ready")
|
||||
|
||||
if err := s.SetMasterService(masterService); err != nil {
|
||||
panic(err)
|
||||
}
|
||||
|
||||
// --- IndexService ---
|
||||
log.Debug("Index service", zap.String("address", Params.IndexServiceAddress))
|
||||
indexService := isc.NewClient(Params.IndexServiceAddress, qn.Params.MetaRootPath, []string{qn.Params.EtcdAddress}, 10)
|
||||
log.Debug("QueryNode start to new IndexServiceClient", zap.Any("IndexServiceAddress", Params.IndexServiceAddress))
|
||||
indexService := isc.NewClient(Params.IndexServiceAddress, qn.Params.MetaRootPath, []string{qn.Params.EtcdAddress}, 10*time.Second)
|
||||
|
||||
if err := indexService.Init(); err != nil {
|
||||
log.Debug("QueryNode IndexServiceClient Init failed", zap.Error(err))
|
||||
panic(err)
|
||||
}
|
||||
|
||||
if err := indexService.Start(); err != nil {
|
||||
log.Debug("QueryNode IndexServiceClient Start failed", zap.Error(err))
|
||||
panic(err)
|
||||
}
|
||||
// wait indexservice healthy
|
||||
// wait IndexService healthy
|
||||
log.Debug("QueryNode start to wait for IndexService ready")
|
||||
err = funcutil.WaitForComponentHealthy(ctx, indexService, "IndexService", 1000000, time.Millisecond*200)
|
||||
if err != nil {
|
||||
log.Debug("QueryNode wait for IndexService ready failed", zap.Error(err))
|
||||
panic(err)
|
||||
}
|
||||
log.Debug("QueryNode report IndexService is ready")
|
||||
|
||||
if err := s.SetIndexService(indexService); err != nil {
|
||||
panic(err)
|
||||
}
|
||||
|
||||
// --- DataService ---
|
||||
log.Debug("Data service", zap.String("address", Params.DataServiceAddress))
|
||||
log.Debug("QueryNode Init data service client ...")
|
||||
|
||||
dataService := dsc.NewClient(Params.DataServiceAddress, qn.Params.MetaRootPath, []string{qn.Params.EtcdAddress}, 10)
|
||||
log.Debug("QueryNode start to new DataServiceClient", zap.Any("DataServiceAddress", Params.DataServiceAddress))
|
||||
dataService := dsc.NewClient(Params.DataServiceAddress, qn.Params.MetaRootPath, []string{qn.Params.EtcdAddress}, 10*time.Second)
|
||||
if err = dataService.Init(); err != nil {
|
||||
log.Debug("QueryNode DataServiceClient Init failed", zap.Error(err))
|
||||
panic(err)
|
||||
}
|
||||
if err = dataService.Start(); err != nil {
|
||||
log.Debug("QueryNode DataServiceClient Start failed", zap.Error(err))
|
||||
panic(err)
|
||||
}
|
||||
log.Debug("QueryNode start to wait for DataService ready")
|
||||
err = funcutil.WaitForComponentInitOrHealthy(ctx, dataService, "DataService", 1000000, time.Millisecond*200)
|
||||
if err != nil {
|
||||
log.Debug("QueryNode wait for DataService ready failed", zap.Error(err))
|
||||
panic(err)
|
||||
}
|
||||
log.Debug("QueryNode report DataService is ready")
|
||||
|
||||
if err := s.SetDataService(dataService); err != nil {
|
||||
panic(err)
|
||||
}
|
||||
|
||||
s.querynode.UpdateStateCode(internalpb.StateCode_Initializing)
|
||||
|
||||
log.Debug("QueryNode", zap.Any("State", internalpb.StateCode_Initializing))
|
||||
if err := s.querynode.Init(); err != nil {
|
||||
log.Error("querynode init error: ", zap.Error(err))
|
||||
log.Error("QueryNode init error: ", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
return nil
|
||||
|
|
|
@ -47,16 +47,18 @@ func getQueryServiceAddress(sess *sessionutil.Session) (string, error) {
|
|||
key := typeutil.QueryServiceRole
|
||||
msess, _, err := sess.GetSessions(key)
|
||||
if err != nil {
|
||||
log.Debug("QueryServiceClient GetSessions failed", zap.Error(err))
|
||||
return "", err
|
||||
}
|
||||
ms, ok := msess[key]
|
||||
if !ok {
|
||||
log.Debug("QueryServiceClient msess key not existed", zap.Any("key", key))
|
||||
return "", fmt.Errorf("number of master service is incorrect, %d", len(msess))
|
||||
}
|
||||
return ms.Address, nil
|
||||
}
|
||||
|
||||
// NewClient creates a client for queryservice grpc call.
|
||||
// NewClient creates a client for QueryService grpc call.
|
||||
func NewClient(ctx context.Context, address, metaRootPath string, etcdAddr []string, timeout time.Duration) (*Client, error) {
|
||||
sess := sessionutil.NewSession(context.Background(), metaRootPath, etcdAddr)
|
||||
|
||||
|
@ -74,11 +76,12 @@ func NewClient(ctx context.Context, address, metaRootPath string, etcdAddr []str
|
|||
|
||||
func (c *Client) Init() error {
|
||||
tracer := opentracing.GlobalTracer()
|
||||
ctx, cancel := context.WithTimeout(context.Background(), c.timeout)
|
||||
defer cancel()
|
||||
log.Debug("QueryServiceClient try connect QueryService", zap.Any("c.addr", c.addr))
|
||||
if c.addr != "" {
|
||||
connectGrpcFunc := func() error {
|
||||
log.Debug("queryservice connect ", zap.String("address", c.addr))
|
||||
ctx, cancelFunc := context.WithTimeout(c.ctx, c.timeout)
|
||||
defer cancelFunc()
|
||||
log.Debug("QueryServiceClient try connect ", zap.String("address", c.addr))
|
||||
conn, err := grpc.DialContext(ctx, c.addr, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
grpc.WithUnaryInterceptor(
|
||||
otgrpc.OpenTracingClientInterceptor(tracer)),
|
||||
|
@ -93,14 +96,14 @@ func (c *Client) Init() error {
|
|||
|
||||
err := retry.Retry(100000, time.Millisecond*200, connectGrpcFunc)
|
||||
if err != nil {
|
||||
log.Debug("QueryServiceClient try connect failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
} else {
|
||||
return c.reconnect()
|
||||
}
|
||||
|
||||
log.Debug("QueryServiceClient try connect success", zap.Any("QueryServiceAddr", c.addr))
|
||||
c.grpcClient = querypb.NewQueryServiceClient(c.conn)
|
||||
log.Debug("connected to queryService", zap.String("queryService", c.addr))
|
||||
return nil
|
||||
}
|
||||
func (c *Client) reconnect() error {
|
||||
|
@ -115,11 +118,14 @@ func (c *Client) reconnect() error {
|
|||
}
|
||||
err = retry.Retry(c.reconnTry, 3*time.Second, getQueryServiceAddressFn)
|
||||
if err != nil {
|
||||
log.Debug("QueryServiceClient getQueryServiceAddress failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
connectGrpcFunc := func() error {
|
||||
log.Debug("QueryService connect ", zap.String("address", c.addr))
|
||||
conn, err := grpc.DialContext(c.ctx, c.addr, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
ctx, cancelFunc := context.WithTimeout(c.ctx, c.timeout)
|
||||
defer cancelFunc()
|
||||
log.Debug("QueryServiceClient try reconnect ", zap.String("address", c.addr))
|
||||
conn, err := grpc.DialContext(ctx, c.addr, grpc.WithInsecure(), grpc.WithBlock(),
|
||||
grpc.WithUnaryInterceptor(
|
||||
otgrpc.OpenTracingClientInterceptor(tracer)),
|
||||
grpc.WithStreamInterceptor(
|
||||
|
@ -133,8 +139,10 @@ func (c *Client) reconnect() error {
|
|||
|
||||
err = retry.Retry(c.reconnTry, 500*time.Millisecond, connectGrpcFunc)
|
||||
if err != nil {
|
||||
log.Debug("QueryServiceClient try reconnect failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
log.Debug("QueryServiceClient try reconnect success")
|
||||
c.grpcClient = querypb.NewQueryServiceClient(c.conn)
|
||||
return nil
|
||||
}
|
||||
|
|
|
@ -79,7 +79,7 @@ func (s *Server) Run() error {
|
|||
if err := s.init(); err != nil {
|
||||
return err
|
||||
}
|
||||
log.Debug("queryservice init done ...")
|
||||
log.Debug("QueryService init done ...")
|
||||
|
||||
if err := s.start(); err != nil {
|
||||
return err
|
||||
|
@ -108,53 +108,60 @@ func (s *Server) init() error {
|
|||
}
|
||||
|
||||
// --- Master Server Client ---
|
||||
log.Debug("Master service", zap.String("address", Params.MasterAddress))
|
||||
log.Debug("Init master service client ...")
|
||||
|
||||
log.Debug("QueryService try to new MasterService client", zap.Any("MasterServiceAddress", Params.MasterAddress))
|
||||
masterService, err := msc.NewClient(Params.MasterAddress, qs.Params.MetaRootPath, []string{qs.Params.EtcdAddress}, 20*time.Second)
|
||||
|
||||
if err != nil {
|
||||
log.Debug("QueryService try to new MasterService client failed", zap.Error(err))
|
||||
panic(err)
|
||||
}
|
||||
|
||||
if err = masterService.Init(); err != nil {
|
||||
log.Debug("QueryService MasterServiceClient Init failed", zap.Error(err))
|
||||
panic(err)
|
||||
}
|
||||
|
||||
if err = masterService.Start(); err != nil {
|
||||
log.Debug("QueryService MasterServiceClient Start failed", zap.Error(err))
|
||||
panic(err)
|
||||
}
|
||||
// wait for master init or healthy
|
||||
log.Debug("QueryService try to wait for MasterService ready")
|
||||
err = funcutil.WaitForComponentInitOrHealthy(ctx, masterService, "MasterService", 1000000, time.Millisecond*200)
|
||||
if err != nil {
|
||||
log.Debug("QueryService wait for MasterService ready failed", zap.Error(err))
|
||||
panic(err)
|
||||
}
|
||||
|
||||
if err := s.SetMasterService(masterService); err != nil {
|
||||
panic(err)
|
||||
}
|
||||
log.Debug("QueryService report MasterService ready")
|
||||
|
||||
// --- Data service client ---
|
||||
log.Debug("DataService", zap.String("Address", Params.DataServiceAddress))
|
||||
log.Debug("QueryService Init data service client ...")
|
||||
log.Debug("QueryService try to new DataService client", zap.Any("DataServiceAddress", Params.DataServiceAddress))
|
||||
|
||||
dataService := dsc.NewClient(Params.DataServiceAddress, qs.Params.MetaRootPath, []string{qs.Params.EtcdAddress}, 10)
|
||||
dataService := dsc.NewClient(Params.DataServiceAddress, qs.Params.MetaRootPath, []string{qs.Params.EtcdAddress}, 10*time.Second)
|
||||
if err = dataService.Init(); err != nil {
|
||||
log.Debug("QueryService DataServiceClient Init failed", zap.Error(err))
|
||||
panic(err)
|
||||
}
|
||||
if err = dataService.Start(); err != nil {
|
||||
log.Debug("QueryService DataServiceClient Start failed", zap.Error(err))
|
||||
panic(err)
|
||||
}
|
||||
log.Debug("QueryService try to wait for DataService ready")
|
||||
err = funcutil.WaitForComponentInitOrHealthy(ctx, dataService, "DataService", 1000000, time.Millisecond*200)
|
||||
if err != nil {
|
||||
log.Debug("QueryService wait for DataService ready failed", zap.Error(err))
|
||||
panic(err)
|
||||
}
|
||||
if err := s.SetDataService(dataService); err != nil {
|
||||
panic(err)
|
||||
}
|
||||
log.Debug("QueryService report DataService ready")
|
||||
|
||||
s.queryservice.UpdateStateCode(internalpb.StateCode_Initializing)
|
||||
|
||||
log.Debug("QueryService", zap.Any("State", internalpb.StateCode_Initializing))
|
||||
if err := s.queryservice.Init(); err != nil {
|
||||
return err
|
||||
}
|
||||
|
|
|
@ -17,6 +17,7 @@ import (
|
|||
"io"
|
||||
"math/rand"
|
||||
"strconv"
|
||||
"sync/atomic"
|
||||
"time"
|
||||
|
||||
"github.com/milvus-io/milvus/internal/util/retry"
|
||||
|
@ -42,7 +43,7 @@ type UniqueID = typeutil.UniqueID
|
|||
type Timestamp = typeutil.Timestamp
|
||||
|
||||
type IndexNode struct {
|
||||
stateCode internalpb.StateCode
|
||||
stateCode atomic.Value
|
||||
|
||||
loopCtx context.Context
|
||||
loopCancel func()
|
||||
|
@ -98,13 +99,18 @@ func (i *IndexNode) Init() error {
|
|||
}
|
||||
err := retry.Retry(100000, time.Millisecond*200, connectEtcdFn)
|
||||
if err != nil {
|
||||
log.Debug("IndexNode try connect etcd failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
log.Debug("IndexNode try connect etcd success")
|
||||
log.Debug("IndexNode start to wait for IndexService ready")
|
||||
|
||||
err = funcutil.WaitForComponentHealthy(ctx, i.serviceClient, "IndexService", 1000000, time.Millisecond*200)
|
||||
if err != nil {
|
||||
log.Debug("IndexNode wait for IndexService ready failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
log.Debug("IndexNode report IndexService is ready")
|
||||
request := &indexpb.RegisterNodeRequest{
|
||||
Base: nil,
|
||||
Address: &commonpb.Address{
|
||||
|
@ -116,18 +122,21 @@ func (i *IndexNode) Init() error {
|
|||
|
||||
resp, err2 := i.serviceClient.RegisterNode(ctx, request)
|
||||
if err2 != nil {
|
||||
log.Debug("indexnode", zap.String("Index NodeImpl connect to IndexService failed", err.Error()))
|
||||
log.Debug("IndexNode RegisterNode failed", zap.Error(err2))
|
||||
return err2
|
||||
}
|
||||
|
||||
if resp.Status.ErrorCode != commonpb.ErrorCode_Success {
|
||||
log.Debug("IndexNode RegisterNode failed", zap.String("Reason", resp.Status.Reason))
|
||||
return errors.New(resp.Status.Reason)
|
||||
}
|
||||
|
||||
err = Params.LoadConfigFromInitParams(resp.InitParams)
|
||||
if err != nil {
|
||||
log.Debug("IndexNode LoadConfigFromInitParams failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
log.Debug("IndexNode LoadConfigFromInitParams success")
|
||||
|
||||
option := &miniokv.Option{
|
||||
Address: Params.MinIOAddress,
|
||||
|
@ -139,11 +148,13 @@ func (i *IndexNode) Init() error {
|
|||
}
|
||||
i.kv, err = miniokv.NewMinIOKV(i.loopCtx, option)
|
||||
if err != nil {
|
||||
log.Debug("IndexNode NewMinIOKV failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
log.Debug("IndexNode NewMinIOKV success")
|
||||
|
||||
i.UpdateStateCode(internalpb.StateCode_Healthy)
|
||||
|
||||
log.Debug("IndexNode", zap.Any("State", i.stateCode.Load()))
|
||||
return nil
|
||||
}
|
||||
|
||||
|
@ -157,7 +168,7 @@ func (i *IndexNode) Start() error {
|
|||
return nil
|
||||
}
|
||||
|
||||
// Close closes the server.
|
||||
// Stop Close closes the server.
|
||||
func (i *IndexNode) Stop() error {
|
||||
i.loopCancel()
|
||||
if i.sched != nil {
|
||||
|
@ -171,7 +182,7 @@ func (i *IndexNode) Stop() error {
|
|||
}
|
||||
|
||||
func (i *IndexNode) UpdateStateCode(code internalpb.StateCode) {
|
||||
i.stateCode = code
|
||||
i.stateCode.Store(code)
|
||||
}
|
||||
|
||||
func (i *IndexNode) SetIndexServiceClient(serviceClient types.IndexService) {
|
||||
|
@ -179,7 +190,7 @@ func (i *IndexNode) SetIndexServiceClient(serviceClient types.IndexService) {
|
|||
}
|
||||
|
||||
func (i *IndexNode) CreateIndex(ctx context.Context, request *indexpb.CreateIndexRequest) (*commonpb.Status, error) {
|
||||
log.Debug("indexnode building index ...",
|
||||
log.Debug("IndexNode building index ...",
|
||||
zap.Int64("IndexBuildID", request.IndexBuildID),
|
||||
zap.String("Indexname", request.IndexName),
|
||||
zap.Int64("IndexID", request.IndexID),
|
||||
|
@ -227,11 +238,11 @@ func (i *IndexNode) AddCloseCallback(callbacks ...func()) {
|
|||
}
|
||||
|
||||
func (i *IndexNode) GetComponentStates(ctx context.Context) (*internalpb.ComponentStates, error) {
|
||||
log.Debug("get indexnode components states ...")
|
||||
log.Debug("get IndexNode components states ...")
|
||||
stateInfo := &internalpb.ComponentInfo{
|
||||
NodeID: Params.NodeID,
|
||||
Role: "NodeImpl",
|
||||
StateCode: i.stateCode,
|
||||
StateCode: i.stateCode.Load().(internalpb.StateCode),
|
||||
}
|
||||
|
||||
ret := &internalpb.ComponentStates{
|
||||
|
@ -242,7 +253,7 @@ func (i *IndexNode) GetComponentStates(ctx context.Context) (*internalpb.Compone
|
|||
},
|
||||
}
|
||||
|
||||
log.Debug("indexnode compoents states",
|
||||
log.Debug("IndexNode compoents states",
|
||||
zap.Any("State", ret.State),
|
||||
zap.Any("Status", ret.Status),
|
||||
zap.Any("SubcomponentStates", ret.SubcomponentStates))
|
||||
|
|
|
@ -17,6 +17,7 @@ import (
|
|||
"math/rand"
|
||||
"strconv"
|
||||
"sync"
|
||||
"sync/atomic"
|
||||
"time"
|
||||
|
||||
"github.com/coreos/etcd/mvcc/mvccpb"
|
||||
|
@ -49,7 +50,7 @@ const (
|
|||
type IndexService struct {
|
||||
nodeClients *PriorityQueue
|
||||
nodeStates map[UniqueID]*internalpb.ComponentStates
|
||||
stateCode internalpb.StateCode
|
||||
stateCode atomic.Value
|
||||
|
||||
ID UniqueID
|
||||
|
||||
|
@ -104,7 +105,7 @@ func (i *IndexService) Register() error {
|
|||
}
|
||||
|
||||
func (i *IndexService) Init() error {
|
||||
log.Debug("indexservice", zap.String("etcd address", Params.EtcdAddress))
|
||||
log.Debug("IndexService", zap.String("etcd address", Params.EtcdAddress))
|
||||
|
||||
i.assignChan = make(chan []UniqueID, 1024)
|
||||
connectEtcdFn := func() error {
|
||||
|
@ -120,15 +121,19 @@ func (i *IndexService) Init() error {
|
|||
i.metaTable = metakv
|
||||
return err
|
||||
}
|
||||
log.Debug("IndexService try to connect etcd")
|
||||
err := retry.Retry(100000, time.Millisecond*200, connectEtcdFn)
|
||||
if err != nil {
|
||||
log.Debug("IndexService try to connect etcd failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
log.Debug("IndexService try to connect etcd success")
|
||||
|
||||
//init idAllocator
|
||||
kvRootPath := Params.KvRootPath
|
||||
i.idAllocator = allocator.NewGlobalIDAllocator("idTimestamp", tsoutil.NewTSOKVBase([]string{Params.EtcdAddress}, kvRootPath, "index_gid"))
|
||||
if err := i.idAllocator.Initialize(); err != nil {
|
||||
log.Debug("IndexService idAllocator initialize failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
|
||||
|
@ -148,21 +153,28 @@ func (i *IndexService) Init() error {
|
|||
|
||||
i.kv, err = miniokv.NewMinIOKV(i.loopCtx, option)
|
||||
if err != nil {
|
||||
log.Debug("IndexService new minio kv failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
log.Debug("IndexService new minio kv success")
|
||||
|
||||
i.sched, err = NewTaskScheduler(i.loopCtx, i.idAllocator, i.kv, i.metaTable)
|
||||
if err != nil {
|
||||
log.Debug("IndexService new task scheduler failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
log.Debug("IndexService new task scheduler success")
|
||||
i.UpdateStateCode(internalpb.StateCode_Healthy)
|
||||
log.Debug("IndexService", zap.Any("State", i.stateCode.Load()))
|
||||
|
||||
i.nodeTasks = NewNodeTasks()
|
||||
|
||||
err = i.assignTasksServerStart()
|
||||
if err != nil {
|
||||
log.Debug("IndexService assign tasks server start failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
log.Debug("IndexService assign tasks server success", zap.Error(err))
|
||||
return nil
|
||||
}
|
||||
|
||||
|
@ -202,15 +214,15 @@ func (i *IndexService) Stop() error {
|
|||
}
|
||||
|
||||
func (i *IndexService) UpdateStateCode(code internalpb.StateCode) {
|
||||
i.stateCode = code
|
||||
i.stateCode.Store(code)
|
||||
}
|
||||
|
||||
func (i *IndexService) GetComponentStates(ctx context.Context) (*internalpb.ComponentStates, error) {
|
||||
log.Debug("get indexservice component states ...")
|
||||
log.Debug("get IndexService component states ...")
|
||||
stateInfo := &internalpb.ComponentInfo{
|
||||
NodeID: i.ID,
|
||||
Role: "IndexService",
|
||||
StateCode: i.stateCode,
|
||||
StateCode: i.stateCode.Load().(internalpb.StateCode),
|
||||
}
|
||||
|
||||
ret := &internalpb.ComponentStates{
|
||||
|
@ -224,7 +236,7 @@ func (i *IndexService) GetComponentStates(ctx context.Context) (*internalpb.Comp
|
|||
}
|
||||
|
||||
func (i *IndexService) GetTimeTickChannel(ctx context.Context) (*milvuspb.StringResponse, error) {
|
||||
log.Debug("get indexservice time tick channel ...")
|
||||
log.Debug("get IndexService time tick channel ...")
|
||||
return &milvuspb.StringResponse{
|
||||
Status: &commonpb.Status{
|
||||
ErrorCode: commonpb.ErrorCode_Success,
|
||||
|
@ -235,7 +247,7 @@ func (i *IndexService) GetTimeTickChannel(ctx context.Context) (*milvuspb.String
|
|||
}
|
||||
|
||||
func (i *IndexService) GetStatisticsChannel(ctx context.Context) (*milvuspb.StringResponse, error) {
|
||||
log.Debug("get indexservice statistics channel ...")
|
||||
log.Debug("get IndexService statistics channel ...")
|
||||
return &milvuspb.StringResponse{
|
||||
Status: &commonpb.Status{
|
||||
ErrorCode: commonpb.ErrorCode_Success,
|
||||
|
@ -319,7 +331,7 @@ func (i *IndexService) BuildIndex(ctx context.Context, req *indexpb.BuildIndexRe
|
|||
}
|
||||
|
||||
func (i *IndexService) GetIndexStates(ctx context.Context, req *indexpb.GetIndexStatesRequest) (*indexpb.GetIndexStatesResponse, error) {
|
||||
log.Debug("get index states ...", zap.Int64s("IndexBuildIDs", req.IndexBuildIDs))
|
||||
log.Debug("IndexService get index states ...", zap.Int64s("IndexBuildIDs", req.IndexBuildIDs))
|
||||
var indexStates []*indexpb.IndexInfo
|
||||
for _, indexID := range req.IndexBuildIDs {
|
||||
indexState, err := i.metaTable.GetIndexState(indexID)
|
||||
|
@ -369,7 +381,7 @@ func (i *IndexService) DropIndex(ctx context.Context, req *indexpb.DropIndexRequ
|
|||
}
|
||||
|
||||
func (i *IndexService) GetIndexFilePaths(ctx context.Context, req *indexpb.GetIndexFilePathsRequest) (*indexpb.GetIndexFilePathsResponse, error) {
|
||||
log.Debug("indexservice", zap.Int64s("get index file paths", req.IndexBuildIDs))
|
||||
log.Debug("IndexService", zap.Int64s("get index file paths", req.IndexBuildIDs))
|
||||
var indexPaths []*indexpb.IndexFilePathInfo = nil
|
||||
|
||||
for _, indexID := range req.IndexBuildIDs {
|
||||
|
@ -379,7 +391,7 @@ func (i *IndexService) GetIndexFilePaths(ctx context.Context, req *indexpb.GetIn
|
|||
}
|
||||
indexPaths = append(indexPaths, indexPathInfo)
|
||||
}
|
||||
log.Debug("indexservice, get index file paths success")
|
||||
log.Debug("IndexService, get index file paths success")
|
||||
|
||||
ret := &indexpb.GetIndexFilePathsResponse{
|
||||
Status: &commonpb.Status{
|
||||
|
@ -387,7 +399,7 @@ func (i *IndexService) GetIndexFilePaths(ctx context.Context, req *indexpb.GetIn
|
|||
},
|
||||
FilePaths: indexPaths,
|
||||
}
|
||||
log.Debug("indexservice", zap.Any("index file paths", ret.FilePaths))
|
||||
log.Debug("IndexService", zap.Any("index file paths", ret.FilePaths))
|
||||
|
||||
return ret, nil
|
||||
}
|
||||
|
@ -402,7 +414,7 @@ func (i *IndexService) tsLoop() {
|
|||
select {
|
||||
case <-tsoTicker.C:
|
||||
if err := i.idAllocator.UpdateID(); err != nil {
|
||||
log.Debug("indexservice", zap.String("failed to update id", err.Error()))
|
||||
log.Debug("IndexService", zap.String("failed to update id", err.Error()))
|
||||
return
|
||||
}
|
||||
case <-ctx.Done():
|
||||
|
|
|
@ -15,6 +15,7 @@ import (
|
|||
"context"
|
||||
"errors"
|
||||
"strconv"
|
||||
"time"
|
||||
|
||||
"go.uber.org/zap"
|
||||
|
||||
|
@ -41,7 +42,7 @@ func (i *IndexService) addNode(nodeID UniqueID, req *indexpb.RegisterNodeRequest
|
|||
}
|
||||
|
||||
nodeAddress := req.Address.Ip + ":" + strconv.FormatInt(req.Address.Port, 10)
|
||||
nodeClient, err := grpcindexnodeclient.NewClient(nodeAddress, 10)
|
||||
nodeClient, err := grpcindexnodeclient.NewClient(nodeAddress, 10*time.Second)
|
||||
if err != nil {
|
||||
return err
|
||||
}
|
||||
|
|
|
@ -81,7 +81,7 @@ func metricProxyNode(v int64) string {
|
|||
return fmt.Sprintf("client_%d", v)
|
||||
}
|
||||
|
||||
// master core
|
||||
// Core master core
|
||||
type Core struct {
|
||||
/*
|
||||
ProxyServiceClient Interface:
|
||||
|
@ -150,7 +150,7 @@ type Core struct {
|
|||
//dd request scheduler
|
||||
ddReqQueue chan reqTask //dd request will be push into this chan
|
||||
|
||||
//proxynode manager
|
||||
//ProxyNode manager
|
||||
proxyNodeManager *proxyNodeManager
|
||||
|
||||
// proxy clients
|
||||
|
@ -1155,18 +1155,21 @@ func (c *Core) reSendDdMsg(ctx context.Context) error {
|
|||
|
||||
func (c *Core) Start() error {
|
||||
if err := c.checkInit(); err != nil {
|
||||
log.Debug("MasterService Start checkInit failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
|
||||
log.Debug("master", zap.Int64("node id", c.session.ServerID))
|
||||
log.Debug("master", zap.String("dd channel name", Params.DdChannel))
|
||||
log.Debug("master", zap.String("time tick channel name", Params.TimeTickChannel))
|
||||
log.Debug("MasterService", zap.Int64("node id", c.session.ServerID))
|
||||
log.Debug("MasterService", zap.String("dd channel name", Params.DdChannel))
|
||||
log.Debug("MasterService", zap.String("time tick channel name", Params.TimeTickChannel))
|
||||
|
||||
c.startOnce.Do(func() {
|
||||
if err := c.proxyNodeManager.WatchProxyNode(); err != nil {
|
||||
log.Debug("MasterService Start WatchProxyNode failed", zap.Error(err))
|
||||
return
|
||||
}
|
||||
if err := c.reSendDdMsg(c.ctx); err != nil {
|
||||
log.Debug("MasterService Start reSendDdMsg failed", zap.Error(err))
|
||||
return
|
||||
}
|
||||
go c.startDdScheduler()
|
||||
|
@ -1178,7 +1181,7 @@ func (c *Core) Start() error {
|
|||
go c.chanTimeTick.StartWatch()
|
||||
c.stateCode.Store(internalpb.StateCode_Healthy)
|
||||
})
|
||||
log.Debug("Master service", zap.String("State Code", internalpb.StateCode_name[int32(internalpb.StateCode_Healthy)]))
|
||||
log.Debug("MasterService", zap.String("State Code", internalpb.StateCode_name[int32(internalpb.StateCode_Healthy)]))
|
||||
return nil
|
||||
}
|
||||
|
||||
|
|
|
@ -79,8 +79,7 @@ func NewProxyNode(ctx context.Context, factory msgstream.Factory) (*ProxyNode, e
|
|||
msFactory: factory,
|
||||
}
|
||||
node.UpdateStateCode(internalpb.StateCode_Abnormal)
|
||||
log.Debug("ProxyNode",
|
||||
zap.Any("State", "Abnormal"))
|
||||
log.Debug("ProxyNode", zap.Any("State", node.stateCode.Load()))
|
||||
return node, nil
|
||||
|
||||
}
|
||||
|
@ -265,8 +264,7 @@ func (node *ProxyNode) Start() error {
|
|||
}
|
||||
|
||||
node.UpdateStateCode(internalpb.StateCode_Healthy)
|
||||
log.Debug("ProxyNode",
|
||||
zap.Any("State", internalpb.StateCode_Healthy))
|
||||
log.Debug("ProxyNode", zap.Any("State", node.stateCode.Load()))
|
||||
|
||||
return nil
|
||||
}
|
||||
|
|
|
@ -108,9 +108,10 @@ func (s *ProxyService) fillNodeInitParams() error {
|
|||
func (s *ProxyService) Init() error {
|
||||
err := s.fillNodeInitParams()
|
||||
if err != nil {
|
||||
log.Debug("ProxyService fillNodeInitParams failed", zap.Error(err))
|
||||
return err
|
||||
}
|
||||
log.Debug("fill node init params ...")
|
||||
log.Debug("ProxyService fillNodeInitParams success ...")
|
||||
|
||||
m := map[string]interface{}{
|
||||
"PulsarAddress": Params.PulsarAddress,
|
||||
|
@ -123,8 +124,7 @@ func (s *ProxyService) Init() error {
|
|||
|
||||
serviceTimeTickMsgStream, _ := s.msFactory.NewTtMsgStream(s.ctx)
|
||||
serviceTimeTickMsgStream.AsProducer([]string{Params.ServiceTimeTickChannel})
|
||||
log.Debug("proxyservice", zap.Strings("proxyservice AsProducer", []string{Params.ServiceTimeTickChannel}))
|
||||
log.Debug("proxyservice", zap.Strings("create service time tick producer channel", []string{Params.ServiceTimeTickChannel}))
|
||||
log.Debug("ProxyService AsProducer", zap.Strings("channels", []string{Params.ServiceTimeTickChannel}))
|
||||
|
||||
channels := make([]string, Params.InsertChannelNum)
|
||||
var i int64 = 0
|
||||
|
@ -133,27 +133,26 @@ func (s *ProxyService) Init() error {
|
|||
}
|
||||
insertTickMsgStream, _ := s.msFactory.NewMsgStream(s.ctx)
|
||||
insertTickMsgStream.AsProducer(channels)
|
||||
log.Debug("proxyservice", zap.Strings("create insert time tick producer channels", channels))
|
||||
log.Debug("ProxyService", zap.Strings("create insert time tick producer channels", channels))
|
||||
|
||||
nodeTimeTickMsgStream, _ := s.msFactory.NewMsgStream(s.ctx)
|
||||
nodeTimeTickMsgStream.AsConsumer(Params.NodeTimeTickChannel,
|
||||
"proxyservicesub") // TODO: add config
|
||||
log.Debug("proxyservice", zap.Strings("create node time tick consumer channel", Params.NodeTimeTickChannel))
|
||||
nodeTimeTickMsgStream.AsConsumer(Params.NodeTimeTickChannel, "proxyservicesub") // TODO: add config
|
||||
log.Debug("ProxyService", zap.Strings("NodeTimeTickChannel", Params.NodeTimeTickChannel))
|
||||
|
||||
ttBarrier := timesync.NewSoftTimeTickBarrier(s.ctx, nodeTimeTickMsgStream, []UniqueID{1}, 10)
|
||||
log.Debug("create soft time tick barrier ...")
|
||||
log.Debug("ProxyService create soft time tick barrier ...")
|
||||
s.tick = newTimeTick(s.ctx, ttBarrier, serviceTimeTickMsgStream, insertTickMsgStream)
|
||||
log.Debug("create time tick ...")
|
||||
log.Debug("ProxyService create time tick ...")
|
||||
|
||||
return nil
|
||||
}
|
||||
|
||||
func (s *ProxyService) Start() error {
|
||||
s.sched.Start()
|
||||
log.Debug("start scheduler ...")
|
||||
log.Debug("ProxyService start scheduler ...")
|
||||
defer func() {
|
||||
s.UpdateStateCode(internalpb.StateCode_Healthy)
|
||||
log.Debug("proxyservice", zap.Any("state of proxyservice", internalpb.StateCode_Healthy))
|
||||
log.Debug("ProxyService", zap.Any("State", s.stateCode.Load()))
|
||||
}()
|
||||
return s.tick.Start()
|
||||
}
|
||||
|
@ -179,7 +178,7 @@ func (s *ProxyService) GetComponentStates(ctx context.Context) (*internalpb.Comp
|
|||
stateInfo := &internalpb.ComponentInfo{
|
||||
NodeID: UniqueID(0),
|
||||
Role: "ProxyService",
|
||||
StateCode: s.stateCode,
|
||||
StateCode: s.stateCode.Load().(internalpb.StateCode),
|
||||
}
|
||||
|
||||
ret := &internalpb.ComponentStates{
|
||||
|
@ -193,7 +192,7 @@ func (s *ProxyService) GetComponentStates(ctx context.Context) (*internalpb.Comp
|
|||
}
|
||||
|
||||
func (s *ProxyService) UpdateStateCode(code internalpb.StateCode) {
|
||||
s.stateCode = code
|
||||
s.stateCode.Store(code)
|
||||
}
|
||||
|
||||
func (s *ProxyService) GetTimeTickChannel(ctx context.Context) (*milvuspb.StringResponse, error) {
|
||||
|
@ -210,7 +209,7 @@ func (s *ProxyService) GetStatisticsChannel(ctx context.Context) (*milvuspb.Stri
|
|||
}
|
||||
|
||||
func (s *ProxyService) RegisterLink(ctx context.Context) (*milvuspb.RegisterLinkResponse, error) {
|
||||
log.Debug("register link")
|
||||
log.Debug("ProxyService RegisterLink")
|
||||
|
||||
t := ®isterLinkTask{
|
||||
ctx: ctx,
|
||||
|
@ -222,6 +221,7 @@ func (s *ProxyService) RegisterLink(ctx context.Context) (*milvuspb.RegisterLink
|
|||
|
||||
err = s.sched.RegisterLinkTaskQueue.Enqueue(t)
|
||||
if err != nil {
|
||||
log.Debug("ProxyService RegisterLink Enqueue failed", zap.Error(err))
|
||||
return &milvuspb.RegisterLinkResponse{
|
||||
Status: &commonpb.Status{
|
||||
ErrorCode: commonpb.ErrorCode_UnexpectedError,
|
||||
|
@ -233,6 +233,7 @@ func (s *ProxyService) RegisterLink(ctx context.Context) (*milvuspb.RegisterLink
|
|||
|
||||
err = t.WaitToFinish()
|
||||
if err != nil {
|
||||
log.Debug("ProxyService RegisterLink WaitToFinish failed", zap.Error(err))
|
||||
return &milvuspb.RegisterLinkResponse{
|
||||
Status: &commonpb.Status{
|
||||
ErrorCode: commonpb.ErrorCode_UnexpectedError,
|
||||
|
@ -241,12 +242,12 @@ func (s *ProxyService) RegisterLink(ctx context.Context) (*milvuspb.RegisterLink
|
|||
Address: nil,
|
||||
}, nil
|
||||
}
|
||||
|
||||
log.Debug("ProxyService rRegisterLink WaitToFinish failed", zap.Error(err))
|
||||
return t.response, nil
|
||||
}
|
||||
|
||||
func (s *ProxyService) RegisterNode(ctx context.Context, request *proxypb.RegisterNodeRequest) (*proxypb.RegisterNodeResponse, error) {
|
||||
log.Debug("proxyservice receive RegisterNode request",
|
||||
log.Debug("ProxyService receive RegisterNode request",
|
||||
zap.String("ip", request.Address.Ip),
|
||||
zap.Int64("port", request.Address.Port))
|
||||
|
||||
|
@ -263,6 +264,7 @@ func (s *ProxyService) RegisterNode(ctx context.Context, request *proxypb.Regist
|
|||
|
||||
err = s.sched.RegisterNodeTaskQueue.Enqueue(t)
|
||||
if err != nil {
|
||||
log.Debug("ProxyService RegisterNode Enqueue failed", zap.Error(err))
|
||||
return &proxypb.RegisterNodeResponse{
|
||||
Status: &commonpb.Status{
|
||||
ErrorCode: commonpb.ErrorCode_UnexpectedError,
|
||||
|
@ -274,6 +276,7 @@ func (s *ProxyService) RegisterNode(ctx context.Context, request *proxypb.Regist
|
|||
|
||||
err = t.WaitToFinish()
|
||||
if err != nil {
|
||||
log.Debug("ProxyService RegisterNode WaitToFinish failed", zap.Error(err))
|
||||
return &proxypb.RegisterNodeResponse{
|
||||
Status: &commonpb.Status{
|
||||
ErrorCode: commonpb.ErrorCode_UnexpectedError,
|
||||
|
@ -287,7 +290,7 @@ func (s *ProxyService) RegisterNode(ctx context.Context, request *proxypb.Regist
|
|||
}
|
||||
|
||||
func (s *ProxyService) InvalidateCollectionMetaCache(ctx context.Context, request *proxypb.InvalidateCollMetaCacheRequest) (*commonpb.Status, error) {
|
||||
log.Debug("proxyservice receive InvalidateCollectionMetaCache request",
|
||||
log.Debug("ProxyService receive InvalidateCollectionMetaCache request",
|
||||
zap.String("db", request.DbName),
|
||||
zap.String("collection", request.CollectionName))
|
||||
|
||||
|
|
|
@ -17,6 +17,7 @@ import (
|
|||
"math/rand"
|
||||
"strconv"
|
||||
"sync"
|
||||
"time"
|
||||
|
||||
"github.com/milvus-io/milvus/internal/util/funcutil"
|
||||
|
||||
|
@ -86,7 +87,7 @@ func (table *globalNodeInfoTable) createClients() error {
|
|||
for nodeID, info := range table.infos {
|
||||
_, ok := table.ProxyNodes[nodeID]
|
||||
if !ok {
|
||||
table.ProxyNodes[nodeID] = grpcproxynodeclient.NewClient(context.Background(), info.ip+":"+strconv.Itoa(int(info.port)), 10)
|
||||
table.ProxyNodes[nodeID] = grpcproxynodeclient.NewClient(context.Background(), info.ip+":"+strconv.Itoa(int(info.port)), 10*time.Second)
|
||||
var err error
|
||||
err = table.ProxyNodes[nodeID].Init()
|
||||
if err != nil {
|
||||
|
|
|
@ -14,6 +14,7 @@ package proxyservice
|
|||
import (
|
||||
"context"
|
||||
"math/rand"
|
||||
"sync/atomic"
|
||||
"time"
|
||||
|
||||
"go.uber.org/zap"
|
||||
|
@ -30,9 +31,7 @@ type ProxyService struct {
|
|||
sched *taskScheduler
|
||||
tick *TimeTick
|
||||
nodeInfos *globalNodeInfoTable
|
||||
stateCode internalpb.StateCode
|
||||
|
||||
//subStates *internalpb.ComponentStates
|
||||
stateCode atomic.Value
|
||||
|
||||
nodeStartParams []*commonpb.KeyValuePair
|
||||
|
||||
|
@ -55,7 +54,7 @@ func NewProxyService(ctx context.Context, factory msgstream.Factory) (*ProxyServ
|
|||
s.sched = newTaskScheduler(ctx1)
|
||||
s.nodeInfos = newGlobalNodeInfoTable()
|
||||
s.UpdateStateCode(internalpb.StateCode_Abnormal)
|
||||
log.Debug("proxyservice", zap.Any("state of proxyservice: ", internalpb.StateCode_Abnormal))
|
||||
log.Debug("ProxyService", zap.Any("State", s.stateCode.Load()))
|
||||
|
||||
return s, nil
|
||||
}
|
||||
|
|
|
@ -137,11 +137,14 @@ func (node *QueryNode) Init() error {
|
|||
|
||||
resp, err := node.queryService.RegisterNode(ctx, registerReq)
|
||||
if err != nil {
|
||||
log.Debug("QueryNode RegisterNode failed", zap.Error(err))
|
||||
panic(err)
|
||||
}
|
||||
if resp.Status.ErrorCode != commonpb.ErrorCode_Success {
|
||||
log.Debug("QueryNode RegisterNode failed", zap.Any("Reason", resp.Status.Reason))
|
||||
panic(resp.Status.Reason)
|
||||
}
|
||||
log.Debug("QueryNode RegisterNode success")
|
||||
|
||||
for _, kv := range resp.InitParams.StartParams {
|
||||
switch kv.Key {
|
||||
|
|
|
@ -19,6 +19,8 @@ import (
|
|||
"sync/atomic"
|
||||
"time"
|
||||
|
||||
"go.uber.org/zap"
|
||||
|
||||
"github.com/milvus-io/milvus/internal/log"
|
||||
"github.com/milvus-io/milvus/internal/msgstream"
|
||||
"github.com/milvus-io/milvus/internal/proto/internalpb"
|
||||
|
@ -119,6 +121,7 @@ func NewQueryService(ctx context.Context, factory msgstream.Factory) (*QueryServ
|
|||
}
|
||||
|
||||
service.UpdateStateCode(internalpb.StateCode_Abnormal)
|
||||
log.Debug("QueryService", zap.Any("queryChannels", queryChannels))
|
||||
return service, nil
|
||||
}
|
||||
|
||||
|
|
Loading…
Reference in New Issue