2021-10-15 18:07:09 +08:00
|
|
|
// 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
|
2021-04-19 15:16:33 +08:00
|
|
|
// with the License. You may obtain a copy of the License at
|
|
|
|
//
|
2021-10-15 18:07:09 +08:00
|
|
|
// http://www.apache.org/licenses/LICENSE-2.0
|
2021-04-19 15:16:33 +08:00
|
|
|
//
|
2021-10-15 18:07:09 +08:00
|
|
|
// 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.
|
2021-04-19 15:16:33 +08:00
|
|
|
|
2021-01-19 11:37:16 +08:00
|
|
|
package datanode
|
|
|
|
|
|
|
|
import (
|
|
|
|
"context"
|
2021-09-09 15:00:00 +08:00
|
|
|
"errors"
|
2022-02-28 19:11:55 +08:00
|
|
|
"fmt"
|
2022-09-04 09:05:09 +08:00
|
|
|
"math"
|
2022-05-24 21:11:59 +08:00
|
|
|
"reflect"
|
2021-03-23 18:50:13 +08:00
|
|
|
"sync"
|
2021-01-19 11:37:16 +08:00
|
|
|
|
2021-09-11 11:36:22 +08:00
|
|
|
"github.com/golang/protobuf/proto"
|
2021-09-17 16:27:56 +08:00
|
|
|
"github.com/opentracing/opentracing-go"
|
2021-11-04 15:40:14 +08:00
|
|
|
"go.uber.org/atomic"
|
2021-02-26 10:13:36 +08:00
|
|
|
"go.uber.org/zap"
|
2021-01-19 11:37:16 +08:00
|
|
|
|
2022-09-16 16:56:49 +08:00
|
|
|
"github.com/milvus-io/milvus/api/commonpb"
|
|
|
|
"github.com/milvus-io/milvus/api/schemapb"
|
2021-04-22 14:45:57 +08:00
|
|
|
"github.com/milvus-io/milvus/internal/log"
|
2022-02-28 19:11:55 +08:00
|
|
|
"github.com/milvus-io/milvus/internal/metrics"
|
2022-03-03 21:57:56 +08:00
|
|
|
"github.com/milvus-io/milvus/internal/mq/msgstream"
|
2021-05-26 12:09:03 +08:00
|
|
|
"github.com/milvus-io/milvus/internal/proto/datapb"
|
2021-04-22 14:45:57 +08:00
|
|
|
"github.com/milvus-io/milvus/internal/proto/internalpb"
|
2022-02-28 19:11:55 +08:00
|
|
|
"github.com/milvus-io/milvus/internal/storage"
|
2022-03-15 21:51:21 +08:00
|
|
|
"github.com/milvus-io/milvus/internal/util/funcutil"
|
2022-05-24 21:11:59 +08:00
|
|
|
"github.com/milvus-io/milvus/internal/util/retry"
|
2022-02-28 19:11:55 +08:00
|
|
|
"github.com/milvus-io/milvus/internal/util/trace"
|
|
|
|
"github.com/milvus-io/milvus/internal/util/tsoutil"
|
2021-01-19 11:37:16 +08:00
|
|
|
)
|
|
|
|
|
|
|
|
type (
|
2021-09-23 19:05:55 +08:00
|
|
|
// InsertData of storage
|
2021-01-19 11:37:16 +08:00
|
|
|
InsertData = storage.InsertData
|
2021-09-23 19:05:55 +08:00
|
|
|
|
|
|
|
// Blob of storage
|
|
|
|
Blob = storage.Blob
|
2021-01-19 11:37:16 +08:00
|
|
|
)
|
2021-09-17 16:27:56 +08:00
|
|
|
|
2021-03-25 14:41:46 +08:00
|
|
|
type insertBufferNode struct {
|
|
|
|
BaseNode
|
2022-05-24 21:11:59 +08:00
|
|
|
|
|
|
|
ctx context.Context
|
2021-06-08 19:25:37 +08:00
|
|
|
channelName string
|
2021-09-26 20:55:59 +08:00
|
|
|
insertBuffer sync.Map // SegmentID to BufferData
|
|
|
|
replica Replica
|
|
|
|
idAllocator allocatorInterface
|
2021-09-18 14:25:50 +08:00
|
|
|
|
2021-09-23 16:03:54 +08:00
|
|
|
flushMap sync.Map
|
2021-10-18 12:34:34 +08:00
|
|
|
flushChan <-chan flushMsg
|
2022-05-25 14:34:00 +08:00
|
|
|
resendTTChan <-chan resendTTMsg
|
2021-09-23 16:03:54 +08:00
|
|
|
flushingSegCache *Cache
|
2021-10-19 11:04:34 +08:00
|
|
|
flushManager flushManager
|
2021-03-25 14:41:46 +08:00
|
|
|
|
2021-12-15 10:53:16 +08:00
|
|
|
timeTickStream msgstream.MsgStream
|
2021-12-30 10:09:48 +08:00
|
|
|
ttLogger *timeTickLogger
|
2021-12-15 10:53:16 +08:00
|
|
|
ttMerger *mergedTimeTickerSender
|
2021-12-22 16:59:05 +08:00
|
|
|
|
|
|
|
lastTimestamp Timestamp
|
2021-11-04 15:40:14 +08:00
|
|
|
}
|
|
|
|
|
|
|
|
type timeTickLogger struct {
|
2021-12-30 10:09:48 +08:00
|
|
|
start atomic.Uint64
|
|
|
|
counter atomic.Int32
|
|
|
|
vChannelName string
|
2021-11-04 15:40:14 +08:00
|
|
|
}
|
|
|
|
|
|
|
|
func (l *timeTickLogger) LogTs(ts Timestamp) {
|
|
|
|
if l.counter.Load() == 0 {
|
|
|
|
l.start.Store(ts)
|
|
|
|
}
|
|
|
|
l.counter.Inc()
|
|
|
|
if l.counter.Load() == 1000 {
|
|
|
|
min := l.start.Load()
|
|
|
|
l.start.Store(ts)
|
|
|
|
l.counter.Store(0)
|
|
|
|
go l.printLogs(min, ts)
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
func (l *timeTickLogger) printLogs(start, end Timestamp) {
|
|
|
|
t1, _ := tsoutil.ParseTS(start)
|
|
|
|
t2, _ := tsoutil.ParseTS(end)
|
2021-12-30 10:09:48 +08:00
|
|
|
log.Debug("IBN timetick log", zap.Time("from", t1), zap.Time("to", t2), zap.Duration("elapsed", t2.Sub(t1)), zap.Uint64("start", start), zap.Uint64("end", end), zap.String("vChannelName", l.vChannelName))
|
2021-06-04 16:31:34 +08:00
|
|
|
}
|
|
|
|
|
2021-06-06 13:21:37 +08:00
|
|
|
type segmentCheckPoint struct {
|
|
|
|
numRows int64
|
|
|
|
pos internalpb.MsgPosition
|
|
|
|
}
|
|
|
|
|
2021-09-23 20:15:54 +08:00
|
|
|
// BufferData buffers insert data, monitoring buffer size and limit
|
2021-09-26 20:55:59 +08:00
|
|
|
// size and limit both indicate numOfRows
|
2021-09-18 14:25:50 +08:00
|
|
|
type BufferData struct {
|
|
|
|
buffer *InsertData
|
|
|
|
size int64
|
2021-09-26 20:55:59 +08:00
|
|
|
limit int64
|
2022-09-04 09:05:09 +08:00
|
|
|
tsFrom Timestamp
|
|
|
|
tsTo Timestamp
|
2021-09-18 14:25:50 +08:00
|
|
|
}
|
|
|
|
|
2021-09-26 20:55:59 +08:00
|
|
|
// newBufferData needs an input dimension to calculate the limit of this buffer
|
|
|
|
//
|
|
|
|
// `limit` is the segment numOfRows a buffer can buffer at most.
|
|
|
|
//
|
|
|
|
// For a float32 vector field:
|
|
|
|
// limit = 16 * 2^20 Byte [By default] / (dimension * 4 Byte)
|
|
|
|
//
|
|
|
|
// For a binary vector field:
|
|
|
|
// limit = 16 * 2^20 Byte [By default]/ (dimension / 8 Byte)
|
|
|
|
//
|
|
|
|
// But since the buffer of binary vector fields is larger than the float32 one
|
|
|
|
// with the same dimension, newBufferData takes the smaller buffer limit
|
|
|
|
// to fit in both types of vector fields
|
|
|
|
//
|
|
|
|
// * This need to change for string field support and multi-vector fields support.
|
2021-09-18 14:25:50 +08:00
|
|
|
func newBufferData(dimension int64) (*BufferData, error) {
|
|
|
|
if dimension == 0 {
|
|
|
|
return nil, errors.New("Invalid dimension")
|
|
|
|
}
|
|
|
|
|
2021-12-23 18:39:11 +08:00
|
|
|
limit := Params.DataNodeCfg.FlushInsertBufferSize / (dimension * 4)
|
2021-09-18 14:25:50 +08:00
|
|
|
|
2022-03-25 14:27:25 +08:00
|
|
|
//TODO::xige-16 eval vec and string field
|
2022-09-04 09:05:09 +08:00
|
|
|
return &BufferData{
|
|
|
|
buffer: &InsertData{Data: make(map[UniqueID]storage.FieldData)},
|
|
|
|
size: 0,
|
|
|
|
limit: limit,
|
|
|
|
tsFrom: math.MaxUint64,
|
|
|
|
tsTo: 0}, nil
|
2021-03-25 14:41:46 +08:00
|
|
|
}
|
2021-01-19 11:37:16 +08:00
|
|
|
|
2021-09-26 20:55:59 +08:00
|
|
|
func (bd *BufferData) effectiveCap() int64 {
|
|
|
|
return bd.limit - bd.size
|
2021-01-19 11:37:16 +08:00
|
|
|
}
|
|
|
|
|
2021-09-26 20:55:59 +08:00
|
|
|
func (bd *BufferData) updateSize(no int64) {
|
|
|
|
bd.size += no
|
2021-01-19 11:37:16 +08:00
|
|
|
}
|
|
|
|
|
2022-09-04 09:05:09 +08:00
|
|
|
// updateTimeRange update BufferData tsFrom, tsTo range according to input time range
|
|
|
|
func (bd *BufferData) updateTimeRange(tr TimeRange) {
|
|
|
|
if tr.timestampMin < bd.tsFrom {
|
|
|
|
bd.tsFrom = tr.timestampMin
|
|
|
|
}
|
|
|
|
if tr.timestampMax > bd.tsTo {
|
|
|
|
bd.tsTo = tr.timestampMax
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2021-01-19 11:37:16 +08:00
|
|
|
func (ibNode *insertBufferNode) Name() string {
|
2021-12-30 10:33:46 +08:00
|
|
|
return "ibNode-" + ibNode.channelName
|
2021-01-19 11:37:16 +08:00
|
|
|
}
|
|
|
|
|
2021-09-17 16:27:56 +08:00
|
|
|
func (ibNode *insertBufferNode) Close() {
|
2021-11-05 14:59:32 +08:00
|
|
|
ibNode.ttMerger.close()
|
|
|
|
|
2021-09-17 16:27:56 +08:00
|
|
|
if ibNode.timeTickStream != nil {
|
|
|
|
ibNode.timeTickStream.Close()
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
func (ibNode *insertBufferNode) Operate(in []Msg) []Msg {
|
2021-06-08 19:25:37 +08:00
|
|
|
// log.Debug("InsertBufferNode Operating")
|
|
|
|
|
2021-01-19 11:37:16 +08:00
|
|
|
if len(in) != 1 {
|
2021-02-26 10:13:36 +08:00
|
|
|
log.Error("Invalid operate message input in insertBufferNode", zap.Int("input length", len(in)))
|
2021-09-17 16:27:56 +08:00
|
|
|
return []Msg{}
|
2021-01-19 11:37:16 +08:00
|
|
|
}
|
|
|
|
|
2021-09-26 10:43:57 +08:00
|
|
|
fgMsg, ok := in[0].(*flowGraphMsg)
|
2021-01-19 11:37:16 +08:00
|
|
|
if !ok {
|
2022-05-24 21:11:59 +08:00
|
|
|
if in[0] == nil {
|
|
|
|
log.Debug("type assertion failed for flowGraphMsg because it's nil")
|
|
|
|
} else {
|
|
|
|
log.Warn("type assertion failed for flowGraphMsg", zap.String("name", reflect.TypeOf(in[0]).Name()))
|
|
|
|
}
|
2021-03-25 14:41:46 +08:00
|
|
|
return []Msg{}
|
|
|
|
}
|
|
|
|
|
2021-12-02 16:39:33 +08:00
|
|
|
if fgMsg.dropCollection {
|
|
|
|
ibNode.flushManager.startDropping()
|
|
|
|
}
|
|
|
|
|
2021-03-25 14:41:46 +08:00
|
|
|
var spans []opentracing.Span
|
2021-09-26 10:43:57 +08:00
|
|
|
for _, msg := range fgMsg.insertMessages {
|
2021-03-25 14:41:46 +08:00
|
|
|
sp, ctx := trace.StartSpanFromContext(msg.TraceCtx())
|
|
|
|
spans = append(spans, sp)
|
|
|
|
msg.SetTraceCtx(ctx)
|
2021-03-23 01:49:50 +08:00
|
|
|
}
|
|
|
|
|
2021-06-18 16:02:05 +08:00
|
|
|
// replace pchannel with vchannel
|
2021-09-26 10:43:57 +08:00
|
|
|
startPositions := make([]*internalpb.MsgPosition, 0, len(fgMsg.startPositions))
|
|
|
|
for idx := range fgMsg.startPositions {
|
|
|
|
pos := proto.Clone(fgMsg.startPositions[idx]).(*internalpb.MsgPosition)
|
2021-06-18 16:02:05 +08:00
|
|
|
pos.ChannelName = ibNode.channelName
|
2021-09-11 11:36:22 +08:00
|
|
|
startPositions = append(startPositions, pos)
|
2021-06-18 16:02:05 +08:00
|
|
|
}
|
2021-09-26 10:43:57 +08:00
|
|
|
endPositions := make([]*internalpb.MsgPosition, 0, len(fgMsg.endPositions))
|
|
|
|
for idx := range fgMsg.endPositions {
|
|
|
|
pos := proto.Clone(fgMsg.endPositions[idx]).(*internalpb.MsgPosition)
|
2021-06-18 16:02:05 +08:00
|
|
|
pos.ChannelName = ibNode.channelName
|
2021-09-11 11:36:22 +08:00
|
|
|
endPositions = append(endPositions, pos)
|
2021-06-18 16:02:05 +08:00
|
|
|
}
|
|
|
|
|
2021-12-22 16:59:05 +08:00
|
|
|
if startPositions[0].Timestamp < ibNode.lastTimestamp {
|
2022-05-24 21:11:59 +08:00
|
|
|
// message stream should guarantee that this should not happen
|
|
|
|
err := fmt.Errorf("insert buffer node consumed old messages, channel = %s, timestamp = %d, lastTimestamp = %d",
|
|
|
|
ibNode.channelName, startPositions[0].Timestamp, ibNode.lastTimestamp)
|
|
|
|
log.Error(err.Error())
|
|
|
|
panic(err)
|
2021-12-22 16:59:05 +08:00
|
|
|
}
|
|
|
|
|
|
|
|
ibNode.lastTimestamp = endPositions[0].Timestamp
|
|
|
|
|
2021-09-17 16:27:56 +08:00
|
|
|
// Updating segment statistics in replica
|
2021-09-26 10:43:57 +08:00
|
|
|
seg2Upload, err := ibNode.updateSegStatesInReplica(fgMsg.insertMessages, startPositions[0], endPositions[0])
|
2021-09-17 16:27:56 +08:00
|
|
|
if err != nil {
|
2022-05-24 21:11:59 +08:00
|
|
|
// Occurs only if the collectionID is mismatch, should not happen
|
|
|
|
err = fmt.Errorf("update segment states in Replica wrong, err = %s", err)
|
|
|
|
log.Error(err.Error())
|
|
|
|
panic(err)
|
2021-01-21 09:55:25 +08:00
|
|
|
}
|
2021-02-04 11:19:48 +08:00
|
|
|
|
2021-09-17 16:27:56 +08:00
|
|
|
// insert messages -> buffer
|
2021-09-26 10:43:57 +08:00
|
|
|
for _, msg := range fgMsg.insertMessages {
|
2021-09-18 14:25:50 +08:00
|
|
|
err := ibNode.bufferInsertMsg(msg, endPositions[0])
|
2021-01-19 11:37:16 +08:00
|
|
|
if err != nil {
|
2022-05-24 21:11:59 +08:00
|
|
|
// error occurs when missing schema info or data is misaligned, should not happen
|
|
|
|
err = fmt.Errorf("insertBufferNode msg to buffer failed, err = %s", err)
|
|
|
|
log.Error(err.Error())
|
|
|
|
panic(err)
|
2021-06-05 16:21:36 +08:00
|
|
|
}
|
2021-01-19 11:37:16 +08:00
|
|
|
}
|
|
|
|
|
2021-09-26 20:55:59 +08:00
|
|
|
// Find and return the smaller input
|
|
|
|
min := func(former, latter int) (smaller int) {
|
|
|
|
if former <= latter {
|
|
|
|
return former
|
2021-01-19 11:37:16 +08:00
|
|
|
}
|
2021-09-26 20:55:59 +08:00
|
|
|
return latter
|
|
|
|
}
|
|
|
|
|
|
|
|
displaySize := min(10, len(seg2Upload))
|
|
|
|
|
2021-10-06 22:20:05 +08:00
|
|
|
// Log the segment statistics in mem
|
2021-09-26 20:55:59 +08:00
|
|
|
for k, segID := range seg2Upload[:displaySize] {
|
|
|
|
bd, ok := ibNode.insertBuffer.Load(segID)
|
|
|
|
if !ok {
|
|
|
|
continue
|
|
|
|
}
|
|
|
|
|
2022-03-02 15:35:55 +08:00
|
|
|
log.Info("insert seg buffer status", zap.Int("No.", k),
|
2021-09-26 20:55:59 +08:00
|
|
|
zap.Int64("segmentID", segID),
|
2021-11-17 10:07:13 +08:00
|
|
|
zap.String("vchannel name", ibNode.channelName),
|
2021-09-26 20:55:59 +08:00
|
|
|
zap.Int64("buffer size", bd.(*BufferData).size),
|
|
|
|
zap.Int64("buffer limit", bd.(*BufferData).limit))
|
2021-01-19 11:37:16 +08:00
|
|
|
}
|
|
|
|
|
2021-11-11 20:56:49 +08:00
|
|
|
// Flush
|
2021-10-25 18:03:42 +08:00
|
|
|
type flushTask struct {
|
|
|
|
buffer *BufferData
|
|
|
|
segmentID UniqueID
|
|
|
|
flushed bool
|
2021-11-11 20:56:49 +08:00
|
|
|
dropped bool
|
2022-04-27 23:03:47 +08:00
|
|
|
auto bool
|
2021-11-11 20:56:49 +08:00
|
|
|
}
|
|
|
|
|
|
|
|
var (
|
|
|
|
flushTaskList []flushTask
|
|
|
|
segmentsToFlush []UniqueID
|
|
|
|
)
|
|
|
|
|
|
|
|
if fgMsg.dropCollection {
|
|
|
|
segmentsToFlush := ibNode.replica.listAllSegmentIDs()
|
2022-03-02 15:35:55 +08:00
|
|
|
log.Info("Receive drop collection req and flushing all segments",
|
2021-11-17 10:07:13 +08:00
|
|
|
zap.Any("segments", segmentsToFlush),
|
|
|
|
zap.String("vchannel name", ibNode.channelName),
|
|
|
|
)
|
2021-11-11 20:56:49 +08:00
|
|
|
flushTaskList = make([]flushTask, 0, len(segmentsToFlush))
|
|
|
|
|
|
|
|
for _, seg2Flush := range segmentsToFlush {
|
|
|
|
var buf *BufferData
|
|
|
|
bd, ok := ibNode.insertBuffer.Load(seg2Flush)
|
|
|
|
if !ok {
|
|
|
|
buf = nil
|
|
|
|
} else {
|
|
|
|
buf = bd.(*BufferData)
|
|
|
|
}
|
|
|
|
flushTaskList = append(flushTaskList, flushTask{
|
|
|
|
buffer: buf,
|
|
|
|
segmentID: seg2Flush,
|
|
|
|
flushed: false,
|
|
|
|
dropped: true,
|
|
|
|
})
|
|
|
|
}
|
2021-11-15 15:25:09 +08:00
|
|
|
} else {
|
|
|
|
segmentsToFlush = make([]UniqueID, 0, len(seg2Upload)+1) //auto flush number + possible manual flush
|
|
|
|
flushTaskList = make([]flushTask, 0, len(seg2Upload)+1)
|
|
|
|
|
|
|
|
// Auto Flush
|
|
|
|
for _, segToFlush := range seg2Upload {
|
|
|
|
// If full, auto flush
|
|
|
|
if bd, ok := ibNode.insertBuffer.Load(segToFlush); ok && bd.(*BufferData).effectiveCap() <= 0 {
|
2022-07-08 14:50:21 +08:00
|
|
|
log.Info("(Auto Flush)",
|
2021-11-17 10:07:13 +08:00
|
|
|
zap.Int64("segment id", segToFlush),
|
|
|
|
zap.String("vchannel name", ibNode.channelName),
|
|
|
|
)
|
2021-11-15 15:25:09 +08:00
|
|
|
ibuffer := bd.(*BufferData)
|
|
|
|
|
|
|
|
flushTaskList = append(flushTaskList, flushTask{
|
|
|
|
buffer: ibuffer,
|
|
|
|
segmentID: segToFlush,
|
|
|
|
flushed: false,
|
|
|
|
dropped: false,
|
2022-04-27 23:03:47 +08:00
|
|
|
auto: true,
|
2021-11-15 15:25:09 +08:00
|
|
|
})
|
2022-02-28 19:11:55 +08:00
|
|
|
|
2021-11-15 15:25:09 +08:00
|
|
|
}
|
2021-05-25 15:35:37 +08:00
|
|
|
}
|
|
|
|
|
2022-07-15 17:12:27 +08:00
|
|
|
mergeFlushTask := func(segmentID UniqueID, setupTask func(task *flushTask)) {
|
2022-07-08 14:50:21 +08:00
|
|
|
// Merge auto & manual flush tasks with the same segment ID.
|
2021-11-15 15:25:09 +08:00
|
|
|
dup := false
|
|
|
|
for i, task := range flushTaskList {
|
2022-07-15 17:12:27 +08:00
|
|
|
if task.segmentID == segmentID {
|
2022-07-08 14:50:21 +08:00
|
|
|
log.Info("merging flush task, updating flushed flag",
|
2022-07-15 17:12:27 +08:00
|
|
|
zap.Int64("segment ID", segmentID))
|
|
|
|
setupTask(&flushTaskList[i])
|
2021-11-15 15:25:09 +08:00
|
|
|
dup = true
|
|
|
|
break
|
|
|
|
}
|
2021-10-25 18:03:42 +08:00
|
|
|
}
|
2022-07-08 14:50:21 +08:00
|
|
|
// Load buffer and create new flush task if there's no existing flush task for this segment.
|
2021-11-15 15:25:09 +08:00
|
|
|
if !dup {
|
2022-07-15 17:12:27 +08:00
|
|
|
bd, ok := ibNode.insertBuffer.Load(segmentID)
|
2021-11-15 15:25:09 +08:00
|
|
|
var buf *BufferData
|
|
|
|
if ok {
|
|
|
|
buf = bd.(*BufferData)
|
|
|
|
}
|
2022-07-15 17:12:27 +08:00
|
|
|
task := flushTask{
|
2021-11-15 15:25:09 +08:00
|
|
|
buffer: buf,
|
2022-07-15 17:12:27 +08:00
|
|
|
segmentID: segmentID,
|
2021-11-15 15:25:09 +08:00
|
|
|
dropped: false,
|
2022-07-15 17:12:27 +08:00
|
|
|
}
|
|
|
|
setupTask(&task)
|
|
|
|
flushTaskList = append(flushTaskList, task)
|
2021-10-25 18:03:42 +08:00
|
|
|
}
|
2022-07-15 17:12:27 +08:00
|
|
|
}
|
|
|
|
|
|
|
|
// Manual Flush
|
|
|
|
select {
|
|
|
|
case fmsg := <-ibNode.flushChan:
|
|
|
|
log.Info("(Manual Flush) receiving flush message",
|
|
|
|
zap.Int64("segmentID", fmsg.segmentID),
|
|
|
|
zap.Int64("collectionID", fmsg.collectionID),
|
|
|
|
zap.Bool("flushed", fmsg.flushed),
|
|
|
|
zap.String("v-channel name", ibNode.channelName),
|
|
|
|
)
|
|
|
|
mergeFlushTask(fmsg.segmentID, func(task *flushTask) {
|
|
|
|
task.flushed = fmsg.flushed
|
|
|
|
})
|
2021-11-15 15:25:09 +08:00
|
|
|
default:
|
2021-10-19 11:04:34 +08:00
|
|
|
}
|
2022-07-15 17:12:27 +08:00
|
|
|
|
|
|
|
// process drop partition
|
|
|
|
for _, partitionDrop := range fgMsg.dropPartitions {
|
|
|
|
segmentIDs := ibNode.replica.listPartitionSegments(partitionDrop)
|
|
|
|
log.Info("(Drop Partition) process drop partition",
|
|
|
|
zap.Int64("collectionID", ibNode.replica.getCollectionID()),
|
|
|
|
zap.Int64("partitionID", partitionDrop),
|
|
|
|
zap.Int64s("segmentIDs", segmentIDs),
|
|
|
|
zap.String("v-channel name", ibNode.channelName),
|
|
|
|
)
|
|
|
|
for _, segID := range segmentIDs {
|
|
|
|
mergeFlushTask(segID, func(task *flushTask) {
|
|
|
|
task.flushed = true
|
|
|
|
task.dropped = true
|
|
|
|
})
|
|
|
|
}
|
|
|
|
}
|
2021-10-25 18:03:42 +08:00
|
|
|
}
|
|
|
|
|
|
|
|
for _, task := range flushTaskList {
|
2022-07-08 14:50:21 +08:00
|
|
|
log.Debug("insertBufferNode flushing BufferData",
|
|
|
|
zap.Int64("segment ID", task.segmentID),
|
|
|
|
zap.Bool("flushed", task.flushed),
|
|
|
|
zap.Bool("dropped", task.dropped),
|
|
|
|
zap.Any("pos", endPositions[0]),
|
|
|
|
)
|
2022-09-02 10:34:59 +08:00
|
|
|
|
|
|
|
segStats, err := ibNode.replica.getSegmentStatslog(task.segmentID)
|
|
|
|
if err != nil {
|
|
|
|
log.Error("failed to get segment stats log", zap.Int64("segmentID", task.segmentID), zap.Error(err))
|
|
|
|
panic(err)
|
|
|
|
}
|
|
|
|
|
2022-05-24 21:11:59 +08:00
|
|
|
err = retry.Do(ibNode.ctx, func() error {
|
|
|
|
return ibNode.flushManager.flushBufferData(task.buffer,
|
2022-09-02 10:34:59 +08:00
|
|
|
segStats,
|
2022-05-24 21:11:59 +08:00
|
|
|
task.segmentID,
|
|
|
|
task.flushed,
|
|
|
|
task.dropped,
|
|
|
|
endPositions[0])
|
2022-09-13 14:12:31 +08:00
|
|
|
}, getFlowGraphRetryOpt())
|
2021-10-19 11:04:34 +08:00
|
|
|
if err != nil {
|
2022-04-27 23:03:47 +08:00
|
|
|
metrics.DataNodeFlushBufferCount.WithLabelValues(fmt.Sprint(Params.DataNodeCfg.GetNodeID()), metrics.FailLabel).Inc()
|
2022-05-24 21:11:59 +08:00
|
|
|
metrics.DataNodeFlushBufferCount.WithLabelValues(fmt.Sprint(Params.DataNodeCfg.GetNodeID()), metrics.TotalLabel).Inc()
|
2022-04-27 23:03:47 +08:00
|
|
|
if task.auto {
|
|
|
|
metrics.DataNodeAutoFlushBufferCount.WithLabelValues(fmt.Sprint(Params.DataNodeCfg.GetNodeID()), metrics.FailLabel).Inc()
|
2022-05-24 21:11:59 +08:00
|
|
|
metrics.DataNodeAutoFlushBufferCount.WithLabelValues(fmt.Sprint(Params.DataNodeCfg.GetNodeID()), metrics.TotalLabel).Inc()
|
2022-04-27 23:03:47 +08:00
|
|
|
}
|
2022-05-24 21:11:59 +08:00
|
|
|
err = fmt.Errorf("insertBufferNode flushBufferData failed, err = %s", err)
|
|
|
|
log.Error(err.Error())
|
|
|
|
panic(err)
|
2022-04-27 23:03:47 +08:00
|
|
|
}
|
2022-05-24 21:11:59 +08:00
|
|
|
segmentsToFlush = append(segmentsToFlush, task.segmentID)
|
|
|
|
ibNode.insertBuffer.Delete(task.segmentID)
|
|
|
|
metrics.DataNodeFlushBufferCount.WithLabelValues(fmt.Sprint(Params.DataNodeCfg.GetNodeID()), metrics.SuccessLabel).Inc()
|
2022-04-27 23:03:47 +08:00
|
|
|
metrics.DataNodeFlushBufferCount.WithLabelValues(fmt.Sprint(Params.DataNodeCfg.GetNodeID()), metrics.TotalLabel).Inc()
|
|
|
|
if task.auto {
|
|
|
|
metrics.DataNodeAutoFlushBufferCount.WithLabelValues(fmt.Sprint(Params.DataNodeCfg.GetNodeID()), metrics.TotalLabel).Inc()
|
2022-05-24 21:11:59 +08:00
|
|
|
metrics.DataNodeAutoFlushBufferCount.WithLabelValues(fmt.Sprint(Params.DataNodeCfg.GetNodeID()), metrics.FailLabel).Inc()
|
2021-05-18 19:45:00 +08:00
|
|
|
}
|
2021-01-19 11:37:16 +08:00
|
|
|
}
|
|
|
|
|
2022-07-08 14:50:21 +08:00
|
|
|
select {
|
|
|
|
case resendTTMsg := <-ibNode.resendTTChan:
|
|
|
|
log.Info("resend TT msg received in insertBufferNode",
|
|
|
|
zap.Int64s("segment IDs", resendTTMsg.segmentIDs))
|
|
|
|
seg2Upload = append(seg2Upload, resendTTMsg.segmentIDs...)
|
|
|
|
default:
|
|
|
|
}
|
2022-05-24 21:11:59 +08:00
|
|
|
ibNode.writeHardTimeTick(fgMsg.timeRange.timestampMax, seg2Upload)
|
2021-01-19 11:37:16 +08:00
|
|
|
|
2021-10-11 16:31:44 +08:00
|
|
|
res := flowGraphMsg{
|
2021-10-18 12:34:34 +08:00
|
|
|
deleteMessages: fgMsg.deleteMessages,
|
|
|
|
timeRange: fgMsg.timeRange,
|
|
|
|
startPositions: fgMsg.startPositions,
|
|
|
|
endPositions: fgMsg.endPositions,
|
|
|
|
segmentsToFlush: segmentsToFlush,
|
2021-11-11 20:56:49 +08:00
|
|
|
dropCollection: fgMsg.dropCollection,
|
2021-10-11 16:31:44 +08:00
|
|
|
}
|
|
|
|
|
2021-03-25 14:41:46 +08:00
|
|
|
for _, sp := range spans {
|
|
|
|
sp.Finish()
|
|
|
|
}
|
2021-01-19 11:37:16 +08:00
|
|
|
|
2021-10-11 16:31:44 +08:00
|
|
|
// send delete msg to DeleteNode
|
|
|
|
return []Msg{&res}
|
2021-01-19 11:37:16 +08:00
|
|
|
}
|
|
|
|
|
2021-10-06 22:20:05 +08:00
|
|
|
// updateSegStatesInReplica updates statistics in replica for the segments in insertMsgs.
|
|
|
|
// If the segment doesn't exist, a new segment will be created.
|
|
|
|
// The segment number of rows will be updated in mem, waiting to be uploaded to DataCoord.
|
2021-09-17 16:27:56 +08:00
|
|
|
func (ibNode *insertBufferNode) updateSegStatesInReplica(insertMsgs []*msgstream.InsertMsg, startPos, endPos *internalpb.MsgPosition) (seg2Upload []UniqueID, err error) {
|
|
|
|
uniqueSeg := make(map[UniqueID]int64)
|
|
|
|
for _, msg := range insertMsgs {
|
|
|
|
|
|
|
|
currentSegID := msg.GetSegmentID()
|
|
|
|
collID := msg.GetCollectionID()
|
|
|
|
partitionID := msg.GetPartitionID()
|
|
|
|
|
|
|
|
if !ibNode.replica.hasSegment(currentSegID, true) {
|
2021-09-27 10:01:59 +08:00
|
|
|
err = ibNode.replica.addNewSegment(currentSegID, collID, partitionID, msg.GetShardName(),
|
2021-09-17 16:27:56 +08:00
|
|
|
startPos, endPos)
|
|
|
|
if err != nil {
|
2021-09-18 09:13:50 +08:00
|
|
|
log.Error("add segment wrong",
|
|
|
|
zap.Int64("segID", currentSegID),
|
|
|
|
zap.Int64("collID", collID),
|
|
|
|
zap.Int64("partID", partitionID),
|
2021-09-27 10:01:59 +08:00
|
|
|
zap.String("chanName", msg.GetShardName()),
|
2021-09-18 09:13:50 +08:00
|
|
|
zap.Error(err))
|
2021-09-17 16:27:56 +08:00
|
|
|
return
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
segNum := uniqueSeg[currentSegID]
|
|
|
|
uniqueSeg[currentSegID] = segNum + int64(len(msg.RowIDs))
|
|
|
|
}
|
|
|
|
|
|
|
|
seg2Upload = make([]UniqueID, 0, len(uniqueSeg))
|
|
|
|
for id, num := range uniqueSeg {
|
|
|
|
seg2Upload = append(seg2Upload, id)
|
|
|
|
ibNode.replica.updateStatistics(id, num)
|
|
|
|
}
|
|
|
|
|
|
|
|
return
|
|
|
|
}
|
|
|
|
|
2021-09-18 14:45:50 +08:00
|
|
|
/* #nosec G103 */
|
2021-09-09 15:00:00 +08:00
|
|
|
// bufferInsertMsg put InsertMsg into buffer
|
|
|
|
// 1.1 fetch related schema from replica
|
|
|
|
// 1.2 Get buffer data and put data into each field buffer
|
|
|
|
// 1.3 Put back into buffer
|
|
|
|
// 1.4 Update related statistics
|
2021-09-18 14:25:50 +08:00
|
|
|
func (ibNode *insertBufferNode) bufferInsertMsg(msg *msgstream.InsertMsg, endPos *internalpb.MsgPosition) error {
|
2022-03-25 14:27:25 +08:00
|
|
|
if err := msg.CheckAligned(); err != nil {
|
|
|
|
return err
|
2021-09-09 15:00:00 +08:00
|
|
|
}
|
|
|
|
currentSegID := msg.GetSegmentID()
|
|
|
|
collectionID := msg.GetCollectionID()
|
|
|
|
|
2021-09-26 20:55:59 +08:00
|
|
|
collSchema, err := ibNode.replica.getCollectionSchema(collectionID, msg.EndTs())
|
|
|
|
if err != nil {
|
|
|
|
log.Error("Get schema wrong:", zap.Error(err))
|
|
|
|
return err
|
|
|
|
}
|
|
|
|
|
|
|
|
// Get Dimension
|
|
|
|
// TODO GOOSE: under assumption that there's only 1 Vector field in one collection schema
|
|
|
|
var dimension int
|
|
|
|
for _, field := range collSchema.Fields {
|
|
|
|
if field.DataType == schemapb.DataType_FloatVector ||
|
|
|
|
field.DataType == schemapb.DataType_BinaryVector {
|
|
|
|
|
2022-03-04 15:09:56 +08:00
|
|
|
dimension, err = storage.GetDimFromParams(field.TypeParams)
|
|
|
|
if err != nil {
|
|
|
|
log.Error("failed to get dim from field", zap.Error(err))
|
|
|
|
return err
|
2021-09-26 20:55:59 +08:00
|
|
|
}
|
|
|
|
break
|
2021-09-09 15:00:00 +08:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2021-09-26 20:55:59 +08:00
|
|
|
newbd, err := newBufferData(int64(dimension))
|
2021-09-09 15:00:00 +08:00
|
|
|
if err != nil {
|
|
|
|
return err
|
|
|
|
}
|
2021-09-26 20:55:59 +08:00
|
|
|
bd, _ := ibNode.insertBuffer.LoadOrStore(currentSegID, newbd)
|
|
|
|
|
|
|
|
buffer := bd.(*BufferData)
|
2022-03-04 15:09:56 +08:00
|
|
|
// idata := buffer.buffer
|
2021-09-09 15:00:00 +08:00
|
|
|
|
2022-03-04 15:09:56 +08:00
|
|
|
addedBuffer, err := storage.InsertMsgToInsertData(msg, collSchema)
|
|
|
|
if err != nil {
|
|
|
|
log.Error("failed to transfer insert msg to insert data", zap.Error(err))
|
|
|
|
return err
|
2021-10-13 21:32:33 +08:00
|
|
|
}
|
|
|
|
|
2022-03-04 15:09:56 +08:00
|
|
|
addedPfData, err := storage.GetPkFromInsertData(collSchema, addedBuffer)
|
|
|
|
if err != nil {
|
|
|
|
log.Warn("no primary field found in insert msg", zap.Error(err))
|
|
|
|
} else {
|
|
|
|
ibNode.replica.updateSegmentPKRange(currentSegID, addedPfData)
|
2021-09-09 15:00:00 +08:00
|
|
|
}
|
|
|
|
|
2022-03-04 15:09:56 +08:00
|
|
|
// Maybe there are large write zoom if frequent insert requests are met.
|
|
|
|
buffer.buffer = storage.MergeInsertData(buffer.buffer, addedBuffer)
|
|
|
|
|
2022-09-04 09:05:09 +08:00
|
|
|
tsData, err := storage.GetTimestampFromInsertData(addedBuffer)
|
|
|
|
if err != nil {
|
|
|
|
log.Warn("no timestamp field found in insert msg", zap.Error(err))
|
|
|
|
return err
|
|
|
|
}
|
|
|
|
|
2021-09-26 20:55:59 +08:00
|
|
|
// update buffer size
|
2022-03-04 15:09:56 +08:00
|
|
|
buffer.updateSize(int64(msg.NRows()))
|
2022-09-04 09:05:09 +08:00
|
|
|
// update timestamp range
|
|
|
|
buffer.updateTimeRange(ibNode.getTimestampRange(tsData))
|
|
|
|
|
2022-04-24 22:03:44 +08:00
|
|
|
metrics.DataNodeConsumeMsgRowsCount.WithLabelValues(fmt.Sprint(Params.DataNodeCfg.GetNodeID()), metrics.InsertLabel).Add(float64(len(msg.RowData)))
|
2021-09-26 20:55:59 +08:00
|
|
|
|
|
|
|
// store in buffer
|
|
|
|
ibNode.insertBuffer.Store(currentSegID, buffer)
|
2021-09-09 15:00:00 +08:00
|
|
|
|
|
|
|
// store current endPositions as Segment->EndPostion
|
2021-09-18 14:25:50 +08:00
|
|
|
ibNode.replica.updateSegmentEndPosition(currentSegID, endPos)
|
|
|
|
|
2021-09-09 15:00:00 +08:00
|
|
|
return nil
|
|
|
|
}
|
|
|
|
|
2022-09-04 09:05:09 +08:00
|
|
|
func (ibNode *insertBufferNode) getTimestampRange(tsData *storage.Int64FieldData) TimeRange {
|
|
|
|
tr := TimeRange{
|
|
|
|
timestampMin: math.MaxUint64,
|
|
|
|
timestampMax: 0,
|
|
|
|
}
|
|
|
|
|
|
|
|
for _, data := range tsData.Data {
|
|
|
|
if data < int64(tr.timestampMin) {
|
|
|
|
tr.timestampMin = Timestamp(data)
|
|
|
|
}
|
|
|
|
if data > int64(tr.timestampMax) {
|
|
|
|
tr.timestampMax = Timestamp(data)
|
|
|
|
}
|
|
|
|
}
|
|
|
|
return tr
|
|
|
|
}
|
|
|
|
|
2021-10-06 23:30:21 +08:00
|
|
|
// writeHardTimeTick writes timetick once insertBufferNode operates.
|
2022-05-24 21:11:59 +08:00
|
|
|
func (ibNode *insertBufferNode) writeHardTimeTick(ts Timestamp, segmentIDs []int64) {
|
2021-11-04 15:40:14 +08:00
|
|
|
ibNode.ttLogger.LogTs(ts)
|
2021-12-15 10:53:16 +08:00
|
|
|
ibNode.ttMerger.bufferTs(ts, segmentIDs)
|
2022-09-16 09:56:47 +08:00
|
|
|
rateCol.updateFlowGraphTt(ibNode.channelName, ts)
|
2021-01-19 11:37:16 +08:00
|
|
|
}
|
2022-05-24 21:11:59 +08:00
|
|
|
|
2021-05-18 19:45:00 +08:00
|
|
|
func (ibNode *insertBufferNode) getCollectionandPartitionIDbySegID(segmentID UniqueID) (collID, partitionID UniqueID, err error) {
|
2021-06-21 16:00:22 +08:00
|
|
|
return ibNode.replica.getCollectionAndPartitionID(segmentID)
|
2021-05-18 19:45:00 +08:00
|
|
|
}
|
|
|
|
|
2022-05-25 14:34:00 +08:00
|
|
|
func newInsertBufferNode(ctx context.Context, collID UniqueID, flushCh <-chan flushMsg, resendTTCh <-chan resendTTMsg,
|
|
|
|
fm flushManager, flushingSegCache *Cache, config *nodeConfig) (*insertBufferNode, error) {
|
2021-01-19 11:37:16 +08:00
|
|
|
|
|
|
|
baseNode := BaseNode{}
|
2021-10-13 11:16:32 +08:00
|
|
|
baseNode.SetMaxQueueLength(config.maxQueueLength)
|
|
|
|
baseNode.SetMaxParallelism(config.maxParallelism)
|
2021-01-19 11:37:16 +08:00
|
|
|
|
2021-01-21 09:55:25 +08:00
|
|
|
//input stream, data node time tick
|
2021-10-13 11:16:32 +08:00
|
|
|
wTt, err := config.msFactory.NewMsgStream(ctx)
|
2021-08-30 10:03:58 +08:00
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
2022-03-04 11:17:56 +08:00
|
|
|
wTt.AsProducer([]string{Params.CommonCfg.DataCoordTimeTick})
|
2022-04-24 22:03:44 +08:00
|
|
|
metrics.DataNodeNumProducers.WithLabelValues(fmt.Sprint(Params.DataNodeCfg.GetNodeID())).Inc()
|
2022-03-04 11:17:56 +08:00
|
|
|
log.Debug("datanode AsProducer", zap.String("TimeTickChannelName", Params.CommonCfg.DataCoordTimeTick))
|
2021-01-21 09:55:25 +08:00
|
|
|
var wTtMsgStream msgstream.MsgStream = wTt
|
2021-01-24 21:20:11 +08:00
|
|
|
wTtMsgStream.Start()
|
2021-01-21 09:55:25 +08:00
|
|
|
|
2021-12-15 10:53:16 +08:00
|
|
|
mt := newMergedTimeTickerSender(func(ts Timestamp, segmentIDs []int64) error {
|
|
|
|
stats := make([]*datapb.SegmentStats, 0, len(segmentIDs))
|
|
|
|
for _, sid := range segmentIDs {
|
|
|
|
stat, err := config.replica.getSegmentStatisticsUpdates(sid)
|
|
|
|
if err != nil {
|
|
|
|
log.Warn("failed to get segment statistics info", zap.Int64("segmentID", sid), zap.Error(err))
|
|
|
|
continue
|
|
|
|
}
|
|
|
|
stats = append(stats, stat)
|
|
|
|
}
|
2021-11-05 14:59:32 +08:00
|
|
|
msgPack := msgstream.MsgPack{}
|
|
|
|
timeTickMsg := msgstream.DataNodeTtMsg{
|
|
|
|
BaseMsg: msgstream.BaseMsg{
|
|
|
|
BeginTimestamp: ts,
|
|
|
|
EndTimestamp: ts,
|
|
|
|
HashValues: []uint32{0},
|
|
|
|
},
|
|
|
|
DataNodeTtMsg: datapb.DataNodeTtMsg{
|
|
|
|
Base: &commonpb.MsgBase{
|
|
|
|
MsgType: commonpb.MsgType_DataNodeTt,
|
|
|
|
MsgID: 0,
|
|
|
|
Timestamp: ts,
|
|
|
|
},
|
2021-12-15 10:53:16 +08:00
|
|
|
ChannelName: config.vChannelName,
|
|
|
|
Timestamp: ts,
|
|
|
|
SegmentsStats: stats,
|
2021-11-05 14:59:32 +08:00
|
|
|
},
|
|
|
|
}
|
|
|
|
msgPack.Msgs = append(msgPack.Msgs, &timeTickMsg)
|
2022-02-28 19:11:55 +08:00
|
|
|
pt, _ := tsoutil.ParseHybridTs(ts)
|
2022-03-15 21:51:21 +08:00
|
|
|
pChan := funcutil.ToPhysicalChannel(config.vChannelName)
|
2022-04-24 22:03:44 +08:00
|
|
|
metrics.DataNodeTimeSync.WithLabelValues(fmt.Sprint(Params.DataNodeCfg.GetNodeID()), pChan).Set(float64(pt))
|
2021-11-05 14:59:32 +08:00
|
|
|
return wTtMsgStream.Produce(&msgPack)
|
|
|
|
})
|
|
|
|
|
2021-01-19 11:37:16 +08:00
|
|
|
return &insertBufferNode{
|
2022-05-24 21:11:59 +08:00
|
|
|
ctx: ctx,
|
2021-06-02 15:58:33 +08:00
|
|
|
BaseNode: baseNode,
|
2021-09-26 20:55:59 +08:00
|
|
|
insertBuffer: sync.Map{},
|
2021-06-02 15:58:33 +08:00
|
|
|
|
2021-12-15 10:53:16 +08:00
|
|
|
timeTickStream: wTtMsgStream,
|
2021-09-23 16:03:54 +08:00
|
|
|
flushMap: sync.Map{},
|
|
|
|
flushChan: flushCh,
|
2022-05-25 14:34:00 +08:00
|
|
|
resendTTChan: resendTTCh,
|
2021-09-23 16:03:54 +08:00
|
|
|
flushingSegCache: flushingSegCache,
|
2021-10-19 11:04:34 +08:00
|
|
|
flushManager: fm,
|
2021-10-13 11:16:32 +08:00
|
|
|
|
|
|
|
replica: config.replica,
|
|
|
|
idAllocator: config.allocator,
|
|
|
|
channelName: config.vChannelName,
|
2021-11-05 14:59:32 +08:00
|
|
|
ttMerger: mt,
|
2021-12-30 10:09:48 +08:00
|
|
|
ttLogger: &timeTickLogger{vChannelName: config.vChannelName},
|
2021-08-30 10:03:58 +08:00
|
|
|
}, nil
|
2021-01-19 11:37:16 +08:00
|
|
|
}
|