提交 cd52adc1 编写于 作者: D dragondriver 提交者: yefu.chen

Add proxy service to ci workflow

Signed-off-by: Ndragondriver <jiquan.long@zilliz.com>
上级 90a0b8b2
......@@ -21,6 +21,10 @@ dir ('build/docker/deploy') {
sh 'docker-compose build --force-rm indexbuilder'
sh 'docker-compose push indexbuilder'
sh 'docker pull ${SOURCE_REPO}/proxyservice:${SOURCE_TAG} || true'
sh 'docker-compose build --force-rm proxyservice'
sh 'docker-compose push proxyservice'
sh 'docker pull ${SOURCE_REPO}/proxynode:${SOURCE_TAG} || true'
sh 'docker-compose build --force-rm proxynode'
sh 'docker-compose push proxynode'
......
......@@ -7,6 +7,7 @@ try {
sh 'docker-compose -p ${DOCKER_COMPOSE_PROJECT_NAME} pull'
sh 'docker-compose -p ${DOCKER_COMPOSE_PROJECT_NAME} up -d master'
sh 'docker-compose -p ${DOCKER_COMPOSE_PROJECT_NAME} up -d indexbuilder'
sh 'docker-compose -p ${DOCKER_COMPOSE_PROJECT_NAME} up -d proxyservice'
sh 'docker-compose -p ${DOCKER_COMPOSE_PROJECT_NAME} up -d proxynode'
sh 'docker-compose -p ${DOCKER_COMPOSE_PROJECT_NAME} run -e QUERY_NODE_ID=1 -d querynode'
sh 'docker-compose -p ${DOCKER_COMPOSE_PROJECT_NAME} run -e QUERY_NODE_ID=2 -d querynode'
......
......@@ -113,6 +113,8 @@ build-go: build-cpp
@echo "Building each component's binary to './bin'"
@echo "Building master ..."
@mkdir -p $(INSTALL_PATH) && go env -w CGO_ENABLED="0" && GO111MODULE=on $(GO) build -o $(INSTALL_PATH)/master $(PWD)/cmd/master/main.go 1>/dev/null
@echo "Building proxy service ..."
@mkdir -p $(INSTALL_PATH) && go env -w CGO_ENABLED="0" && GO111MODULE=on $(GO) build -o $(INSTALL_PATH)/proxyservice $(PWD)/cmd/proxy/service/proxy_service.go 1>/dev/null
@echo "Building proxy node ..."
@mkdir -p $(INSTALL_PATH) && go env -w CGO_ENABLED="0" && GO111MODULE=on $(GO) build -o $(INSTALL_PATH)/proxynode $(PWD)/cmd/proxy/node/proxy_node.go 1>/dev/null
@echo "Building query node ..."
......@@ -166,6 +168,7 @@ install: all
@mkdir -p $(GOPATH)/bin && cp -f $(PWD)/bin/querynode $(GOPATH)/bin/querynode
@mkdir -p $(GOPATH)/bin && cp -f $(PWD)/bin/master $(GOPATH)/bin/master
@mkdir -p $(GOPATH)/bin && cp -f $(PWD)/bin/proxynode $(GOPATH)/bin/proxynode
@mkdir -p $(GOPATH)/bin && cp -f $(PWD)/bin/proxyservice $(GOPATH)/bin/proxyservice
@mkdir -p $(GOPATH)/bin && cp -f $(PWD)/bin/writenode $(GOPATH)/bin/writenode
@mkdir -p $(GOPATH)/bin && cp -f $(PWD)/bin/indexbuilder $(GOPATH)/bin/indexbuilder
@mkdir -p $(GOPATH)/bin && cp -f $(PWD)/bin/singlenode $(GOPATH)/bin/singlenode
......@@ -180,6 +183,7 @@ clean:
@rm -rf lib/
@rm -rf $(GOPATH)/bin/master
@rm -rf $(GOPATH)/bin/proxynode
@rm -rf $(GOPATH)/bin/proxyservice
@rm -rf $(GOPATH)/bin/querynode
@rm -rf $(GOPATH)/bin/writenode
@rm -rf $(GOPATH)/bin/indexbuilder
......
......@@ -7,3 +7,5 @@ ETCD_ADDRESS=etcd:2379
MASTER_ADDRESS=master:53100
MINIO_ADDRESS=minio:9000
INDEX_BUILDER_ADDRESS=indexbuilder:31000
PROXY_NODE_HOST=proxynode
PROXY_SERVICE_ADDRESS=proxyservice:19530
......@@ -15,6 +15,16 @@ services:
networks:
- milvus
proxyservice:
image: ${TARGET_REPO}/proxyservice:${TARGET_TAG}
build:
context: ../../../
dockerfile: build/docker/deploy/proxyservice/DockerFile
cache_from:
- ${SOURCE_REPO}/proxyservice:${SOURCE_TAG}
networks:
- milvus
proxynode:
image: ${TARGET_REPO}/proxynode:${TARGET_TAG}
build:
......@@ -25,7 +35,9 @@ services:
environment:
PULSAR_ADDRESS: ${PULSAR_ADDRESS}
MASTER_ADDRESS: ${MASTER_ADDRESS}
networks:
PROXY_NODE_HOST: ${PROXY_NODE_HOST}
PROXY_SERVICE_ADDRESS: ${PROXY_SERVICE_ADDRESS}
networks:
- milvus
indexbuilder:
......
# 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.
FROM alpine:3.12.1
COPY ./bin/proxyservice /milvus-distributed/bin/proxyservice
COPY ./configs/ /milvus-distributed/configs/
WORKDIR /milvus-distributed/
CMD ["./bin/proxyservice"]
EXPOSE 19530
......@@ -12,7 +12,7 @@ services:
- ../../..:/milvus-distributed:delegated
working_dir: "/milvus-distributed/tests/python"
command: >
/bin/bash -c "pytest --ip proxynode"
/bin/bash -c "pytest --ip proxyservice"
networks:
- milvus
......
package main
import (
"context"
"log"
"os"
"os/signal"
"syscall"
grpcproxyservice "github.com/zilliztech/milvus-distributed/internal/distributed/proxyservice"
"go.uber.org/zap"
)
func main() {
ctx, cancel := context.WithCancel(context.Background())
svr, err := grpcproxyservice.CreateProxyServiceServer()
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.Init(); err != nil {
log.Fatal("init server failed", zap.Error(err))
}
if err := svr.Start(); err != nil {
log.Fatal("run server failed", zap.Error(err))
}
<-ctx.Done()
log.Print("Got signal to exit", zap.String("signal", sig.String()))
if err := svr.Stop(); err != nil {
log.Fatal("stop server failed", zap.Error(err))
}
switch sig {
case syscall.SIGTERM:
exit(0)
default:
exit(1)
}
}
func exit(code int) {
os.Exit(code)
}
......@@ -16,7 +16,7 @@ import (
"github.com/zilliztech/milvus-distributed/internal/indexnode"
"github.com/zilliztech/milvus-distributed/internal/master"
proxynodeimpl "github.com/zilliztech/milvus-distributed/internal/proxynode"
"github.com/zilliztech/milvus-distributed/internal/proxynode"
"github.com/zilliztech/milvus-distributed/internal/querynode"
"github.com/zilliztech/milvus-distributed/internal/writenode"
)
......@@ -62,10 +62,10 @@ func InitMaster(cpuprofile *string, wg *sync.WaitGroup) {
func InitProxy(wg *sync.WaitGroup) {
defer wg.Done()
//proxynodeimpl.Init()
//fmt.Println("ProxyID is", proxynodeimpl.Params.ProxyID())
//proxynode.Init()
//fmt.Println("ProxyID is", proxynode.Params.ProxyID())
ctx, cancel := context.WithCancel(context.Background())
svr, err := proxynodeimpl.CreateProxyNodeImpl(ctx)
svr, err := proxynode.CreateProxyNodeImpl(ctx)
if err != nil {
log.Print("create server failed", zap.Error(err))
}
......@@ -220,9 +220,13 @@ func main() {
wg.Add(1)
go InitMaster(cpuprofile, &wg)
time.Sleep(time.Second * 1)
wg.Add(1)
go InitProxy(&wg)
wg.Add(1)
go InitQueryNode(&wg)
wg.Add(1)
go InitIndexBuilder(&wg)
wg.Add(1)
go InitWriteNode(&wg)
wg.Wait()
}
......
......@@ -51,7 +51,6 @@ proxyNode:
address: localhost
port: 19530
# not used now
proxyService:
address: localhost
port: 19530
......
此差异已折叠。
......@@ -50,7 +50,7 @@ struct TableStruct_milvus_2eproto {
PROTOBUF_SECTION_VARIABLE(protodesc_cold);
static const ::PROTOBUF_NAMESPACE_ID::internal::AuxillaryParseTableField aux[]
PROTOBUF_SECTION_VARIABLE(protodesc_cold);
static const ::PROTOBUF_NAMESPACE_ID::internal::ParseTable schema[40]
static const ::PROTOBUF_NAMESPACE_ID::internal::ParseTable schema[41]
PROTOBUF_SECTION_VARIABLE(protodesc_cold);
static const ::PROTOBUF_NAMESPACE_ID::internal::FieldMetadata field_metadata[];
static const ::PROTOBUF_NAMESPACE_ID::internal::SerializationTable serialization_table[];
......@@ -147,6 +147,9 @@ extern PlaceholderGroupDefaultTypeInternal _PlaceholderGroup_default_instance_;
class PlaceholderValue;
class PlaceholderValueDefaultTypeInternal;
extern PlaceholderValueDefaultTypeInternal _PlaceholderValue_default_instance_;
class RegisterLinkResponse;
class RegisterLinkResponseDefaultTypeInternal;
extern RegisterLinkResponseDefaultTypeInternal _RegisterLinkResponse_default_instance_;
class ReleaseCollectionRequest;
class ReleaseCollectionRequestDefaultTypeInternal;
extern ReleaseCollectionRequestDefaultTypeInternal _ReleaseCollectionRequest_default_instance_;
......@@ -213,6 +216,7 @@ template<> ::milvus::proto::milvus::PartitionStatsRequest* Arena::CreateMaybeMes
template<> ::milvus::proto::milvus::PartitionStatsResponse* Arena::CreateMaybeMessage<::milvus::proto::milvus::PartitionStatsResponse>(Arena*);
template<> ::milvus::proto::milvus::PlaceholderGroup* Arena::CreateMaybeMessage<::milvus::proto::milvus::PlaceholderGroup>(Arena*);
template<> ::milvus::proto::milvus::PlaceholderValue* Arena::CreateMaybeMessage<::milvus::proto::milvus::PlaceholderValue>(Arena*);
template<> ::milvus::proto::milvus::RegisterLinkResponse* Arena::CreateMaybeMessage<::milvus::proto::milvus::RegisterLinkResponse>(Arena*);
template<> ::milvus::proto::milvus::ReleaseCollectionRequest* Arena::CreateMaybeMessage<::milvus::proto::milvus::ReleaseCollectionRequest>(Arena*);
template<> ::milvus::proto::milvus::ReleasePartitionRequest* Arena::CreateMaybeMessage<::milvus::proto::milvus::ReleasePartitionRequest>(Arena*);
template<> ::milvus::proto::milvus::SearchRequest* Arena::CreateMaybeMessage<::milvus::proto::milvus::SearchRequest>(Arena*);
......@@ -6716,6 +6720,150 @@ class FlushRequest :
mutable ::PROTOBUF_NAMESPACE_ID::internal::CachedSize _cached_size_;
friend struct ::TableStruct_milvus_2eproto;
};
// -------------------------------------------------------------------
class RegisterLinkResponse :
public ::PROTOBUF_NAMESPACE_ID::Message /* @@protoc_insertion_point(class_definition:milvus.proto.milvus.RegisterLinkResponse) */ {
public:
RegisterLinkResponse();
virtual ~RegisterLinkResponse();
RegisterLinkResponse(const RegisterLinkResponse& from);
RegisterLinkResponse(RegisterLinkResponse&& from) noexcept
: RegisterLinkResponse() {
*this = ::std::move(from);
}
inline RegisterLinkResponse& operator=(const RegisterLinkResponse& from) {
CopyFrom(from);
return *this;
}
inline RegisterLinkResponse& operator=(RegisterLinkResponse&& from) noexcept {
if (GetArenaNoVirtual() == from.GetArenaNoVirtual()) {
if (this != &from) InternalSwap(&from);
} else {
CopyFrom(from);
}
return *this;
}
static const ::PROTOBUF_NAMESPACE_ID::Descriptor* descriptor() {
return GetDescriptor();
}
static const ::PROTOBUF_NAMESPACE_ID::Descriptor* GetDescriptor() {
return GetMetadataStatic().descriptor;
}
static const ::PROTOBUF_NAMESPACE_ID::Reflection* GetReflection() {
return GetMetadataStatic().reflection;
}
static const RegisterLinkResponse& default_instance();
static void InitAsDefaultInstance(); // FOR INTERNAL USE ONLY
static inline const RegisterLinkResponse* internal_default_instance() {
return reinterpret_cast<const RegisterLinkResponse*>(
&_RegisterLinkResponse_default_instance_);
}
static constexpr int kIndexInFileMessages =
40;
friend void swap(RegisterLinkResponse& a, RegisterLinkResponse& b) {
a.Swap(&b);
}
inline void Swap(RegisterLinkResponse* other) {
if (other == this) return;
InternalSwap(other);
}
// implements Message ----------------------------------------------
inline RegisterLinkResponse* New() const final {
return CreateMaybeMessage<RegisterLinkResponse>(nullptr);
}
RegisterLinkResponse* New(::PROTOBUF_NAMESPACE_ID::Arena* arena) const final {
return CreateMaybeMessage<RegisterLinkResponse>(arena);
}
void CopyFrom(const ::PROTOBUF_NAMESPACE_ID::Message& from) final;
void MergeFrom(const ::PROTOBUF_NAMESPACE_ID::Message& from) final;
void CopyFrom(const RegisterLinkResponse& from);
void MergeFrom(const RegisterLinkResponse& from);
PROTOBUF_ATTRIBUTE_REINITIALIZES void Clear() final;
bool IsInitialized() const final;
size_t ByteSizeLong() const final;
#if GOOGLE_PROTOBUF_ENABLE_EXPERIMENTAL_PARSER
const char* _InternalParse(const char* ptr, ::PROTOBUF_NAMESPACE_ID::internal::ParseContext* ctx) final;
#else
bool MergePartialFromCodedStream(
::PROTOBUF_NAMESPACE_ID::io::CodedInputStream* input) final;
#endif // GOOGLE_PROTOBUF_ENABLE_EXPERIMENTAL_PARSER
void SerializeWithCachedSizes(
::PROTOBUF_NAMESPACE_ID::io::CodedOutputStream* output) const final;
::PROTOBUF_NAMESPACE_ID::uint8* InternalSerializeWithCachedSizesToArray(
::PROTOBUF_NAMESPACE_ID::uint8* target) const final;
int GetCachedSize() const final { return _cached_size_.Get(); }
private:
inline void SharedCtor();
inline void SharedDtor();
void SetCachedSize(int size) const final;
void InternalSwap(RegisterLinkResponse* other);
friend class ::PROTOBUF_NAMESPACE_ID::internal::AnyMetadata;
static ::PROTOBUF_NAMESPACE_ID::StringPiece FullMessageName() {
return "milvus.proto.milvus.RegisterLinkResponse";
}
private:
inline ::PROTOBUF_NAMESPACE_ID::Arena* GetArenaNoVirtual() const {
return nullptr;
}
inline void* MaybeArenaPtr() const {
return nullptr;
}
public:
::PROTOBUF_NAMESPACE_ID::Metadata GetMetadata() const final;
private:
static ::PROTOBUF_NAMESPACE_ID::Metadata GetMetadataStatic() {
::PROTOBUF_NAMESPACE_ID::internal::AssignDescriptors(&::descriptor_table_milvus_2eproto);
return ::descriptor_table_milvus_2eproto.file_level_metadata[kIndexInFileMessages];
}
public:
// nested types ----------------------------------------------------
// accessors -------------------------------------------------------
enum : int {
kAddressFieldNumber = 1,
kStatusFieldNumber = 2,
};
// .milvus.proto.common.Address address = 1;
bool has_address() const;
void clear_address();
const ::milvus::proto::common::Address& address() const;
::milvus::proto::common::Address* release_address();
::milvus::proto::common::Address* mutable_address();
void set_allocated_address(::milvus::proto::common::Address* address);
// .milvus.proto.common.Status status = 2;
bool has_status() const;
void clear_status();
const ::milvus::proto::common::Status& status() const;
::milvus::proto::common::Status* release_status();
::milvus::proto::common::Status* mutable_status();
void set_allocated_status(::milvus::proto::common::Status* status);
// @@protoc_insertion_point(class_scope:milvus.proto.milvus.RegisterLinkResponse)
private:
class _Internal;
::PROTOBUF_NAMESPACE_ID::internal::InternalMetadataWithArena _internal_metadata_;
::milvus::proto::common::Address* address_;
::milvus::proto::common::Status* status_;
mutable ::PROTOBUF_NAMESPACE_ID::internal::CachedSize _cached_size_;
friend struct ::TableStruct_milvus_2eproto;
};
// ===================================================================
......@@ -12474,6 +12622,100 @@ inline void FlushRequest::set_allocated_collection_name(std::string* collection_
// @@protoc_insertion_point(field_set_allocated:milvus.proto.milvus.FlushRequest.collection_name)
}
// -------------------------------------------------------------------
// RegisterLinkResponse
// .milvus.proto.common.Address address = 1;
inline bool RegisterLinkResponse::has_address() const {
return this != internal_default_instance() && address_ != nullptr;
}
inline const ::milvus::proto::common::Address& RegisterLinkResponse::address() const {
const ::milvus::proto::common::Address* p = address_;
// @@protoc_insertion_point(field_get:milvus.proto.milvus.RegisterLinkResponse.address)
return p != nullptr ? *p : *reinterpret_cast<const ::milvus::proto::common::Address*>(
&::milvus::proto::common::_Address_default_instance_);
}
inline ::milvus::proto::common::Address* RegisterLinkResponse::release_address() {
// @@protoc_insertion_point(field_release:milvus.proto.milvus.RegisterLinkResponse.address)
::milvus::proto::common::Address* temp = address_;
address_ = nullptr;
return temp;
}
inline ::milvus::proto::common::Address* RegisterLinkResponse::mutable_address() {
if (address_ == nullptr) {
auto* p = CreateMaybeMessage<::milvus::proto::common::Address>(GetArenaNoVirtual());
address_ = p;
}
// @@protoc_insertion_point(field_mutable:milvus.proto.milvus.RegisterLinkResponse.address)
return address_;
}
inline void RegisterLinkResponse::set_allocated_address(::milvus::proto::common::Address* address) {
::PROTOBUF_NAMESPACE_ID::Arena* message_arena = GetArenaNoVirtual();
if (message_arena == nullptr) {
delete reinterpret_cast< ::PROTOBUF_NAMESPACE_ID::MessageLite*>(address_);
}
if (address) {
::PROTOBUF_NAMESPACE_ID::Arena* submessage_arena = nullptr;
if (message_arena != submessage_arena) {
address = ::PROTOBUF_NAMESPACE_ID::internal::GetOwnedMessage(
message_arena, address, submessage_arena);
}
} else {
}
address_ = address;
// @@protoc_insertion_point(field_set_allocated:milvus.proto.milvus.RegisterLinkResponse.address)
}
// .milvus.proto.common.Status status = 2;
inline bool RegisterLinkResponse::has_status() const {
return this != internal_default_instance() && status_ != nullptr;
}
inline const ::milvus::proto::common::Status& RegisterLinkResponse::status() const {
const ::milvus::proto::common::Status* p = status_;
// @@protoc_insertion_point(field_get:milvus.proto.milvus.RegisterLinkResponse.status)
return p != nullptr ? *p : *reinterpret_cast<const ::milvus::proto::common::Status*>(
&::milvus::proto::common::_Status_default_instance_);
}
inline ::milvus::proto::common::Status* RegisterLinkResponse::release_status() {
// @@protoc_insertion_point(field_release:milvus.proto.milvus.RegisterLinkResponse.status)
::milvus::proto::common::Status* temp = status_;
status_ = nullptr;
return temp;
}
inline ::milvus::proto::common::Status* RegisterLinkResponse::mutable_status() {
if (status_ == nullptr) {
auto* p = CreateMaybeMessage<::milvus::proto::common::Status>(GetArenaNoVirtual());
status_ = p;
}
// @@protoc_insertion_point(field_mutable:milvus.proto.milvus.RegisterLinkResponse.status)
return status_;
}
inline void RegisterLinkResponse::set_allocated_status(::milvus::proto::common::Status* status) {
::PROTOBUF_NAMESPACE_ID::Arena* message_arena = GetArenaNoVirtual();
if (message_arena == nullptr) {
delete reinterpret_cast< ::PROTOBUF_NAMESPACE_ID::MessageLite*>(status_);
}
if (status) {
::PROTOBUF_NAMESPACE_ID::Arena* submessage_arena = nullptr;
if (message_arena != submessage_arena) {
status = ::PROTOBUF_NAMESPACE_ID::internal::GetOwnedMessage(
message_arena, status, submessage_arena);
}
} else {
}
status_ = status;
// @@protoc_insertion_point(field_set_allocated:milvus.proto.milvus.RegisterLinkResponse.status)
}
#ifdef __GNUC__
#pragma GCC diagnostic pop
#endif // __GNUC__
......@@ -12555,6 +12797,8 @@ inline void FlushRequest::set_allocated_collection_name(std::string* collection_
// -------------------------------------------------------------------
// -------------------------------------------------------------------
// @@protoc_insertion_point(namespace_scope)
......
......@@ -3,13 +3,18 @@ package grpcproxynode
import (
"context"
"net"
"os"
"strconv"
"sync"
"github.com/go-basic/ipv4"
grpcproxyservice "github.com/zilliztech/milvus-distributed/internal/distributed/proxyservice"
"github.com/zilliztech/milvus-distributed/internal/proto/commonpb"
"github.com/zilliztech/milvus-distributed/internal/proto/milvuspb"
proxynodeimpl "github.com/zilliztech/milvus-distributed/internal/proxynode"
"github.com/zilliztech/milvus-distributed/internal/proxynode"
"github.com/zilliztech/milvus-distributed/internal/proto/proxypb"
......@@ -19,12 +24,12 @@ import (
type Server struct {
ctx context.Context
wg sync.WaitGroup
impl proxynodeimpl.ProxyNode
impl proxynode.ProxyNode
grpcServer *grpc.Server
ip string
port int
proxyServiceAddress string
//proxyServiceClient *proxyservice.Client
proxyServiceClient *grpcproxyservice.Client
}
func CreateProxyNodeServer() (*Server, error) {
......@@ -32,51 +37,56 @@ func CreateProxyNodeServer() (*Server, error) {
}
func (s *Server) connectProxyService() error {
proxynodeimpl.Params.Init()
//s.proxyServiceAddress = proxynodeimpl.Params.ProxyServiceAddress()
//s.proxyServiceClient = proxyservice.NewClient(s.ctx, s.proxyServiceAddress)
//
//getAvailablePort := func() int {
// listener, err := net.Listen("tcp", ":0")
// if err != nil {
// panic(err)
// }
// defer listener.Close()
//
// return listener.Addr().(*net.TCPAddr).Port
//}
//getLocalIp := func() string {
// return ipv4.LocalIP()
//}
//s.ip = getLocalIp()
//s.port = getAvailablePort()
//
//request := &proxypb.RegisterNodeRequest{
// Address: &commonpb.Address{
// Ip: s.ip,
// Port: int64(s.port),
// },
//}
//response, err := s.proxyServiceClient.RegisterNode(request)
//if err != nil {
// panic(err)
//}
//
//proxynodeimpl.Params.Save("_proxyID", strconv.Itoa(int(response.InitParams.NodeID)))
//
//for _, params := range response.InitParams.StartParams {
// proxynodeimpl.Params.Save(params.Key, params.Value)
//}
//
//return err
return nil
proxynode.Params.Init()
s.proxyServiceAddress = proxynode.Params.ProxyServiceAddress()
s.proxyServiceClient = grpcproxyservice.NewClient(s.ctx, s.proxyServiceAddress)
getAvailablePort := func() int {
listener, err := net.Listen("tcp", ":0")
if err != nil {
panic(err)
}
defer listener.Close()
return listener.Addr().(*net.TCPAddr).Port
}
getLocalIP := func() string {
localIP := ipv4.LocalIP()
host := os.Getenv("PROXY_NODE_HOST")
// TODO: shall we write this to ParamTable?
if len(host) <= 0 {
return localIP
}
return host
}
s.ip = getLocalIP()
s.port = getAvailablePort()
request := &proxypb.RegisterNodeRequest{
Address: &commonpb.Address{
Ip: s.ip,
Port: int64(s.port),
},
}
response, err := s.proxyServiceClient.RegisterNode(request)
if err != nil {
panic(err)
}
proxynode.Params.Save("_proxyID", strconv.Itoa(int(response.InitParams.NodeID)))
for _, params := range response.InitParams.StartParams {
proxynode.Params.Save(params.Key, params.Value)
}
return err
}
func (s *Server) Init() error {
s.ctx = context.Background()
var err error
s.impl, err = proxynodeimpl.CreateProxyNodeImpl(s.ctx)
s.impl, err = proxynode.CreateProxyNodeImpl(s.ctx)
if err != nil {
return err
}
......@@ -93,7 +103,7 @@ func (s *Server) Start() error {
defer s.wg.Done()
// TODO: use config
lis, err := net.Listen("tcp", ":"+strconv.Itoa(proxynodeimpl.Params.NetworkPort()))
lis, err := net.Listen("tcp", ":"+strconv.Itoa(s.port))
if err != nil {
panic(err)
}
......
package grpcproxyservice
import (
"context"
"google.golang.org/grpc"
"github.com/zilliztech/milvus-distributed/internal/proto/proxypb"
)
type Client struct {
proxyServiceClient proxypb.ProxyServiceClient
address string
ctx context.Context
}
func (c *Client) tryConnect() error {
if c.proxyServiceClient != nil {
return nil
}
conn, err := grpc.DialContext(c.ctx, c.address, grpc.WithInsecure(), grpc.WithBlock())
if err != nil {
return err
}
c.proxyServiceClient = proxypb.NewProxyServiceClient(conn)
return nil
}
func (c *Client) RegisterNode(request *proxypb.RegisterNodeRequest) (*proxypb.RegisterNodeResponse, error) {
err := c.tryConnect()
if err != nil {
return nil, err
}
return c.proxyServiceClient.RegisterNode(c.ctx, request)
}
func (c *Client) InvalidateCollectionMetaCache(request *proxypb.InvalidateCollMetaCacheRequest) error {
var err error
err = c.tryConnect()
if err != nil {
return err
}
_, err = c.proxyServiceClient.InvalidateCollectionMetaCache(c.ctx, request)
return err
}
func NewClient(ctx context.Context, address string) *Client {
return &Client{
address: address,
ctx: ctx,
}
}
package grpcproxyservice
import (
"net"
"strconv"
"github.com/zilliztech/milvus-distributed/internal/util/paramtable"
)
type ParamTable struct {
paramtable.BaseTable
}
var Params ParamTable
func (pt *ParamTable) Init() {
pt.BaseTable.Init()
}
func (pt *ParamTable) NetworkPort() int {
return pt.ParseInt("proxyService.port")
}
func (pt *ParamTable) NetworkAddress() string {
addr, err := pt.Load("proxyService.address")
if err != nil {
panic(err)
}
hostName, _ := net.LookupHost(addr)
if len(hostName) <= 0 {
if ip := net.ParseIP(addr); ip == nil {
panic("invalid ip proxyService.address")
}
}
port, err := pt.Load("proxyService.port")
if err != nil {
panic(err)
}
_, err = strconv.Atoi(port)
if err != nil {
panic(err)
}
return addr + ":" + port
}
package grpcproxyservice
import (
"context"
"fmt"
"net"
"strconv"
"sync"
"github.com/zilliztech/milvus-distributed/internal/proto/milvuspb"
"github.com/zilliztech/milvus-distributed/internal/proto/commonpb"
"github.com/zilliztech/milvus-distributed/internal/proto/proxypb"
"github.com/zilliztech/milvus-distributed/internal/proxyservice"
"google.golang.org/grpc"
)
type Server struct {
ctx context.Context
wg sync.WaitGroup
impl proxyservice.ProxyService
grpcServer *grpc.Server
}
func CreateProxyServiceServer() (*Server, error) {
return &Server{}, nil
}
func (s *Server) Init() error {
s.ctx = context.Background()
Params.Init()
s.impl, _ = proxyservice.CreateProxyService(s.ctx)
s.impl.Init()
return nil
}
func (s *Server) Start() error {
fmt.Println("proxy service start ...")
s.wg.Add(1)
go func() {
defer s.wg.Done()
// TODO: use config
fmt.Println("network port: ", Params.NetworkPort())
lis, err := net.Listen("tcp", ":"+strconv.Itoa(Params.NetworkPort()))
if err != nil {
panic(err)
}
s.grpcServer = grpc.NewServer()
proxypb.RegisterProxyServiceServer(s.grpcServer, s)
milvuspb.RegisterProxyServiceServer(s.grpcServer, s)
if err = s.grpcServer.Serve(lis); err != nil {
panic(err)
}
}()
s.impl.Start()
return nil
}
func (s *Server) Stop() error {
s.impl.Stop()
if s.grpcServer != nil {
s.grpcServer.GracefulStop()
}
s.wg.Wait()
return nil
}
func (s *Server) RegisterLink(ctx context.Context, empty *commonpb.Empty) (*milvuspb.RegisterLinkResponse, error) {
return s.impl.RegisterLink()
}
func (s *Server) RegisterNode(ctx context.Context, request *proxypb.RegisterNodeRequest) (*proxypb.RegisterNodeResponse, error) {
return s.impl.RegisterNode(request)
}
func (s *Server) InvalidateCollectionMetaCache(ctx context.Context, request *proxypb.InvalidateCollMetaCacheRequest) (*commonpb.Status, error) {
return &commonpb.Status{}, s.impl.InvalidateCollectionMetaCache(request)
}
......@@ -292,3 +292,12 @@ service MilvusService {
rpc GetDdChannel(common.Empty) returns (StringResponse) {}
}
message RegisterLinkResponse {
common.Address address = 1;
common.Status status = 2;
}
service ProxyService {
rpc RegisterLink(common.Empty) returns (RegisterLinkResponse) {}
}
......@@ -14,10 +14,7 @@ message RegisterNodeRequest {
message RegisterNodeResponse {
internal.InitParams init_params = 1;
}
message RegisterLinkResponse {
common.Address address = 1;
common.Status status = 2;
}
message InvalidateCollMetaCacheRequest {
......@@ -27,7 +24,6 @@ message InvalidateCollMetaCacheRequest {
}
service ProxyService {
rpc RegisterLink(common.Empty) returns (RegisterLinkResponse) {}
rpc RegisterNode(RegisterNodeRequest) returns (RegisterNodeResponse) {}
rpc InvalidateCollectionMetaCache(InvalidateCollMetaCacheRequest) returns (common.Status) {}
}
......
......@@ -75,6 +75,7 @@ func (m *RegisterNodeRequest) GetAddress() *commonpb.Address {
type RegisterNodeResponse struct {
InitParams *internalpb2.InitParams `protobuf:"bytes,1,opt,name=init_params,json=initParams,proto3" json:"init_params,omitempty"`
Status *commonpb.Status `protobuf:"bytes,2,opt,name=status,proto3" json:"status,omitempty"`
XXX_NoUnkeyedLiteral struct{} `json:"-"`
XXX_unrecognized []byte `json:"-"`
XXX_sizecache int32 `json:"-"`
......@@ -112,41 +113,9 @@ func (m *RegisterNodeResponse) GetInitParams() *internalpb2.InitParams {
return nil
}
type RegisterLinkResponse struct {
Address *commonpb.Address `protobuf:"bytes,1,opt,name=address,proto3" json:"address,omitempty"`
XXX_NoUnkeyedLiteral struct{} `json:"-"`
XXX_unrecognized []byte `json:"-"`
XXX_sizecache int32 `json:"-"`
}
func (m *RegisterLinkResponse) Reset() { *m = RegisterLinkResponse{} }
func (m *RegisterLinkResponse) String() string { return proto.CompactTextString(m) }
func (*RegisterLinkResponse) ProtoMessage() {}
func (*RegisterLinkResponse) Descriptor() ([]byte, []int) {
return fileDescriptor_34ca2fbc94d169de, []int{2}
}
func (m *RegisterLinkResponse) XXX_Unmarshal(b []byte) error {
return xxx_messageInfo_RegisterLinkResponse.Unmarshal(m, b)
}
func (m *RegisterLinkResponse) XXX_Marshal(b []byte, deterministic bool) ([]byte, error) {
return xxx_messageInfo_RegisterLinkResponse.Marshal(b, m, deterministic)
}
func (m *RegisterLinkResponse) XXX_Merge(src proto.Message) {
xxx_messageInfo_RegisterLinkResponse.Merge(m, src)
}
func (m *RegisterLinkResponse) XXX_Size() int {
return xxx_messageInfo_RegisterLinkResponse.Size(m)
}
func (m *RegisterLinkResponse) XXX_DiscardUnknown() {
xxx_messageInfo_RegisterLinkResponse.DiscardUnknown(m)
}
var xxx_messageInfo_RegisterLinkResponse proto.InternalMessageInfo
func (m *RegisterLinkResponse) GetAddress() *commonpb.Address {
func (m *RegisterNodeResponse) GetStatus() *commonpb.Status {
if m != nil {
return m.Address
return m.Status
}
return nil
}
......@@ -164,7 +133,7 @@ func (m *InvalidateCollMetaCacheRequest) Reset() { *m = InvalidateCollMe
func (m *InvalidateCollMetaCacheRequest) String() string { return proto.CompactTextString(m) }
func (*InvalidateCollMetaCacheRequest) ProtoMessage() {}
func (*InvalidateCollMetaCacheRequest) Descriptor() ([]byte, []int) {
return fileDescriptor_34ca2fbc94d169de, []int{3}
return fileDescriptor_34ca2fbc94d169de, []int{2}
}
func (m *InvalidateCollMetaCacheRequest) XXX_Unmarshal(b []byte) error {
......@@ -209,41 +178,39 @@ func (m *InvalidateCollMetaCacheRequest) GetCollectionName() string {
func init() {
proto.RegisterType((*RegisterNodeRequest)(nil), "milvus.proto.proxy.RegisterNodeRequest")
proto.RegisterType((*RegisterNodeResponse)(nil), "milvus.proto.proxy.RegisterNodeResponse")
proto.RegisterType((*RegisterLinkResponse)(nil), "milvus.proto.proxy.RegisterLinkResponse")
proto.RegisterType((*InvalidateCollMetaCacheRequest)(nil), "milvus.proto.proxy.InvalidateCollMetaCacheRequest")
}
func init() { proto.RegisterFile("proxy_service.proto", fileDescriptor_34ca2fbc94d169de) }
var fileDescriptor_34ca2fbc94d169de = []byte{
// 429 bytes of a gzipped FileDescriptorProto
0x1f, 0x8b, 0x08, 0x00, 0x00, 0x00, 0x00, 0x00, 0x02, 0xff, 0xc4, 0x52, 0x4d, 0x6b, 0x14, 0x41,
0x10, 0xdd, 0x89, 0x92, 0x60, 0x67, 0x89, 0xd2, 0x11, 0x0c, 0xe3, 0x07, 0x3a, 0x97, 0xe4, 0xe2,
0x8c, 0xac, 0xe0, 0x55, 0xb2, 0xc1, 0x43, 0xc0, 0x2c, 0x61, 0x02, 0x1e, 0x72, 0x59, 0x7a, 0xa6,
0x8b, 0xdd, 0xc2, 0x9e, 0xee, 0xb1, 0xbb, 0x66, 0x31, 0xb9, 0x78, 0xf3, 0x17, 0xf8, 0x7f, 0xfc,
0x6b, 0x32, 0xdd, 0xd9, 0x2f, 0x1c, 0x06, 0xc4, 0x43, 0x6e, 0x53, 0x3d, 0xaf, 0x5e, 0xbd, 0x7a,
0xf5, 0xd8, 0x61, 0x6d, 0xcd, 0xf7, 0x9b, 0xa9, 0x03, 0xbb, 0xc0, 0x12, 0xd2, 0xda, 0x1a, 0x32,
0x9c, 0x57, 0xa8, 0x16, 0x8d, 0x0b, 0x55, 0xea, 0x11, 0xf1, 0xb0, 0x34, 0x55, 0x65, 0x74, 0x78,
0x8b, 0x0f, 0x50, 0x13, 0x58, 0x2d, 0x54, 0xa8, 0x93, 0x1f, 0xec, 0x30, 0x87, 0x19, 0x3a, 0x02,
0x3b, 0x31, 0x12, 0x72, 0xf8, 0xd6, 0x80, 0x23, 0xfe, 0x8e, 0x3d, 0x2c, 0x84, 0x83, 0xa3, 0xe8,
0x75, 0x74, 0xb2, 0x3f, 0x7a, 0x91, 0x6e, 0xf1, 0xde, 0x11, 0x5e, 0xb8, 0xd9, 0x58, 0x38, 0xc8,
0x3d, 0x92, 0x7f, 0x60, 0x7b, 0x42, 0x4a, 0x0b, 0xce, 0x1d, 0xed, 0xf4, 0x34, 0x9d, 0x06, 0x4c,
0xbe, 0x04, 0x27, 0xd7, 0xec, 0xe9, 0xb6, 0x00, 0x57, 0x1b, 0xed, 0x80, 0x8f, 0xd9, 0x3e, 0x6a,
0xa4, 0x69, 0x2d, 0xac, 0xa8, 0xdc, 0x9d, 0x90, 0x37, 0xdb, 0x9c, 0xab, 0x5d, 0xce, 0x35, 0xd2,
0xa5, 0x07, 0xe6, 0x0c, 0x57, 0xdf, 0xc9, 0x64, 0xcd, 0xfd, 0x19, 0xf5, 0xd7, 0x15, 0xf7, 0x86,
0xd6, 0xe8, 0x5f, 0xb4, 0xfe, 0x8a, 0xd8, 0xab, 0x73, 0xbd, 0x10, 0x0a, 0xa5, 0x20, 0x38, 0x33,
0x4a, 0x5d, 0x00, 0x89, 0x33, 0x51, 0xce, 0xff, 0xc3, 0xb8, 0x67, 0x6c, 0x4f, 0x16, 0x53, 0x2d,
0x2a, 0xf0, 0xc6, 0x3d, 0xca, 0x77, 0x65, 0x31, 0x11, 0x15, 0xf0, 0x63, 0xf6, 0xb8, 0x34, 0x4a,
0x41, 0x49, 0x68, 0x74, 0x00, 0x3c, 0xf0, 0x80, 0x83, 0xf5, 0x73, 0x0b, 0x1c, 0xfd, 0xde, 0x61,
0xc3, 0xcb, 0xf6, 0xd6, 0x57, 0x21, 0x0c, 0xfc, 0x0b, 0x1b, 0x6e, 0xee, 0xcd, 0xe3, 0x4e, 0x19,
0x9f, 0xaa, 0x9a, 0x6e, 0xe2, 0x93, 0xf4, 0xef, 0xcc, 0xa4, 0x5d, 0xae, 0x25, 0x03, 0x5e, 0xae,
0x79, 0xdb, 0x5b, 0xf1, 0xe3, 0xbe, 0xde, 0x8d, 0x38, 0xf5, 0x0f, 0xd9, 0x3c, 0x7b, 0x32, 0xe0,
0x96, 0xbd, 0xdc, 0xf6, 0x38, 0x6c, 0xba, 0x72, 0x9a, 0x8f, 0xba, 0xc8, 0xfa, 0xcf, 0x12, 0x3f,
0xef, 0x74, 0xe0, 0x8a, 0x04, 0x35, 0x2e, 0x19, 0x8c, 0x7e, 0x46, 0xec, 0x89, 0x77, 0xb0, 0xd5,
0xb2, 0x74, 0xf1, 0x1e, 0x84, 0x8c, 0x4f, 0xaf, 0x3f, 0xce, 0x90, 0xe6, 0x4d, 0xd1, 0xfe, 0xc9,
0x6e, 0x51, 0x29, 0xbc, 0x25, 0x28, 0xe7, 0x59, 0xe8, 0x7a, 0x2b, 0xd1, 0x91, 0xc5, 0xa2, 0x21,
0x90, 0xd9, 0x32, 0xfd, 0x99, 0xa7, 0xca, 0xfc, 0xf8, 0xba, 0x28, 0x76, 0x7d, 0xf9, 0xfe, 0x4f,
0x00, 0x00, 0x00, 0xff, 0xff, 0x25, 0x02, 0x4c, 0xbc, 0x21, 0x04, 0x00, 0x00,
// 403 bytes of a gzipped FileDescriptorProto
0x1f, 0x8b, 0x08, 0x00, 0x00, 0x00, 0x00, 0x00, 0x02, 0xff, 0xc4, 0x92, 0x3f, 0xaf, 0xda, 0x30,
0x14, 0xc5, 0x49, 0x5b, 0x81, 0x6a, 0x10, 0xad, 0x4c, 0xa5, 0xa2, 0xf4, 0x8f, 0xda, 0x2c, 0xb0,
0x34, 0xa9, 0x82, 0xd4, 0xb5, 0x02, 0x26, 0x06, 0x10, 0x0a, 0x5b, 0x17, 0xe4, 0xc4, 0x57, 0x60,
0xc9, 0xb1, 0x53, 0xdb, 0x41, 0x2d, 0x4b, 0xb7, 0xae, 0x5d, 0xfa, 0x1d, 0xfb, 0x35, 0xaa, 0xd8,
0x84, 0xf7, 0xa2, 0x87, 0x78, 0xc3, 0x1b, 0xde, 0x96, 0xeb, 0xfc, 0x7c, 0x72, 0xee, 0x39, 0x41,
0x83, 0x42, 0xc9, 0x1f, 0x3f, 0xb7, 0x1a, 0xd4, 0x81, 0x65, 0x10, 0x16, 0x4a, 0x1a, 0x89, 0x71,
0xce, 0xf8, 0xa1, 0xd4, 0x6e, 0x0a, 0x2d, 0xe1, 0xf7, 0x32, 0x99, 0xe7, 0x52, 0xb8, 0x33, 0xbf,
0xcf, 0x84, 0x01, 0x25, 0x08, 0x77, 0x73, 0xf0, 0x0b, 0x0d, 0x12, 0xd8, 0x31, 0x6d, 0x40, 0xad,
0x24, 0x85, 0x04, 0xbe, 0x97, 0xa0, 0x0d, 0xfe, 0x8c, 0x9e, 0xa5, 0x44, 0xc3, 0xd0, 0xfb, 0xe0,
0x8d, 0xbb, 0xf1, 0xdb, 0xb0, 0xa1, 0x7b, 0x12, 0x5c, 0xea, 0xdd, 0x8c, 0x68, 0x48, 0x2c, 0x89,
0xbf, 0xa0, 0x0e, 0xa1, 0x54, 0x81, 0xd6, 0xc3, 0x27, 0x57, 0x2e, 0x4d, 0x1d, 0x93, 0xd4, 0x70,
0xf0, 0xc7, 0x43, 0xaf, 0x9a, 0x0e, 0x74, 0x21, 0x85, 0x06, 0x3c, 0x43, 0x5d, 0x26, 0x98, 0xd9,
0x16, 0x44, 0x91, 0x5c, 0x9f, 0x9c, 0x7c, 0x6c, 0x8a, 0x9e, 0x97, 0x59, 0x08, 0x66, 0xd6, 0x16,
0x4c, 0x10, 0x3b, 0x3f, 0xe3, 0x09, 0x6a, 0x6b, 0x43, 0x4c, 0x59, 0x7b, 0x7a, 0x73, 0xd1, 0xd3,
0xc6, 0x22, 0xc9, 0x09, 0x0d, 0xfe, 0x7a, 0xe8, 0xfd, 0x42, 0x1c, 0x08, 0x67, 0x94, 0x18, 0x98,
0x4b, 0xce, 0x97, 0x60, 0xc8, 0x9c, 0x64, 0xfb, 0x07, 0xc4, 0xf3, 0x1a, 0x75, 0x68, 0xba, 0x15,
0x24, 0x07, 0x6b, 0xe5, 0x79, 0xd2, 0xa6, 0xe9, 0x8a, 0xe4, 0x80, 0x47, 0xe8, 0x45, 0x26, 0x39,
0x87, 0xcc, 0x30, 0x29, 0x1c, 0xf0, 0xd4, 0x02, 0xfd, 0x9b, 0xe3, 0x0a, 0x8c, 0xff, 0x79, 0xa8,
0xb7, 0xae, 0x1a, 0xdd, 0xb8, 0xca, 0x71, 0x86, 0x7a, 0xb7, 0x83, 0xc3, 0xa3, 0xf0, 0x6e, 0xfb,
0xe1, 0x85, 0x72, 0xfd, 0xf1, 0xfd, 0xa0, 0xeb, 0x20, 0x68, 0x61, 0x85, 0xde, 0x35, 0xb3, 0x70,
0x8e, 0xce, 0x89, 0xe0, 0xf8, 0x92, 0xd8, 0xf5, 0xf8, 0xfc, 0x6b, 0x35, 0x04, 0xad, 0xf8, 0xb7,
0x87, 0x5e, 0xda, 0x4d, 0x2b, 0x2f, 0xf5, 0xb6, 0x8f, 0x60, 0x64, 0x36, 0xfd, 0xf6, 0x75, 0xc7,
0xcc, 0xbe, 0x4c, 0xab, 0x37, 0xd1, 0x91, 0x71, 0xce, 0x8e, 0x06, 0xb2, 0x7d, 0xe4, 0x6e, 0x7d,
0xa2, 0x4c, 0x1b, 0xc5, 0xd2, 0xd2, 0x00, 0x8d, 0xea, 0x5f, 0x31, 0xb2, 0x52, 0x91, 0xfd, 0x7c,
0x91, 0xa6, 0x6d, 0x3b, 0x4e, 0xfe, 0x07, 0x00, 0x00, 0xff, 0xff, 0xa8, 0x2b, 0x26, 0x14, 0xaf,
0x03, 0x00, 0x00,
}
// Reference imports to suppress errors if they are not otherwise used.
......@@ -258,7 +225,6 @@ const _ = grpc.SupportPackageIsVersion4
//
// For semantics around ctx use and closing/ending streaming RPCs, please refer to https://godoc.org/google.golang.org/grpc#ClientConn.NewStream.
type ProxyServiceClient interface {
RegisterLink(ctx context.Context, in *commonpb.Empty, opts ...grpc.CallOption) (*RegisterLinkResponse, error)
RegisterNode(ctx context.Context, in *RegisterNodeRequest, opts ...grpc.CallOption) (*RegisterNodeResponse, error)
InvalidateCollectionMetaCache(ctx context.Context, in *InvalidateCollMetaCacheRequest, opts ...grpc.CallOption) (*commonpb.Status, error)
}
......@@ -271,15 +237,6 @@ func NewProxyServiceClient(cc *grpc.ClientConn) ProxyServiceClient {
return &proxyServiceClient{cc}
}
func (c *proxyServiceClient) RegisterLink(ctx context.Context, in *commonpb.Empty, opts ...grpc.CallOption) (*RegisterLinkResponse, error) {
out := new(RegisterLinkResponse)
err := c.cc.Invoke(ctx, "/milvus.proto.proxy.ProxyService/RegisterLink", in, out, opts...)
if err != nil {
return nil, err
}
return out, nil
}
func (c *proxyServiceClient) RegisterNode(ctx context.Context, in *RegisterNodeRequest, opts ...grpc.CallOption) (*RegisterNodeResponse, error) {
out := new(RegisterNodeResponse)
err := c.cc.Invoke(ctx, "/milvus.proto.proxy.ProxyService/RegisterNode", in, out, opts...)
......@@ -300,7 +257,6 @@ func (c *proxyServiceClient) InvalidateCollectionMetaCache(ctx context.Context,
// ProxyServiceServer is the server API for ProxyService service.
type ProxyServiceServer interface {
RegisterLink(context.Context, *commonpb.Empty) (*RegisterLinkResponse, error)
RegisterNode(context.Context, *RegisterNodeRequest) (*RegisterNodeResponse, error)
InvalidateCollectionMetaCache(context.Context, *InvalidateCollMetaCacheRequest) (*commonpb.Status, error)
}
......@@ -309,9 +265,6 @@ type ProxyServiceServer interface {
type UnimplementedProxyServiceServer struct {
}
func (*UnimplementedProxyServiceServer) RegisterLink(ctx context.Context, req *commonpb.Empty) (*RegisterLinkResponse, error) {
return nil, status.Errorf(codes.Unimplemented, "method RegisterLink not implemented")
}
func (*UnimplementedProxyServiceServer) RegisterNode(ctx context.Context, req *RegisterNodeRequest) (*RegisterNodeResponse, error) {
return nil, status.Errorf(codes.Unimplemented, "method RegisterNode not implemented")
}
......@@ -323,24 +276,6 @@ func RegisterProxyServiceServer(s *grpc.Server, srv ProxyServiceServer) {
s.RegisterService(&_ProxyService_serviceDesc, srv)
}
func _ProxyService_RegisterLink_Handler(srv interface{}, ctx context.Context, dec func(interface{}) error, interceptor grpc.UnaryServerInterceptor) (interface{}, error) {
in := new(commonpb.Empty)
if err := dec(in); err != nil {
return nil, err
}
if interceptor == nil {
return srv.(ProxyServiceServer).RegisterLink(ctx, in)
}
info := &grpc.UnaryServerInfo{
Server: srv,
FullMethod: "/milvus.proto.proxy.ProxyService/RegisterLink",
}
handler := func(ctx context.Context, req interface{}) (interface{}, error) {
return srv.(ProxyServiceServer).RegisterLink(ctx, req.(*commonpb.Empty))
}
return interceptor(ctx, in, info, handler)
}
func _ProxyService_RegisterNode_Handler(srv interface{}, ctx context.Context, dec func(interface{}) error, interceptor grpc.UnaryServerInterceptor) (interface{}, error) {
in := new(RegisterNodeRequest)
if err := dec(in); err != nil {
......@@ -381,10 +316,6 @@ var _ProxyService_serviceDesc = grpc.ServiceDesc{
ServiceName: "milvus.proto.proxy.ProxyService",
HandlerType: (*ProxyServiceServer)(nil),
Methods: []grpc.MethodDesc{
{
MethodName: "RegisterLink",
Handler: _ProxyService_RegisterLink_Handler,
},
{
MethodName: "RegisterNode",
Handler: _ProxyService_RegisterNode_Handler,
......
......@@ -66,6 +66,12 @@ func (pt *ParamTable) NetworkAddress() string {
}
func (pt *ParamTable) ProxyServiceAddress() string {
addressFromEnv := os.Getenv("PROXY_SERVICE_ADDRESS")
if len(addressFromEnv) > 0 {
// TODO: or write to param table?
return addressFromEnv
}
addr, err := pt.Load("proxyService.address")
if err != nil {
panic(err)
......
......@@ -11,6 +11,8 @@ import (
"github.com/zilliztech/milvus-distributed/internal/msgstream/pulsarms"
grpcproxyservice "github.com/zilliztech/milvus-distributed/internal/distributed/proxyservice"
"github.com/zilliztech/milvus-distributed/internal/proto/internalpb2"
"github.com/opentracing/opentracing-go"
......@@ -32,9 +34,10 @@ type NodeImpl struct {
cancel func()
wg sync.WaitGroup
initParams *internalpb2.InitParams
ip string
port int
proxyServiceClient *grpcproxyservice.Client
initParams *internalpb2.InitParams
ip string
port int
masterConn *grpc.ClientConn
masterClient masterpb.MasterServiceClient
......
package proxyservice
import (
"context"
"fmt"
"time"
"github.com/zilliztech/milvus-distributed/internal/proto/commonpb"
"github.com/zilliztech/milvus-distributed/internal/proto/milvuspb"
"github.com/zilliztech/milvus-distributed/internal/proto/internalpb2"
"github.com/zilliztech/milvus-distributed/internal/proto/proxypb"
)
const (
timeoutInterval = time.Second * 10
)
func (s ServiceImpl) Init() error {
return nil
}
func (s *ServiceImpl) Start() error {
s.sched.Start()
return nil
}
func (s *ServiceImpl) Stop() error {
s.sched.Close()
return nil
}
func (s *ServiceImpl) GetComponentStates() (*internalpb2.ComponentStates, error) {
panic("implement me")
}
func (s *ServiceImpl) GetTimeTickChannel() (string, error) {
panic("implement me")
}
func (s *ServiceImpl) GetStatisticsChannel() (string, error) {
panic("implement me")
}
func (s *ServiceImpl) RegisterLink() (*milvuspb.RegisterLinkResponse, error) {
fmt.Println("register link")
ctx, cancel := context.WithTimeout(s.ctx, timeoutInterval)
defer cancel()
t := &RegisterLinkTask{
Condition: NewTaskCondition(ctx),
nodeInfos: s.nodeInfos,
}
var err error
err = s.sched.RegisterLinkTaskQueue.Enqueue(t)
if err != nil {
return &milvuspb.RegisterLinkResponse{
Status: &commonpb.Status{
ErrorCode: commonpb.ErrorCode_UNEXPECTED_ERROR,
Reason: err.Error(),
},
Address: nil,
}, nil
}
err = t.WaitToFinish()
if err != nil {
return &milvuspb.RegisterLinkResponse{
Status: &commonpb.Status{
ErrorCode: commonpb.ErrorCode_UNEXPECTED_ERROR,
Reason: err.Error(),
},
Address: nil,
}, nil
}
return t.response, nil
}
func (s *ServiceImpl) RegisterNode(request *proxypb.RegisterNodeRequest) (*proxypb.RegisterNodeResponse, error) {
fmt.Println("RegisterNode: ", request)
ctx, cancel := context.WithTimeout(s.ctx, timeoutInterval)
defer cancel()
t := &RegisterNodeTask{
request: request,
Condition: NewTaskCondition(ctx),
allocator: s.allocator,
nodeInfos: s.nodeInfos,
}
var err error
err = s.sched.RegisterNodeTaskQueue.Enqueue(t)
if err != nil {
return &proxypb.RegisterNodeResponse{
Status: &commonpb.Status{
ErrorCode: commonpb.ErrorCode_UNEXPECTED_ERROR,
Reason: err.Error(),
},
InitParams: nil,
}, nil
}
err = t.WaitToFinish()
if err != nil {
return &proxypb.RegisterNodeResponse{
Status: &commonpb.Status{
ErrorCode: commonpb.ErrorCode_UNEXPECTED_ERROR,
Reason: err.Error(),
},
InitParams: nil,
}, nil
}
return t.response, nil
}
func (s *ServiceImpl) InvalidateCollectionMetaCache(request *proxypb.InvalidateCollMetaCacheRequest) error {
fmt.Println("InvalidateCollectionMetaCache")
ctx, cancel := context.WithTimeout(s.ctx, timeoutInterval)
defer cancel()
t := &InvalidateCollectionMetaCacheTask{
request: request,
Condition: NewTaskCondition(ctx),
}
var err error
err = s.sched.RegisterNodeTaskQueue.Enqueue(t)
if err != nil {
return err
}
err = t.WaitToFinish()
if err != nil {
return err
}
return nil
}
package proxyservice
import (
"github.com/zilliztech/milvus-distributed/internal/proto/milvuspb"
"github.com/zilliztech/milvus-distributed/internal/proto/proxypb"
"github.com/zilliztech/milvus-distributed/internal/util/typeutil"
)
type Component = typeutil.Component
type Service = typeutil.Service
type ProxyService interface {
Component
Service
RegisterLink() (*milvuspb.RegisterLinkResponse, error)
RegisterNode(request *proxypb.RegisterNodeRequest) (*proxypb.RegisterNodeResponse, error)
// TODO: i'm sure it's not a best way to keep consistency, fix me
InvalidateCollectionMetaCache(request *proxypb.InvalidateCollMetaCacheRequest) error
}
package proxyservice
import (
"fmt"
"math/rand"
"sync"
"time"
"github.com/zilliztech/milvus-distributed/internal/errors"
"github.com/zilliztech/milvus-distributed/internal/proto/proxypb"
)
type NodeInfo struct {
ip string
port int64
}
// TODO: replace as real node client impl
type NodeClient interface {
InvalidateCollectionMetaCache(request *proxypb.InvalidateCollMetaCacheRequest) error
}
type FakeNodeClient struct {
}
func (c *FakeNodeClient) InvalidateCollectionMetaCache(request *proxypb.InvalidateCollMetaCacheRequest) error {
panic("implement me")
}
type GlobalNodeInfoTable struct {
mtx sync.RWMutex
nodeIDs []UniqueID
infos map[UniqueID]*NodeInfo
createClientMtx sync.RWMutex
// lazy creating, so len(clients) <= len(infos)
clients map[UniqueID]NodeClient
}
func (table *GlobalNodeInfoTable) randomPick() UniqueID {
rand.Seed(time.Now().UnixNano())
l := len(table.nodeIDs)
choice := rand.Intn(l)
return table.nodeIDs[choice]
}
func (table *GlobalNodeInfoTable) Pick() (*NodeInfo, error) {
table.mtx.RLock()
defer table.mtx.RUnlock()
if len(table.nodeIDs) <= 0 || len(table.infos) <= 0 {
return nil, errors.New("no available server node")
}
id := table.randomPick()
info, ok := table.infos[id]
if !ok {
// though impossible
return nil, errors.New("fix me, something wrong in pick algorithm")
}
return info, nil
}
func (table *GlobalNodeInfoTable) Register(id UniqueID, info *NodeInfo) error {
table.mtx.Lock()
defer table.mtx.Unlock()
_, ok := table.infos[id]
if !ok {
table.infos[id] = info
}
if !SliceContain(table.nodeIDs, id) {
table.nodeIDs = append(table.nodeIDs, id)
}
return nil
}
func (table *GlobalNodeInfoTable) createClients() error {
if len(table.clients) == len(table.infos) {
return nil
}
for nodeID, info := range table.infos {
_, ok := table.clients[nodeID]
if !ok {
// TODO: use info to create client
fmt.Println(info)
table.clients[nodeID] = &FakeNodeClient{}
}
}
return nil
}
func (table *GlobalNodeInfoTable) ObtainAllClients() (map[UniqueID]NodeClient, error) {
table.mtx.RLock()
defer table.mtx.RUnlock()
table.createClientMtx.Lock()
defer table.createClientMtx.Unlock()
err := table.createClients()
return table.clients, err
}
func NewGlobalNodeInfoTable() *GlobalNodeInfoTable {
return &GlobalNodeInfoTable{
nodeIDs: make([]UniqueID, 0),
infos: make(map[UniqueID]*NodeInfo),
clients: make(map[UniqueID]NodeClient),
}
}
package proxyservice
import (
"sync"
"github.com/zilliztech/milvus-distributed/internal/util/typeutil"
)
type UniqueID = typeutil.UniqueID
type NodeIDAllocator interface {
AllocOne() UniqueID
}
type NaiveNodeIDAllocatorImpl struct {
mtx sync.Mutex
now UniqueID
}
func (allocator *NaiveNodeIDAllocatorImpl) AllocOne() UniqueID {
allocator.mtx.Lock()
defer func() {
allocator.now++
allocator.mtx.Unlock()
}()
return allocator.now
}
func NewNodeIDAllocator() NodeIDAllocator {
return &NaiveNodeIDAllocatorImpl{
now: 0,
}
}
package proxyservice
import (
"github.com/zilliztech/milvus-distributed/internal/proto/internalpb2"
"github.com/zilliztech/milvus-distributed/internal/proto/proxypb"
"context"
"math/rand"
"time"
)
type ProxyService struct {
// implement Service
//nodeClients [] .Interface
// factory method
type ServiceImpl struct {
allocator NodeIDAllocator
sched *TaskScheduler
nodeInfos *GlobalNodeInfoTable
ctx context.Context
cancel context.CancelFunc
}
func (s ProxyService) Init() error {
panic("implement me")
}
func CreateProxyService(ctx context.Context) (ProxyService, error) {
rand.Seed(time.Now().UnixNano())
ctx1, cancel := context.WithCancel(ctx)
s := &ServiceImpl{
ctx: ctx1,
cancel: cancel,
}
func (s ProxyService) Start() error {
panic("implement me")
}
func (s ProxyService) Stop() error {
panic("implement me")
}
func (s ProxyService) GetComponentStates() (*internalpb2.ComponentStates, error) {
panic("implement me")
}
func (s ProxyService) GetTimeTickChannel() (string, error) {
panic("implement me")
}
func (s ProxyService) GetStatisticsChannel() (string, error) {
panic("implement me")
}
func (s ProxyService) RegisterLink() (proxypb.RegisterLinkResponse, error) {
panic("implement me")
}
func (s ProxyService) RegisterNode(request proxypb.RegisterNodeRequest) (proxypb.RegisterNodeResponse, error) {
panic("implement me")
}
func (s ProxyService) InvalidateCollectionMetaCache(request proxypb.InvalidateCollMetaCacheRequest) error {
panic("implement me")
}
s.allocator = NewNodeIDAllocator()
s.sched = NewTaskScheduler(ctx1)
s.nodeInfos = NewGlobalNodeInfoTable()
func NewProxyServiceImpl() interface{} {
return &ProxyService{}
return s, nil
}
package proxyservice
import (
"context"
"fmt"
"github.com/zilliztech/milvus-distributed/internal/proto/commonpb"
"github.com/zilliztech/milvus-distributed/internal/proto/internalpb2"
"github.com/zilliztech/milvus-distributed/internal/proto/milvuspb"
"github.com/zilliztech/milvus-distributed/internal/proto/proxypb"
"github.com/zilliztech/milvus-distributed/internal/errors"
)
type TaskEnum = int
const (
FromSDK TaskEnum = 0
FromMaster TaskEnum = 1
FromNode TaskEnum = 2
)
type task interface {
PreExecute() error
Execute() error
PostExecute() error
WaitToFinish() error
Notify(err error)
}
type Condition interface {
WaitToFinish() error
Notify(err error)
}
type TaskCondition struct {
done chan error
ctx context.Context
}
func (c *TaskCondition) WaitToFinish() error {
select {
case <-c.ctx.Done():
return errors.New("timeout")
case err := <-c.done:
return err
}
}
func (c *TaskCondition) Notify(err error) {
c.done <- err
}
func NewTaskCondition(ctx context.Context) Condition {
return &TaskCondition{
done: make(chan error),
ctx: ctx,
}
}
type RegisterLinkTask struct {
Condition
response *milvuspb.RegisterLinkResponse
nodeInfos *GlobalNodeInfoTable
}
func (t *RegisterLinkTask) PreExecute() error {
return nil
}
func (t *RegisterLinkTask) Execute() error {
info, err := t.nodeInfos.Pick()
fmt.Println("info: ", info)
if err != nil {
return err
}
t.response = &milvuspb.RegisterLinkResponse{
Address: &commonpb.Address{
Ip: info.ip,
Port: info.port,
},
Status: &commonpb.Status{
ErrorCode: commonpb.ErrorCode_SUCCESS,
Reason: "",
},
}
return nil
}
func (t *RegisterLinkTask) PostExecute() error {
return nil
}
type RegisterNodeTask struct {
Condition
request *proxypb.RegisterNodeRequest
response *proxypb.RegisterNodeResponse
allocator NodeIDAllocator
nodeInfos *GlobalNodeInfoTable
}
func (t *RegisterNodeTask) PreExecute() error {
return nil
}
func (t *RegisterNodeTask) Execute() error {
nodeID := t.allocator.AllocOne()
info := NodeInfo{
ip: t.request.Address.Ip,
port: t.request.Address.Port,
}
err := t.nodeInfos.Register(nodeID, &info)
// TODO: fill init params
t.response = &proxypb.RegisterNodeResponse{
InitParams: &internalpb2.InitParams{
NodeID: nodeID,
StartParams: nil,
},
}
return err
}
func (t *RegisterNodeTask) PostExecute() error {
return nil
}
type InvalidateCollectionMetaCacheTask struct {
Condition
request *proxypb.InvalidateCollMetaCacheRequest
nodeInfos *GlobalNodeInfoTable
}
func (t *InvalidateCollectionMetaCacheTask) PreExecute() error {
return nil
}
func (t *InvalidateCollectionMetaCacheTask) Execute() error {
var err error
clients, err := t.nodeInfos.ObtainAllClients()
if err != nil {
return err
}
for _, c := range clients {
err = c.InvalidateCollectionMetaCache(t.request)
if err != nil {
return err
}
}
return nil
}
func (t *InvalidateCollectionMetaCacheTask) PostExecute() error {
return nil
}
package proxyservice
import (
"container/list"
"log"
"sync"
"github.com/zilliztech/milvus-distributed/internal/errors"
)
type TaskQueue interface {
Chan() <-chan int
Empty() bool
Full() bool
addTask(t task) error
FrontTask() task
PopTask() task
Enqueue(t task) error
}
type BaseTaskQueue struct {
tasks *list.List
mtx sync.Mutex
// maxTaskNum should keep still
maxTaskNum int64
bufChan chan int // to block scheduler
}
func (queue *BaseTaskQueue) Chan() <-chan int {
return queue.bufChan
}
func (queue *BaseTaskQueue) Empty() bool {
return queue.tasks.Len() <= 0
}
func (queue *BaseTaskQueue) Full() bool {
return int64(queue.tasks.Len()) >= queue.maxTaskNum
}
func (queue *BaseTaskQueue) addTask(t task) error {
queue.mtx.Lock()
defer queue.mtx.Unlock()
if queue.Full() {
return errors.New("task queue is full")
}
queue.tasks.PushBack(t)
queue.bufChan <- 1
return nil
}
func (queue *BaseTaskQueue) FrontTask() task {
queue.mtx.Lock()
defer queue.mtx.Unlock()
if queue.tasks.Len() <= 0 {
log.Panic("sorry, but the task list is empty!")
return nil
}
return queue.tasks.Front().Value.(task)
}
func (queue *BaseTaskQueue) PopTask() task {
queue.mtx.Lock()
defer queue.mtx.Unlock()
if queue.tasks.Len() <= 0 {
log.Panic("sorry, but the task list is empty!")
return nil
}
ft := queue.tasks.Front()
queue.tasks.Remove(ft)
return ft.Value.(task)
}
func (queue *BaseTaskQueue) Enqueue(t task) error {
return queue.addTask(t)
}
func NewBaseTaskQueue() TaskQueue {
return &BaseTaskQueue{
tasks: list.New(),
maxTaskNum: 1024,
bufChan: make(chan int, 1024),
}
}
package proxyservice
import (
"context"
"sync"
)
type TaskScheduler struct {
RegisterLinkTaskQueue TaskQueue
RegisterNodeTaskQueue TaskQueue
InvalidateCollectionMetaCacheTaskQueue TaskQueue
wg sync.WaitGroup
ctx context.Context
cancel context.CancelFunc
}
func NewTaskScheduler(ctx context.Context) *TaskScheduler {
ctx1, cancel := context.WithCancel(ctx)
return &TaskScheduler{
RegisterLinkTaskQueue: NewBaseTaskQueue(),
RegisterNodeTaskQueue: NewBaseTaskQueue(),
InvalidateCollectionMetaCacheTaskQueue: NewBaseTaskQueue(),
ctx: ctx1,
cancel: cancel,
}
}
func (sched *TaskScheduler) scheduleRegisterLinkTask() task {
return sched.RegisterLinkTaskQueue.PopTask()
}
func (sched *TaskScheduler) scheduleRegisterNodeTask() task {
return sched.RegisterNodeTaskQueue.PopTask()
}
func (sched *TaskScheduler) scheduleInvalidateCollectionMetaCacheTask() task {
return sched.InvalidateCollectionMetaCacheTaskQueue.PopTask()
}
func (sched *TaskScheduler) processTask(t task, q TaskQueue) {
var err error
err = t.PreExecute()
defer func() {
t.Notify(err)
}()
if err != nil {
return
}
err = t.Execute()
if err != nil {
return
}
err = t.PostExecute()
}
func (sched *TaskScheduler) registerLinkLoop() {
defer sched.wg.Done()
for {
select {
case <-sched.ctx.Done():
return
case <-sched.RegisterLinkTaskQueue.Chan():
if !sched.RegisterLinkTaskQueue.Empty() {
t := sched.scheduleRegisterLinkTask()
go sched.processTask(t, sched.RegisterLinkTaskQueue)
}
}
}
}
func (sched *TaskScheduler) registerNodeLoop() {
defer sched.wg.Done()
for {
select {
case <-sched.ctx.Done():
return
case <-sched.RegisterNodeTaskQueue.Chan():
if !sched.RegisterNodeTaskQueue.Empty() {
t := sched.scheduleRegisterNodeTask()
go sched.processTask(t, sched.RegisterNodeTaskQueue)
}
}
}
}
func (sched *TaskScheduler) invalidateCollectionMetaCacheLoop() {
defer sched.wg.Done()
for {
select {
case <-sched.ctx.Done():
return
case <-sched.InvalidateCollectionMetaCacheTaskQueue.Chan():
if !sched.InvalidateCollectionMetaCacheTaskQueue.Empty() {
t := sched.scheduleInvalidateCollectionMetaCacheTask()
go sched.processTask(t, sched.InvalidateCollectionMetaCacheTaskQueue)
}
}
}
}
func (sched *TaskScheduler) Start() {
sched.wg.Add(1)
go sched.registerLinkLoop()
sched.wg.Add(1)
go sched.registerNodeLoop()
sched.wg.Add(1)
go sched.invalidateCollectionMetaCacheLoop()
}
func (sched *TaskScheduler) Close() {
sched.cancel()
sched.wg.Wait()
}
package proxyservice
import (
"reflect"
)
// what if golang support generic programming
func SliceContain(s interface{}, item interface{}) bool {
ss := reflect.ValueOf(s)
if ss.Kind() != reflect.Slice {
panic("SliceContain expect a slice")
}
for i := 0; i < ss.Len(); i++ {
if ss.Index(i).Interface() == item {
return true
}
}
return false
}
......@@ -4,5 +4,5 @@ numpy==1.18.1
pytest==5.3.4
pytest-cov==2.8.1
pytest-timeout==1.3.4
pymilvus-distributed==0.0.16
pymilvus-distributed==0.0.17
sklearn==0.0
Markdown is supported
0% .
You are about to add 0 people to the discussion. Proceed with caution.
先完成此消息的编辑!
想要评论请 注册