enhance: support cchannel for streaming service (#44143)

issue: #43897

- add cchannel as a special vchannel to hold some ddl and dcl.

Signed-off-by: chyezh <chyezh@outlook.com>
This commit is contained in:
Zhen Ye 2025-09-02 10:05:52 +08:00 committed by GitHub
parent 1e704ecf9f
commit 9e2d1963d4
No known key found for this signature in database
GPG Key ID: B5690EEEBB952194
30 changed files with 1656 additions and 1234 deletions

View File

@ -157,18 +157,15 @@ func (s *StreamingNodeManager) SetBalancerReady(b balancer.Balancer) {
func (s *StreamingNodeManager) execute() (err error) { func (s *StreamingNodeManager) execute() (err error) {
defer s.notifier.Finish(struct{}{}) defer s.notifier.Finish(struct{}{})
balancer, err := s.balancer.GetWithContext(s.notifier.Context()) b, err := s.balancer.GetWithContext(s.notifier.Context())
if err != nil { if err != nil {
return errors.Wrap(err, "failed to wait balancer ready") return errors.Wrap(err, "failed to wait balancer ready")
} }
for { for {
if err := balancer.WatchChannelAssignments(s.notifier.Context(), func( if err := b.WatchChannelAssignments(s.notifier.Context(), func(param balancer.WatchChannelAssignmentsCallbackParam) error {
version typeutil.VersionInt64Pair,
relations []types.PChannelInfoAssigned,
) error {
s.cond.LockAndBroadcast() s.cond.LockAndBroadcast()
s.latestAssignments = make(map[string]types.PChannelInfoAssigned) s.latestAssignments = make(map[string]types.PChannelInfoAssigned)
for _, relation := range relations { for _, relation := range param.Relations {
s.latestAssignments[relation.Channel.Name] = relation s.latestAssignments[relation.Channel.Name] = relation
} }
s.nodeChangedNotifier.NotifyAll() s.nodeChangedNotifier.NotifyAll()

View File

@ -8,6 +8,8 @@ import (
"github.com/stretchr/testify/mock" "github.com/stretchr/testify/mock"
"github.com/milvus-io/milvus/internal/mocks/streamingcoord/server/mock_balancer" "github.com/milvus-io/milvus/internal/mocks/streamingcoord/server/mock_balancer"
"github.com/milvus-io/milvus/internal/streamingcoord/server/balancer"
"github.com/milvus-io/milvus/pkg/v2/proto/streamingpb"
"github.com/milvus-io/milvus/pkg/v2/streaming/util/types" "github.com/milvus-io/milvus/pkg/v2/streaming/util/types"
"github.com/milvus-io/milvus/pkg/v2/util/typeutil" "github.com/milvus-io/milvus/pkg/v2/util/typeutil"
) )
@ -24,13 +26,17 @@ func TestStreamingNodeManager(t *testing.T) {
ch := make(chan pChannelInfoAssigned, 1) ch := make(chan pChannelInfoAssigned, 1)
b.EXPECT().GetAllStreamingNodes(mock.Anything).Return(map[int64]*types.StreamingNodeInfo{}, nil) b.EXPECT().GetAllStreamingNodes(mock.Anything).Return(map[int64]*types.StreamingNodeInfo{}, nil)
b.EXPECT().WatchChannelAssignments(mock.Anything, mock.Anything).Run( b.EXPECT().WatchChannelAssignments(mock.Anything, mock.Anything).Run(
func(ctx context.Context, cb func(typeutil.VersionInt64Pair, []types.PChannelInfoAssigned) error) { func(ctx context.Context, cb balancer.WatchChannelAssignmentsCallback) {
for { for {
select { select {
case <-ctx.Done(): case <-ctx.Done():
return return
case p := <-ch: case p := <-ch:
cb(p.version, p.pchannels) cb(balancer.WatchChannelAssignmentsCallbackParam{
Version: p.version,
CChannelAssignment: &streamingpb.CChannelAssignment{Meta: &streamingpb.CChannelMeta{Pchannel: "pchannel"}},
Relations: p.pchannels,
})
} }
} }
}) })

View File

@ -10,8 +10,8 @@ import (
"github.com/stretchr/testify/mock" "github.com/stretchr/testify/mock"
"github.com/milvus-io/milvus/internal/mocks/streamingcoord/server/mock_balancer" "github.com/milvus-io/milvus/internal/mocks/streamingcoord/server/mock_balancer"
"github.com/milvus-io/milvus/internal/streamingcoord/server/balancer"
"github.com/milvus-io/milvus/pkg/v2/streaming/util/types" "github.com/milvus-io/milvus/pkg/v2/streaming/util/types"
"github.com/milvus-io/milvus/pkg/v2/util/typeutil"
) )
func ResetStreamingNodeManager() { func ResetStreamingNodeManager() {
@ -21,7 +21,7 @@ func ResetStreamingNodeManager() {
func ResetDoNothingStreamingNodeManager(t *testing.T) { func ResetDoNothingStreamingNodeManager(t *testing.T) {
ResetStreamingNodeManager() ResetStreamingNodeManager()
b := mock_balancer.NewMockBalancer(t) b := mock_balancer.NewMockBalancer(t)
b.EXPECT().WatchChannelAssignments(mock.Anything, mock.Anything).RunAndReturn(func(ctx context.Context, cb func(version typeutil.VersionInt64Pair, relations []types.PChannelInfoAssigned) error) error { b.EXPECT().WatchChannelAssignments(mock.Anything, mock.Anything).RunAndReturn(func(ctx context.Context, cb balancer.WatchChannelAssignmentsCallback) error {
<-ctx.Done() <-ctx.Done()
return ctx.Err() return ctx.Err()
}).Maybe() }).Maybe()

View File

@ -2,7 +2,6 @@ package datacoord
import ( import (
"context" "context"
"github.com/milvus-io/milvus/pkg/v2/taskcommon"
"testing" "testing"
"time" "time"
@ -14,6 +13,7 @@ import (
"github.com/milvus-io/milvus/internal/datacoord/allocator" "github.com/milvus-io/milvus/internal/datacoord/allocator"
"github.com/milvus-io/milvus/internal/datacoord/session" "github.com/milvus-io/milvus/internal/datacoord/session"
"github.com/milvus-io/milvus/pkg/v2/proto/datapb" "github.com/milvus-io/milvus/pkg/v2/proto/datapb"
"github.com/milvus-io/milvus/pkg/v2/taskcommon"
"github.com/milvus-io/milvus/pkg/v2/util/merr" "github.com/milvus-io/milvus/pkg/v2/util/merr"
) )

View File

@ -9,9 +9,28 @@ import (
"github.com/milvus-io/milvus/pkg/v2/util/funcutil" "github.com/milvus-io/milvus/pkg/v2/util/funcutil"
) )
// routePChannel routes the pchannel of the vchannel.
// If the vchannel is control channel, it will return the pchannel of the cchannel.
// Otherwise, it will return the pchannel of the vchannel.
// TODO: support cross-cluster replication, so the remote vchannel should be mapping to the pchannel of the local cluster.
func (w *walAccesserImpl) routePChannel(ctx context.Context, vchannel string) (string, error) {
if vchannel == message.ControlChannel {
assignments, err := w.streamingCoordClient.Assignment().GetLatestAssignments(ctx)
if err != nil {
return "", err
}
return assignments.PChannelOfCChannel(), nil
}
pchannel := funcutil.ToPhysicalChannel(vchannel)
return pchannel, nil
}
// appendToWAL appends the message to the wal. // appendToWAL appends the message to the wal.
func (w *walAccesserImpl) appendToWAL(ctx context.Context, msg message.MutableMessage) (*types.AppendResult, error) { func (w *walAccesserImpl) appendToWAL(ctx context.Context, msg message.MutableMessage) (*types.AppendResult, error) {
pchannel := funcutil.ToPhysicalChannel(msg.VChannel()) pchannel, err := w.routePChannel(ctx, msg.VChannel())
if err != nil {
return nil, err
}
// get producer of pchannel. // get producer of pchannel.
p := w.getProducer(pchannel) p := w.getProducer(pchannel)
return p.Produce(ctx, msg) return p.Produce(ctx, msg)

View File

@ -103,6 +103,9 @@ func (m *delegatorMsgstreamAdaptor) Seek(ctx context.Context, msgPositions []*ms
zap.Uint64("timestamp", position.GetTimestamp()), zap.Uint64("timestamp", position.GetTimestamp()),
) )
handler := adaptor.NewMsgPackAdaptorHandler() handler := adaptor.NewMsgPackAdaptorHandler()
if position.GetChannelName() == message.ControlChannel {
panic("should never seek from control channel at delegator msgstream adaptor")
}
pchannel := funcutil.ToPhysicalChannel(position.GetChannelName()) pchannel := funcutil.ToPhysicalChannel(position.GetChannelName())
m.scanner = WAL().Read(ctx, ReadOption{ m.scanner = WAL().Read(ctx, ReadOption{
PChannel: pchannel, PChannel: pchannel,

View File

@ -17,7 +17,6 @@ import (
"github.com/milvus-io/milvus/pkg/v2/streaming/util/message" "github.com/milvus-io/milvus/pkg/v2/streaming/util/message"
"github.com/milvus-io/milvus/pkg/v2/streaming/util/types" "github.com/milvus-io/milvus/pkg/v2/streaming/util/types"
"github.com/milvus-io/milvus/pkg/v2/util/conc" "github.com/milvus-io/milvus/pkg/v2/util/conc"
"github.com/milvus-io/milvus/pkg/v2/util/funcutil"
"github.com/milvus-io/milvus/pkg/v2/util/typeutil" "github.com/milvus-io/milvus/pkg/v2/util/typeutil"
) )
@ -84,7 +83,7 @@ func (w *walAccesserImpl) RawAppend(ctx context.Context, msg message.MutableMess
} }
// Read returns a scanner for reading records from the wal. // Read returns a scanner for reading records from the wal.
func (w *walAccesserImpl) Read(_ context.Context, opts ReadOption) Scanner { func (w *walAccesserImpl) Read(ctx context.Context, opts ReadOption) Scanner {
if !w.lifetime.Add(typeutil.LifetimeStateWorking) { if !w.lifetime.Add(typeutil.LifetimeStateWorking) {
newErrScanner(ErrWALAccesserClosed) newErrScanner(ErrWALAccesserClosed)
} }
@ -95,7 +94,10 @@ func (w *walAccesserImpl) Read(_ context.Context, opts ReadOption) Scanner {
} }
if opts.VChannel != "" { if opts.VChannel != "" {
pchannel := funcutil.ToPhysicalChannel(opts.VChannel) pchannel, err := w.routePChannel(ctx, opts.VChannel)
if err != nil {
panic(err)
}
if opts.PChannel != "" && opts.PChannel != pchannel { if opts.PChannel != "" && opts.PChannel != pchannel {
panic("pchannel is not match with vchannel") panic("pchannel is not match with vchannel")
} }

View File

@ -209,6 +209,12 @@ type QueryCoordCatalog interface {
// StreamingCoordCataLog is the interface for streamingcoord catalog // StreamingCoordCataLog is the interface for streamingcoord catalog
type StreamingCoordCataLog interface { type StreamingCoordCataLog interface {
// GetCChannel get the control channel from metastore.
GetCChannel(ctx context.Context) (*streamingpb.CChannelMeta, error)
// SaveCChannel save the control channel to metastore.
SaveCChannel(ctx context.Context, info *streamingpb.CChannelMeta) error
// GetVersion get the streaming version from metastore. // GetVersion get the streaming version from metastore.
GetVersion(ctx context.Context) (*streamingpb.StreamingVersion, error) GetVersion(ctx context.Context) (*streamingpb.StreamingVersion, error)

View File

@ -5,4 +5,5 @@ const (
PChannelMetaPrefix = MetaPrefix + "pchannel/" PChannelMetaPrefix = MetaPrefix + "pchannel/"
BroadcastTaskPrefix = MetaPrefix + "broadcast-task/" BroadcastTaskPrefix = MetaPrefix + "broadcast-task/"
VersionPrefix = MetaPrefix + "version/" VersionPrefix = MetaPrefix + "version/"
CChannelMetaPrefix = MetaPrefix + "cchannel/"
) )

View File

@ -18,6 +18,7 @@ import (
// NewCataLog creates a new catalog instance // NewCataLog creates a new catalog instance
// streamingcoord-meta // streamingcoord-meta
// ├── version // ├── version
// ├── cchannel
// ├── broadcast // ├── broadcast
// │   ├── task-1 // │   ├── task-1
// │   └── task-2 // │   └── task-2
@ -36,6 +37,31 @@ type catalog struct {
metaKV kv.MetaKv metaKV kv.MetaKv
} }
// GetCChannel returns the control channel
func (c *catalog) GetCChannel(ctx context.Context) (*streamingpb.CChannelMeta, error) {
value, err := c.metaKV.Load(ctx, CChannelMetaPrefix)
if err != nil {
if errors.Is(err, merr.ErrIoKeyNotFound) {
return nil, nil
}
return nil, err
}
info := &streamingpb.CChannelMeta{}
if err = proto.Unmarshal([]byte(value), info); err != nil {
return nil, errors.Wrapf(err, "unmarshal cchannel meta failed")
}
return info, nil
}
// SaveCChannel saves the control channel
func (c *catalog) SaveCChannel(ctx context.Context, info *streamingpb.CChannelMeta) error {
v, err := proto.Marshal(info)
if err != nil {
return errors.Wrapf(err, "marshal cchannel meta failed")
}
return c.metaKV.Save(ctx, CChannelMetaPrefix, string(v))
}
// GetVersion returns the streaming version // GetVersion returns the streaming version
func (c *catalog) GetVersion(ctx context.Context) (*streamingpb.StreamingVersion, error) { func (c *catalog) GetVersion(ctx context.Context) (*streamingpb.StreamingVersion, error) {
value, err := c.metaKV.Load(ctx, VersionPrefix) value, err := c.metaKV.Load(ctx, VersionPrefix)

View File

@ -61,6 +61,16 @@ func TestCatalog(t *testing.T) {
assert.NoError(t, err) assert.NoError(t, err)
assert.Equal(t, v.Version, int64(1)) assert.Equal(t, v.Version, int64(1))
// CChannel test
err = catalog.SaveCChannel(context.Background(), &streamingpb.CChannelMeta{
Pchannel: "test",
})
assert.NoError(t, err)
assignments, err := catalog.GetCChannel(context.Background())
assert.NoError(t, err)
assert.Equal(t, assignments.Pchannel, "test")
// PChannel test // PChannel test
err = catalog.SavePChannels(context.Background(), []*streamingpb.PChannelMeta{ err = catalog.SavePChannels(context.Background(), []*streamingpb.PChannelMeta{
{ {

View File

@ -23,6 +23,64 @@ func (_m *MockStreamingCoordCataLog) EXPECT() *MockStreamingCoordCataLog_Expecte
return &MockStreamingCoordCataLog_Expecter{mock: &_m.Mock} return &MockStreamingCoordCataLog_Expecter{mock: &_m.Mock}
} }
// GetCChannel provides a mock function with given fields: ctx
func (_m *MockStreamingCoordCataLog) GetCChannel(ctx context.Context) (*streamingpb.CChannelMeta, error) {
ret := _m.Called(ctx)
if len(ret) == 0 {
panic("no return value specified for GetCChannel")
}
var r0 *streamingpb.CChannelMeta
var r1 error
if rf, ok := ret.Get(0).(func(context.Context) (*streamingpb.CChannelMeta, error)); ok {
return rf(ctx)
}
if rf, ok := ret.Get(0).(func(context.Context) *streamingpb.CChannelMeta); ok {
r0 = rf(ctx)
} else {
if ret.Get(0) != nil {
r0 = ret.Get(0).(*streamingpb.CChannelMeta)
}
}
if rf, ok := ret.Get(1).(func(context.Context) error); ok {
r1 = rf(ctx)
} else {
r1 = ret.Error(1)
}
return r0, r1
}
// MockStreamingCoordCataLog_GetCChannel_Call is a *mock.Call that shadows Run/Return methods with type explicit version for method 'GetCChannel'
type MockStreamingCoordCataLog_GetCChannel_Call struct {
*mock.Call
}
// GetCChannel is a helper method to define mock.On call
// - ctx context.Context
func (_e *MockStreamingCoordCataLog_Expecter) GetCChannel(ctx interface{}) *MockStreamingCoordCataLog_GetCChannel_Call {
return &MockStreamingCoordCataLog_GetCChannel_Call{Call: _e.mock.On("GetCChannel", ctx)}
}
func (_c *MockStreamingCoordCataLog_GetCChannel_Call) Run(run func(ctx context.Context)) *MockStreamingCoordCataLog_GetCChannel_Call {
_c.Call.Run(func(args mock.Arguments) {
run(args[0].(context.Context))
})
return _c
}
func (_c *MockStreamingCoordCataLog_GetCChannel_Call) Return(_a0 *streamingpb.CChannelMeta, _a1 error) *MockStreamingCoordCataLog_GetCChannel_Call {
_c.Call.Return(_a0, _a1)
return _c
}
func (_c *MockStreamingCoordCataLog_GetCChannel_Call) RunAndReturn(run func(context.Context) (*streamingpb.CChannelMeta, error)) *MockStreamingCoordCataLog_GetCChannel_Call {
_c.Call.Return(run)
return _c
}
// GetVersion provides a mock function with given fields: ctx // GetVersion provides a mock function with given fields: ctx
func (_m *MockStreamingCoordCataLog) GetVersion(ctx context.Context) (*streamingpb.StreamingVersion, error) { func (_m *MockStreamingCoordCataLog) GetVersion(ctx context.Context) (*streamingpb.StreamingVersion, error) {
ret := _m.Called(ctx) ret := _m.Called(ctx)
@ -245,6 +303,53 @@ func (_c *MockStreamingCoordCataLog_SaveBroadcastTask_Call) RunAndReturn(run fun
return _c return _c
} }
// SaveCChannel provides a mock function with given fields: ctx, info
func (_m *MockStreamingCoordCataLog) SaveCChannel(ctx context.Context, info *streamingpb.CChannelMeta) error {
ret := _m.Called(ctx, info)
if len(ret) == 0 {
panic("no return value specified for SaveCChannel")
}
var r0 error
if rf, ok := ret.Get(0).(func(context.Context, *streamingpb.CChannelMeta) error); ok {
r0 = rf(ctx, info)
} else {
r0 = ret.Error(0)
}
return r0
}
// MockStreamingCoordCataLog_SaveCChannel_Call is a *mock.Call that shadows Run/Return methods with type explicit version for method 'SaveCChannel'
type MockStreamingCoordCataLog_SaveCChannel_Call struct {
*mock.Call
}
// SaveCChannel is a helper method to define mock.On call
// - ctx context.Context
// - info *streamingpb.CChannelMeta
func (_e *MockStreamingCoordCataLog_Expecter) SaveCChannel(ctx interface{}, info interface{}) *MockStreamingCoordCataLog_SaveCChannel_Call {
return &MockStreamingCoordCataLog_SaveCChannel_Call{Call: _e.mock.On("SaveCChannel", ctx, info)}
}
func (_c *MockStreamingCoordCataLog_SaveCChannel_Call) Run(run func(ctx context.Context, info *streamingpb.CChannelMeta)) *MockStreamingCoordCataLog_SaveCChannel_Call {
_c.Call.Run(func(args mock.Arguments) {
run(args[0].(context.Context), args[1].(*streamingpb.CChannelMeta))
})
return _c
}
func (_c *MockStreamingCoordCataLog_SaveCChannel_Call) Return(_a0 error) *MockStreamingCoordCataLog_SaveCChannel_Call {
_c.Call.Return(_a0)
return _c
}
func (_c *MockStreamingCoordCataLog_SaveCChannel_Call) RunAndReturn(run func(context.Context, *streamingpb.CChannelMeta) error) *MockStreamingCoordCataLog_SaveCChannel_Call {
_c.Call.Return(run)
return _c
}
// SavePChannels provides a mock function with given fields: ctx, info // SavePChannels provides a mock function with given fields: ctx, info
func (_m *MockStreamingCoordCataLog) SavePChannels(ctx context.Context, info []*streamingpb.PChannelMeta) error { func (_m *MockStreamingCoordCataLog) SavePChannels(ctx context.Context, info []*streamingpb.PChannelMeta) error {
ret := _m.Called(ctx, info) ret := _m.Called(ctx, info)

View File

@ -5,14 +5,15 @@ package mock_balancer
import ( import (
context "context" context "context"
streamingpb "github.com/milvus-io/milvus/pkg/v2/proto/streamingpb" balancer "github.com/milvus-io/milvus/internal/streamingcoord/server/balancer"
mock "github.com/stretchr/testify/mock" mock "github.com/stretchr/testify/mock"
streamingpb "github.com/milvus-io/milvus/pkg/v2/proto/streamingpb"
syncutil "github.com/milvus-io/milvus/pkg/v2/util/syncutil" syncutil "github.com/milvus-io/milvus/pkg/v2/util/syncutil"
types "github.com/milvus-io/milvus/pkg/v2/streaming/util/types" types "github.com/milvus-io/milvus/pkg/v2/streaming/util/types"
typeutil "github.com/milvus-io/milvus/pkg/v2/util/typeutil"
) )
// MockBalancer is an autogenerated mock type for the Balancer type // MockBalancer is an autogenerated mock type for the Balancer type
@ -361,7 +362,7 @@ func (_c *MockBalancer_UpdateBalancePolicy_Call) RunAndReturn(run func(context.C
} }
// WatchChannelAssignments provides a mock function with given fields: ctx, cb // WatchChannelAssignments provides a mock function with given fields: ctx, cb
func (_m *MockBalancer) WatchChannelAssignments(ctx context.Context, cb func(typeutil.VersionInt64Pair, []types.PChannelInfoAssigned) error) error { func (_m *MockBalancer) WatchChannelAssignments(ctx context.Context, cb balancer.WatchChannelAssignmentsCallback) error {
ret := _m.Called(ctx, cb) ret := _m.Called(ctx, cb)
if len(ret) == 0 { if len(ret) == 0 {
@ -369,7 +370,7 @@ func (_m *MockBalancer) WatchChannelAssignments(ctx context.Context, cb func(typ
} }
var r0 error var r0 error
if rf, ok := ret.Get(0).(func(context.Context, func(typeutil.VersionInt64Pair, []types.PChannelInfoAssigned) error) error); ok { if rf, ok := ret.Get(0).(func(context.Context, balancer.WatchChannelAssignmentsCallback) error); ok {
r0 = rf(ctx, cb) r0 = rf(ctx, cb)
} else { } else {
r0 = ret.Error(0) r0 = ret.Error(0)
@ -385,14 +386,14 @@ type MockBalancer_WatchChannelAssignments_Call struct {
// WatchChannelAssignments is a helper method to define mock.On call // WatchChannelAssignments is a helper method to define mock.On call
// - ctx context.Context // - ctx context.Context
// - cb func(typeutil.VersionInt64Pair , []types.PChannelInfoAssigned) error // - cb balancer.WatchChannelAssignmentsCallback
func (_e *MockBalancer_Expecter) WatchChannelAssignments(ctx interface{}, cb interface{}) *MockBalancer_WatchChannelAssignments_Call { func (_e *MockBalancer_Expecter) WatchChannelAssignments(ctx interface{}, cb interface{}) *MockBalancer_WatchChannelAssignments_Call {
return &MockBalancer_WatchChannelAssignments_Call{Call: _e.mock.On("WatchChannelAssignments", ctx, cb)} return &MockBalancer_WatchChannelAssignments_Call{Call: _e.mock.On("WatchChannelAssignments", ctx, cb)}
} }
func (_c *MockBalancer_WatchChannelAssignments_Call) Run(run func(ctx context.Context, cb func(typeutil.VersionInt64Pair, []types.PChannelInfoAssigned) error)) *MockBalancer_WatchChannelAssignments_Call { func (_c *MockBalancer_WatchChannelAssignments_Call) Run(run func(ctx context.Context, cb balancer.WatchChannelAssignmentsCallback)) *MockBalancer_WatchChannelAssignments_Call {
_c.Call.Run(func(args mock.Arguments) { _c.Call.Run(func(args mock.Arguments) {
run(args[0].(context.Context), args[1].(func(typeutil.VersionInt64Pair, []types.PChannelInfoAssigned) error)) run(args[0].(context.Context), args[1].(balancer.WatchChannelAssignmentsCallback))
}) })
return _c return _c
} }
@ -402,7 +403,7 @@ func (_c *MockBalancer_WatchChannelAssignments_Call) Return(_a0 error) *MockBala
return _c return _c
} }
func (_c *MockBalancer_WatchChannelAssignments_Call) RunAndReturn(run func(context.Context, func(typeutil.VersionInt64Pair, []types.PChannelInfoAssigned) error) error) *MockBalancer_WatchChannelAssignments_Call { func (_c *MockBalancer_WatchChannelAssignments_Call) RunAndReturn(run func(context.Context, balancer.WatchChannelAssignmentsCallback) error) *MockBalancer_WatchChannelAssignments_Call {
_c.Call.Return(run) _c.Call.Return(run)
return _c return _c
} }

View File

@ -11,14 +11,16 @@ import (
"github.com/milvus-io/milvus/internal/mocks/streamingcoord/server/mock_balancer" "github.com/milvus-io/milvus/internal/mocks/streamingcoord/server/mock_balancer"
"github.com/milvus-io/milvus/internal/querycoordv2/meta" "github.com/milvus-io/milvus/internal/querycoordv2/meta"
"github.com/milvus-io/milvus/internal/querycoordv2/session" "github.com/milvus-io/milvus/internal/querycoordv2/session"
"github.com/milvus-io/milvus/internal/streamingcoord/server/balancer"
"github.com/milvus-io/milvus/pkg/v2/proto/datapb" "github.com/milvus-io/milvus/pkg/v2/proto/datapb"
"github.com/milvus-io/milvus/pkg/v2/proto/streamingpb"
"github.com/milvus-io/milvus/pkg/v2/streaming/util/types" "github.com/milvus-io/milvus/pkg/v2/streaming/util/types"
"github.com/milvus-io/milvus/pkg/v2/util/typeutil" "github.com/milvus-io/milvus/pkg/v2/util/typeutil"
) )
func TestAssignChannelToWALLocatedFirst(t *testing.T) { func TestAssignChannelToWALLocatedFirst(t *testing.T) {
balancer := mock_balancer.NewMockBalancer(t) b := mock_balancer.NewMockBalancer(t)
balancer.EXPECT().WatchChannelAssignments(mock.Anything, mock.Anything).RunAndReturn(func(ctx context.Context, cb func(typeutil.VersionInt64Pair, []types.PChannelInfoAssigned) error) error { b.EXPECT().WatchChannelAssignments(mock.Anything, mock.Anything).RunAndReturn(func(ctx context.Context, cb balancer.WatchChannelAssignmentsCallback) error {
versions := []typeutil.VersionInt64Pair{ versions := []typeutil.VersionInt64Pair{
{Global: 1, Local: 2}, {Global: 1, Local: 2},
} }
@ -39,12 +41,16 @@ func TestAssignChannelToWALLocatedFirst(t *testing.T) {
}, },
} }
for i := 0; i < len(versions); i++ { for i := 0; i < len(versions); i++ {
cb(versions[i], pchans[i]) cb(balancer.WatchChannelAssignmentsCallbackParam{
Version: versions[i],
CChannelAssignment: &streamingpb.CChannelAssignment{Meta: &streamingpb.CChannelMeta{Pchannel: "pchannel"}},
Relations: pchans[i],
})
} }
<-ctx.Done() <-ctx.Done()
return context.Cause(ctx) return context.Cause(ctx)
}) })
snmanager.StaticStreamingNodeManager.SetBalancerReady(balancer) snmanager.StaticStreamingNodeManager.SetBalancerReady(b)
channels := []*meta.DmChannel{ channels := []*meta.DmChannel{
{VchannelInfo: &datapb.VchannelInfo{ChannelName: "pchannel_v1"}}, {VchannelInfo: &datapb.VchannelInfo{ChannelName: "pchannel_v1"}},

View File

@ -33,6 +33,7 @@ import (
. "github.com/milvus-io/milvus/internal/querycoordv2/params" . "github.com/milvus-io/milvus/internal/querycoordv2/params"
"github.com/milvus-io/milvus/internal/querycoordv2/session" "github.com/milvus-io/milvus/internal/querycoordv2/session"
"github.com/milvus-io/milvus/internal/querycoordv2/utils" "github.com/milvus-io/milvus/internal/querycoordv2/utils"
"github.com/milvus-io/milvus/internal/streamingcoord/server/balancer"
"github.com/milvus-io/milvus/internal/util/streamingutil" "github.com/milvus-io/milvus/internal/util/streamingutil"
"github.com/milvus-io/milvus/pkg/v2/kv" "github.com/milvus-io/milvus/pkg/v2/kv"
"github.com/milvus-io/milvus/pkg/v2/proto/datapb" "github.com/milvus-io/milvus/pkg/v2/proto/datapb"
@ -216,13 +217,13 @@ func (suite *ReplicaObserverSuite) TestCheckNodesInReplica() {
func (suite *ReplicaObserverSuite) TestCheckSQnodesInReplica() { func (suite *ReplicaObserverSuite) TestCheckSQnodesInReplica() {
suite.observer.Stop() suite.observer.Stop()
snmanager.ResetStreamingNodeManager() snmanager.ResetStreamingNodeManager()
balancer := mock_balancer.NewMockBalancer(suite.T()) b := mock_balancer.NewMockBalancer(suite.T())
change := make(chan struct{}) change := make(chan struct{})
balancer.EXPECT().WatchChannelAssignments(mock.Anything, mock.Anything).RunAndReturn(func(ctx context.Context, cb func(typeutil.VersionInt64Pair, []types.PChannelInfoAssigned) error) error { b.EXPECT().WatchChannelAssignments(mock.Anything, mock.Anything).RunAndReturn(func(ctx context.Context, wcac balancer.WatchChannelAssignmentsCallback) error {
<-ctx.Done() <-ctx.Done()
return ctx.Err() return ctx.Err()
}) })
balancer.EXPECT().GetAllStreamingNodes(mock.Anything).RunAndReturn(func(ctx context.Context) (map[int64]*types.StreamingNodeInfo, error) { b.EXPECT().GetAllStreamingNodes(mock.Anything).RunAndReturn(func(ctx context.Context) (map[int64]*types.StreamingNodeInfo, error) {
pchans := []map[int64]*types.StreamingNodeInfo{ pchans := []map[int64]*types.StreamingNodeInfo{
{ {
1: {ServerID: 1, Address: "localhost:1"}, 1: {ServerID: 1, Address: "localhost:1"},
@ -241,7 +242,7 @@ func (suite *ReplicaObserverSuite) TestCheckSQnodesInReplica() {
return pchans[0], nil return pchans[0], nil
} }
}) })
snmanager.StaticStreamingNodeManager.SetBalancerReady(balancer) snmanager.StaticStreamingNodeManager.SetBalancerReady(b)
suite.observer = NewReplicaObserver(suite.meta, suite.distMgr) suite.observer = NewReplicaObserver(suite.meta, suite.distMgr)
suite.observer.Start() suite.observer.Start()

View File

@ -31,6 +31,7 @@ import (
"github.com/milvus-io/milvus/internal/mocks/distributed/mock_streaming" "github.com/milvus-io/milvus/internal/mocks/distributed/mock_streaming"
"github.com/milvus-io/milvus/internal/mocks/streamingcoord/server/mock_balancer" "github.com/milvus-io/milvus/internal/mocks/streamingcoord/server/mock_balancer"
mockrootcoord "github.com/milvus-io/milvus/internal/rootcoord/mocks" mockrootcoord "github.com/milvus-io/milvus/internal/rootcoord/mocks"
"github.com/milvus-io/milvus/internal/streamingcoord/server/balancer"
mocktso "github.com/milvus-io/milvus/internal/tso/mocks" mocktso "github.com/milvus-io/milvus/internal/tso/mocks"
"github.com/milvus-io/milvus/internal/util/streamingutil" "github.com/milvus-io/milvus/internal/util/streamingutil"
"github.com/milvus-io/milvus/pkg/v2/common" "github.com/milvus-io/milvus/pkg/v2/common"
@ -38,7 +39,6 @@ import (
"github.com/milvus-io/milvus/pkg/v2/streaming/util/types" "github.com/milvus-io/milvus/pkg/v2/streaming/util/types"
"github.com/milvus-io/milvus/pkg/v2/util/merr" "github.com/milvus-io/milvus/pkg/v2/util/merr"
"github.com/milvus-io/milvus/pkg/v2/util/syncutil" "github.com/milvus-io/milvus/pkg/v2/util/syncutil"
"github.com/milvus-io/milvus/pkg/v2/util/typeutil"
) )
func TestGarbageCollectorCtx_ReDropCollection(t *testing.T) { func TestGarbageCollectorCtx_ReDropCollection(t *testing.T) {
@ -552,7 +552,7 @@ func TestGcPartitionData(t *testing.T) {
snmanager.ResetStreamingNodeManager() snmanager.ResetStreamingNodeManager()
b := mock_balancer.NewMockBalancer(t) b := mock_balancer.NewMockBalancer(t)
b.EXPECT().WatchChannelAssignments(mock.Anything, mock.Anything).Run( b.EXPECT().WatchChannelAssignments(mock.Anything, mock.Anything).Run(
func(ctx context.Context, cb func(typeutil.VersionInt64Pair, []types.PChannelInfoAssigned) error) { func(ctx context.Context, cb balancer.WatchChannelAssignmentsCallback) {
<-ctx.Done() <-ctx.Done()
}) })
b.EXPECT().RegisterStreamingEnabledNotifier(mock.Anything).Run(func(notifier *syncutil.AsyncTaskNotifier[struct{}]) { b.EXPECT().RegisterStreamingEnabledNotifier(mock.Anything).Run(func(notifier *syncutil.AsyncTaskNotifier[struct{}]) {

View File

@ -161,6 +161,7 @@ func (c *assignmentDiscoverClient) recvLoop() (err error) {
c.w.Update(types.VersionedStreamingNodeAssignments{ c.w.Update(types.VersionedStreamingNodeAssignments{
Version: newIncomingVersion, Version: newIncomingVersion,
Assignments: newIncomingAssignments, Assignments: newIncomingAssignments,
CChannel: resp.FullAssignment.Cchannel,
}) })
case *streamingpb.AssignmentDiscoverResponse_Close: case *streamingpb.AssignmentDiscoverResponse_Close:
// nothing to do now, just wait io.EOF. // nothing to do now, just wait io.EOF.

View File

@ -5,10 +5,10 @@ import (
"github.com/cockroachdb/errors" "github.com/cockroachdb/errors"
"github.com/milvus-io/milvus/internal/streamingcoord/server/balancer/channel"
"github.com/milvus-io/milvus/pkg/v2/proto/streamingpb" "github.com/milvus-io/milvus/pkg/v2/proto/streamingpb"
"github.com/milvus-io/milvus/pkg/v2/streaming/util/types" "github.com/milvus-io/milvus/pkg/v2/streaming/util/types"
"github.com/milvus-io/milvus/pkg/v2/util/syncutil" "github.com/milvus-io/milvus/pkg/v2/util/syncutil"
"github.com/milvus-io/milvus/pkg/v2/util/typeutil"
) )
var ( var (
@ -16,6 +16,11 @@ var (
ErrBalancerClosed = errors.New("balancer is closed") ErrBalancerClosed = errors.New("balancer is closed")
) )
type (
WatchChannelAssignmentsCallbackParam = channel.WatchChannelAssignmentsCallbackParam
WatchChannelAssignmentsCallback = channel.WatchChannelAssignmentsCallback
)
// Balancer is a load balancer to balance the load of log node. // Balancer is a load balancer to balance the load of log node.
// Given the balance result to assign or remove channels to corresponding log node. // Given the balance result to assign or remove channels to corresponding log node.
// Balancer is a local component, it should promise all channel can be assigned, and reach the final consistency. // Balancer is a local component, it should promise all channel can be assigned, and reach the final consistency.
@ -41,7 +46,7 @@ type Balancer interface {
GetLatestWALLocated(ctx context.Context, pchannel string) (int64, bool) GetLatestWALLocated(ctx context.Context, pchannel string) (int64, bool)
// WatchChannelAssignments watches the balance result. // WatchChannelAssignments watches the balance result.
WatchChannelAssignments(ctx context.Context, cb func(version typeutil.VersionInt64Pair, relations []types.PChannelInfoAssigned) error) error WatchChannelAssignments(ctx context.Context, cb WatchChannelAssignmentsCallback) error
// MarkAsAvailable marks the pchannels as available, and trigger a rebalance. // MarkAsAvailable marks the pchannels as available, and trigger a rebalance.
MarkAsUnavailable(ctx context.Context, pChannels []types.PChannelInfo) error MarkAsUnavailable(ctx context.Context, pChannels []types.PChannelInfo) error

View File

@ -2,6 +2,7 @@ package balancer
import ( import (
"context" "context"
"sort"
"time" "time"
"github.com/cockroachdb/errors" "github.com/cockroachdb/errors"
@ -31,6 +32,8 @@ func RecoverBalancer(
incomingNewChannel ...string, // Concurrent incoming new channel directly from the configuration. incomingNewChannel ...string, // Concurrent incoming new channel directly from the configuration.
// we should add a rpc interface for creating new incoming new channel. // we should add a rpc interface for creating new incoming new channel.
) (Balancer, error) { ) (Balancer, error) {
sort.Strings(incomingNewChannel)
policyBuilder := mustGetPolicy(paramtable.Get().StreamingCfg.WALBalancerPolicyName.GetValue()) policyBuilder := mustGetPolicy(paramtable.Get().StreamingCfg.WALBalancerPolicyName.GetValue())
policy := policyBuilder.Build() policy := policyBuilder.Build()
logger := resource.Resource().Logger().With(log.FieldComponent("balancer"), zap.String("policy", policyBuilder.Name())) logger := resource.Resource().Logger().With(log.FieldComponent("balancer"), zap.String("policy", policyBuilder.Name()))
@ -91,7 +94,7 @@ func (b *balancerImpl) GetLatestWALLocated(ctx context.Context, pchannel string)
} }
// WatchChannelAssignments watches the balance result. // WatchChannelAssignments watches the balance result.
func (b *balancerImpl) WatchChannelAssignments(ctx context.Context, cb func(version typeutil.VersionInt64Pair, relations []types.PChannelInfoAssigned) error) error { func (b *balancerImpl) WatchChannelAssignments(ctx context.Context, cb WatchChannelAssignmentsCallback) error {
if !b.lifetime.Add(typeutil.LifetimeStateWorking) { if !b.lifetime.Add(typeutil.LifetimeStateWorking) {
return status.NewOnShutdownError("balancer is closing") return status.NewOnShutdownError("balancer is closing")
} }

View File

@ -69,6 +69,8 @@ func TestBalancer(t *testing.T) {
catalog := mock_metastore.NewMockStreamingCoordCataLog(t) catalog := mock_metastore.NewMockStreamingCoordCataLog(t)
resource.InitForTest(resource.OptETCD(etcdClient), resource.OptStreamingCatalog(catalog), resource.OptStreamingManagerClient(streamingNodeManager)) resource.InitForTest(resource.OptETCD(etcdClient), resource.OptStreamingCatalog(catalog), resource.OptStreamingManagerClient(streamingNodeManager))
catalog.EXPECT().GetCChannel(mock.Anything).Return(nil, nil)
catalog.EXPECT().SaveCChannel(mock.Anything, mock.Anything).Return(nil)
catalog.EXPECT().GetVersion(mock.Anything).Return(nil, nil) catalog.EXPECT().GetVersion(mock.Anything).Return(nil, nil)
catalog.EXPECT().SaveVersion(mock.Anything, mock.Anything).Return(nil) catalog.EXPECT().SaveVersion(mock.Anything, mock.Anything).Return(nil)
catalog.EXPECT().ListPChannel(mock.Anything).Unset() catalog.EXPECT().ListPChannel(mock.Anything).Unset()
@ -120,16 +122,16 @@ func TestBalancer(t *testing.T) {
resource.Resource().ETCD().Put(context.Background(), dataNodePath, string(data)) resource.Resource().ETCD().Put(context.Background(), dataNodePath, string(data))
ctx := context.Background() ctx := context.Background()
b, err := balancer.RecoverBalancer(ctx) b, err := balancer.RecoverBalancer(ctx, "test-channel-1")
assert.NoError(t, err) assert.NoError(t, err)
assert.NotNil(t, b) assert.NotNil(t, b)
doneErr := errors.New("done") doneErr := errors.New("done")
err = b.WatchChannelAssignments(context.Background(), func(version typeutil.VersionInt64Pair, relations []types.PChannelInfoAssigned) error { err = b.WatchChannelAssignments(context.Background(), func(param balancer.WatchChannelAssignmentsCallbackParam) error {
for _, relation := range relations { for _, relation := range param.Relations {
assert.Equal(t, relation.Channel.AccessMode, types.AccessModeRO) assert.Equal(t, relation.Channel.AccessMode, types.AccessModeRO)
} }
if len(relations) == 3 { if len(param.Relations) == 3 {
return doneErr return doneErr
} }
return nil return nil
@ -140,12 +142,12 @@ func TestBalancer(t *testing.T) {
resource.Resource().ETCD().Delete(context.Background(), dataNodePath) resource.Resource().ETCD().Delete(context.Background(), dataNodePath)
checkReady := func() { checkReady := func() {
err = b.WatchChannelAssignments(ctx, func(version typeutil.VersionInt64Pair, relations []types.PChannelInfoAssigned) error { err = b.WatchChannelAssignments(ctx, func(param balancer.WatchChannelAssignmentsCallbackParam) error {
// should one pchannel be assigned to per nodes // should one pchannel be assigned to per nodes
nodeIDs := typeutil.NewSet[int64]() nodeIDs := typeutil.NewSet[int64]()
if len(relations) == 3 { if len(param.Relations) == 3 {
rwCount := types.AccessModeRW rwCount := types.AccessModeRW
for _, relation := range relations { for _, relation := range param.Relations {
if relation.Channel.AccessMode == types.AccessModeRW { if relation.Channel.AccessMode == types.AccessModeRW {
rwCount++ rwCount++
} }
@ -172,7 +174,7 @@ func TestBalancer(t *testing.T) {
// create a inifite block watcher and can be interrupted by close of balancer. // create a inifite block watcher and can be interrupted by close of balancer.
f := syncutil.NewFuture[error]() f := syncutil.NewFuture[error]()
go func() { go func() {
err := b.WatchChannelAssignments(context.Background(), func(version typeutil.VersionInt64Pair, relations []types.PChannelInfoAssigned) error { err := b.WatchChannelAssignments(context.Background(), func(param balancer.WatchChannelAssignmentsCallbackParam) error {
return nil return nil
}) })
f.Set(err) f.Set(err)
@ -195,8 +197,8 @@ func TestBalancer(t *testing.T) {
assert.False(t, resp.Config.AllowRebalance) assert.False(t, resp.Config.AllowRebalance)
assert.False(t, paramtable.Get().StreamingCfg.WALBalancerPolicyAllowRebalance.GetAsBool()) assert.False(t, paramtable.Get().StreamingCfg.WALBalancerPolicyAllowRebalance.GetAsBool())
b.Trigger(ctx) b.Trigger(ctx)
err = b.WatchChannelAssignments(ctx, func(version typeutil.VersionInt64Pair, relations []types.PChannelInfoAssigned) error { err = b.WatchChannelAssignments(ctx, func(param balancer.WatchChannelAssignmentsCallbackParam) error {
for _, relation := range relations { for _, relation := range param.Relations {
if relation.Node.ServerID == 1 { if relation.Node.ServerID == 1 {
return nil return nil
} }
@ -283,6 +285,8 @@ func TestBalancer_WithRecoveryLag(t *testing.T) {
catalog := mock_metastore.NewMockStreamingCoordCataLog(t) catalog := mock_metastore.NewMockStreamingCoordCataLog(t)
resource.InitForTest(resource.OptETCD(etcdClient), resource.OptStreamingCatalog(catalog), resource.OptStreamingManagerClient(streamingNodeManager)) resource.InitForTest(resource.OptETCD(etcdClient), resource.OptStreamingCatalog(catalog), resource.OptStreamingManagerClient(streamingNodeManager))
catalog.EXPECT().GetCChannel(mock.Anything).Return(nil, nil)
catalog.EXPECT().SaveCChannel(mock.Anything, mock.Anything).Return(nil)
catalog.EXPECT().GetVersion(mock.Anything).Return(nil, nil) catalog.EXPECT().GetVersion(mock.Anything).Return(nil, nil)
catalog.EXPECT().SaveVersion(mock.Anything, mock.Anything).Return(nil) catalog.EXPECT().SaveVersion(mock.Anything, mock.Anything).Return(nil)
catalog.EXPECT().ListPChannel(mock.Anything).Unset() catalog.EXPECT().ListPChannel(mock.Anything).Unset()
@ -329,16 +333,16 @@ func TestBalancer_WithRecoveryLag(t *testing.T) {
catalog.EXPECT().SavePChannels(mock.Anything, mock.Anything).Return(nil).Maybe() catalog.EXPECT().SavePChannels(mock.Anything, mock.Anything).Return(nil).Maybe()
ctx := context.Background() ctx := context.Background()
b, err := balancer.RecoverBalancer(ctx) b, err := balancer.RecoverBalancer(ctx, "test-channel-1")
assert.NoError(t, err) assert.NoError(t, err)
assert.NotNil(t, b) assert.NotNil(t, b)
b.Trigger(context.Background()) b.Trigger(context.Background())
ctx2, cancel := context.WithTimeout(ctx, 2*time.Second) ctx2, cancel := context.WithTimeout(ctx, 2*time.Second)
defer cancel() defer cancel()
b.WatchChannelAssignments(ctx2, func(version typeutil.VersionInt64Pair, relations []types.PChannelInfoAssigned) error { b.WatchChannelAssignments(ctx2, func(param balancer.WatchChannelAssignmentsCallbackParam) error {
counts := map[int64]int{} counts := map[int64]int{}
for _, relation := range relations { for _, relation := range param.Relations {
assert.Equal(t, relation.Channel.AccessMode, types.AccessModeRW) assert.Equal(t, relation.Channel.AccessMode, types.AccessModeRW)
counts[relation.Node.ServerID]++ counts[relation.Node.ServerID]++
} }
@ -351,9 +355,9 @@ func TestBalancer_WithRecoveryLag(t *testing.T) {
lag.Store(false) lag.Store(false)
b.Trigger(context.Background()) b.Trigger(context.Background())
doneErr := errors.New("done") doneErr := errors.New("done")
b.WatchChannelAssignments(context.Background(), func(version typeutil.VersionInt64Pair, relations []types.PChannelInfoAssigned) error { b.WatchChannelAssignments(context.Background(), func(param balancer.WatchChannelAssignmentsCallbackParam) error {
counts := map[int64]int{} counts := map[int64]int{}
for _, relation := range relations { for _, relation := range param.Relations {
assert.Equal(t, relation.Channel.AccessMode, types.AccessModeRW) assert.Equal(t, relation.Channel.AccessMode, types.AccessModeRW)
counts[relation.Node.ServerID]++ counts[relation.Node.ServerID]++
} }

View File

@ -5,6 +5,7 @@ import (
"sync" "sync"
"github.com/cockroachdb/errors" "github.com/cockroachdb/errors"
"google.golang.org/protobuf/proto"
"github.com/milvus-io/milvus/internal/streamingcoord/server/resource" "github.com/milvus-io/milvus/internal/streamingcoord/server/resource"
"github.com/milvus-io/milvus/pkg/v2/proto/streamingpb" "github.com/milvus-io/milvus/pkg/v2/proto/streamingpb"
@ -17,6 +18,15 @@ import (
var ErrChannelNotExist = errors.New("channel not exist") var ErrChannelNotExist = errors.New("channel not exist")
type (
WatchChannelAssignmentsCallbackParam struct {
Version typeutil.VersionInt64Pair
CChannelAssignment *streamingpb.CChannelAssignment
Relations []types.PChannelInfoAssigned
}
WatchChannelAssignmentsCallback func(param WatchChannelAssignmentsCallbackParam) error
)
// RecoverChannelManager creates a new channel manager. // RecoverChannelManager creates a new channel manager.
func RecoverChannelManager(ctx context.Context, incomingChannel ...string) (*ChannelManager, error) { func RecoverChannelManager(ctx context.Context, incomingChannel ...string) (*ChannelManager, error) {
// streamingVersion is used to identify current streaming service version. // streamingVersion is used to identify current streaming service version.
@ -25,6 +35,11 @@ func RecoverChannelManager(ctx context.Context, incomingChannel ...string) (*Cha
if err != nil { if err != nil {
return nil, err return nil, err
} }
cchannelMeta, err := recoverCChannelMeta(ctx, incomingChannel...)
if err != nil {
return nil, err
}
channels, metrics, err := recoverFromConfigurationAndMeta(ctx, streamingVersion, incomingChannel...) channels, metrics, err := recoverFromConfigurationAndMeta(ctx, streamingVersion, incomingChannel...)
if err != nil { if err != nil {
return nil, err return nil, err
@ -39,10 +54,32 @@ func RecoverChannelManager(ctx context.Context, incomingChannel ...string) (*Cha
Local: 0, Local: 0,
}, },
metrics: metrics, metrics: metrics,
cchannelMeta: cchannelMeta,
streamingVersion: streamingVersion, streamingVersion: streamingVersion,
}, nil }, nil
} }
// recoverCChannelMeta recovers the control channel meta.
func recoverCChannelMeta(ctx context.Context, incomingChannel ...string) (*streamingpb.CChannelMeta, error) {
cchannelMeta, err := resource.Resource().StreamingCatalog().GetCChannel(ctx)
if err != nil {
return nil, err
}
if cchannelMeta == nil {
if len(incomingChannel) == 0 {
return nil, errors.New("no incoming channel while no control channel meta found")
}
cchannelMeta = &streamingpb.CChannelMeta{
Pchannel: incomingChannel[0],
}
if err := resource.Resource().StreamingCatalog().SaveCChannel(ctx, cchannelMeta); err != nil {
return nil, err
}
return cchannelMeta, nil
}
return cchannelMeta, nil
}
// recoverFromConfigurationAndMeta recovers the channel manager from configuration and meta. // recoverFromConfigurationAndMeta recovers the channel manager from configuration and meta.
func recoverFromConfigurationAndMeta(ctx context.Context, streamingVersion *streamingpb.StreamingVersion, incomingChannel ...string) (map[ChannelID]*PChannelMeta, *channelMetrics, error) { func recoverFromConfigurationAndMeta(ctx context.Context, streamingVersion *streamingpb.StreamingVersion, incomingChannel ...string) (map[ChannelID]*PChannelMeta, *channelMetrics, error) {
// Recover metrics. // Recover metrics.
@ -87,6 +124,7 @@ type ChannelManager struct {
channels map[ChannelID]*PChannelMeta channels map[ChannelID]*PChannelMeta
version typeutil.VersionInt64Pair version typeutil.VersionInt64Pair
metrics *channelMetrics metrics *channelMetrics
cchannelMeta *streamingpb.CChannelMeta
streamingVersion *streamingpb.StreamingVersion // used to identify the current streaming service version. streamingVersion *streamingpb.StreamingVersion // used to identify the current streaming service version.
// null if no streaming service has been run. // null if no streaming service has been run.
// 1 if streaming service has been run once. // 1 if streaming service has been run once.
@ -282,7 +320,7 @@ func (cm *ChannelManager) GetLatestWALLocated(ctx context.Context, pchannel stri
return 0, false return 0, false
} }
func (cm *ChannelManager) WatchAssignmentResult(ctx context.Context, cb func(version typeutil.VersionInt64Pair, assignments []types.PChannelInfoAssigned) error) error { func (cm *ChannelManager) WatchAssignmentResult(ctx context.Context, cb WatchChannelAssignmentsCallback) error {
// push the first balance result to watcher callback function if balance result is ready. // push the first balance result to watcher callback function if balance result is ready.
version, err := cm.applyAssignments(cb) version, err := cm.applyAssignments(cb)
if err != nil { if err != nil {
@ -300,7 +338,7 @@ func (cm *ChannelManager) WatchAssignmentResult(ctx context.Context, cb func(ver
} }
// applyAssignments applies the assignments. // applyAssignments applies the assignments.
func (cm *ChannelManager) applyAssignments(cb func(version typeutil.VersionInt64Pair, assignments []types.PChannelInfoAssigned) error) (typeutil.VersionInt64Pair, error) { func (cm *ChannelManager) applyAssignments(cb WatchChannelAssignmentsCallback) (typeutil.VersionInt64Pair, error) {
cm.cond.L.Lock() cm.cond.L.Lock()
assignments := make([]types.PChannelInfoAssigned, 0, len(cm.channels)) assignments := make([]types.PChannelInfoAssigned, 0, len(cm.channels))
for _, c := range cm.channels { for _, c := range cm.channels {
@ -309,8 +347,15 @@ func (cm *ChannelManager) applyAssignments(cb func(version typeutil.VersionInt64
} }
} }
version := cm.version version := cm.version
cchannelAssignment := proto.Clone(cm.cchannelMeta).(*streamingpb.CChannelMeta)
cm.cond.L.Unlock() cm.cond.L.Unlock()
return version, cb(version, assignments) return version, cb(WatchChannelAssignmentsCallbackParam{
Version: version,
CChannelAssignment: &streamingpb.CChannelAssignment{
Meta: cchannelAssignment,
},
Relations: assignments,
})
} }
// waitChanges waits for the layout to be updated. // waitChanges waits for the layout to be updated.

View File

@ -14,7 +14,6 @@ import (
"github.com/milvus-io/milvus/pkg/v2/proto/streamingpb" "github.com/milvus-io/milvus/pkg/v2/proto/streamingpb"
"github.com/milvus-io/milvus/pkg/v2/streaming/util/types" "github.com/milvus-io/milvus/pkg/v2/streaming/util/types"
"github.com/milvus-io/milvus/pkg/v2/util/syncutil" "github.com/milvus-io/milvus/pkg/v2/util/syncutil"
"github.com/milvus-io/milvus/pkg/v2/util/typeutil"
) )
func TestChannelManager(t *testing.T) { func TestChannelManager(t *testing.T) {
@ -26,6 +25,9 @@ func TestChannelManager(t *testing.T) {
ctx := context.Background() ctx := context.Background()
// Test recover failure. // Test recover failure.
catalog.EXPECT().GetCChannel(mock.Anything).Return(&streamingpb.CChannelMeta{
Pchannel: "test",
}, nil)
catalog.EXPECT().GetVersion(mock.Anything).Return(&streamingpb.StreamingVersion{ catalog.EXPECT().GetVersion(mock.Anything).Return(&streamingpb.StreamingVersion{
Version: 1, Version: 1,
}, nil) }, nil)
@ -123,6 +125,9 @@ func TestStreamingEnableChecker(t *testing.T) {
catalog := mock_metastore.NewMockStreamingCoordCataLog(t) catalog := mock_metastore.NewMockStreamingCoordCataLog(t)
resource.InitForTest(resource.OptStreamingCatalog(catalog)) resource.InitForTest(resource.OptStreamingCatalog(catalog))
// Test recover failure. // Test recover failure.
catalog.EXPECT().GetCChannel(mock.Anything).Return(&streamingpb.CChannelMeta{
Pchannel: "test-channel",
}, nil)
catalog.EXPECT().GetVersion(mock.Anything).Return(nil, nil) catalog.EXPECT().GetVersion(mock.Anything).Return(nil, nil)
catalog.EXPECT().SaveVersion(mock.Anything, mock.Anything).Return(nil) catalog.EXPECT().SaveVersion(mock.Anything, mock.Anything).Return(nil)
catalog.EXPECT().ListPChannel(mock.Anything).Return(nil, nil) catalog.EXPECT().ListPChannel(mock.Anything).Return(nil, nil)
@ -156,6 +161,9 @@ func TestChannelManagerWatch(t *testing.T) {
catalog := mock_metastore.NewMockStreamingCoordCataLog(t) catalog := mock_metastore.NewMockStreamingCoordCataLog(t)
resource.InitForTest(resource.OptStreamingCatalog(catalog)) resource.InitForTest(resource.OptStreamingCatalog(catalog))
catalog.EXPECT().GetCChannel(mock.Anything).Return(&streamingpb.CChannelMeta{
Pchannel: "test-channel",
}, nil)
catalog.EXPECT().GetVersion(mock.Anything).Return(&streamingpb.StreamingVersion{ catalog.EXPECT().GetVersion(mock.Anything).Return(&streamingpb.StreamingVersion{
Version: 1, Version: 1,
}, nil) }, nil)
@ -184,7 +192,7 @@ func TestChannelManagerWatch(t *testing.T) {
called := make(chan struct{}, 1) called := make(chan struct{}, 1)
go func() { go func() {
defer close(done) defer close(done)
err := manager.WatchAssignmentResult(ctx, func(version typeutil.VersionInt64Pair, assignments []types.PChannelInfoAssigned) error { err := manager.WatchAssignmentResult(ctx, func(param WatchChannelAssignmentsCallbackParam) error {
select { select {
case called <- struct{}{}: case called <- struct{}{}:
default: default:

View File

@ -1,9 +1,9 @@
package discover package discover
import ( import (
"github.com/milvus-io/milvus/internal/streamingcoord/server/balancer"
"github.com/milvus-io/milvus/pkg/v2/proto/streamingpb" "github.com/milvus-io/milvus/pkg/v2/proto/streamingpb"
"github.com/milvus-io/milvus/pkg/v2/streaming/util/types" "github.com/milvus-io/milvus/pkg/v2/streaming/util/types"
"github.com/milvus-io/milvus/pkg/v2/util/typeutil"
) )
// discoverGrpcServerHelper is a wrapped discover server of log messages. // discoverGrpcServerHelper is a wrapped discover server of log messages.
@ -12,9 +12,9 @@ type discoverGrpcServerHelper struct {
} }
// SendFullAssignment sends the full assignment to client. // SendFullAssignment sends the full assignment to client.
func (h *discoverGrpcServerHelper) SendFullAssignment(v typeutil.VersionInt64Pair, relations []types.PChannelInfoAssigned) error { func (h *discoverGrpcServerHelper) SendFullAssignment(param balancer.WatchChannelAssignmentsCallbackParam) error {
assignmentsMap := make(map[int64]*streamingpb.StreamingNodeAssignment) assignmentsMap := make(map[int64]*streamingpb.StreamingNodeAssignment)
for _, relation := range relations { for _, relation := range param.Relations {
if assignmentsMap[relation.Node.ServerID] == nil { if assignmentsMap[relation.Node.ServerID] == nil {
assignmentsMap[relation.Node.ServerID] = &streamingpb.StreamingNodeAssignment{ assignmentsMap[relation.Node.ServerID] = &streamingpb.StreamingNodeAssignment{
Node: types.NewProtoFromStreamingNodeInfo(relation.Node), Node: types.NewProtoFromStreamingNodeInfo(relation.Node),
@ -33,10 +33,11 @@ func (h *discoverGrpcServerHelper) SendFullAssignment(v typeutil.VersionInt64Pai
Response: &streamingpb.AssignmentDiscoverResponse_FullAssignment{ Response: &streamingpb.AssignmentDiscoverResponse_FullAssignment{
FullAssignment: &streamingpb.FullStreamingNodeAssignmentWithVersion{ FullAssignment: &streamingpb.FullStreamingNodeAssignmentWithVersion{
Version: &streamingpb.VersionPair{ Version: &streamingpb.VersionPair{
Global: v.Global, Global: param.Version.Global,
Local: v.Local, Local: param.Version.Local,
}, },
Assignments: assignments, Assignments: assignments,
Cchannel: param.CChannelAssignment,
}, },
}, },
}) })

View File

@ -8,6 +8,7 @@ import (
"github.com/stretchr/testify/mock" "github.com/stretchr/testify/mock"
"github.com/milvus-io/milvus/internal/mocks/streamingcoord/server/mock_balancer" "github.com/milvus-io/milvus/internal/mocks/streamingcoord/server/mock_balancer"
"github.com/milvus-io/milvus/internal/streamingcoord/server/balancer"
"github.com/milvus-io/milvus/internal/streamingcoord/server/resource" "github.com/milvus-io/milvus/internal/streamingcoord/server/resource"
"github.com/milvus-io/milvus/pkg/v2/mocks/proto/mock_streamingpb" "github.com/milvus-io/milvus/pkg/v2/mocks/proto/mock_streamingpb"
"github.com/milvus-io/milvus/pkg/v2/proto/streamingpb" "github.com/milvus-io/milvus/pkg/v2/proto/streamingpb"
@ -18,7 +19,7 @@ import (
func TestAssignmentDiscover(t *testing.T) { func TestAssignmentDiscover(t *testing.T) {
resource.InitForTest() resource.InitForTest()
b := mock_balancer.NewMockBalancer(t) b := mock_balancer.NewMockBalancer(t)
b.EXPECT().WatchChannelAssignments(mock.Anything, mock.Anything).RunAndReturn(func(ctx context.Context, cb func(typeutil.VersionInt64Pair, []types.PChannelInfoAssigned) error) error { b.EXPECT().WatchChannelAssignments(mock.Anything, mock.Anything).RunAndReturn(func(ctx context.Context, cb balancer.WatchChannelAssignmentsCallback) error {
versions := []typeutil.VersionInt64Pair{ versions := []typeutil.VersionInt64Pair{
{Global: 1, Local: 2}, {Global: 1, Local: 2},
{Global: 1, Local: 3}, {Global: 1, Local: 3},
@ -42,7 +43,11 @@ func TestAssignmentDiscover(t *testing.T) {
}, },
} }
for i := 0; i < len(versions); i++ { for i := 0; i < len(versions); i++ {
cb(versions[i], pchans[i]) cb(balancer.WatchChannelAssignmentsCallbackParam{
Version: versions[i],
CChannelAssignment: &streamingpb.CChannelAssignment{Meta: &streamingpb.CChannelMeta{Pchannel: "pchannel"}},
Relations: pchans[i],
})
} }
<-ctx.Done() <-ctx.Done()
return context.Cause(ctx) return context.Cause(ctx)

View File

@ -238,7 +238,7 @@ func (r *recoveryStorageImpl) observeMessage(msg message.ImmutableMessage) {
// The incoming message id is always sorted with timetick. // The incoming message id is always sorted with timetick.
func (r *recoveryStorageImpl) handleMessage(msg message.ImmutableMessage) { func (r *recoveryStorageImpl) handleMessage(msg message.ImmutableMessage) {
if msg.VChannel() != "" && msg.MessageType() != message.MessageTypeCreateCollection && if msg.VChannel() != "" && msg.MessageType() != message.MessageTypeCreateCollection &&
msg.MessageType() != message.MessageTypeDropCollection && r.vchannels[msg.VChannel()] == nil { msg.MessageType() != message.MessageTypeDropCollection && r.vchannels[msg.VChannel()] == nil && msg.VChannel() != message.ControlChannel {
r.detectInconsistency(msg, "vchannel not found") r.detectInconsistency(msg, "vchannel not found")
} }

View File

@ -14,13 +14,15 @@ import (
kvfactory "github.com/milvus-io/milvus/internal/util/dependency/kv" kvfactory "github.com/milvus-io/milvus/internal/util/dependency/kv"
"github.com/milvus-io/milvus/internal/util/sessionutil" "github.com/milvus-io/milvus/internal/util/sessionutil"
"github.com/milvus-io/milvus/internal/util/streamingutil/service/attributes" "github.com/milvus-io/milvus/internal/util/streamingutil/service/attributes"
"github.com/milvus-io/milvus/pkg/v2/util/funcutil"
"github.com/milvus-io/milvus/pkg/v2/util/typeutil" "github.com/milvus-io/milvus/pkg/v2/util/typeutil"
) )
func TestSessionDiscoverer(t *testing.T) { func TestSessionDiscoverer(t *testing.T) {
etcdClient, _ := kvfactory.GetEtcdAndPath() etcdClient, _ := kvfactory.GetEtcdAndPath()
targetVersion := "0.1.0" targetVersion := "0.1.0"
d := NewSessionDiscoverer(etcdClient, "session/", false, ">="+targetVersion) prefix := funcutil.RandomString(10) + "/"
d := NewSessionDiscoverer(etcdClient, prefix, false, ">="+targetVersion)
expected := []map[int64]*sessionutil.SessionRaw{ expected := []map[int64]*sessionutil.SessionRaw{
{}, {},
@ -71,7 +73,7 @@ func TestSessionDiscoverer(t *testing.T) {
for k, v := range expected[idx+1] { for k, v := range expected[idx+1] {
sessionStr, err := json.Marshal(v) sessionStr, err := json.Marshal(v)
assert.NoError(t, err) assert.NoError(t, err)
ops = append(ops, clientv3.OpPut(fmt.Sprintf("session/%d", k), string(sessionStr))) ops = append(ops, clientv3.OpPut(fmt.Sprintf("%s%d", prefix, k), string(sessionStr)))
} }
resp, err := etcdClient.Txn(ctx).Then( resp, err := etcdClient.Txn(ctx).Then(
@ -87,7 +89,7 @@ func TestSessionDiscoverer(t *testing.T) {
assert.ErrorIs(t, err, io.EOF) assert.ErrorIs(t, err, io.EOF)
// Do a init discover here. // Do a init discover here.
d = NewSessionDiscoverer(etcdClient, "session/", false, ">="+targetVersion) d = NewSessionDiscoverer(etcdClient, prefix, false, ">="+targetVersion)
err = d.Discover(ctx, func(state VersionedState) error { err = d.Discover(ctx, func(state VersionedState) error {
// balance attributes // balance attributes
sessions := state.Sessions() sessions := state.Sessions()

View File

@ -66,6 +66,11 @@ message PChannelMeta {
uint64 last_assign_timestamp_seconds = 5; // The last assigned timestamp in seconds. uint64 last_assign_timestamp_seconds = 5; // The last assigned timestamp in seconds.
} }
// CChannelMeta is the meta information of a control channel.
message CChannelMeta {
string pchannel = 1; // the pchannel that control channel locate on.
}
// StreamingVersion is the version of the streaming service. // StreamingVersion is the version of the streaming service.
message StreamingVersion { message StreamingVersion {
int64 version = 1; // version of the streaming, int64 version = 1; // version of the streaming,
@ -211,6 +216,12 @@ message AssignmentDiscoverResponse {
message FullStreamingNodeAssignmentWithVersion { message FullStreamingNodeAssignmentWithVersion {
VersionPair version = 1; VersionPair version = 1;
repeated StreamingNodeAssignment assignments = 2; repeated StreamingNodeAssignment assignments = 2;
CChannelAssignment cchannel = 3; // Where the control channel located.
}
// CChannelAssignment is the assignment info of a control channel.
message CChannelAssignment {
CChannelMeta meta = 1;
} }
message CloseAssignmentDiscoverResponse {} message CloseAssignmentDiscoverResponse {}

File diff suppressed because it is too large Load Diff

View File

@ -6,6 +6,10 @@ import (
"google.golang.org/protobuf/proto" "google.golang.org/protobuf/proto"
) )
// ControlChannel is the name of control channel which is used to identify the control channel from other vchannels.
// It's just a hint, which is not the real virtual channel name of control channel.
const ControlChannel string = "__cchan"
// AsImmutableTxnMessage converts an ImmutableMessage to ImmutableTxnMessage // AsImmutableTxnMessage converts an ImmutableMessage to ImmutableTxnMessage
var AsImmutableTxnMessage = func(msg ImmutableMessage) ImmutableTxnMessage { var AsImmutableTxnMessage = func(msg ImmutableMessage) ImmutableTxnMessage {
underlying, ok := msg.(*immutableTxnMessageImpl) underlying, ok := msg.(*immutableTxnMessageImpl)

View File

@ -37,6 +37,12 @@ type AssignmentRebalanceTrigger interface {
type VersionedStreamingNodeAssignments struct { type VersionedStreamingNodeAssignments struct {
Version typeutil.VersionInt64Pair Version typeutil.VersionInt64Pair
Assignments map[int64]StreamingNodeAssignment Assignments map[int64]StreamingNodeAssignment
CChannel *streamingpb.CChannelAssignment
}
// PChannelOfCChannel returns the pchannel of the cchannel.
func (v *VersionedStreamingNodeAssignments) PChannelOfCChannel() string {
return v.CChannel.Meta.Pchannel
} }
// StreamingNodeAssignment is the relation between server and channels. // StreamingNodeAssignment is the relation between server and channels.