mirror of
https://gitee.com/milvus-io/milvus.git
synced 2026-01-07 19:31:51 +08:00
issue: https://github.com/milvus-io/milvus/issues/44123 <!-- This is an auto-generated comment: release notes by coderabbit.ai --> - Core invariant: legacy in-cluster CDC/replication plumbing (ReplicateMsg types, ReplicateID-based guards and flags) is obsolete — the system relies on standard msgstream positions, subPos/end-ts semantics and timetick ordering as the single source of truth for message ordering and skipping, so replication-specific channels/types/guards can be removed safely. - Removed/simplified logic (what and why): removed replication feature flags and params (ReplicateMsgChannel, TTMsgEnabled, CollectionReplicateEnable), ReplicateMsg type and its tests, ReplicateID constants/helpers and MergeProperties hooks, ReplicateConfig and its propagation (streamPipeline, StreamConfig, dispatcher, target), replicate-aware dispatcher/pipeline branches, and replicate-mode pre-checks/timestamp-allocation in proxy tasks — these implemented a redundant alternate “replicate-mode” pathway that duplicated position/end-ts and timetick logic. - Why this does NOT cause data loss or regression (concrete code paths): no persistence or core write paths were removed — proxy PreExecute flows (internal/proxy/task_*.go) still perform the same schema/ID/size validations and then follow the normal non-replicate execution path; dispatcher and pipeline continue to use position/subPos and pullback/end-ts in Seek/grouping (pkg/mq/msgdispatcher/dispatcher.go, internal/util/pipeline/stream_pipeline.go), so skipping and ordering behavior remains unchanged; timetick emission in rootcoord (sendMinDdlTsAsTt) is now ungated (no silent suppression), preserving or increasing timetick delivery rather than removing it. - PR type and net effect: Enhancement/Refactor — removes deprecated replication API surface (types, helpers, config, tests) and replication branches, simplifies public APIs and constructor signatures, and reduces surface area for future maintenance while keeping DML/DDL persistence, ordering, and seek semantics intact. <!-- end of auto-generated comment: release notes by coderabbit.ai --> --------- Signed-off-by: bigsheeper <yihao.dai@zilliz.com>
112 lines
3.8 KiB
Go
112 lines
3.8 KiB
Go
// Licensed to the LF AI & Data foundation under one
|
|
// or more contributor license agreements. See the NOTICE file
|
|
// distributed with this work for additional information
|
|
// regarding copyright ownership. The ASF licenses this file
|
|
// to you under the Apache License, Version 2.0 (the
|
|
// "License"); you may not use this file except in compliance
|
|
// with the License. You may obtain a copy of the License at
|
|
//
|
|
// http://www.apache.org/licenses/LICENSE-2.0
|
|
//
|
|
// Unless required by applicable law or agreed to in writing, software
|
|
// distributed under the License is distributed on an "AS IS" BASIS,
|
|
// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
|
|
// See the License for the specific language governing permissions and
|
|
// limitations under the License.
|
|
|
|
package pipeline
|
|
|
|
import (
|
|
"context"
|
|
"fmt"
|
|
"time"
|
|
|
|
"go.uber.org/zap"
|
|
|
|
"github.com/milvus-io/milvus-proto/go-api/v2/commonpb"
|
|
"github.com/milvus-io/milvus-proto/go-api/v2/msgpb"
|
|
"github.com/milvus-io/milvus-proto/go-api/v2/schemapb"
|
|
"github.com/milvus-io/milvus/internal/util/flowgraph"
|
|
"github.com/milvus-io/milvus/pkg/v2/log"
|
|
"github.com/milvus-io/milvus/pkg/v2/metrics"
|
|
"github.com/milvus-io/milvus/pkg/v2/mq/common"
|
|
"github.com/milvus-io/milvus/pkg/v2/mq/msgdispatcher"
|
|
"github.com/milvus-io/milvus/pkg/v2/mq/msgstream"
|
|
"github.com/milvus-io/milvus/pkg/v2/util/paramtable"
|
|
"github.com/milvus-io/milvus/pkg/v2/util/tsoutil"
|
|
"github.com/milvus-io/milvus/pkg/v2/util/typeutil"
|
|
)
|
|
|
|
// DmInputNode receives messages from message streams, packs messages between two timeticks, and passes all
|
|
//
|
|
// messages between two timeticks to the following flowgraph node. In DataNode, the following flow graph node is
|
|
// flowgraph ddNode.
|
|
func newDmInputNode(dmNodeConfig *nodeConfig, input <-chan *msgstream.MsgPack) *flowgraph.InputNode {
|
|
if input == nil {
|
|
panic("unreachable: input channel is nil for input node")
|
|
}
|
|
name := fmt.Sprintf("dmInputNode-data-%s", dmNodeConfig.vChannelName)
|
|
node := flowgraph.NewInputNode(
|
|
input,
|
|
name,
|
|
paramtable.Get().DataNodeCfg.FlowGraphMaxQueueLength.GetAsInt32(),
|
|
paramtable.Get().DataNodeCfg.FlowGraphMaxParallelism.GetAsInt32(),
|
|
typeutil.DataNodeRole,
|
|
dmNodeConfig.serverID,
|
|
dmNodeConfig.collectionID,
|
|
metrics.AllLabel,
|
|
)
|
|
return node
|
|
}
|
|
|
|
func createNewInputFromDispatcher(initCtx context.Context,
|
|
dispatcherClient msgdispatcher.Client,
|
|
vchannel string,
|
|
seekPos *msgpb.MsgPosition,
|
|
schema *schemapb.CollectionSchema,
|
|
dbProperties []*commonpb.KeyValuePair,
|
|
) (<-chan *msgstream.MsgPack, error) {
|
|
log := log.With(zap.Int64("nodeID", paramtable.GetNodeID()),
|
|
zap.String("vchannel", vchannel))
|
|
|
|
var (
|
|
input <-chan *msgstream.MsgPack
|
|
err error
|
|
start = time.Now()
|
|
)
|
|
|
|
if seekPos != nil && len(seekPos.MsgID) != 0 {
|
|
input, err = dispatcherClient.Register(initCtx, &msgdispatcher.StreamConfig{
|
|
VChannel: vchannel,
|
|
Pos: seekPos,
|
|
SubPos: common.SubscriptionPositionUnknown,
|
|
})
|
|
if err != nil {
|
|
log.Warn("datanode consume failed after retried", zap.Error(err))
|
|
dispatcherClient.Deregister(vchannel)
|
|
return nil, err
|
|
}
|
|
|
|
log.Info("datanode seek successfully when register to msgDispatcher",
|
|
zap.ByteString("msgID", seekPos.GetMsgID()),
|
|
zap.Time("tsTime", tsoutil.PhysicalTime(seekPos.GetTimestamp())),
|
|
zap.Duration("tsLag", time.Since(tsoutil.PhysicalTime(seekPos.GetTimestamp()))),
|
|
zap.Duration("dur", time.Since(start)))
|
|
return input, err
|
|
}
|
|
|
|
input, err = dispatcherClient.Register(initCtx, &msgdispatcher.StreamConfig{
|
|
VChannel: vchannel,
|
|
Pos: nil,
|
|
SubPos: common.SubscriptionPositionEarliest,
|
|
})
|
|
if err != nil {
|
|
log.Warn("datanode consume failed after retried", zap.Error(err))
|
|
dispatcherClient.Deregister(vchannel)
|
|
return nil, err
|
|
}
|
|
|
|
log.Info("datanode consume successfully when register to msgDispatcher")
|
|
return input, err
|
|
}
|