Add index_builder

Signed-off-by: zhenshan.cao <zhenshan.cao@zilliz.com>
This commit is contained in:
zhenshan.cao 2020-12-10 17:55:55 +08:00 committed by yefu.chen
parent 762b1e1f1f
commit fc054874d3
13 changed files with 1573 additions and 0 deletions

View File

@ -0,0 +1,53 @@
package main
import (
"context"
"log"
"os"
"os/signal"
"syscall"
"github.com/zilliztech/milvus-distributed/internal/indexbuilder"
"go.uber.org/zap"
)
func main() {
indexbuilder.Init()
ctx, cancel := context.WithCancel(context.Background())
svr, err := indexbuilder.CreateBuilder(ctx)
if err != nil {
log.Print("create server failed", zap.Error(err))
}
sc := make(chan os.Signal, 1)
signal.Notify(sc,
syscall.SIGHUP,
syscall.SIGINT,
syscall.SIGTERM,
syscall.SIGQUIT)
var sig os.Signal
go func() {
sig = <-sc
cancel()
}()
if err := svr.Start(); err != nil {
log.Fatal("run builder server failed", zap.Error(err))
}
<-ctx.Done()
log.Print("Got signal to exit", zap.String("signal", sig.String()))
svr.Close()
switch sig {
case syscall.SIGTERM:
exit(0)
default:
exit(1)
}
}
func exit(code int) {
os.Exit(code)
}

View File

@ -47,3 +47,7 @@ proxy:
queryNode: queryNode:
gracefulTime: 5000 #ms gracefulTime: 5000 #ms
indexBuilder:
address: localhost
port: 310310

View File

@ -0,0 +1,24 @@
package indexbuilder
import (
"context"
"time"
"github.com/zilliztech/milvus-distributed/internal/proto/indexbuilderpb"
)
const (
reqTimeoutInterval = time.Second * 10
)
func (b *Builder) BuildIndex(ctx context.Context, request *indexbuilderpb.BuildIndexRequest) (*indexbuilderpb.BuildIndexResponse, error) {
panic("implement me")
}
func (b *Builder) DescribeIndex(ctx context.Context, request *indexbuilderpb.DescribleIndexRequest) (*indexbuilderpb.DescribleIndexResponse, error) {
panic("implement me")
}
func (b *Builder) GetIndexFilePaths(ctx context.Context, request *indexbuilderpb.GetIndexFilePathsRequest) (*indexbuilderpb.GetIndexFilePathsResponse, error) {
panic("implement me")
}

View File

@ -0,0 +1,141 @@
package indexbuilder
import (
"context"
etcdkv "github.com/zilliztech/milvus-distributed/internal/kv/etcd"
"go.etcd.io/etcd/clientv3"
"log"
"math/rand"
"net"
"strconv"
"sync"
"time"
"github.com/zilliztech/milvus-distributed/internal/allocator"
"github.com/zilliztech/milvus-distributed/internal/proto/indexbuilderpb"
"github.com/zilliztech/milvus-distributed/internal/util/typeutil"
"google.golang.org/grpc"
)
type UniqueID = typeutil.UniqueID
type Builder struct {
loopCtx context.Context
loopCancel func()
loopWg sync.WaitGroup
grpcServer *grpc.Server
sched *TaskScheduler
idAllocator *allocator.IDAllocator
metaTable *metaTable
// Add callback functions at different stages
startCallbacks []func()
closeCallbacks []func()
}
func Init() {
rand.Seed(time.Now().UnixNano())
Params.Init()
}
func CreateBuilder(ctx context.Context) (*Builder, error) {
ctx1, cancel := context.WithCancel(ctx)
b := &Builder{
loopCtx: ctx1,
loopCancel: cancel,
}
etcdAddress := Params.EtcdAddress
etcdClient, err := clientv3.New(clientv3.Config{Endpoints: []string{etcdAddress}})
if err != nil {
return nil, err
}
etcdKV := etcdkv.NewEtcdKV(etcdClient, Params.MetaRootPath)
metakv, err := NewMetaTable(etcdKV)
if err != nil {
return nil, err
}
b.metaTable = metakv
idAllocator, err := allocator.NewIDAllocator(b.loopCtx, Params.MasterAddress)
if err != nil {
return nil, err
}
b.idAllocator = idAllocator
b.sched, err = NewTaskScheduler(b.loopCtx, b.idAllocator)
if err != nil {
return nil, err
}
return b, nil
}
// AddStartCallback adds a callback in the startServer phase.
func (b *Builder) AddStartCallback(callbacks ...func()) {
b.startCallbacks = append(b.startCallbacks, callbacks...)
}
func (b *Builder) startBuilder() error {
b.sched.Start()
// Start callbacks
for _, cb := range b.startCallbacks {
cb()
}
b.loopWg.Add(1)
go b.grpcLoop()
return nil
}
// AddCloseCallback adds a callback in the Close phase.
func (b *Builder) AddCloseCallback(callbacks ...func()) {
b.closeCallbacks = append(b.closeCallbacks, callbacks...)
}
func (b *Builder) grpcLoop() {
defer b.loopWg.Done()
lis, err := net.Listen("tcp", ":"+strconv.Itoa(Params.Port))
if err != nil {
log.Fatalf("Builder grpc server fatal error=%v", err)
}
b.grpcServer = grpc.NewServer()
indexbuilderpb.RegisterIndexBuildServiceServer(b.grpcServer, b)
if err = b.grpcServer.Serve(lis); err != nil {
log.Fatalf("Builder grpc server fatal error=%v", err)
}
}
func (b *Builder) Start() error {
return b.startBuilder()
}
func (b *Builder) stopBuilderLoop() {
b.loopCancel()
if b.grpcServer != nil {
b.grpcServer.GracefulStop()
}
b.sched.Close()
b.loopWg.Wait()
}
// Close closes the server.
func (b *Builder) Close() {
b.stopBuilderLoop()
for _, cb := range b.closeCallbacks {
cb()
}
log.Print("builder closed.")
}

View File

@ -0,0 +1,62 @@
package indexbuilder
import (
"context"
"github.com/zilliztech/milvus-distributed/internal/proto/indexbuilderpb"
"log"
"os"
"testing"
"go.uber.org/zap"
"google.golang.org/grpc"
)
var ctx context.Context
var cancel func()
var clientConn *grpc.ClientConn
var buildClient indexbuilderpb.IndexBuildServiceClient
var builderServer *Builder
var testNum = 10
func startBuilder(ctx context.Context) {
builderServer, err := CreateBuilder(ctx)
if err != nil {
log.Print("create builder failed", zap.Error(err))
}
// TODO: change to wait until master is ready
if err := builderServer.Start(); err != nil {
log.Fatal("run builder failed", zap.Error(err))
}
}
func setup() {
Params.Init()
ctx, cancel = context.WithCancel(context.Background())
startBuilder(ctx)
addr := Params.Address
conn, err := grpc.DialContext(ctx, addr, grpc.WithInsecure(), grpc.WithBlock())
if err != nil {
log.Fatalf("Connect to builder server failed, error= %v", err)
}
clientConn = conn
buildClient = indexbuilderpb.NewIndexBuildServiceClient(clientConn)
}
func shutdown() {
cancel()
builderServer.Close()
}
func TestMain(m *testing.M) {
setup()
code := m.Run()
shutdown()
os.Exit(code)
}

View File

@ -0,0 +1,99 @@
package indexbuilder
import (
"fmt"
"strconv"
"sync"
"github.com/golang/protobuf/proto"
"github.com/zilliztech/milvus-distributed/internal/errors"
"github.com/zilliztech/milvus-distributed/internal/kv"
pb "github.com/zilliztech/milvus-distributed/internal/proto/indexbuilderpb"
)
type metaTable struct {
client kv.TxnBase // client of a reliable kv service, i.e. etcd client
indexID2Meta map[UniqueID]pb.IndexMeta // index id to index meta
lock sync.RWMutex
}
func NewMetaTable(kv kv.TxnBase) (*metaTable, error) {
mt := &metaTable{
client: kv,
lock: sync.RWMutex{},
}
err := mt.reloadFromKV()
if err != nil {
return nil, err
}
return mt, nil
}
func (mt *metaTable) reloadFromKV() error {
mt.indexID2Meta = make(map[UniqueID]pb.IndexMeta)
_, values, err := mt.client.LoadWithPrefix("indexes")
if err != nil {
return err
}
for _, value := range values {
indexMeta := pb.IndexMeta{}
err = proto.UnmarshalText(value, &indexMeta)
if err != nil {
return err
}
mt.indexID2Meta[indexMeta.IndexID] = indexMeta
}
return nil
}
// metaTable.lock.Lock() before call this function
func (mt *metaTable) saveIndexMeta(meta *pb.IndexMeta) error {
value := proto.MarshalTextString(meta)
mt.indexID2Meta[meta.IndexID] = *meta
return mt.client.Save("/indexes/"+strconv.FormatInt(meta.IndexID, 10), value)
}
func (mt *metaTable) AddIndex(meta *pb.IndexMeta) error {
mt.lock.Lock()
defer mt.lock.Unlock()
return nil
}
func (mt *metaTable) UpdateIndex(meta *pb.IndexMeta) error {
mt.lock.Lock()
defer mt.lock.Unlock()
return nil
}
func (mt *metaTable) GetIndexByID(indexID UniqueID) (*pb.IndexMeta, error) {
mt.lock.RLock()
defer mt.lock.RUnlock()
sm, ok := mt.indexID2Meta[indexID]
if !ok {
return nil, errors.Errorf("can't find index id = %d", indexID)
}
return &sm, nil
}
func (mt *metaTable) DeleteIndex(indexID UniqueID) error {
mt.lock.Lock()
defer mt.lock.Unlock()
indexMeta, ok := mt.indexID2Meta[indexID]
if !ok {
return errors.Errorf("can't find index. id = " + strconv.FormatInt(indexID, 10))
}
fmt.Print(indexMeta)
return nil
}

View File

@ -0,0 +1,85 @@
package indexbuilder
import (
"net"
"strconv"
"github.com/zilliztech/milvus-distributed/internal/util/paramtable"
)
type ParamTable struct {
paramtable.BaseTable
Address string
Port int
MasterAddress string
EtcdAddress string
MetaRootPath string
}
var Params ParamTable
func (pt *ParamTable) Init() {
pt.BaseTable.Init()
pt.initAddress()
pt.initPort()
}
func (pt *ParamTable) initAddress() {
addr, err := pt.Load("indexBuilder.address")
if err != nil {
panic(err)
}
hostName, _ := net.LookupHost(addr)
if len(hostName) <= 0 {
if ip := net.ParseIP(addr); ip == nil {
panic("invalid ip indexBuilder.address")
}
}
port, err := pt.Load("indexBuilder.port")
if err != nil {
panic(err)
}
_, err = strconv.Atoi(port)
if err != nil {
panic(err)
}
pt.Address = addr + ":" + port
}
func (pt *ParamTable) initPort() {
pt.Port = pt.ParseInt("indexBuilder.port")
}
func (pt *ParamTable) initEtcdAddress() {
addr, err := pt.Load("_EtcdAddress")
if err != nil {
panic(err)
}
pt.EtcdAddress = addr
}
func (pt *ParamTable) initMetaRootPath() {
rootPath, err := pt.Load("etcd.rootPath")
if err != nil {
panic(err)
}
subPath, err := pt.Load("etcd.metaSubPath")
if err != nil {
panic(err)
}
pt.MetaRootPath = rootPath + "/" + subPath
}
func (pt *ParamTable) initMasterAddress() {
ret, err := pt.Load("_MasterAddress")
if err != nil {
panic(err)
}
pt.MasterAddress = ret
}

View File

@ -0,0 +1,26 @@
package indexbuilder
import (
"testing"
"github.com/stretchr/testify/assert"
)
func TestParamTable_Init(t *testing.T) {
Params.Init()
}
func TestParamTable_Address(t *testing.T) {
address := Params.Address
assert.Equal(t, address, "localhost")
}
func TestParamTable_Port(t *testing.T) {
port := Params.Port
assert.Equal(t, port, 310310)
}
func TestParamTable_MetaRootPath(t *testing.T) {
path := Params.MetaRootPath
assert.Equal(t, path, "by-dev/meta")
}

View File

@ -0,0 +1,82 @@
package indexbuilder
import (
"context"
"github.com/zilliztech/milvus-distributed/internal/allocator"
"github.com/zilliztech/milvus-distributed/internal/errors"
)
type task interface {
ID() UniqueID // return ReqID
SetID(uid UniqueID) // set ReqID
PreExecute() error
Execute() error
PostExecute() error
WaitToFinish() error
Notify(err error)
}
type BaseTask struct {
done chan error
ctx context.Context
id UniqueID
}
func (bt *BaseTask) ID() UniqueID {
return bt.id
}
func (bt *BaseTask) setID(id UniqueID) {
bt.id = id
}
func (bt *BaseTask) WaitToFinish() error {
for {
select {
case <-bt.ctx.Done():
return errors.New("timeout")
case err := <-bt.done:
return err
}
}
}
func (bt *BaseTask) Notify(err error) {
bt.done <- err
}
type IndexBuildTask struct {
BaseTask
rowIDAllocator *allocator.IDAllocator
}
func (it *IndexBuildTask) PreExecute() error {
return nil
}
func (it *IndexBuildTask) Execute() error {
return nil
}
func (it *IndexBuildTask) PostExecute() error {
return nil
}
type DescribeIndexTask struct {
BaseTask
ctx context.Context
}
func (dct *DescribeIndexTask) PreExecute() error {
return nil
}
func (dct *DescribeIndexTask) Execute() error {
return nil
}
func (dct *DescribeIndexTask) PostExecute() error {
return nil
}

View File

@ -0,0 +1,271 @@
package indexbuilder
import (
"container/list"
"context"
"errors"
"log"
"sync"
"github.com/zilliztech/milvus-distributed/internal/allocator"
)
type TaskQueue interface {
utChan() <-chan int
utEmpty() bool
utFull() bool
addUnissuedTask(t task) error
FrontUnissuedTask() task
PopUnissuedTask() task
AddActiveTask(t task)
PopActiveTask(tID UniqueID) task
Enqueue(t task) error
}
type BaseTaskQueue struct {
unissuedTasks *list.List
activeTasks map[UniqueID]task
utLock sync.Mutex
atLock sync.Mutex
// maxTaskNum should keep still
maxTaskNum int64
utBufChan chan int // to block scheduler
sched *TaskScheduler
}
func (queue *BaseTaskQueue) utChan() <-chan int {
return queue.utBufChan
}
func (queue *BaseTaskQueue) utEmpty() bool {
return queue.unissuedTasks.Len() == 0
}
func (queue *BaseTaskQueue) utFull() bool {
return int64(queue.unissuedTasks.Len()) >= queue.maxTaskNum
}
func (queue *BaseTaskQueue) addUnissuedTask(t task) error {
queue.utLock.Lock()
defer queue.utLock.Unlock()
if queue.utFull() {
return errors.New("task queue is full")
}
queue.unissuedTasks.PushBack(t)
queue.utBufChan <- 1
return nil
}
func (queue *BaseTaskQueue) FrontUnissuedTask() task {
queue.utLock.Lock()
defer queue.utLock.Unlock()
if queue.unissuedTasks.Len() <= 0 {
log.Panic("sorry, but the unissued task list is empty!")
return nil
}
return queue.unissuedTasks.Front().Value.(task)
}
func (queue *BaseTaskQueue) PopUnissuedTask() task {
queue.utLock.Lock()
defer queue.utLock.Unlock()
if queue.unissuedTasks.Len() <= 0 {
log.Fatal("sorry, but the unissued task list is empty!")
return nil
}
ft := queue.unissuedTasks.Front()
queue.unissuedTasks.Remove(ft)
return ft.Value.(task)
}
func (queue *BaseTaskQueue) AddActiveTask(t task) {
queue.atLock.Lock()
defer queue.atLock.Unlock()
tID := t.ID()
_, ok := queue.activeTasks[tID]
if ok {
log.Fatalf("task with ID %v already in active task list!", tID)
}
queue.activeTasks[tID] = t
}
func (queue *BaseTaskQueue) PopActiveTask(tID UniqueID) task {
queue.atLock.Lock()
defer queue.atLock.Unlock()
t, ok := queue.activeTasks[tID]
if ok {
delete(queue.activeTasks, tID)
return t
}
log.Fatalf("sorry, but the ID %d was not found in the active task list!", tID)
return nil
}
func (queue *BaseTaskQueue) Enqueue(t task) error {
tID, _ := queue.sched.idAllocator.AllocOne()
log.Printf("[Builder] allocate reqID: %v", tID)
t.SetID(tID)
return queue.addUnissuedTask(t)
}
type DdTaskQueue struct {
BaseTaskQueue
lock sync.Mutex
}
type DescribleTaskQueue struct {
BaseTaskQueue
}
type IndexBuildTaskQueue struct {
BaseTaskQueue
}
func (queue *DdTaskQueue) Enqueue(t task) error {
queue.lock.Lock()
defer queue.lock.Unlock()
return queue.BaseTaskQueue.Enqueue(t)
}
func NewDescribleTaskQueue(sched *TaskScheduler) *DescribleTaskQueue {
return &DescribleTaskQueue{
BaseTaskQueue: BaseTaskQueue{
unissuedTasks: list.New(),
activeTasks: make(map[UniqueID]task),
maxTaskNum: 1024,
utBufChan: make(chan int, 1024),
sched: sched,
},
}
}
func NewIndexBuildTaskQueue(sched *TaskScheduler) *IndexBuildTaskQueue {
return &IndexBuildTaskQueue{
BaseTaskQueue: BaseTaskQueue{
unissuedTasks: list.New(),
activeTasks: make(map[UniqueID]task),
maxTaskNum: 1024,
utBufChan: make(chan int, 1024),
sched: sched,
},
}
}
type TaskScheduler struct {
DescribeQueue TaskQueue
IndexBuildQueue TaskQueue
idAllocator *allocator.IDAllocator
wg sync.WaitGroup
ctx context.Context
cancel context.CancelFunc
}
func NewTaskScheduler(ctx context.Context,
idAllocator *allocator.IDAllocator) (*TaskScheduler, error) {
ctx1, cancel := context.WithCancel(ctx)
s := &TaskScheduler{
idAllocator: idAllocator,
ctx: ctx1,
cancel: cancel,
}
s.DescribeQueue = NewDescribleTaskQueue(s)
s.IndexBuildQueue = NewIndexBuildTaskQueue(s)
return s, nil
}
func (sched *TaskScheduler) scheduleDescribleTask() task {
return sched.DescribeQueue.PopUnissuedTask()
}
func (sched *TaskScheduler) scheduleIndexBuildTask() task {
return sched.IndexBuildQueue.PopUnissuedTask()
}
func (sched *TaskScheduler) processTask(t task, q TaskQueue) {
err := t.PreExecute()
defer func() {
t.Notify(err)
log.Printf("notify with error: %v", err)
}()
if err != nil {
return
}
q.AddActiveTask(t)
log.Printf("task add to active list ...")
defer func() {
q.PopActiveTask(t.ID())
log.Printf("pop from active list ...")
}()
err = t.Execute()
if err != nil {
log.Printf("execute definition task failed, error = %v", err)
return
}
log.Printf("task execution done ...")
err = t.PostExecute()
log.Printf("post execute task done ...")
}
func (sched *TaskScheduler) indexBuildingLoop() {
defer sched.wg.Done()
for {
select {
case <-sched.ctx.Done():
return
case <-sched.IndexBuildQueue.utChan():
if !sched.IndexBuildQueue.utEmpty() {
t := sched.scheduleIndexBuildTask()
sched.processTask(t, sched.IndexBuildQueue)
}
}
}
}
func (sched *TaskScheduler) describeLoop() {
defer sched.wg.Done()
for {
select {
case <-sched.ctx.Done():
return
case <-sched.DescribeQueue.utChan():
if !sched.DescribeQueue.utEmpty() {
t := sched.scheduleDescribleTask()
go sched.processTask(t, sched.DescribeQueue)
}
}
}
}
func (sched *TaskScheduler) Start() error {
sched.wg.Add(1)
go sched.indexBuildingLoop()
sched.wg.Add(1)
go sched.describeLoop()
return nil
}
func (sched *TaskScheduler) Close() {
sched.cancel()
sched.wg.Wait()
}

View File

@ -0,0 +1,70 @@
syntax = "proto3";
package milvus.proto.service;
option go_package="github.com/zilliztech/milvus-distributed/internal/proto/indexbuilderpb";
import "common.proto";
enum IndexStatus {
NONE = 0;
UNISSUED = 1;
INPROGRESS = 2;
FINISHED = 3;
}
message BuildIndexRequest {
repeated string data_paths=2;
repeated common.KeyValuePair type_params = 3;
repeated common.KeyValuePair index_params = 4;
}
message BuildIndexResponse {
common.Status status = 1;
int64 indexID = 2;
}
message GetIndexFilePathsRequest {
int64 indexID = 1;
}
message GetIndexFilePathsResponse {
common.Status status = 1;
int64 indexID = 2;
repeated string index_file_paths=3;
}
message DescribleIndexRequest {
int64 indexID = 1;
}
message DescribleIndexResponse {
common.Status status = 1;
IndexStatus index_status =2;
int64 indexID = 3;
repeated string index_file_paths=4;
uint64 enque_time = 5;
uint64 schedule_time = 6;
uint64 build_complete_time = 7;
}
message IndexMeta {
IndexStatus status =1;
int64 indexID = 2;
BuildIndexRequest req = 3;
}
service IndexBuildService {
/**
* @brief This method is used to create collection
*
* @param CollectionSchema, use to provide collection information to be created.
*
* @return Status
*/
rpc BuildIndex(BuildIndexRequest) returns (BuildIndexResponse){}
rpc DescribeIndex(DescribleIndexRequest) returns (DescribleIndexResponse){}
rpc GetIndexFilePaths(GetIndexFilePathsRequest) returns (GetIndexFilePathsResponse){}
}

View File

@ -0,0 +1,654 @@
// Code generated by protoc-gen-go. DO NOT EDIT.
// source: index_builder.proto
package indexbuilderpb
import (
context "context"
fmt "fmt"
proto "github.com/golang/protobuf/proto"
commonpb "github.com/zilliztech/milvus-distributed/internal/proto/commonpb"
grpc "google.golang.org/grpc"
codes "google.golang.org/grpc/codes"
status "google.golang.org/grpc/status"
math "math"
)
// Reference imports to suppress errors if they are not otherwise used.
var _ = proto.Marshal
var _ = fmt.Errorf
var _ = math.Inf
// This is a compile-time assertion to ensure that this generated file
// is compatible with the proto package it is being compiled against.
// A compilation error at this line likely means your copy of the
// proto package needs to be updated.
const _ = proto.ProtoPackageIsVersion3 // please upgrade the proto package
type IndexStatus int32
const (
IndexStatus_NONE IndexStatus = 0
IndexStatus_UNISSUED IndexStatus = 1
IndexStatus_INPROGRESS IndexStatus = 2
IndexStatus_FINISHED IndexStatus = 3
)
var IndexStatus_name = map[int32]string{
0: "NONE",
1: "UNISSUED",
2: "INPROGRESS",
3: "FINISHED",
}
var IndexStatus_value = map[string]int32{
"NONE": 0,
"UNISSUED": 1,
"INPROGRESS": 2,
"FINISHED": 3,
}
func (x IndexStatus) String() string {
return proto.EnumName(IndexStatus_name, int32(x))
}
func (IndexStatus) EnumDescriptor() ([]byte, []int) {
return fileDescriptor_c1d6a79d693ba681, []int{0}
}
type BuildIndexRequest struct {
DataPaths []string `protobuf:"bytes,2,rep,name=data_paths,json=dataPaths,proto3" json:"data_paths,omitempty"`
TypeParams []*commonpb.KeyValuePair `protobuf:"bytes,3,rep,name=type_params,json=typeParams,proto3" json:"type_params,omitempty"`
IndexParams []*commonpb.KeyValuePair `protobuf:"bytes,4,rep,name=index_params,json=indexParams,proto3" json:"index_params,omitempty"`
XXX_NoUnkeyedLiteral struct{} `json:"-"`
XXX_unrecognized []byte `json:"-"`
XXX_sizecache int32 `json:"-"`
}
func (m *BuildIndexRequest) Reset() { *m = BuildIndexRequest{} }
func (m *BuildIndexRequest) String() string { return proto.CompactTextString(m) }
func (*BuildIndexRequest) ProtoMessage() {}
func (*BuildIndexRequest) Descriptor() ([]byte, []int) {
return fileDescriptor_c1d6a79d693ba681, []int{0}
}
func (m *BuildIndexRequest) XXX_Unmarshal(b []byte) error {
return xxx_messageInfo_BuildIndexRequest.Unmarshal(m, b)
}
func (m *BuildIndexRequest) XXX_Marshal(b []byte, deterministic bool) ([]byte, error) {
return xxx_messageInfo_BuildIndexRequest.Marshal(b, m, deterministic)
}
func (m *BuildIndexRequest) XXX_Merge(src proto.Message) {
xxx_messageInfo_BuildIndexRequest.Merge(m, src)
}
func (m *BuildIndexRequest) XXX_Size() int {
return xxx_messageInfo_BuildIndexRequest.Size(m)
}
func (m *BuildIndexRequest) XXX_DiscardUnknown() {
xxx_messageInfo_BuildIndexRequest.DiscardUnknown(m)
}
var xxx_messageInfo_BuildIndexRequest proto.InternalMessageInfo
func (m *BuildIndexRequest) GetDataPaths() []string {
if m != nil {
return m.DataPaths
}
return nil
}
func (m *BuildIndexRequest) GetTypeParams() []*commonpb.KeyValuePair {
if m != nil {
return m.TypeParams
}
return nil
}
func (m *BuildIndexRequest) GetIndexParams() []*commonpb.KeyValuePair {
if m != nil {
return m.IndexParams
}
return nil
}
type BuildIndexResponse struct {
Status *commonpb.Status `protobuf:"bytes,1,opt,name=status,proto3" json:"status,omitempty"`
IndexID int64 `protobuf:"varint,2,opt,name=indexID,proto3" json:"indexID,omitempty"`
XXX_NoUnkeyedLiteral struct{} `json:"-"`
XXX_unrecognized []byte `json:"-"`
XXX_sizecache int32 `json:"-"`
}
func (m *BuildIndexResponse) Reset() { *m = BuildIndexResponse{} }
func (m *BuildIndexResponse) String() string { return proto.CompactTextString(m) }
func (*BuildIndexResponse) ProtoMessage() {}
func (*BuildIndexResponse) Descriptor() ([]byte, []int) {
return fileDescriptor_c1d6a79d693ba681, []int{1}
}
func (m *BuildIndexResponse) XXX_Unmarshal(b []byte) error {
return xxx_messageInfo_BuildIndexResponse.Unmarshal(m, b)
}
func (m *BuildIndexResponse) XXX_Marshal(b []byte, deterministic bool) ([]byte, error) {
return xxx_messageInfo_BuildIndexResponse.Marshal(b, m, deterministic)
}
func (m *BuildIndexResponse) XXX_Merge(src proto.Message) {
xxx_messageInfo_BuildIndexResponse.Merge(m, src)
}
func (m *BuildIndexResponse) XXX_Size() int {
return xxx_messageInfo_BuildIndexResponse.Size(m)
}
func (m *BuildIndexResponse) XXX_DiscardUnknown() {
xxx_messageInfo_BuildIndexResponse.DiscardUnknown(m)
}
var xxx_messageInfo_BuildIndexResponse proto.InternalMessageInfo
func (m *BuildIndexResponse) GetStatus() *commonpb.Status {
if m != nil {
return m.Status
}
return nil
}
func (m *BuildIndexResponse) GetIndexID() int64 {
if m != nil {
return m.IndexID
}
return 0
}
type GetIndexFilePathsRequest struct {
IndexID int64 `protobuf:"varint,1,opt,name=indexID,proto3" json:"indexID,omitempty"`
XXX_NoUnkeyedLiteral struct{} `json:"-"`
XXX_unrecognized []byte `json:"-"`
XXX_sizecache int32 `json:"-"`
}
func (m *GetIndexFilePathsRequest) Reset() { *m = GetIndexFilePathsRequest{} }
func (m *GetIndexFilePathsRequest) String() string { return proto.CompactTextString(m) }
func (*GetIndexFilePathsRequest) ProtoMessage() {}
func (*GetIndexFilePathsRequest) Descriptor() ([]byte, []int) {
return fileDescriptor_c1d6a79d693ba681, []int{2}
}
func (m *GetIndexFilePathsRequest) XXX_Unmarshal(b []byte) error {
return xxx_messageInfo_GetIndexFilePathsRequest.Unmarshal(m, b)
}
func (m *GetIndexFilePathsRequest) XXX_Marshal(b []byte, deterministic bool) ([]byte, error) {
return xxx_messageInfo_GetIndexFilePathsRequest.Marshal(b, m, deterministic)
}
func (m *GetIndexFilePathsRequest) XXX_Merge(src proto.Message) {
xxx_messageInfo_GetIndexFilePathsRequest.Merge(m, src)
}
func (m *GetIndexFilePathsRequest) XXX_Size() int {
return xxx_messageInfo_GetIndexFilePathsRequest.Size(m)
}
func (m *GetIndexFilePathsRequest) XXX_DiscardUnknown() {
xxx_messageInfo_GetIndexFilePathsRequest.DiscardUnknown(m)
}
var xxx_messageInfo_GetIndexFilePathsRequest proto.InternalMessageInfo
func (m *GetIndexFilePathsRequest) GetIndexID() int64 {
if m != nil {
return m.IndexID
}
return 0
}
type GetIndexFilePathsResponse struct {
Status *commonpb.Status `protobuf:"bytes,1,opt,name=status,proto3" json:"status,omitempty"`
IndexID int64 `protobuf:"varint,2,opt,name=indexID,proto3" json:"indexID,omitempty"`
IndexFilePaths []string `protobuf:"bytes,3,rep,name=index_file_paths,json=indexFilePaths,proto3" json:"index_file_paths,omitempty"`
XXX_NoUnkeyedLiteral struct{} `json:"-"`
XXX_unrecognized []byte `json:"-"`
XXX_sizecache int32 `json:"-"`
}
func (m *GetIndexFilePathsResponse) Reset() { *m = GetIndexFilePathsResponse{} }
func (m *GetIndexFilePathsResponse) String() string { return proto.CompactTextString(m) }
func (*GetIndexFilePathsResponse) ProtoMessage() {}
func (*GetIndexFilePathsResponse) Descriptor() ([]byte, []int) {
return fileDescriptor_c1d6a79d693ba681, []int{3}
}
func (m *GetIndexFilePathsResponse) XXX_Unmarshal(b []byte) error {
return xxx_messageInfo_GetIndexFilePathsResponse.Unmarshal(m, b)
}
func (m *GetIndexFilePathsResponse) XXX_Marshal(b []byte, deterministic bool) ([]byte, error) {
return xxx_messageInfo_GetIndexFilePathsResponse.Marshal(b, m, deterministic)
}
func (m *GetIndexFilePathsResponse) XXX_Merge(src proto.Message) {
xxx_messageInfo_GetIndexFilePathsResponse.Merge(m, src)
}
func (m *GetIndexFilePathsResponse) XXX_Size() int {
return xxx_messageInfo_GetIndexFilePathsResponse.Size(m)
}
func (m *GetIndexFilePathsResponse) XXX_DiscardUnknown() {
xxx_messageInfo_GetIndexFilePathsResponse.DiscardUnknown(m)
}
var xxx_messageInfo_GetIndexFilePathsResponse proto.InternalMessageInfo
func (m *GetIndexFilePathsResponse) GetStatus() *commonpb.Status {
if m != nil {
return m.Status
}
return nil
}
func (m *GetIndexFilePathsResponse) GetIndexID() int64 {
if m != nil {
return m.IndexID
}
return 0
}
func (m *GetIndexFilePathsResponse) GetIndexFilePaths() []string {
if m != nil {
return m.IndexFilePaths
}
return nil
}
type DescribleIndexRequest struct {
IndexID int64 `protobuf:"varint,1,opt,name=indexID,proto3" json:"indexID,omitempty"`
XXX_NoUnkeyedLiteral struct{} `json:"-"`
XXX_unrecognized []byte `json:"-"`
XXX_sizecache int32 `json:"-"`
}
func (m *DescribleIndexRequest) Reset() { *m = DescribleIndexRequest{} }
func (m *DescribleIndexRequest) String() string { return proto.CompactTextString(m) }
func (*DescribleIndexRequest) ProtoMessage() {}
func (*DescribleIndexRequest) Descriptor() ([]byte, []int) {
return fileDescriptor_c1d6a79d693ba681, []int{4}
}
func (m *DescribleIndexRequest) XXX_Unmarshal(b []byte) error {
return xxx_messageInfo_DescribleIndexRequest.Unmarshal(m, b)
}
func (m *DescribleIndexRequest) XXX_Marshal(b []byte, deterministic bool) ([]byte, error) {
return xxx_messageInfo_DescribleIndexRequest.Marshal(b, m, deterministic)
}
func (m *DescribleIndexRequest) XXX_Merge(src proto.Message) {
xxx_messageInfo_DescribleIndexRequest.Merge(m, src)
}
func (m *DescribleIndexRequest) XXX_Size() int {
return xxx_messageInfo_DescribleIndexRequest.Size(m)
}
func (m *DescribleIndexRequest) XXX_DiscardUnknown() {
xxx_messageInfo_DescribleIndexRequest.DiscardUnknown(m)
}
var xxx_messageInfo_DescribleIndexRequest proto.InternalMessageInfo
func (m *DescribleIndexRequest) GetIndexID() int64 {
if m != nil {
return m.IndexID
}
return 0
}
type DescribleIndexResponse struct {
Status *commonpb.Status `protobuf:"bytes,1,opt,name=status,proto3" json:"status,omitempty"`
IndexStatus IndexStatus `protobuf:"varint,2,opt,name=index_status,json=indexStatus,proto3,enum=milvus.proto.service.IndexStatus" json:"index_status,omitempty"`
IndexID int64 `protobuf:"varint,3,opt,name=indexID,proto3" json:"indexID,omitempty"`
IndexFilePaths []string `protobuf:"bytes,4,rep,name=index_file_paths,json=indexFilePaths,proto3" json:"index_file_paths,omitempty"`
EnqueTime uint64 `protobuf:"varint,5,opt,name=enque_time,json=enqueTime,proto3" json:"enque_time,omitempty"`
ScheduleTime uint64 `protobuf:"varint,6,opt,name=schedule_time,json=scheduleTime,proto3" json:"schedule_time,omitempty"`
BuildCompleteTime uint64 `protobuf:"varint,7,opt,name=build_complete_time,json=buildCompleteTime,proto3" json:"build_complete_time,omitempty"`
XXX_NoUnkeyedLiteral struct{} `json:"-"`
XXX_unrecognized []byte `json:"-"`
XXX_sizecache int32 `json:"-"`
}
func (m *DescribleIndexResponse) Reset() { *m = DescribleIndexResponse{} }
func (m *DescribleIndexResponse) String() string { return proto.CompactTextString(m) }
func (*DescribleIndexResponse) ProtoMessage() {}
func (*DescribleIndexResponse) Descriptor() ([]byte, []int) {
return fileDescriptor_c1d6a79d693ba681, []int{5}
}
func (m *DescribleIndexResponse) XXX_Unmarshal(b []byte) error {
return xxx_messageInfo_DescribleIndexResponse.Unmarshal(m, b)
}
func (m *DescribleIndexResponse) XXX_Marshal(b []byte, deterministic bool) ([]byte, error) {
return xxx_messageInfo_DescribleIndexResponse.Marshal(b, m, deterministic)
}
func (m *DescribleIndexResponse) XXX_Merge(src proto.Message) {
xxx_messageInfo_DescribleIndexResponse.Merge(m, src)
}
func (m *DescribleIndexResponse) XXX_Size() int {
return xxx_messageInfo_DescribleIndexResponse.Size(m)
}
func (m *DescribleIndexResponse) XXX_DiscardUnknown() {
xxx_messageInfo_DescribleIndexResponse.DiscardUnknown(m)
}
var xxx_messageInfo_DescribleIndexResponse proto.InternalMessageInfo
func (m *DescribleIndexResponse) GetStatus() *commonpb.Status {
if m != nil {
return m.Status
}
return nil
}
func (m *DescribleIndexResponse) GetIndexStatus() IndexStatus {
if m != nil {
return m.IndexStatus
}
return IndexStatus_NONE
}
func (m *DescribleIndexResponse) GetIndexID() int64 {
if m != nil {
return m.IndexID
}
return 0
}
func (m *DescribleIndexResponse) GetIndexFilePaths() []string {
if m != nil {
return m.IndexFilePaths
}
return nil
}
func (m *DescribleIndexResponse) GetEnqueTime() uint64 {
if m != nil {
return m.EnqueTime
}
return 0
}
func (m *DescribleIndexResponse) GetScheduleTime() uint64 {
if m != nil {
return m.ScheduleTime
}
return 0
}
func (m *DescribleIndexResponse) GetBuildCompleteTime() uint64 {
if m != nil {
return m.BuildCompleteTime
}
return 0
}
type IndexMeta struct {
Status IndexStatus `protobuf:"varint,1,opt,name=status,proto3,enum=milvus.proto.service.IndexStatus" json:"status,omitempty"`
IndexID int64 `protobuf:"varint,2,opt,name=indexID,proto3" json:"indexID,omitempty"`
Req *BuildIndexRequest `protobuf:"bytes,3,opt,name=req,proto3" json:"req,omitempty"`
XXX_NoUnkeyedLiteral struct{} `json:"-"`
XXX_unrecognized []byte `json:"-"`
XXX_sizecache int32 `json:"-"`
}
func (m *IndexMeta) Reset() { *m = IndexMeta{} }
func (m *IndexMeta) String() string { return proto.CompactTextString(m) }
func (*IndexMeta) ProtoMessage() {}
func (*IndexMeta) Descriptor() ([]byte, []int) {
return fileDescriptor_c1d6a79d693ba681, []int{6}
}
func (m *IndexMeta) XXX_Unmarshal(b []byte) error {
return xxx_messageInfo_IndexMeta.Unmarshal(m, b)
}
func (m *IndexMeta) XXX_Marshal(b []byte, deterministic bool) ([]byte, error) {
return xxx_messageInfo_IndexMeta.Marshal(b, m, deterministic)
}
func (m *IndexMeta) XXX_Merge(src proto.Message) {
xxx_messageInfo_IndexMeta.Merge(m, src)
}
func (m *IndexMeta) XXX_Size() int {
return xxx_messageInfo_IndexMeta.Size(m)
}
func (m *IndexMeta) XXX_DiscardUnknown() {
xxx_messageInfo_IndexMeta.DiscardUnknown(m)
}
var xxx_messageInfo_IndexMeta proto.InternalMessageInfo
func (m *IndexMeta) GetStatus() IndexStatus {
if m != nil {
return m.Status
}
return IndexStatus_NONE
}
func (m *IndexMeta) GetIndexID() int64 {
if m != nil {
return m.IndexID
}
return 0
}
func (m *IndexMeta) GetReq() *BuildIndexRequest {
if m != nil {
return m.Req
}
return nil
}
func init() {
proto.RegisterEnum("milvus.proto.service.IndexStatus", IndexStatus_name, IndexStatus_value)
proto.RegisterType((*BuildIndexRequest)(nil), "milvus.proto.service.BuildIndexRequest")
proto.RegisterType((*BuildIndexResponse)(nil), "milvus.proto.service.BuildIndexResponse")
proto.RegisterType((*GetIndexFilePathsRequest)(nil), "milvus.proto.service.GetIndexFilePathsRequest")
proto.RegisterType((*GetIndexFilePathsResponse)(nil), "milvus.proto.service.GetIndexFilePathsResponse")
proto.RegisterType((*DescribleIndexRequest)(nil), "milvus.proto.service.DescribleIndexRequest")
proto.RegisterType((*DescribleIndexResponse)(nil), "milvus.proto.service.DescribleIndexResponse")
proto.RegisterType((*IndexMeta)(nil), "milvus.proto.service.IndexMeta")
}
func init() { proto.RegisterFile("index_builder.proto", fileDescriptor_c1d6a79d693ba681) }
var fileDescriptor_c1d6a79d693ba681 = []byte{
// 611 bytes of a gzipped FileDescriptorProto
0x1f, 0x8b, 0x08, 0x00, 0x00, 0x00, 0x00, 0x00, 0x02, 0xff, 0xb4, 0x55, 0xcb, 0x6e, 0xd3, 0x4c,
0x14, 0xae, 0xe3, 0xfc, 0x6d, 0x73, 0x92, 0x56, 0xc9, 0xf4, 0x07, 0x99, 0x20, 0xa4, 0x10, 0x16,
0x58, 0x5c, 0x62, 0x91, 0xb2, 0xe9, 0xb6, 0x4d, 0xda, 0x5a, 0x88, 0x34, 0xb2, 0x29, 0x0b, 0x36,
0x91, 0x2f, 0x07, 0x32, 0xd2, 0xf8, 0x52, 0xcf, 0xb8, 0xa2, 0x7d, 0x0e, 0x76, 0xbc, 0x0a, 0x6f,
0xc0, 0x2b, 0xf0, 0x30, 0xc8, 0xe3, 0x49, 0x69, 0x5a, 0x57, 0x8d, 0x84, 0x58, 0xfa, 0xcc, 0x77,
0x99, 0xf9, 0xce, 0x39, 0x09, 0xec, 0xd0, 0x38, 0xc4, 0xaf, 0x33, 0x3f, 0xa7, 0x2c, 0xc4, 0x6c,
0x90, 0x66, 0x89, 0x48, 0xc8, 0xff, 0x11, 0x65, 0xe7, 0x39, 0x2f, 0xbf, 0x06, 0x1c, 0xb3, 0x73,
0x1a, 0x60, 0xb7, 0x15, 0x24, 0x51, 0x94, 0xc4, 0x65, 0xb5, 0xff, 0x43, 0x83, 0xce, 0x7e, 0xc1,
0xb2, 0x0b, 0x01, 0x07, 0xcf, 0x72, 0xe4, 0x82, 0x3c, 0x01, 0x08, 0x3d, 0xe1, 0xcd, 0x52, 0x4f,
0xcc, 0xb9, 0x51, 0xeb, 0xe9, 0x66, 0xc3, 0x69, 0x14, 0x95, 0x69, 0x51, 0x20, 0xfb, 0xd0, 0x14,
0x17, 0x29, 0xce, 0x52, 0x2f, 0xf3, 0x22, 0x6e, 0xe8, 0x3d, 0xdd, 0x6c, 0x0e, 0x9f, 0x0e, 0x96,
0xec, 0x94, 0xcb, 0x3b, 0xbc, 0xf8, 0xe8, 0xb1, 0x1c, 0xa7, 0x1e, 0xcd, 0x1c, 0x28, 0x58, 0x53,
0x49, 0x22, 0x23, 0x68, 0x95, 0x77, 0x56, 0x22, 0xf5, 0x55, 0x45, 0x9a, 0x92, 0x56, 0xaa, 0xf4,
0x03, 0x20, 0xd7, 0x6f, 0xcf, 0xd3, 0x24, 0xe6, 0x48, 0x76, 0x61, 0x9d, 0x0b, 0x4f, 0xe4, 0xdc,
0xd0, 0x7a, 0x9a, 0xd9, 0x1c, 0x3e, 0xae, 0x54, 0x75, 0x25, 0xc4, 0x51, 0x50, 0x62, 0xc0, 0x86,
0x54, 0xb6, 0x47, 0x46, 0xad, 0xa7, 0x99, 0xba, 0xb3, 0xf8, 0xec, 0xbf, 0x05, 0xe3, 0x08, 0x85,
0xb4, 0x38, 0xa4, 0x0c, 0x65, 0x06, 0x8b, 0xa4, 0xae, 0xb1, 0xb4, 0x65, 0xd6, 0x37, 0x0d, 0x1e,
0x55, 0xd0, 0xfe, 0xc9, 0x15, 0x89, 0x09, 0xed, 0x32, 0xcd, 0xcf, 0x94, 0xa1, 0x6a, 0x9b, 0x2e,
0xdb, 0xb6, 0x4d, 0x97, 0x2e, 0xd0, 0x7f, 0x03, 0x0f, 0x46, 0xc8, 0x83, 0x8c, 0xfa, 0x0c, 0x97,
0x7a, 0x7e, 0xf7, 0x4b, 0x7e, 0xd6, 0xe0, 0xe1, 0x4d, 0xce, 0xdf, 0x3c, 0xe3, 0xaa, 0xf5, 0x8a,
0x5a, 0xbc, 0x65, 0xfb, 0x66, 0xeb, 0xd5, 0xb8, 0x0e, 0xa4, 0x9f, 0x12, 0x28, 0x5b, 0xef, 0xde,
0x0a, 0x43, 0xbf, 0x3f, 0x8c, 0x7a, 0x55, 0x18, 0xc5, 0x9c, 0x63, 0x7c, 0x96, 0xe3, 0x4c, 0xd0,
0x08, 0x8d, 0xff, 0x7a, 0x9a, 0x59, 0x77, 0x1a, 0xb2, 0xf2, 0x81, 0x46, 0x48, 0x9e, 0xc1, 0x16,
0x0f, 0xe6, 0x18, 0xe6, 0x4c, 0x21, 0xd6, 0x25, 0xa2, 0xb5, 0x28, 0x4a, 0xd0, 0x00, 0x76, 0xe4,
0xda, 0xcd, 0x82, 0x24, 0x4a, 0x19, 0x0a, 0x05, 0xdd, 0x90, 0xd0, 0x8e, 0x3c, 0x3a, 0x50, 0x27,
0x05, 0xbe, 0xff, 0x5d, 0x83, 0x86, 0x7c, 0xd4, 0x7b, 0x14, 0x1e, 0xd9, 0x5b, 0x0a, 0x70, 0xa5,
0x14, 0xee, 0x9f, 0x86, 0x3d, 0xd0, 0x33, 0x3c, 0x93, 0xb1, 0x34, 0x87, 0xcf, 0xab, 0x15, 0x6f,
0x2d, 0xbd, 0x53, 0x70, 0x5e, 0x1c, 0x40, 0xf3, 0x9a, 0x17, 0xd9, 0x84, 0xfa, 0xe4, 0x64, 0x32,
0x6e, 0xaf, 0x91, 0x16, 0x6c, 0x9e, 0x4e, 0x6c, 0xd7, 0x3d, 0x1d, 0x8f, 0xda, 0x1a, 0xd9, 0x06,
0xb0, 0x27, 0x53, 0xe7, 0xe4, 0xc8, 0x19, 0xbb, 0x6e, 0xbb, 0x56, 0x9c, 0x1e, 0xda, 0x13, 0xdb,
0x3d, 0x1e, 0x8f, 0xda, 0xfa, 0xf0, 0x57, 0x0d, 0x3a, 0x52, 0x45, 0x9a, 0xb8, 0xa5, 0x23, 0xf1,
0x00, 0xfe, 0x98, 0x92, 0x55, 0xaf, 0xd5, 0x35, 0xef, 0x07, 0x96, 0xc3, 0xd8, 0x5f, 0x23, 0x0c,
0xb6, 0xd4, 0xa0, 0x96, 0x73, 0x4a, 0x5e, 0x56, 0x93, 0x2b, 0x37, 0xa0, 0xfb, 0x6a, 0x35, 0xf0,
0x95, 0xdb, 0x39, 0x74, 0x6e, 0x2d, 0x38, 0x19, 0x54, 0x8b, 0xdc, 0xf5, 0x03, 0xd2, 0xb5, 0x56,
0xc6, 0x2f, 0x7c, 0xf7, 0x8f, 0x3f, 0x1d, 0x7e, 0xa1, 0x62, 0x9e, 0xfb, 0xc5, 0x7e, 0x59, 0x97,
0x94, 0x31, 0x7a, 0x29, 0x30, 0x98, 0x5b, 0xa5, 0xd2, 0xeb, 0x90, 0x72, 0x91, 0x51, 0x3f, 0x17,
0x18, 0x5a, 0x34, 0x16, 0x98, 0xc5, 0x1e, 0xb3, 0xa4, 0xbc, 0x25, 0x27, 0x44, 0xfd, 0x4b, 0xa4,
0xbe, 0xbf, 0x2e, 0xab, 0xbb, 0xbf, 0x03, 0x00, 0x00, 0xff, 0xff, 0x6f, 0xad, 0xe3, 0x08, 0x3f,
0x06, 0x00, 0x00,
}
// Reference imports to suppress errors if they are not otherwise used.
var _ context.Context
var _ grpc.ClientConn
// This is a compile-time assertion to ensure that this generated file
// is compatible with the grpc package it is being compiled against.
const _ = grpc.SupportPackageIsVersion4
// IndexBuildServiceClient is the client API for IndexBuildService service.
//
// For semantics around ctx use and closing/ending streaming RPCs, please refer to https://godoc.org/google.golang.org/grpc#ClientConn.NewStream.
type IndexBuildServiceClient interface {
//*
// @brief This method is used to create collection
//
// @param CollectionSchema, use to provide collection information to be created.
//
// @return Status
BuildIndex(ctx context.Context, in *BuildIndexRequest, opts ...grpc.CallOption) (*BuildIndexResponse, error)
DescribeIndex(ctx context.Context, in *DescribleIndexRequest, opts ...grpc.CallOption) (*DescribleIndexResponse, error)
GetIndexFilePaths(ctx context.Context, in *GetIndexFilePathsRequest, opts ...grpc.CallOption) (*GetIndexFilePathsResponse, error)
}
type indexBuildServiceClient struct {
cc *grpc.ClientConn
}
func NewIndexBuildServiceClient(cc *grpc.ClientConn) IndexBuildServiceClient {
return &indexBuildServiceClient{cc}
}
func (c *indexBuildServiceClient) BuildIndex(ctx context.Context, in *BuildIndexRequest, opts ...grpc.CallOption) (*BuildIndexResponse, error) {
out := new(BuildIndexResponse)
err := c.cc.Invoke(ctx, "/milvus.proto.service.IndexBuildService/BuildIndex", in, out, opts...)
if err != nil {
return nil, err
}
return out, nil
}
func (c *indexBuildServiceClient) DescribeIndex(ctx context.Context, in *DescribleIndexRequest, opts ...grpc.CallOption) (*DescribleIndexResponse, error) {
out := new(DescribleIndexResponse)
err := c.cc.Invoke(ctx, "/milvus.proto.service.IndexBuildService/DescribeIndex", in, out, opts...)
if err != nil {
return nil, err
}
return out, nil
}
func (c *indexBuildServiceClient) GetIndexFilePaths(ctx context.Context, in *GetIndexFilePathsRequest, opts ...grpc.CallOption) (*GetIndexFilePathsResponse, error) {
out := new(GetIndexFilePathsResponse)
err := c.cc.Invoke(ctx, "/milvus.proto.service.IndexBuildService/GetIndexFilePaths", in, out, opts...)
if err != nil {
return nil, err
}
return out, nil
}
// IndexBuildServiceServer is the server API for IndexBuildService service.
type IndexBuildServiceServer interface {
//*
// @brief This method is used to create collection
//
// @param CollectionSchema, use to provide collection information to be created.
//
// @return Status
BuildIndex(context.Context, *BuildIndexRequest) (*BuildIndexResponse, error)
DescribeIndex(context.Context, *DescribleIndexRequest) (*DescribleIndexResponse, error)
GetIndexFilePaths(context.Context, *GetIndexFilePathsRequest) (*GetIndexFilePathsResponse, error)
}
// UnimplementedIndexBuildServiceServer can be embedded to have forward compatible implementations.
type UnimplementedIndexBuildServiceServer struct {
}
func (*UnimplementedIndexBuildServiceServer) BuildIndex(ctx context.Context, req *BuildIndexRequest) (*BuildIndexResponse, error) {
return nil, status.Errorf(codes.Unimplemented, "method BuildIndex not implemented")
}
func (*UnimplementedIndexBuildServiceServer) DescribeIndex(ctx context.Context, req *DescribleIndexRequest) (*DescribleIndexResponse, error) {
return nil, status.Errorf(codes.Unimplemented, "method DescribeIndex not implemented")
}
func (*UnimplementedIndexBuildServiceServer) GetIndexFilePaths(ctx context.Context, req *GetIndexFilePathsRequest) (*GetIndexFilePathsResponse, error) {
return nil, status.Errorf(codes.Unimplemented, "method GetIndexFilePaths not implemented")
}
func RegisterIndexBuildServiceServer(s *grpc.Server, srv IndexBuildServiceServer) {
s.RegisterService(&_IndexBuildService_serviceDesc, srv)
}
func _IndexBuildService_BuildIndex_Handler(srv interface{}, ctx context.Context, dec func(interface{}) error, interceptor grpc.UnaryServerInterceptor) (interface{}, error) {
in := new(BuildIndexRequest)
if err := dec(in); err != nil {
return nil, err
}
if interceptor == nil {
return srv.(IndexBuildServiceServer).BuildIndex(ctx, in)
}
info := &grpc.UnaryServerInfo{
Server: srv,
FullMethod: "/milvus.proto.service.IndexBuildService/BuildIndex",
}
handler := func(ctx context.Context, req interface{}) (interface{}, error) {
return srv.(IndexBuildServiceServer).BuildIndex(ctx, req.(*BuildIndexRequest))
}
return interceptor(ctx, in, info, handler)
}
func _IndexBuildService_DescribeIndex_Handler(srv interface{}, ctx context.Context, dec func(interface{}) error, interceptor grpc.UnaryServerInterceptor) (interface{}, error) {
in := new(DescribleIndexRequest)
if err := dec(in); err != nil {
return nil, err
}
if interceptor == nil {
return srv.(IndexBuildServiceServer).DescribeIndex(ctx, in)
}
info := &grpc.UnaryServerInfo{
Server: srv,
FullMethod: "/milvus.proto.service.IndexBuildService/DescribeIndex",
}
handler := func(ctx context.Context, req interface{}) (interface{}, error) {
return srv.(IndexBuildServiceServer).DescribeIndex(ctx, req.(*DescribleIndexRequest))
}
return interceptor(ctx, in, info, handler)
}
func _IndexBuildService_GetIndexFilePaths_Handler(srv interface{}, ctx context.Context, dec func(interface{}) error, interceptor grpc.UnaryServerInterceptor) (interface{}, error) {
in := new(GetIndexFilePathsRequest)
if err := dec(in); err != nil {
return nil, err
}
if interceptor == nil {
return srv.(IndexBuildServiceServer).GetIndexFilePaths(ctx, in)
}
info := &grpc.UnaryServerInfo{
Server: srv,
FullMethod: "/milvus.proto.service.IndexBuildService/GetIndexFilePaths",
}
handler := func(ctx context.Context, req interface{}) (interface{}, error) {
return srv.(IndexBuildServiceServer).GetIndexFilePaths(ctx, req.(*GetIndexFilePathsRequest))
}
return interceptor(ctx, in, info, handler)
}
var _IndexBuildService_serviceDesc = grpc.ServiceDesc{
ServiceName: "milvus.proto.service.IndexBuildService",
HandlerType: (*IndexBuildServiceServer)(nil),
Methods: []grpc.MethodDesc{
{
MethodName: "BuildIndex",
Handler: _IndexBuildService_BuildIndex_Handler,
},
{
MethodName: "DescribeIndex",
Handler: _IndexBuildService_DescribeIndex_Handler,
},
{
MethodName: "GetIndexFilePaths",
Handler: _IndexBuildService_GetIndexFilePaths_Handler,
},
},
Streams: []grpc.StreamDesc{},
Metadata: "index_builder.proto",
}

View File

@ -23,6 +23,7 @@ mkdir -p etcdpb
mkdir -p internalpb mkdir -p internalpb
mkdir -p servicepb mkdir -p servicepb
mkdir -p masterpb mkdir -p masterpb
mkdir -p indexbuilderpb
${protoc} --go_out=plugins=grpc,paths=source_relative:./commonpb common.proto ${protoc} --go_out=plugins=grpc,paths=source_relative:./commonpb common.proto
${protoc} --go_out=plugins=grpc,paths=source_relative:./schemapb schema.proto ${protoc} --go_out=plugins=grpc,paths=source_relative:./schemapb schema.proto
@ -31,5 +32,6 @@ ${protoc} --go_out=plugins=grpc,paths=source_relative:./internalpb internal_msg.
${protoc} --go_out=plugins=grpc,paths=source_relative:./servicepb service_msg.proto ${protoc} --go_out=plugins=grpc,paths=source_relative:./servicepb service_msg.proto
${protoc} --go_out=plugins=grpc,paths=source_relative:./servicepb service.proto ${protoc} --go_out=plugins=grpc,paths=source_relative:./servicepb service.proto
${protoc} --go_out=plugins=grpc,paths=source_relative:./masterpb master.proto ${protoc} --go_out=plugins=grpc,paths=source_relative:./masterpb master.proto
${protoc} --go_out=plugins=grpc,paths=source_relative:./indexbuilderpb index_builder.proto
popd popd