未验证 提交 761350e4 编写于 作者: C Cai Yudong 提交者: GitHub

Refactor timetick msg handling between proxy node and master (#5321)

Proxy node send channel timetick msg to master, master reduce timetick
msg for each channel and send the min timetick to msg stream.

Resolves: #5276 
Signed-off-by: Nyudong.cai <yudong.cai@zilliz.com>
上级 04eb594d
......@@ -271,3 +271,7 @@ func (m *mockMasterService) GetDdChannel(ctx context.Context) (*milvuspb.StringR
Value: "ddchannel",
}, nil
}
func (m *mockMasterService) UpdateChannelTimeTick(ctx context.Context, req *internalpb.ChannelTimeTickMsg) (*commonpb.Status, error) {
panic("not implemented") // TODO: Implement
}
......@@ -145,10 +145,16 @@ func (c *GrpcClient) DescribeIndex(ctx context.Context, in *milvuspb.DescribeInd
func (c *GrpcClient) AllocTimestamp(ctx context.Context, in *masterpb.AllocTimestampRequest) (*masterpb.AllocTimestampResponse, error) {
return c.grpcClient.AllocTimestamp(ctx, in)
}
func (c *GrpcClient) AllocID(ctx context.Context, in *masterpb.AllocIDRequest) (*masterpb.AllocIDResponse, error) {
return c.grpcClient.AllocID(ctx, in)
}
// UpdateChannelTimeTick used to handle ChannelTimeTickMsg
func (c *GrpcClient) UpdateChannelTimeTick(ctx context.Context, in *internalpb.ChannelTimeTickMsg) (*commonpb.Status, error) {
return c.grpcClient.UpdateChannelTimeTick(ctx, in)
}
//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) {
return c.grpcClient.DescribeSegment(ctx, in)
......
......@@ -383,6 +383,11 @@ func (s *Server) AllocID(ctx context.Context, in *masterpb.AllocIDRequest) (*mas
return s.masterService.AllocID(ctx, in)
}
// UpdateChannelTimeTick used to handle ChannelTimeTickMsg
func (s *Server) UpdateChannelTimeTick(ctx context.Context, in *internalpb.ChannelTimeTickMsg) (*commonpb.Status, error) {
return s.masterService.UpdateChannelTimeTick(ctx, in)
}
func (s *Server) DescribeSegment(ctx context.Context, in *milvuspb.DescribeSegmentRequest) (*milvuspb.DescribeSegmentResponse, error) {
return s.masterService.DescribeSegment(ctx, in)
}
......
......@@ -136,6 +136,9 @@ type Core struct {
//dd request scheduler
ddReqQueue chan reqTask //dd request will be push into this chan
// channel timetick
chanTimeTick *timetickSync
//time tick loop
lastTimeTick typeutil.Timestamp
......@@ -426,16 +429,6 @@ func (c *Core) setMsgStreams() error {
return fmt.Errorf("ProxyTimeTickChannel is empty")
}
var err error
m := map[string]interface{}{
"PulsarAddress": Params.PulsarAddress,
"ReceiveBufSize": 1024,
"PulsarBufSize": 1024}
err = c.msFactory.SetParams(m)
if err != nil {
return err
}
proxyTimeTickStream, _ := c.msFactory.NewMsgStream(c.ctx)
proxyTimeTickStream.AsConsumer([]string{Params.ProxyTimeTickChannel}, Params.MsgChannelSubName)
log.Debug("master AsConsumer: " + Params.ProxyTimeTickChannel + " : " + Params.MsgChannelSubName)
......@@ -878,6 +871,18 @@ func (c *Core) Init() error {
return tsoAllocator.UpdateTSO()
}
m := map[string]interface{}{
"PulsarAddress": Params.PulsarAddress,
"ReceiveBufSize": 1024,
"PulsarBufSize": 1024}
if initError = c.msFactory.SetParams(m); initError != nil {
return
}
c.chanTimeTick, initError = newTimeTickSync(c.ctx, c.msFactory, c.etcdCli)
if initError != nil {
return
}
c.ddReqQueue = make(chan reqTask, 1024)
initError = c.setMsgStreams()
})
......@@ -971,6 +976,7 @@ func (c *Core) Start() error {
go c.startDataServiceSegmentLoop()
go c.startSegmentFlushCompletedLoop()
go c.tsLoop()
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)]))
......@@ -1648,3 +1654,26 @@ func (c *Core) AllocID(ctx context.Context, in *masterpb.AllocIDRequest) (*maste
Count: in.Count,
}, nil
}
// UpdateChannelTimeTick used to handle ChannelTimeTickMsg
func (c *Core) UpdateChannelTimeTick(ctx context.Context, in *internalpb.ChannelTimeTickMsg) (*commonpb.Status, error) {
status := &commonpb.Status{
ErrorCode: commonpb.ErrorCode_Success,
Reason: "",
}
if in.Base.MsgType != commonpb.MsgType_TimeTick {
status.ErrorCode = commonpb.ErrorCode_UnexpectedError
status.Reason = fmt.Sprintf("UpdateChannelTimeTick receive invalid message %d", in.Base.GetMsgType())
return status, nil
}
err := c.chanTimeTick.UpdateTimeTick(in)
if err != nil {
status.ErrorCode = commonpb.ErrorCode_UnexpectedError
status.Reason = err.Error()
return status, nil
}
return &commonpb.Status{
ErrorCode: commonpb.ErrorCode_Success,
Reason: "",
}, nil
}
......@@ -16,6 +16,7 @@ import (
"encoding/json"
"fmt"
"math/rand"
"strconv"
"sync"
"testing"
"time"
......@@ -34,6 +35,8 @@ import (
"github.com/milvus-io/milvus/internal/proto/querypb"
"github.com/milvus-io/milvus/internal/proto/schemapb"
"github.com/milvus-io/milvus/internal/types"
"github.com/milvus-io/milvus/internal/util/funcutil"
"github.com/milvus-io/milvus/internal/util/sessionutil"
"github.com/milvus-io/milvus/internal/util/typeutil"
"github.com/stretchr/testify/assert"
"go.etcd.io/etcd/clientv3"
......@@ -215,6 +218,14 @@ func TestMasterService(t *testing.T) {
Params.KvRootPath = fmt.Sprintf("/%d/%s", randVal, Params.KvRootPath)
Params.MsgChannelSubName = fmt.Sprintf("subname-%d", randVal)
etcdCli, err := clientv3.New(clientv3.Config{Endpoints: []string{Params.EtcdAddress}, DialTimeout: 5 * time.Second})
assert.Nil(t, err)
_, err = etcdCli.Delete(ctx, ProxyNodeSessionPrefix, clientv3.WithPrefix())
assert.Nil(t, err)
defer func() {
_, _ = etcdCli.Delete(ctx, ProxyNodeSessionPrefix, clientv3.WithPrefix())
}()
pm := &proxyMock{
randVal: randVal,
collArray: make([]string, 0, 16),
......@@ -244,6 +255,12 @@ func TestMasterService(t *testing.T) {
err = core.SetQueryService(qm)
assert.Nil(t, err)
// initialize master's session manager before core init
self := sessionutil.NewSession("masterservice", funcutil.GetLocalIP()+":"+strconv.Itoa(53100), true)
sm := sessionutil.NewSessionManager(ctx, Params.EtcdAddress, Params.MetaRootPath, self)
sm.Init()
sessionutil.SetGlobalSessionManager(sm)
err = core.Init()
assert.Nil(t, err)
......@@ -1415,6 +1432,77 @@ func TestMasterService(t *testing.T) {
assert.Nil(t, err)
})
t.Run("channel timetick", func(t *testing.T) {
const (
proxyNodeIDInvalid = 102
proxyNodeName0 = "proxynode_0"
proxyNodeName1 = "proxynode_1"
chanName0 = "c0"
chanName1 = "c1"
chanName2 = "c2"
ts0 = uint64(100)
ts1 = uint64(120)
ts2 = uint64(150)
)
p1 := sessionutil.Session{
ServerID: 100,
}
p2 := sessionutil.Session{
ServerID: 101,
}
ctx2, cancel2 := context.WithTimeout(ctx, RequestTimeout)
defer cancel2()
s1, err := json.Marshal(&p1)
assert.Nil(t, err)
s2, err := json.Marshal(&p2)
assert.Nil(t, err)
_, err = core.etcdCli.Put(ctx2, ProxyNodeSessionPrefix+"-1", string(s1))
assert.Nil(t, err)
_, err = core.etcdCli.Put(ctx2, ProxyNodeSessionPrefix+"-2", string(s2))
assert.Nil(t, err)
time.Sleep(time.Second)
msg0 := &internalpb.ChannelTimeTickMsg{
Base: &commonpb.MsgBase{
MsgType: commonpb.MsgType_TimeTick,
SourceID: 100,
},
ChannelNames: []string{chanName0, chanName1},
Timestamps: []uint64{ts0, ts2},
}
s, _ := core.UpdateChannelTimeTick(ctx, msg0)
assert.Equal(t, commonpb.ErrorCode_Success, s.ErrorCode)
time.Sleep(100 * time.Millisecond)
t.Log(core.chanTimeTick.proxyTimeTick)
msg1 := &internalpb.ChannelTimeTickMsg{
Base: &commonpb.MsgBase{
MsgType: commonpb.MsgType_TimeTick,
SourceID: 101,
},
ChannelNames: []string{chanName1, chanName2},
Timestamps: []uint64{ts1, ts2},
}
s, _ = core.UpdateChannelTimeTick(ctx, msg1)
assert.Equal(t, commonpb.ErrorCode_Success, s.ErrorCode)
time.Sleep(100 * time.Millisecond)
msgInvalid := &internalpb.ChannelTimeTickMsg{
Base: &commonpb.MsgBase{
MsgType: commonpb.MsgType_TimeTick,
SourceID: proxyNodeIDInvalid,
},
}
s, _ = core.UpdateChannelTimeTick(ctx, msgInvalid)
assert.Equal(t, commonpb.ErrorCode_UnexpectedError, s.ErrorCode)
time.Sleep(1 * time.Second)
assert.Equal(t, 2, core.chanTimeTick.GetProxyNodeNum())
assert.Equal(t, 3, core.chanTimeTick.GetChanNum())
})
err = core.Stop()
assert.Nil(t, err)
st, err := core.GetComponentStates(ctx)
......
// Copyright (C) 2019-2020 Zilliz. All rights reserved.
//
// Licensed under the Apache License, Version 2.0 (the "License"); you may not use this file except in compliance
// with the License. You may obtain a copy of the License at
//
// http://www.apache.org/licenses/LICENSE-2.0
//
// Unless required by applicable law or agreed to in writing, software distributed under the License
// is distributed on an "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express
// or implied. See the License for the specific language governing permissions and limitations under the License.
package masterservice
import (
"context"
"encoding/json"
"fmt"
"sync"
"github.com/coreos/etcd/mvcc/mvccpb"
"github.com/milvus-io/milvus/internal/log"
"github.com/milvus-io/milvus/internal/msgstream"
"github.com/milvus-io/milvus/internal/proto/commonpb"
"github.com/milvus-io/milvus/internal/proto/internalpb"
"github.com/milvus-io/milvus/internal/util/sessionutil"
"github.com/milvus-io/milvus/internal/util/typeutil"
"go.etcd.io/etcd/clientv3"
"go.uber.org/zap"
)
type timetickSync struct {
lock sync.Mutex
ctx context.Context
etcdCli *clientv3.Client
msFactory msgstream.Factory
proxyTimeTick map[typeutil.UniqueID]*internalpb.ChannelTimeTickMsg
chanStream map[string]msgstream.MsgStream
sendChan chan map[typeutil.UniqueID]*internalpb.ChannelTimeTickMsg
}
// ProxyNodeSessionPrefix used for etcd watch
const ProxyNodeSessionPrefix = "session/proxynode"
func newTimeTickSync(ctx context.Context, factory msgstream.Factory, cli *clientv3.Client) (*timetickSync, error) {
tss := timetickSync{
lock: sync.Mutex{},
ctx: ctx,
etcdCli: cli,
msFactory: factory,
proxyTimeTick: make(map[typeutil.UniqueID]*internalpb.ChannelTimeTickMsg),
chanStream: make(map[string]msgstream.MsgStream),
sendChan: make(chan map[typeutil.UniqueID]*internalpb.ChannelTimeTickMsg, 16),
}
ctx2, cancel := context.WithTimeout(ctx, RequestTimeout)
defer cancel()
resp, err := cli.Get(ctx2, ProxyMetaPrefix, clientv3.WithPrefix())
if err != nil {
return nil, err
}
for _, v := range resp.Kvs {
var sess sessionutil.Session
err := json.Unmarshal(v.Value, &sess)
if err != nil {
log.Debug("unmarshal SvrSession failed", zap.Error(err))
continue
}
if _, ok := tss.proxyTimeTick[sess.ServerID]; !ok {
tss.proxyTimeTick[sess.ServerID] = nil
}
}
return &tss, nil
}
// sendToChannel send all channels' timetick to sendChan
// lock is needed by the invoker
func (t *timetickSync) sendToChannel() {
for _, v := range t.proxyTimeTick {
if v == nil {
return
}
}
if len(t.proxyTimeTick) == 0 {
return
}
// clear proxyTimeTick and send a clone
ptt := make(map[typeutil.UniqueID]*internalpb.ChannelTimeTickMsg)
for k, v := range t.proxyTimeTick {
ptt[k] = v
t.proxyTimeTick[k] = nil
}
t.sendChan <- ptt
}
// UpdateTimeTick check msg validation and send it to local channel
func (t *timetickSync) UpdateTimeTick(in *internalpb.ChannelTimeTickMsg) error {
t.lock.Lock()
defer t.lock.Unlock()
_, ok := t.proxyTimeTick[in.Base.SourceID]
if !ok {
return fmt.Errorf("Skip ChannelTimeTickMsg from un-recognized proxy node %d", in.Base.SourceID)
}
t.proxyTimeTick[in.Base.SourceID] = in
t.sendToChannel()
return nil
}
// StartWatch watch proxy node change and process all channels' timetick msg
func (t *timetickSync) StartWatch() {
rch := t.etcdCli.Watch(t.ctx, ProxyNodeSessionPrefix, clientv3.WithPrefix(), clientv3.WithCreatedNotify())
for {
select {
case <-t.ctx.Done():
log.Debug("timetickSync context done", zap.Error(t.ctx.Err()))
return
case wresp, ok := <-rch:
if !ok {
log.Debug("time tick sync watch etcd failed")
}
for _, ev := range wresp.Events {
switch ev.Type {
case mvccpb.PUT:
var sess sessionutil.Session
err := json.Unmarshal(ev.Kv.Value, &sess)
if err != nil {
log.Debug("watch proxy node, unmarshal failed", zap.Error(err))
continue
}
func() {
t.lock.Lock()
defer t.lock.Unlock()
t.proxyTimeTick[sess.ServerID] = nil
}()
case mvccpb.DELETE:
var sess sessionutil.Session
err := json.Unmarshal(ev.PrevKv.Value, &sess)
if err != nil {
log.Debug("watch proxy node, unmarshal failed", zap.Error(err))
continue
}
func() {
t.lock.Lock()
defer t.lock.Unlock()
if _, ok := t.proxyTimeTick[sess.ServerID]; ok {
delete(t.proxyTimeTick, sess.ServerID)
t.sendToChannel()
}
}()
}
}
case ptt, ok := <-t.sendChan:
if !ok {
log.Debug("timetickSync sendChan closed", zap.Error(t.ctx.Err()))
return
}
// reduce each channel to get min timestamp
chanName2TimeTickMap := make(map[string]typeutil.Timestamp)
for _, cttMsg := range ptt {
chanNum := len(cttMsg.ChannelNames)
for i := 0; i < chanNum; i++ {
name := cttMsg.ChannelNames[i]
ts := cttMsg.Timestamps[i]
cts, ok := chanName2TimeTickMap[name]
if !ok || ts < cts {
chanName2TimeTickMap[name] = ts
}
}
}
// send timetick msg to msg stream
for chanName, chanTs := range chanName2TimeTickMap {
if err := t.SendChannelTimeTick(chanName, chanTs); err != nil {
log.Debug("SendChannelTimeTick fail", zap.Error(t.ctx.Err()))
}
}
}
}
}
// SendChannelTimeTick send each channel's min timetick to msg stream
func (t *timetickSync) SendChannelTimeTick(chanName string, ts typeutil.Timestamp) error {
msgPack := msgstream.MsgPack{}
baseMsg := msgstream.BaseMsg{
BeginTimestamp: ts,
EndTimestamp: ts,
HashValues: []uint32{0},
}
timeTickResult := internalpb.TimeTickMsg{
Base: &commonpb.MsgBase{
MsgType: commonpb.MsgType_TimeTick,
MsgID: 0,
Timestamp: ts,
SourceID: int64(Params.NodeID),
},
}
timeTickMsg := &msgstream.TimeTickMsg{
BaseMsg: baseMsg,
TimeTickMsg: timeTickResult,
}
msgPack.Msgs = append(msgPack.Msgs, timeTickMsg)
t.lock.Lock()
defer t.lock.Unlock()
// send timetick msg to msg stream
var err error
var stream msgstream.MsgStream
stream, ok := t.chanStream[chanName]
if !ok {
stream, err = t.msFactory.NewMsgStream(t.ctx)
if err != nil {
return err
}
stream.AsProducer([]string{chanName})
t.chanStream[chanName] = stream
}
return stream.Broadcast(&msgPack)
}
// GetProxyNodeNum return the num of detected proxy node
func (t *timetickSync) GetProxyNodeNum() int {
t.lock.Lock()
defer t.lock.Unlock()
return len(t.proxyTimeTick)
}
// GetChanNum return the num of channel
func (t *timetickSync) GetChanNum() int {
t.lock.Lock()
defer t.lock.Unlock()
return len(t.chanStream)
}
......@@ -227,3 +227,9 @@ message MsgPosition {
string msgGroup = 3;
uint64 timestamp = 4;
}
message ChannelTimeTickMsg {
common.MsgBase base = 1;
repeated string channelNames = 2;
repeated uint64 timestamps = 3;
}
......@@ -93,6 +93,7 @@ service MasterService {
rpc AllocTimestamp(AllocTimestampRequest) returns (AllocTimestampResponse) {}
rpc AllocID(AllocIDRequest) returns (AllocIDResponse) {}
rpc UpdateChannelTimeTick(internal.ChannelTimeTickMsg) returns (common.Status) {}
rpc GetDdChannel(internal.GetDdChannelRequest) returns (milvus.StringResponse) {}
}
......
......@@ -241,50 +241,52 @@ func init() {
func init() { proto.RegisterFile("master.proto", fileDescriptor_f9c348dec43a6705) }
var fileDescriptor_f9c348dec43a6705 = []byte{
// 686 bytes of a gzipped FileDescriptorProto
0x1f, 0x8b, 0x08, 0x00, 0x00, 0x00, 0x00, 0x00, 0x02, 0xff, 0xac, 0x96, 0xdb, 0x4f, 0x1a, 0x41,
0x14, 0xc6, 0x05, 0xad, 0x8d, 0x47, 0x40, 0x33, 0xb5, 0x8d, 0xa1, 0x3e, 0x58, 0xec, 0x05, 0xd4,
0x82, 0xd1, 0xf4, 0x0f, 0xa8, 0x90, 0x28, 0x0f, 0x26, 0x2d, 0xd8, 0xa6, 0x97, 0x18, 0xb3, 0xac,
0x27, 0x30, 0x61, 0x77, 0x66, 0xdd, 0x19, 0xb4, 0xaf, 0x4d, 0xfa, 0x87, 0x37, 0x7b, 0x99, 0x61,
0x97, 0xbd, 0x64, 0x49, 0xfb, 0xc6, 0xb0, 0xbf, 0xf9, 0xbe, 0x39, 0xe7, 0x7c, 0xc9, 0x0c, 0x54,
0x6c, 0x43, 0x48, 0x74, 0xdb, 0x8e, 0xcb, 0x25, 0x27, 0xcf, 0x6c, 0x6a, 0x3d, 0xcc, 0x44, 0xb0,
0x6a, 0x07, 0x9f, 0xea, 0x15, 0x93, 0xdb, 0x36, 0x67, 0xc1, 0x9f, 0xf5, 0x4a, 0x14, 0xa9, 0xd7,
0x28, 0x93, 0xe8, 0x32, 0xc3, 0x0a, 0xd6, 0x8d, 0x5b, 0x78, 0xfe, 0xd1, 0xb2, 0xb8, 0x79, 0x4d,
0x6d, 0x14, 0xd2, 0xb0, 0x9d, 0x01, 0xde, 0xcf, 0x50, 0x48, 0x72, 0x02, 0x6b, 0x23, 0x43, 0xe0,
0x6e, 0x69, 0xbf, 0xd4, 0xdc, 0x3c, 0xdd, 0x6b, 0xc7, 0x8c, 0x42, 0x83, 0x2b, 0x31, 0x3e, 0x37,
0x04, 0x0e, 0x7c, 0x92, 0xec, 0xc0, 0x13, 0x93, 0xcf, 0x98, 0xdc, 0x5d, 0xdd, 0x2f, 0x35, 0xab,
0x83, 0x60, 0xd1, 0xf8, 0x5d, 0x82, 0x17, 0x8b, 0x0e, 0xc2, 0xe1, 0x4c, 0x20, 0x39, 0x83, 0x75,
0x21, 0x0d, 0x39, 0x13, 0xa1, 0xc9, 0xcb, 0x54, 0x93, 0xa1, 0x8f, 0x0c, 0x42, 0x94, 0xec, 0xc1,
0x86, 0x54, 0x4a, 0xbb, 0xe5, 0xfd, 0x52, 0x73, 0x6d, 0x30, 0xff, 0x23, 0xe3, 0x0c, 0xdf, 0xa0,
0xe6, 0x1f, 0xa1, 0xdf, 0xfb, 0x0f, 0xd5, 0x95, 0xa3, 0xca, 0x16, 0x6c, 0x69, 0xe5, 0x7f, 0xa9,
0xaa, 0x06, 0xe5, 0x7e, 0xcf, 0x97, 0x5e, 0x1d, 0x94, 0xfb, 0xbd, 0xf4, 0x3a, 0x4e, 0xff, 0x6c,
0x43, 0xf5, 0xca, 0x9f, 0xf1, 0x10, 0xdd, 0x07, 0x6a, 0x22, 0x71, 0x80, 0x5c, 0xa0, 0xec, 0x72,
0xdb, 0xe1, 0x0c, 0x99, 0xf4, 0x54, 0x51, 0x90, 0x93, 0xb8, 0xa5, 0x1e, 0x79, 0x12, 0x0d, 0xfb,
0x51, 0x7f, 0x9b, 0xb1, 0x63, 0x01, 0x6f, 0xac, 0x10, 0xdb, 0x77, 0xf4, 0x86, 0x79, 0x4d, 0xcd,
0x69, 0x77, 0x62, 0x30, 0x86, 0x56, 0x9e, 0xe3, 0x02, 0xaa, 0x1c, 0x0f, 0xe2, 0x3b, 0xc2, 0xc5,
0x50, 0xba, 0x94, 0x8d, 0x55, 0x2f, 0x1b, 0x2b, 0xe4, 0x1e, 0x76, 0x2e, 0xd0, 0x77, 0xa7, 0x42,
0x52, 0x53, 0x28, 0xc3, 0xd3, 0x6c, 0xc3, 0x04, 0xbc, 0xa4, 0xe5, 0x2d, 0x6c, 0x77, 0x5d, 0x34,
0x24, 0x76, 0xb9, 0x65, 0xa1, 0x29, 0x29, 0x67, 0xe4, 0x38, 0x75, 0xeb, 0x22, 0xa6, 0x8c, 0xf2,
0x46, 0xde, 0x58, 0x21, 0x3f, 0xa1, 0xd6, 0x73, 0xb9, 0x13, 0x91, 0x3f, 0x4c, 0x95, 0x8f, 0x43,
0x05, 0xc5, 0x6f, 0xa1, 0x7a, 0x69, 0x88, 0x88, 0x76, 0x2b, 0x55, 0x3b, 0xc6, 0x28, 0xe9, 0x57,
0xa9, 0xe8, 0x39, 0xe7, 0x56, 0xa4, 0x3d, 0x8f, 0x40, 0x7a, 0x28, 0x4c, 0x97, 0x8e, 0xa2, 0x0d,
0x6a, 0xa7, 0x57, 0x90, 0x00, 0x95, 0x55, 0xa7, 0x30, 0xaf, 0x8d, 0x19, 0x6c, 0x0d, 0x27, 0xfc,
0x71, 0xfe, 0x4d, 0x90, 0xa3, 0xf4, 0x89, 0xc6, 0x29, 0x65, 0x79, 0x5c, 0x0c, 0xd6, 0x7e, 0x37,
0xb0, 0x15, 0x0c, 0xf8, 0x93, 0xe1, 0x4a, 0xea, 0x57, 0x79, 0x94, 0x13, 0x03, 0x4d, 0x15, 0x1c,
0xd4, 0x77, 0xa8, 0x7a, 0x03, 0x9e, 0x8b, 0xb7, 0x32, 0x43, 0xb0, 0xac, 0xf4, 0x0d, 0x54, 0x2e,
0x0d, 0x31, 0x57, 0x6e, 0x66, 0x45, 0x20, 0x21, 0x5c, 0x28, 0x01, 0x53, 0xa8, 0x79, 0x5d, 0xd3,
0x9b, 0x45, 0x46, 0x7e, 0xe3, 0x90, 0xb2, 0x38, 0x2a, 0xc4, 0x46, 0xa7, 0xae, 0x52, 0x31, 0xc4,
0xb1, 0x8d, 0x4c, 0x66, 0x4c, 0x61, 0x81, 0xca, 0x9f, 0x7a, 0x02, 0xd6, 0x7e, 0x08, 0x15, 0xef,
0x2c, 0xe1, 0x07, 0x91, 0xd1, 0xbb, 0x28, 0xa2, 0x9c, 0x5a, 0x05, 0x48, 0x6d, 0xf3, 0x05, 0x36,
0x83, 0xd8, 0xf4, 0xd9, 0x1d, 0xfe, 0x22, 0xef, 0x72, 0x82, 0xe5, 0x13, 0x05, 0x27, 0x3f, 0x81,
0xaa, 0x2a, 0x2d, 0x10, 0x6e, 0xe5, 0x96, 0x1f, 0x93, 0x3e, 0x2c, 0x82, 0xea, 0x02, 0x3e, 0xc3,
0x86, 0x17, 0xcd, 0xc0, 0xe5, 0x4d, 0x66, 0x74, 0x97, 0x39, 0xfc, 0x34, 0xbc, 0xa6, 0xf5, 0x4b,
0x21, 0x91, 0xab, 0xe0, 0xe9, 0x93, 0xfa, 0x60, 0x49, 0xe4, 0x2a, 0x9d, 0xd5, 0xe7, 0xff, 0x0a,
0x4f, 0xc3, 0x9b, 0x9b, 0x1c, 0x64, 0xef, 0xd4, 0x2f, 0x86, 0xfa, 0xeb, 0x7c, 0x48, 0xeb, 0x1a,
0x50, 0xb9, 0x40, 0xd9, 0xbb, 0x53, 0x17, 0xd5, 0x61, 0xf6, 0x45, 0xa5, 0xa1, 0xe5, 0x2e, 0xa8,
0xf3, 0x0f, 0x3f, 0xce, 0xc6, 0x54, 0x4e, 0x66, 0x23, 0xaf, 0x83, 0x9d, 0x80, 0x7a, 0x4f, 0x79,
0xf8, 0xab, 0xa3, 0x2c, 0x3a, 0xbe, 0x4a, 0x27, 0x38, 0xa9, 0x33, 0x1a, 0xad, 0xfb, 0xeb, 0xb3,
0xbf, 0x01, 0x00, 0x00, 0xff, 0xff, 0x43, 0xba, 0xb8, 0x0a, 0x42, 0x0a, 0x00, 0x00,
// 710 bytes of a gzipped FileDescriptorProto
0x1f, 0x8b, 0x08, 0x00, 0x00, 0x00, 0x00, 0x00, 0x02, 0xff, 0xac, 0x96, 0x5b, 0x4f, 0xdb, 0x3c,
0x18, 0xc7, 0x69, 0xe1, 0xe5, 0x15, 0x0f, 0x6d, 0x41, 0x1e, 0x4c, 0xa8, 0xe3, 0x82, 0x95, 0x1d,
0x5a, 0x60, 0x2d, 0x02, 0xed, 0x03, 0x8c, 0x56, 0x82, 0x5e, 0x20, 0x6d, 0x2d, 0x4c, 0x3b, 0x08,
0x21, 0x37, 0x58, 0xad, 0x45, 0x62, 0x87, 0xd8, 0x85, 0xdd, 0xee, 0xd3, 0xee, 0x6b, 0x4c, 0x89,
0x63, 0x37, 0x69, 0x0e, 0x4b, 0xb5, 0xdd, 0xc5, 0xf1, 0xcf, 0xff, 0xbf, 0x9f, 0x83, 0xf4, 0x18,
0x2a, 0x0e, 0x16, 0x92, 0x78, 0x6d, 0xd7, 0xe3, 0x92, 0xa3, 0x67, 0x0e, 0xb5, 0x1f, 0xa7, 0x42,
0xad, 0xda, 0x6a, 0xab, 0x5e, 0xb1, 0xb8, 0xe3, 0x70, 0xa6, 0x7e, 0xd6, 0x2b, 0x51, 0xa4, 0x5e,
0xa3, 0x4c, 0x12, 0x8f, 0x61, 0x5b, 0xad, 0x1b, 0xb7, 0xb0, 0xfd, 0xc1, 0xb6, 0xb9, 0x75, 0x45,
0x1d, 0x22, 0x24, 0x76, 0xdc, 0x01, 0x79, 0x98, 0x12, 0x21, 0xd1, 0x31, 0xac, 0x8c, 0xb0, 0x20,
0x3b, 0xa5, 0xbd, 0x52, 0x73, 0xfd, 0x64, 0xb7, 0x1d, 0x33, 0x0a, 0x0d, 0x2e, 0xc5, 0xf8, 0x0c,
0x0b, 0x32, 0x08, 0x48, 0xb4, 0x05, 0xff, 0x59, 0x7c, 0xca, 0xe4, 0xce, 0xf2, 0x5e, 0xa9, 0x59,
0x1d, 0xa8, 0x45, 0xe3, 0x67, 0x09, 0x9e, 0xcf, 0x3b, 0x08, 0x97, 0x33, 0x41, 0xd0, 0x29, 0xac,
0x0a, 0x89, 0xe5, 0x54, 0x84, 0x26, 0x2f, 0x52, 0x4d, 0x86, 0x01, 0x32, 0x08, 0x51, 0xb4, 0x0b,
0x6b, 0x52, 0x2b, 0xed, 0x94, 0xf7, 0x4a, 0xcd, 0x95, 0xc1, 0xec, 0x47, 0xc6, 0x1d, 0xbe, 0x40,
0x2d, 0xb8, 0x42, 0xbf, 0xf7, 0x0f, 0xa2, 0x2b, 0x47, 0x95, 0x6d, 0xd8, 0x30, 0xca, 0x7f, 0x13,
0x55, 0x0d, 0xca, 0xfd, 0x5e, 0x20, 0xbd, 0x3c, 0x28, 0xf7, 0x7b, 0xe9, 0x71, 0x9c, 0xfc, 0xda,
0x84, 0xea, 0x65, 0x50, 0xe3, 0x21, 0xf1, 0x1e, 0xa9, 0x45, 0x90, 0x0b, 0xe8, 0x9c, 0xc8, 0x2e,
0x77, 0x5c, 0xce, 0x08, 0x93, 0xbe, 0x2a, 0x11, 0xe8, 0x38, 0x6e, 0x69, 0x4a, 0x9e, 0x44, 0xc3,
0x7c, 0xd4, 0xdf, 0x64, 0x9c, 0x98, 0xc3, 0x1b, 0x4b, 0xc8, 0x09, 0x1c, 0xfd, 0x62, 0x5e, 0x51,
0xeb, 0xbe, 0x3b, 0xc1, 0x8c, 0x11, 0x3b, 0xcf, 0x71, 0x0e, 0xd5, 0x8e, 0xfb, 0xf1, 0x13, 0xe1,
0x62, 0x28, 0x3d, 0xca, 0xc6, 0x3a, 0x97, 0x8d, 0x25, 0xf4, 0x00, 0x5b, 0xe7, 0x24, 0x70, 0xa7,
0x42, 0x52, 0x4b, 0x68, 0xc3, 0x93, 0x6c, 0xc3, 0x04, 0xbc, 0xa0, 0xe5, 0x2d, 0x6c, 0x76, 0x3d,
0x82, 0x25, 0xe9, 0x72, 0xdb, 0x26, 0x96, 0xa4, 0x9c, 0xa1, 0xa3, 0xd4, 0xa3, 0xf3, 0x98, 0x36,
0xca, 0x2b, 0x79, 0x63, 0x09, 0x7d, 0x87, 0x5a, 0xcf, 0xe3, 0x6e, 0x44, 0xfe, 0x20, 0x55, 0x3e,
0x0e, 0x15, 0x14, 0xbf, 0x85, 0xea, 0x05, 0x16, 0x11, 0xed, 0x56, 0xaa, 0x76, 0x8c, 0xd1, 0xd2,
0x2f, 0x53, 0xd1, 0x33, 0xce, 0xed, 0x48, 0x7a, 0x9e, 0x00, 0xf5, 0x88, 0xb0, 0x3c, 0x3a, 0x8a,
0x26, 0xa8, 0x9d, 0x1e, 0x41, 0x02, 0xd4, 0x56, 0x9d, 0xc2, 0xbc, 0x31, 0x66, 0xb0, 0x31, 0x9c,
0xf0, 0xa7, 0xd9, 0x9e, 0x40, 0x87, 0xe9, 0x15, 0x8d, 0x53, 0xda, 0xf2, 0xa8, 0x18, 0x6c, 0xfc,
0x6e, 0x60, 0x43, 0x15, 0xf8, 0x23, 0xf6, 0x24, 0x0d, 0xa2, 0x3c, 0xcc, 0x69, 0x03, 0x43, 0x15,
0x2c, 0xd4, 0x57, 0xa8, 0xfa, 0x05, 0x9e, 0x89, 0xb7, 0x32, 0x9b, 0x60, 0x51, 0xe9, 0x1b, 0xa8,
0x5c, 0x60, 0x31, 0x53, 0x6e, 0x66, 0xb5, 0x40, 0x42, 0xb8, 0x50, 0x07, 0xdc, 0x43, 0xcd, 0xcf,
0x9a, 0x39, 0x2c, 0x32, 0xfa, 0x37, 0x0e, 0x69, 0x8b, 0xc3, 0x42, 0x6c, 0xb4, 0xea, 0xba, 0x2b,
0x86, 0x64, 0xec, 0x10, 0x26, 0x33, 0xaa, 0x30, 0x47, 0xe5, 0x57, 0x3d, 0x01, 0x1b, 0x3f, 0x02,
0x15, 0xff, 0x2e, 0xe1, 0x86, 0xc8, 0xc8, 0x5d, 0x14, 0xd1, 0x4e, 0xad, 0x02, 0xa4, 0xb1, 0xb9,
0x86, 0x75, 0xd5, 0x36, 0x7d, 0x76, 0x47, 0x7e, 0xa0, 0xb7, 0x39, 0x8d, 0x15, 0x10, 0x05, 0x2b,
0x3f, 0x81, 0xaa, 0x0e, 0x4d, 0x09, 0xb7, 0x72, 0xc3, 0x8f, 0x49, 0x1f, 0x14, 0x41, 0x4d, 0x00,
0x9f, 0x60, 0xcd, 0x6f, 0x4d, 0xe5, 0xf2, 0x3a, 0xb3, 0x75, 0x17, 0xb9, 0xfc, 0x7d, 0x38, 0xa6,
0xcd, 0x4b, 0x21, 0xd1, 0x57, 0xea, 0xe9, 0x93, 0xfa, 0x60, 0x49, 0xf4, 0x55, 0x3a, 0x6b, 0xee,
0xff, 0x19, 0xfe, 0x0f, 0x27, 0x37, 0xda, 0xcf, 0x3e, 0x69, 0x5e, 0x0c, 0xf5, 0x57, 0xf9, 0x90,
0xd1, 0xc5, 0xb0, 0x7d, 0xed, 0xde, 0xf9, 0x63, 0x41, 0x0d, 0x1f, 0x3d, 0xfe, 0xe6, 0x2b, 0x31,
0x1b, 0xb1, 0x71, 0xee, 0x52, 0x8c, 0xff, 0x94, 0x27, 0x0c, 0x95, 0x73, 0x22, 0x7b, 0x77, 0x7a,
0x16, 0x1e, 0x64, 0xcf, 0x42, 0x03, 0x2d, 0x36, 0x03, 0xcf, 0xde, 0x7f, 0x3b, 0x1d, 0x53, 0x39,
0x99, 0x8e, 0x7c, 0xf3, 0x8e, 0xa2, 0xde, 0x51, 0x1e, 0x7e, 0x75, 0xb4, 0x45, 0x27, 0x50, 0xe9,
0xa8, 0x64, 0xb8, 0xa3, 0xd1, 0x6a, 0xb0, 0x3e, 0xfd, 0x1d, 0x00, 0x00, 0xff, 0xff, 0xee, 0x13,
0xfc, 0x7b, 0xa5, 0x0a, 0x00, 0x00,
}
// Reference imports to suppress errors if they are not otherwise used.
......@@ -364,6 +366,7 @@ type MasterServiceClient interface {
DropIndex(ctx context.Context, in *milvuspb.DropIndexRequest, opts ...grpc.CallOption) (*commonpb.Status, error)
AllocTimestamp(ctx context.Context, in *AllocTimestampRequest, opts ...grpc.CallOption) (*AllocTimestampResponse, error)
AllocID(ctx context.Context, in *AllocIDRequest, opts ...grpc.CallOption) (*AllocIDResponse, error)
UpdateChannelTimeTick(ctx context.Context, in *internalpb.ChannelTimeTickMsg, opts ...grpc.CallOption) (*commonpb.Status, error)
GetDdChannel(ctx context.Context, in *internalpb.GetDdChannelRequest, opts ...grpc.CallOption) (*milvuspb.StringResponse, error)
}
......@@ -546,6 +549,15 @@ func (c *masterServiceClient) AllocID(ctx context.Context, in *AllocIDRequest, o
return out, nil
}
func (c *masterServiceClient) UpdateChannelTimeTick(ctx context.Context, in *internalpb.ChannelTimeTickMsg, opts ...grpc.CallOption) (*commonpb.Status, error) {
out := new(commonpb.Status)
err := c.cc.Invoke(ctx, "/milvus.proto.master.MasterService/UpdateChannelTimeTick", in, out, opts...)
if err != nil {
return nil, err
}
return out, nil
}
func (c *masterServiceClient) GetDdChannel(ctx context.Context, in *internalpb.GetDdChannelRequest, opts ...grpc.CallOption) (*milvuspb.StringResponse, error) {
out := new(milvuspb.StringResponse)
err := c.cc.Invoke(ctx, "/milvus.proto.master.MasterService/GetDdChannel", in, out, opts...)
......@@ -622,6 +634,7 @@ type MasterServiceServer interface {
DropIndex(context.Context, *milvuspb.DropIndexRequest) (*commonpb.Status, error)
AllocTimestamp(context.Context, *AllocTimestampRequest) (*AllocTimestampResponse, error)
AllocID(context.Context, *AllocIDRequest) (*AllocIDResponse, error)
UpdateChannelTimeTick(context.Context, *internalpb.ChannelTimeTickMsg) (*commonpb.Status, error)
GetDdChannel(context.Context, *internalpb.GetDdChannelRequest) (*milvuspb.StringResponse, error)
}
......@@ -686,6 +699,9 @@ func (*UnimplementedMasterServiceServer) AllocTimestamp(ctx context.Context, req
func (*UnimplementedMasterServiceServer) AllocID(ctx context.Context, req *AllocIDRequest) (*AllocIDResponse, error) {
return nil, status.Errorf(codes.Unimplemented, "method AllocID not implemented")
}
func (*UnimplementedMasterServiceServer) UpdateChannelTimeTick(ctx context.Context, req *internalpb.ChannelTimeTickMsg) (*commonpb.Status, error) {
return nil, status.Errorf(codes.Unimplemented, "method UpdateChannelTimeTick not implemented")
}
func (*UnimplementedMasterServiceServer) GetDdChannel(ctx context.Context, req *internalpb.GetDdChannelRequest) (*milvuspb.StringResponse, error) {
return nil, status.Errorf(codes.Unimplemented, "method GetDdChannel not implemented")
}
......@@ -1036,6 +1052,24 @@ func _MasterService_AllocID_Handler(srv interface{}, ctx context.Context, dec fu
return interceptor(ctx, in, info, handler)
}
func _MasterService_UpdateChannelTimeTick_Handler(srv interface{}, ctx context.Context, dec func(interface{}) error, interceptor grpc.UnaryServerInterceptor) (interface{}, error) {
in := new(internalpb.ChannelTimeTickMsg)
if err := dec(in); err != nil {
return nil, err
}
if interceptor == nil {
return srv.(MasterServiceServer).UpdateChannelTimeTick(ctx, in)
}
info := &grpc.UnaryServerInfo{
Server: srv,
FullMethod: "/milvus.proto.master.MasterService/UpdateChannelTimeTick",
}
handler := func(ctx context.Context, req interface{}) (interface{}, error) {
return srv.(MasterServiceServer).UpdateChannelTimeTick(ctx, req.(*internalpb.ChannelTimeTickMsg))
}
return interceptor(ctx, in, info, handler)
}
func _MasterService_GetDdChannel_Handler(srv interface{}, ctx context.Context, dec func(interface{}) error, interceptor grpc.UnaryServerInterceptor) (interface{}, error) {
in := new(internalpb.GetDdChannelRequest)
if err := dec(in); err != nil {
......@@ -1134,6 +1168,10 @@ var _MasterService_serviceDesc = grpc.ServiceDesc{
MethodName: "AllocID",
Handler: _MasterService_AllocID_Handler,
},
{
MethodName: "UpdateChannelTimeTick",
Handler: _MasterService_UpdateChannelTimeTick_Handler,
},
{
MethodName: "GetDdChannel",
Handler: _MasterService_GetDdChannel_Handler,
......
......@@ -104,6 +104,7 @@ type MasterService interface {
//global timestamp allocator
AllocTimestamp(ctx context.Context, req *masterpb.AllocTimestampRequest) (*masterpb.AllocTimestampResponse, error)
AllocID(ctx context.Context, req *masterpb.AllocIDRequest) (*masterpb.AllocIDResponse, error)
UpdateChannelTimeTick(ctx context.Context, req *internalpb.ChannelTimeTickMsg) (*commonpb.Status, error)
//segment
DescribeSegment(ctx context.Context, req *milvuspb.DescribeSegmentRequest) (*milvuspb.DescribeSegmentResponse, error)
......
Markdown is supported
0% .
You are about to add 0 people to the discussion. Proceed with caution.
先完成此消息的编辑!
想要评论请 注册