2021-04-19 15:16:33 +08:00
|
|
|
// Copyright (C) 2019-2020 Zilliz. All rights reserved.
|
|
|
|
//
|
|
|
|
// Licensed under the Apache License, Version 2.0 (the "License"); you may not use this file except in compliance
|
|
|
|
// with the License. You may obtain a copy of the License at
|
|
|
|
//
|
|
|
|
// http://www.apache.org/licenses/LICENSE-2.0
|
|
|
|
//
|
|
|
|
// Unless required by applicable law or agreed to in writing, software distributed under the License
|
|
|
|
// is distributed on an "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express
|
|
|
|
// or implied. See the License for the specific language governing permissions and limitations under the License.
|
|
|
|
|
2021-01-19 11:37:16 +08:00
|
|
|
package datanode
|
|
|
|
|
|
|
|
import (
|
|
|
|
"bytes"
|
|
|
|
"context"
|
|
|
|
"encoding/binary"
|
2021-09-09 15:00:00 +08:00
|
|
|
"errors"
|
2021-06-16 19:03:57 +08:00
|
|
|
"fmt"
|
2021-01-19 11:37:16 +08:00
|
|
|
"path"
|
|
|
|
"strconv"
|
2021-03-23 18:50:13 +08:00
|
|
|
"sync"
|
2021-01-19 11:37:16 +08:00
|
|
|
"unsafe"
|
|
|
|
|
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-02-26 10:13:36 +08:00
|
|
|
"go.uber.org/zap"
|
2021-01-19 11:37:16 +08:00
|
|
|
|
2021-04-22 14:45:57 +08:00
|
|
|
"github.com/milvus-io/milvus/internal/kv"
|
|
|
|
miniokv "github.com/milvus-io/milvus/internal/kv/minio"
|
|
|
|
"github.com/milvus-io/milvus/internal/log"
|
|
|
|
"github.com/milvus-io/milvus/internal/msgstream"
|
|
|
|
"github.com/milvus-io/milvus/internal/storage"
|
|
|
|
"github.com/milvus-io/milvus/internal/util/trace"
|
|
|
|
|
|
|
|
"github.com/milvus-io/milvus/internal/proto/commonpb"
|
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/etcdpb"
|
|
|
|
"github.com/milvus-io/milvus/internal/proto/internalpb"
|
|
|
|
"github.com/milvus-io/milvus/internal/proto/schemapb"
|
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
|
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
|
|
|
|
flushChan <-chan *flushMsg
|
|
|
|
flushingSegCache *Cache
|
2021-03-25 14:41:46 +08:00
|
|
|
|
2021-04-12 18:09:28 +08:00
|
|
|
minIOKV kv.BaseKV
|
2021-03-25 14:41:46 +08:00
|
|
|
|
|
|
|
timeTickStream msgstream.MsgStream
|
|
|
|
segmentStatisticsStream msgstream.MsgStream
|
2021-06-04 16:31:34 +08:00
|
|
|
|
2021-09-17 16:25:55 +08:00
|
|
|
dsSaveBinlog func(fu *segmentFlushUnit) error
|
2021-06-04 16:31:34 +08:00
|
|
|
}
|
|
|
|
|
2021-06-06 13:21:37 +08:00
|
|
|
type segmentCheckPoint struct {
|
|
|
|
numRows int64
|
|
|
|
pos internalpb.MsgPosition
|
|
|
|
}
|
|
|
|
|
|
|
|
type segmentFlushUnit struct {
|
2021-06-11 19:15:48 +08:00
|
|
|
collID UniqueID
|
|
|
|
segID UniqueID
|
|
|
|
field2Path map[UniqueID]string
|
|
|
|
checkPoint map[UniqueID]segmentCheckPoint
|
|
|
|
startPositions []*datapb.SegmentStartPosition
|
|
|
|
flushed bool
|
2021-03-25 14:41:46 +08:00
|
|
|
}
|
|
|
|
|
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
|
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-09-26 20:55:59 +08:00
|
|
|
limit := Params.FlushInsertBufferSize / (dimension * 4)
|
2021-09-18 14:25:50 +08:00
|
|
|
|
2021-09-26 20:55:59 +08:00
|
|
|
return &BufferData{&InsertData{Data: make(map[UniqueID]storage.FieldData)}, 0, limit}, 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
|
|
|
}
|
|
|
|
|
|
|
|
func (ibNode *insertBufferNode) Name() string {
|
|
|
|
return "ibNode"
|
|
|
|
}
|
|
|
|
|
2021-09-17 16:27:56 +08:00
|
|
|
func (ibNode *insertBufferNode) Close() {
|
|
|
|
if ibNode.timeTickStream != nil {
|
|
|
|
ibNode.timeTickStream.Close()
|
|
|
|
}
|
|
|
|
|
|
|
|
if ibNode.segmentStatisticsStream != nil {
|
|
|
|
ibNode.segmentStatisticsStream.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 {
|
2021-09-26 10:43:57 +08:00
|
|
|
log.Error("type assertion failed for flowGraphMsg")
|
2021-09-17 16:27:56 +08:00
|
|
|
ibNode.Close()
|
2021-03-25 14:41:46 +08:00
|
|
|
return []Msg{}
|
|
|
|
}
|
|
|
|
|
|
|
|
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-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 {
|
|
|
|
log.Warn("update segment states in Replica wrong", zap.Error(err))
|
|
|
|
return []Msg{}
|
2021-01-21 09:55:25 +08:00
|
|
|
}
|
2021-02-04 11:19:48 +08:00
|
|
|
|
2021-09-17 16:27:56 +08:00
|
|
|
if len(seg2Upload) > 0 {
|
|
|
|
err := ibNode.uploadMemStates2Coord(seg2Upload)
|
2021-03-16 17:55:42 +08:00
|
|
|
if err != nil {
|
2021-09-17 16:27:56 +08:00
|
|
|
log.Error("upload segment statistics to coord error", zap.Error(err))
|
2021-02-04 11:19:48 +08:00
|
|
|
}
|
2021-01-21 09:55:25 +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 {
|
2021-09-09 15:00:00 +08:00
|
|
|
log.Warn("msg to buffer failed", zap.Error(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))
|
|
|
|
|
|
|
|
for k, segID := range seg2Upload[:displaySize] {
|
|
|
|
bd, ok := ibNode.insertBuffer.Load(segID)
|
|
|
|
if !ok {
|
|
|
|
continue
|
|
|
|
}
|
|
|
|
|
|
|
|
log.Debug("insert seg buffer status", zap.Int("No.", k),
|
|
|
|
zap.Int64("segmentID", segID),
|
|
|
|
zap.Int64("buffer size", bd.(*BufferData).size),
|
|
|
|
zap.Int64("buffer limit", bd.(*BufferData).limit))
|
2021-01-19 11:37:16 +08:00
|
|
|
}
|
|
|
|
|
2021-09-17 16:27:56 +08:00
|
|
|
// Auto Flush
|
|
|
|
finishCh := make(chan segmentFlushUnit, len(seg2Upload))
|
2021-06-04 16:31:34 +08:00
|
|
|
finishCnt := sync.WaitGroup{}
|
2021-09-17 16:27:56 +08:00
|
|
|
for _, segToFlush := range seg2Upload {
|
2021-05-25 15:35:37 +08:00
|
|
|
// If full, auto flush
|
2021-09-26 20:55:59 +08:00
|
|
|
if bd, ok := ibNode.insertBuffer.Load(segToFlush); ok && bd.(*BufferData).effectiveCap() <= 0 {
|
|
|
|
|
|
|
|
// Move data from insertBuffer to flushBuffer
|
|
|
|
ibuffer := bd.(*BufferData)
|
|
|
|
ibNode.flushMap.Store(segToFlush, ibuffer.buffer)
|
|
|
|
ibNode.insertBuffer.Delete(segToFlush)
|
|
|
|
|
|
|
|
log.Debug(". Insert Buffer full, auto flushing ", zap.Int64("num of rows", ibuffer.size))
|
2021-01-19 11:37:16 +08:00
|
|
|
|
2021-09-26 10:43:57 +08:00
|
|
|
collMeta, err := ibNode.getCollMetabySegID(segToFlush, fgMsg.timeRange.timestampMax)
|
2021-05-25 15:35:37 +08:00
|
|
|
if err != nil {
|
|
|
|
log.Error("Auto flush failed .. cannot get collection meta ..", zap.Error(err))
|
|
|
|
continue
|
|
|
|
}
|
2021-03-23 18:50:13 +08:00
|
|
|
|
2021-05-25 15:35:37 +08:00
|
|
|
collID, partitionID, err := ibNode.getCollectionandPartitionIDbySegID(segToFlush)
|
|
|
|
if err != nil {
|
|
|
|
log.Error("Auto flush failed .. cannot get collection ID or partition ID..", zap.Error(err))
|
2021-03-23 18:50:13 +08:00
|
|
|
continue
|
2021-01-19 11:37:16 +08:00
|
|
|
}
|
2021-06-04 16:31:34 +08:00
|
|
|
finishCnt.Add(1)
|
2021-03-23 18:50:13 +08:00
|
|
|
|
2021-05-25 15:35:37 +08:00
|
|
|
go flushSegment(collMeta, segToFlush, partitionID, collID,
|
2021-06-06 00:59:36 +08:00
|
|
|
&ibNode.flushMap, ibNode.minIOKV, finishCh, &finishCnt, ibNode, ibNode.idAllocator)
|
2021-06-04 16:31:34 +08:00
|
|
|
}
|
|
|
|
}
|
|
|
|
finishCnt.Wait()
|
|
|
|
close(finishCh)
|
|
|
|
for fu := range finishCh {
|
|
|
|
if fu.field2Path == nil {
|
|
|
|
log.Debug("segment is empty")
|
|
|
|
continue
|
|
|
|
}
|
2021-06-21 16:00:22 +08:00
|
|
|
fu.checkPoint = ibNode.replica.listSegmentsCheckPoints()
|
2021-06-04 16:31:34 +08:00
|
|
|
fu.flushed = false
|
2021-06-04 19:20:34 +08:00
|
|
|
if err := ibNode.dsSaveBinlog(&fu); err != nil {
|
2021-06-04 16:31:34 +08:00
|
|
|
log.Debug("data service save bin log path failed", zap.Error(err))
|
2021-05-25 15:35:37 +08:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2021-09-17 16:27:56 +08:00
|
|
|
// Manul Flush
|
2021-06-02 15:58:33 +08:00
|
|
|
select {
|
|
|
|
case fmsg := <-ibNode.flushChan:
|
|
|
|
currentSegID := fmsg.segmentID
|
2021-06-08 19:25:37 +08:00
|
|
|
log.Debug(". Receiving flush message",
|
|
|
|
zap.Int64("segmentID", currentSegID),
|
|
|
|
zap.Int64("collectionID", fmsg.collectionID),
|
|
|
|
)
|
2021-05-25 15:35:37 +08:00
|
|
|
|
2021-09-26 20:55:59 +08:00
|
|
|
bd, ok := ibNode.insertBuffer.Load(currentSegID)
|
|
|
|
if !ok {
|
|
|
|
break
|
|
|
|
}
|
|
|
|
|
|
|
|
if bd.(*BufferData).size <= 0 { // Buffer empty
|
2021-05-25 15:35:37 +08:00
|
|
|
log.Debug(".. Buffer empty ...")
|
2021-06-06 13:21:37 +08:00
|
|
|
ibNode.dsSaveBinlog(&segmentFlushUnit{
|
|
|
|
collID: fmsg.collectionID,
|
|
|
|
segID: currentSegID,
|
|
|
|
field2Path: map[UniqueID]string{},
|
2021-06-21 16:00:22 +08:00
|
|
|
checkPoint: ibNode.replica.listSegmentsCheckPoints(),
|
2021-06-06 13:21:37 +08:00
|
|
|
flushed: true,
|
2021-06-04 16:31:34 +08:00
|
|
|
})
|
2021-06-21 16:00:22 +08:00
|
|
|
ibNode.replica.segmentFlushed(currentSegID)
|
2021-09-26 20:55:59 +08:00
|
|
|
} else { // Buffer not empty
|
2021-03-23 18:50:13 +08:00
|
|
|
log.Debug(".. Buffer not empty, flushing ..")
|
2021-06-06 13:21:37 +08:00
|
|
|
finishCh := make(chan segmentFlushUnit, 1)
|
2021-06-04 16:31:34 +08:00
|
|
|
|
2021-09-26 20:55:59 +08:00
|
|
|
// Since buffer is not empty, so there must be data for key currentSegID
|
|
|
|
bd, _ := ibNode.insertBuffer.LoadAndDelete(currentSegID)
|
|
|
|
|
|
|
|
ibNode.flushMap.Store(currentSegID, bd.(*BufferData).buffer)
|
2021-03-30 09:47:27 +08:00
|
|
|
clearFn := func() {
|
2021-06-06 13:21:37 +08:00
|
|
|
finishCh <- segmentFlushUnit{field2Path: nil}
|
2021-03-30 09:47:27 +08:00
|
|
|
log.Debug(".. Clearing flush Buffer ..")
|
|
|
|
ibNode.flushMap.Delete(currentSegID)
|
2021-06-04 16:31:34 +08:00
|
|
|
close(finishCh)
|
2021-03-30 09:47:27 +08:00
|
|
|
}
|
2021-03-23 18:50:13 +08:00
|
|
|
|
2021-06-16 19:03:57 +08:00
|
|
|
collID, partitionID, err := ibNode.getCollectionandPartitionIDbySegID(currentSegID)
|
2021-03-23 18:50:13 +08:00
|
|
|
if err != nil {
|
|
|
|
log.Error("Flush failed .. cannot get segment ..", zap.Error(err))
|
2021-03-30 09:47:27 +08:00
|
|
|
clearFn()
|
2021-06-04 16:31:34 +08:00
|
|
|
break
|
2021-05-25 15:35:37 +08:00
|
|
|
// TODO add error handling
|
2021-03-23 18:50:13 +08:00
|
|
|
}
|
|
|
|
|
2021-09-26 10:43:57 +08:00
|
|
|
collMeta, err := ibNode.getCollMetabySegID(currentSegID, fgMsg.timeRange.timestampMax)
|
2021-02-07 17:02:13 +08:00
|
|
|
if err != nil {
|
2021-03-23 18:50:13 +08:00
|
|
|
log.Error("Flush failed .. cannot get collection schema ..", zap.Error(err))
|
2021-03-30 09:47:27 +08:00
|
|
|
clearFn()
|
2021-06-04 16:31:34 +08:00
|
|
|
break
|
2021-05-25 15:35:37 +08:00
|
|
|
// TODO add error handling
|
2021-02-07 17:02:13 +08:00
|
|
|
}
|
2021-03-23 18:50:13 +08:00
|
|
|
|
2021-06-16 19:03:57 +08:00
|
|
|
flushSegment(collMeta, currentSegID, partitionID, collID,
|
2021-06-06 00:59:36 +08:00
|
|
|
&ibNode.flushMap, ibNode.minIOKV, finishCh, nil, ibNode, ibNode.idAllocator)
|
2021-06-04 16:31:34 +08:00
|
|
|
fu := <-finishCh
|
|
|
|
close(finishCh)
|
|
|
|
if fu.field2Path != nil {
|
2021-06-21 16:00:22 +08:00
|
|
|
fu.checkPoint = ibNode.replica.listSegmentsCheckPoints()
|
2021-06-04 16:31:34 +08:00
|
|
|
fu.flushed = true
|
2021-06-08 19:25:37 +08:00
|
|
|
if err := ibNode.dsSaveBinlog(&fu); err != nil {
|
|
|
|
log.Debug("Data service save binlog path failed", zap.Error(err))
|
2021-06-04 16:31:34 +08:00
|
|
|
} else {
|
2021-06-21 16:00:22 +08:00
|
|
|
ibNode.replica.segmentFlushed(fu.segID)
|
2021-09-23 16:03:54 +08:00
|
|
|
ibNode.flushingSegCache.Remove(fu.segID)
|
2021-06-04 16:31:34 +08:00
|
|
|
}
|
|
|
|
}
|
2021-05-18 19:45:00 +08:00
|
|
|
}
|
|
|
|
|
2021-06-02 15:58:33 +08:00
|
|
|
default:
|
2021-01-19 11:37:16 +08:00
|
|
|
}
|
|
|
|
|
2021-09-26 10:43:57 +08:00
|
|
|
if err := ibNode.writeHardTimeTick(fgMsg.timeRange.timestampMax); err != nil {
|
2021-06-04 16:31:34 +08:00
|
|
|
log.Error("send hard time tick into pulsar channel failed", zap.Error(err))
|
|
|
|
}
|
2021-01-19 11:37:16 +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-06-02 15:58:33 +08:00
|
|
|
return nil
|
2021-01-19 11:37:16 +08:00
|
|
|
}
|
|
|
|
|
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 {
|
2021-09-09 15:00:00 +08:00
|
|
|
if len(msg.RowIDs) != len(msg.Timestamps) || len(msg.RowIDs) != len(msg.RowData) {
|
|
|
|
return errors.New("misaligned messages detected")
|
|
|
|
}
|
|
|
|
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 {
|
|
|
|
|
|
|
|
for _, t := range field.TypeParams {
|
|
|
|
if t.Key == "dim" {
|
|
|
|
dimension, err = strconv.Atoi(t.Value)
|
|
|
|
if err != nil {
|
|
|
|
log.Error("strconv wrong on get dim", zap.Error(err))
|
|
|
|
return err
|
|
|
|
}
|
|
|
|
break
|
|
|
|
}
|
|
|
|
}
|
|
|
|
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)
|
|
|
|
idata := buffer.buffer
|
2021-09-09 15:00:00 +08:00
|
|
|
|
|
|
|
// 1.2 Get Fields
|
|
|
|
var pos int = 0 // Record position of blob
|
|
|
|
var fieldIDs []int64
|
|
|
|
var fieldTypes []schemapb.DataType
|
|
|
|
for _, field := range collSchema.Fields {
|
|
|
|
fieldIDs = append(fieldIDs, field.FieldID)
|
|
|
|
fieldTypes = append(fieldTypes, field.DataType)
|
|
|
|
}
|
|
|
|
|
|
|
|
for _, field := range collSchema.Fields {
|
|
|
|
switch field.DataType {
|
|
|
|
case schemapb.DataType_FloatVector:
|
|
|
|
var dim int
|
|
|
|
for _, t := range field.TypeParams {
|
|
|
|
if t.Key == "dim" {
|
|
|
|
dim, err = strconv.Atoi(t.Value)
|
|
|
|
if err != nil {
|
|
|
|
log.Error("strconv wrong on get dim", zap.Error(err))
|
2021-09-26 20:55:59 +08:00
|
|
|
break
|
2021-09-09 15:00:00 +08:00
|
|
|
}
|
|
|
|
break
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
if _, ok := idata.Data[field.FieldID]; !ok {
|
|
|
|
idata.Data[field.FieldID] = &storage.FloatVectorFieldData{
|
|
|
|
NumRows: make([]int64, 0, 1),
|
|
|
|
Data: make([]float32, 0),
|
|
|
|
Dim: dim,
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
fieldData := idata.Data[field.FieldID].(*storage.FloatVectorFieldData)
|
|
|
|
|
|
|
|
var offset int
|
|
|
|
for _, blob := range msg.RowData {
|
|
|
|
offset = 0
|
|
|
|
for j := 0; j < dim; j++ {
|
|
|
|
var v float32
|
|
|
|
readBinary(blob.GetValue()[pos+offset:], &v, field.DataType)
|
|
|
|
fieldData.Data = append(fieldData.Data, v)
|
|
|
|
offset += int(unsafe.Sizeof(*(&v)))
|
|
|
|
}
|
|
|
|
}
|
|
|
|
pos += offset
|
|
|
|
fieldData.NumRows = append(fieldData.NumRows, int64(len(msg.RowData)))
|
|
|
|
|
|
|
|
case schemapb.DataType_BinaryVector:
|
|
|
|
var dim int
|
|
|
|
for _, t := range field.TypeParams {
|
|
|
|
if t.Key == "dim" {
|
|
|
|
dim, err = strconv.Atoi(t.Value)
|
|
|
|
if err != nil {
|
2021-09-26 20:55:59 +08:00
|
|
|
log.Error("strconv wrong on get dim", zap.Error(err))
|
|
|
|
return err
|
2021-09-09 15:00:00 +08:00
|
|
|
}
|
|
|
|
break
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
if _, ok := idata.Data[field.FieldID]; !ok {
|
|
|
|
idata.Data[field.FieldID] = &storage.BinaryVectorFieldData{
|
|
|
|
NumRows: make([]int64, 0, 1),
|
|
|
|
Data: make([]byte, 0),
|
|
|
|
Dim: dim,
|
|
|
|
}
|
|
|
|
}
|
|
|
|
fieldData := idata.Data[field.FieldID].(*storage.BinaryVectorFieldData)
|
|
|
|
|
|
|
|
var offset int
|
|
|
|
for _, blob := range msg.RowData {
|
|
|
|
bv := blob.GetValue()[pos : pos+(dim/8)]
|
|
|
|
fieldData.Data = append(fieldData.Data, bv...)
|
|
|
|
offset = len(bv)
|
|
|
|
}
|
|
|
|
pos += offset
|
|
|
|
fieldData.NumRows = append(fieldData.NumRows, int64(len(msg.RowData)))
|
|
|
|
|
|
|
|
case schemapb.DataType_Bool:
|
|
|
|
if _, ok := idata.Data[field.FieldID]; !ok {
|
|
|
|
idata.Data[field.FieldID] = &storage.BoolFieldData{
|
|
|
|
NumRows: make([]int64, 0, 1),
|
|
|
|
Data: make([]bool, 0),
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
fieldData := idata.Data[field.FieldID].(*storage.BoolFieldData)
|
|
|
|
var v bool
|
|
|
|
for _, blob := range msg.RowData {
|
|
|
|
readBinary(blob.GetValue()[pos:], &v, field.DataType)
|
|
|
|
fieldData.Data = append(fieldData.Data, v)
|
|
|
|
}
|
|
|
|
pos += int(unsafe.Sizeof(*(&v)))
|
|
|
|
fieldData.NumRows = append(fieldData.NumRows, int64(len(msg.RowData)))
|
|
|
|
|
|
|
|
case schemapb.DataType_Int8:
|
|
|
|
if _, ok := idata.Data[field.FieldID]; !ok {
|
|
|
|
idata.Data[field.FieldID] = &storage.Int8FieldData{
|
|
|
|
NumRows: make([]int64, 0, 1),
|
|
|
|
Data: make([]int8, 0),
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
fieldData := idata.Data[field.FieldID].(*storage.Int8FieldData)
|
|
|
|
var v int8
|
|
|
|
for _, blob := range msg.RowData {
|
|
|
|
readBinary(blob.GetValue()[pos:], &v, field.DataType)
|
|
|
|
fieldData.Data = append(fieldData.Data, v)
|
|
|
|
}
|
|
|
|
pos += int(unsafe.Sizeof(*(&v)))
|
|
|
|
fieldData.NumRows = append(fieldData.NumRows, int64(len(msg.RowData)))
|
|
|
|
|
|
|
|
case schemapb.DataType_Int16:
|
|
|
|
if _, ok := idata.Data[field.FieldID]; !ok {
|
|
|
|
idata.Data[field.FieldID] = &storage.Int16FieldData{
|
|
|
|
NumRows: make([]int64, 0, 1),
|
|
|
|
Data: make([]int16, 0),
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
fieldData := idata.Data[field.FieldID].(*storage.Int16FieldData)
|
|
|
|
var v int16
|
|
|
|
for _, blob := range msg.RowData {
|
|
|
|
readBinary(blob.GetValue()[pos:], &v, field.DataType)
|
|
|
|
fieldData.Data = append(fieldData.Data, v)
|
|
|
|
}
|
|
|
|
pos += int(unsafe.Sizeof(*(&v)))
|
|
|
|
fieldData.NumRows = append(fieldData.NumRows, int64(len(msg.RowData)))
|
|
|
|
|
|
|
|
case schemapb.DataType_Int32:
|
|
|
|
if _, ok := idata.Data[field.FieldID]; !ok {
|
|
|
|
idata.Data[field.FieldID] = &storage.Int32FieldData{
|
|
|
|
NumRows: make([]int64, 0, 1),
|
|
|
|
Data: make([]int32, 0),
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
fieldData := idata.Data[field.FieldID].(*storage.Int32FieldData)
|
|
|
|
var v int32
|
|
|
|
for _, blob := range msg.RowData {
|
|
|
|
readBinary(blob.GetValue()[pos:], &v, field.DataType)
|
|
|
|
fieldData.Data = append(fieldData.Data, v)
|
|
|
|
}
|
|
|
|
pos += int(unsafe.Sizeof(*(&v)))
|
|
|
|
fieldData.NumRows = append(fieldData.NumRows, int64(len(msg.RowData)))
|
|
|
|
|
|
|
|
case schemapb.DataType_Int64:
|
|
|
|
if _, ok := idata.Data[field.FieldID]; !ok {
|
|
|
|
idata.Data[field.FieldID] = &storage.Int64FieldData{
|
|
|
|
NumRows: make([]int64, 0, 1),
|
|
|
|
Data: make([]int64, 0),
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
fieldData := idata.Data[field.FieldID].(*storage.Int64FieldData)
|
|
|
|
switch field.FieldID {
|
|
|
|
case 0: // rowIDs
|
|
|
|
fieldData.Data = append(fieldData.Data, msg.RowIDs...)
|
|
|
|
fieldData.NumRows = append(fieldData.NumRows, int64(len(msg.RowData)))
|
|
|
|
case 1: // Timestamps
|
|
|
|
for _, ts := range msg.Timestamps {
|
|
|
|
fieldData.Data = append(fieldData.Data, int64(ts))
|
|
|
|
}
|
|
|
|
fieldData.NumRows = append(fieldData.NumRows, int64(len(msg.RowData)))
|
|
|
|
default:
|
|
|
|
var v int64
|
|
|
|
for _, blob := range msg.RowData {
|
|
|
|
readBinary(blob.GetValue()[pos:], &v, field.DataType)
|
|
|
|
fieldData.Data = append(fieldData.Data, v)
|
|
|
|
}
|
|
|
|
pos += int(unsafe.Sizeof(*(&v)))
|
|
|
|
fieldData.NumRows = append(fieldData.NumRows, int64(len(msg.RowData)))
|
|
|
|
}
|
|
|
|
|
|
|
|
case schemapb.DataType_Float:
|
|
|
|
if _, ok := idata.Data[field.FieldID]; !ok {
|
|
|
|
idata.Data[field.FieldID] = &storage.FloatFieldData{
|
|
|
|
NumRows: make([]int64, 0, 1),
|
|
|
|
Data: make([]float32, 0),
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
fieldData := idata.Data[field.FieldID].(*storage.FloatFieldData)
|
|
|
|
var v float32
|
|
|
|
for _, blob := range msg.RowData {
|
|
|
|
readBinary(blob.GetValue()[pos:], &v, field.DataType)
|
|
|
|
fieldData.Data = append(fieldData.Data, v)
|
|
|
|
}
|
|
|
|
pos += int(unsafe.Sizeof(*(&v)))
|
|
|
|
fieldData.NumRows = append(fieldData.NumRows, int64(len(msg.RowData)))
|
|
|
|
|
|
|
|
case schemapb.DataType_Double:
|
|
|
|
if _, ok := idata.Data[field.FieldID]; !ok {
|
|
|
|
idata.Data[field.FieldID] = &storage.DoubleFieldData{
|
|
|
|
NumRows: make([]int64, 0, 1),
|
|
|
|
Data: make([]float64, 0),
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
fieldData := idata.Data[field.FieldID].(*storage.DoubleFieldData)
|
|
|
|
var v float64
|
|
|
|
for _, blob := range msg.RowData {
|
|
|
|
readBinary(blob.GetValue()[pos:], &v, field.DataType)
|
|
|
|
fieldData.Data = append(fieldData.Data, v)
|
|
|
|
}
|
|
|
|
|
|
|
|
pos += int(unsafe.Sizeof(*(&v)))
|
|
|
|
fieldData.NumRows = append(fieldData.NumRows, int64(len(msg.RowData)))
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2021-09-26 20:55:59 +08:00
|
|
|
// update buffer size
|
|
|
|
buffer.updateSize(int64(len(msg.RowData)))
|
|
|
|
|
|
|
|
// 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
|
|
|
// update segment pk filter
|
|
|
|
ibNode.replica.updateSegmentPKRange(currentSegID, msg.GetRowIDs())
|
|
|
|
return nil
|
|
|
|
}
|
|
|
|
|
|
|
|
func readBinary(data []byte, receiver interface{}, dataType schemapb.DataType) {
|
|
|
|
buf := bytes.NewReader(data)
|
|
|
|
err := binary.Read(buf, binary.LittleEndian, receiver)
|
|
|
|
if err != nil {
|
|
|
|
log.Error("binary.Read failed", zap.Any("data type", dataType), zap.Error(err))
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2021-06-11 17:53:37 +08:00
|
|
|
func flushSegment(
|
|
|
|
collMeta *etcdpb.CollectionMeta,
|
|
|
|
segID, partitionID, collID UniqueID,
|
|
|
|
insertData *sync.Map,
|
|
|
|
kv kv.BaseKV,
|
|
|
|
flushUnit chan<- segmentFlushUnit,
|
|
|
|
wgFinish *sync.WaitGroup,
|
|
|
|
ibNode *insertBufferNode,
|
|
|
|
idAllocator allocatorInterface) {
|
|
|
|
|
2021-06-04 16:31:34 +08:00
|
|
|
if wgFinish != nil {
|
|
|
|
defer wgFinish.Done()
|
|
|
|
}
|
2021-01-22 09:36:40 +08:00
|
|
|
|
2021-03-30 09:47:27 +08:00
|
|
|
clearFn := func(isSuccess bool) {
|
2021-05-18 19:45:00 +08:00
|
|
|
if !isSuccess {
|
2021-06-06 13:21:37 +08:00
|
|
|
flushUnit <- segmentFlushUnit{field2Path: nil}
|
2021-05-18 19:45:00 +08:00
|
|
|
}
|
|
|
|
|
2021-03-23 18:50:13 +08:00
|
|
|
log.Debug(".. Clearing flush Buffer ..")
|
|
|
|
insertData.Delete(segID)
|
2021-03-30 09:47:27 +08:00
|
|
|
}
|
2021-01-22 09:36:40 +08:00
|
|
|
|
|
|
|
inCodec := storage.NewInsertCodec(collMeta)
|
|
|
|
|
|
|
|
// buffer data to binlogs
|
2021-03-23 18:50:13 +08:00
|
|
|
data, ok := insertData.Load(segID)
|
|
|
|
if !ok {
|
|
|
|
log.Error("Flush failed ... cannot load insertData ..")
|
2021-03-30 09:47:27 +08:00
|
|
|
clearFn(false)
|
2021-03-23 18:50:13 +08:00
|
|
|
return
|
|
|
|
}
|
2021-01-22 09:36:40 +08:00
|
|
|
|
2021-05-20 18:38:45 +08:00
|
|
|
binLogs, statsBinlogs, err := inCodec.Serialize(partitionID, segID, data.(*InsertData))
|
2021-01-22 09:36:40 +08:00
|
|
|
if err != nil {
|
2021-03-23 18:50:13 +08:00
|
|
|
log.Error("Flush failed ... cannot generate binlog ..", zap.Error(err))
|
2021-03-30 09:47:27 +08:00
|
|
|
clearFn(false)
|
2021-03-23 18:50:13 +08:00
|
|
|
return
|
2021-01-22 09:36:40 +08:00
|
|
|
}
|
|
|
|
|
2021-03-23 18:50:13 +08:00
|
|
|
log.Debug(".. Saving binlogs to MinIO ..", zap.Int("number", len(binLogs)))
|
|
|
|
field2Path := make(map[UniqueID]string, len(binLogs))
|
|
|
|
kvs := make(map[string]string, len(binLogs))
|
|
|
|
paths := make([]string, 0, len(binLogs))
|
2021-05-20 18:38:45 +08:00
|
|
|
field2Logidx := make(map[UniqueID]UniqueID, len(binLogs))
|
|
|
|
|
|
|
|
// write insert binlog
|
2021-03-23 18:50:13 +08:00
|
|
|
for _, blob := range binLogs {
|
|
|
|
fieldID, err := strconv.ParseInt(blob.GetKey(), 10, 64)
|
2021-01-22 09:36:40 +08:00
|
|
|
if err != nil {
|
2021-03-23 18:50:13 +08:00
|
|
|
log.Error("Flush failed ... cannot parse string to fieldID ..", zap.Error(err))
|
2021-03-30 09:47:27 +08:00
|
|
|
clearFn(false)
|
2021-03-23 18:50:13 +08:00
|
|
|
return
|
2021-01-22 09:36:40 +08:00
|
|
|
}
|
2021-08-30 10:03:58 +08:00
|
|
|
log.Debug("save binlog", zap.Int64("fieldID", fieldID))
|
2021-01-22 09:36:40 +08:00
|
|
|
|
2021-05-20 18:38:45 +08:00
|
|
|
logidx, err := idAllocator.allocID()
|
2021-01-22 09:36:40 +08:00
|
|
|
if err != nil {
|
2021-03-23 18:50:13 +08:00
|
|
|
log.Error("Flush failed ... cannot alloc ID ..", zap.Error(err))
|
2021-03-30 09:47:27 +08:00
|
|
|
clearFn(false)
|
2021-03-23 18:50:13 +08:00
|
|
|
return
|
2021-01-22 09:36:40 +08:00
|
|
|
}
|
|
|
|
|
2021-05-20 18:38:45 +08:00
|
|
|
// no error raise if alloc=false
|
|
|
|
k, _ := idAllocator.genKey(false, collID, partitionID, segID, fieldID, logidx)
|
|
|
|
|
2021-03-23 18:50:13 +08:00
|
|
|
key := path.Join(Params.InsertBinlogRootPath, k)
|
|
|
|
paths = append(paths, key)
|
|
|
|
kvs[key] = string(blob.Value[:])
|
|
|
|
field2Path[fieldID] = key
|
2021-05-20 18:38:45 +08:00
|
|
|
field2Logidx[fieldID] = logidx
|
|
|
|
}
|
|
|
|
|
|
|
|
// write stats binlog
|
|
|
|
for _, blob := range statsBinlogs {
|
|
|
|
fieldID, err := strconv.ParseInt(blob.GetKey(), 10, 64)
|
|
|
|
if err != nil {
|
|
|
|
log.Error("Flush failed ... cannot parse string to fieldID ..", zap.Error(err))
|
|
|
|
clearFn(false)
|
|
|
|
return
|
|
|
|
}
|
|
|
|
|
|
|
|
logidx := field2Logidx[fieldID]
|
|
|
|
|
|
|
|
// no error raise if alloc=false
|
|
|
|
k, _ := idAllocator.genKey(false, collID, partitionID, segID, fieldID, logidx)
|
|
|
|
|
|
|
|
key := path.Join(Params.StatsBinlogRootPath, k)
|
|
|
|
kvs[key] = string(blob.Value[:])
|
2021-03-23 18:50:13 +08:00
|
|
|
}
|
2021-06-04 16:31:34 +08:00
|
|
|
log.Debug("save binlog file to MinIO/S3")
|
2021-03-23 18:50:13 +08:00
|
|
|
|
|
|
|
err = kv.MultiSave(kvs)
|
|
|
|
if err != nil {
|
|
|
|
log.Error("Flush failed ... cannot save to MinIO ..", zap.Error(err))
|
|
|
|
_ = kv.MultiRemove(paths)
|
2021-03-30 09:47:27 +08:00
|
|
|
clearFn(false)
|
2021-03-23 18:50:13 +08:00
|
|
|
return
|
|
|
|
}
|
2021-01-22 09:36:40 +08:00
|
|
|
|
2021-06-21 16:00:22 +08:00
|
|
|
ibNode.replica.updateSegmentCheckPoint(segID)
|
|
|
|
startPos := ibNode.replica.listNewSegmentsStartPositions()
|
2021-06-11 19:15:48 +08:00
|
|
|
flushUnit <- segmentFlushUnit{collID: collID, segID: segID, field2Path: field2Path, startPositions: startPos}
|
2021-05-18 19:45:00 +08:00
|
|
|
clearFn(true)
|
|
|
|
}
|
|
|
|
|
2021-01-19 11:37:16 +08:00
|
|
|
func (ibNode *insertBufferNode) writeHardTimeTick(ts Timestamp) error {
|
|
|
|
msgPack := msgstream.MsgPack{}
|
2021-06-08 19:25:37 +08:00
|
|
|
timeTickMsg := msgstream.DataNodeTtMsg{
|
2021-01-19 11:37:16 +08:00
|
|
|
BaseMsg: msgstream.BaseMsg{
|
|
|
|
BeginTimestamp: ts,
|
|
|
|
EndTimestamp: ts,
|
|
|
|
HashValues: []uint32{0},
|
|
|
|
},
|
2021-06-08 19:25:37 +08:00
|
|
|
DataNodeTtMsg: datapb.DataNodeTtMsg{
|
2021-01-19 11:37:16 +08:00
|
|
|
Base: &commonpb.MsgBase{
|
2021-06-08 19:25:37 +08:00
|
|
|
MsgType: commonpb.MsgType_DataNodeTt,
|
|
|
|
MsgID: 0,
|
|
|
|
Timestamp: ts,
|
2021-01-19 11:37:16 +08:00
|
|
|
},
|
2021-06-08 19:25:37 +08:00
|
|
|
ChannelName: ibNode.channelName,
|
|
|
|
Timestamp: ts,
|
2021-01-19 11:37:16 +08:00
|
|
|
},
|
|
|
|
}
|
|
|
|
msgPack.Msgs = append(msgPack.Msgs, &timeTickMsg)
|
2021-03-25 14:41:46 +08:00
|
|
|
return ibNode.timeTickStream.Produce(&msgPack)
|
2021-01-19 11:37:16 +08:00
|
|
|
}
|
|
|
|
|
2021-09-17 16:27:56 +08:00
|
|
|
func (ibNode *insertBufferNode) uploadMemStates2Coord(segIDs []UniqueID) error {
|
2021-02-26 10:13:36 +08:00
|
|
|
log.Debug("Updating segments statistics...")
|
2021-03-12 14:22:09 +08:00
|
|
|
statsUpdates := make([]*internalpb.SegmentStatisticsUpdates, 0, len(segIDs))
|
2021-01-21 09:55:25 +08:00
|
|
|
for _, segID := range segIDs {
|
|
|
|
updates, err := ibNode.replica.getSegmentStatisticsUpdates(segID)
|
|
|
|
if err != nil {
|
2021-02-26 10:13:36 +08:00
|
|
|
log.Error("get segment statistics updates wrong", zap.Int64("segmentID", segID), zap.Error(err))
|
2021-01-21 09:55:25 +08:00
|
|
|
continue
|
|
|
|
}
|
2021-06-09 17:31:48 +08:00
|
|
|
|
|
|
|
log.Debug("Segment Statistics to Update",
|
|
|
|
zap.Int64("Segment ID", updates.GetSegmentID()),
|
|
|
|
zap.Int64("NumOfRows", updates.GetNumRows()),
|
|
|
|
)
|
|
|
|
|
2021-01-21 09:55:25 +08:00
|
|
|
statsUpdates = append(statsUpdates, updates)
|
|
|
|
}
|
|
|
|
|
2021-03-12 14:22:09 +08:00
|
|
|
segStats := internalpb.SegmentStatistics{
|
2021-01-21 09:55:25 +08:00
|
|
|
Base: &commonpb.MsgBase{
|
2021-03-10 14:45:35 +08:00
|
|
|
MsgType: commonpb.MsgType_SegmentStatistics,
|
2021-01-21 09:55:25 +08:00
|
|
|
MsgID: UniqueID(0), // GOOSE TODO
|
|
|
|
Timestamp: Timestamp(0), // GOOSE TODO
|
2021-01-24 21:20:11 +08:00
|
|
|
SourceID: Params.NodeID,
|
2021-01-21 09:55:25 +08:00
|
|
|
},
|
|
|
|
SegStats: statsUpdates,
|
|
|
|
}
|
|
|
|
|
|
|
|
var msg msgstream.TsMsg = &msgstream.SegmentStatisticsMsg{
|
|
|
|
BaseMsg: msgstream.BaseMsg{
|
2021-02-04 11:19:48 +08:00
|
|
|
HashValues: []uint32{0}, // GOOSE TODO
|
2021-01-21 09:55:25 +08:00
|
|
|
},
|
|
|
|
SegmentStatistics: segStats,
|
|
|
|
}
|
|
|
|
|
|
|
|
var msgPack = msgstream.MsgPack{
|
|
|
|
Msgs: []msgstream.TsMsg{msg},
|
|
|
|
}
|
2021-03-25 14:41:46 +08:00
|
|
|
return ibNode.segmentStatisticsStream.Produce(&msgPack)
|
2021-01-21 09:55:25 +08:00
|
|
|
}
|
|
|
|
|
2021-06-08 19:25:37 +08:00
|
|
|
func (ibNode *insertBufferNode) getCollMetabySegID(segmentID UniqueID, ts Timestamp) (meta *etcdpb.CollectionMeta, err error) {
|
2021-08-11 14:24:09 +08:00
|
|
|
if !ibNode.replica.hasSegment(segmentID, true) {
|
2021-06-16 19:03:57 +08:00
|
|
|
return nil, fmt.Errorf("No such segment %d in the replica", segmentID)
|
2021-05-18 19:45:00 +08:00
|
|
|
}
|
|
|
|
|
2021-06-16 19:03:57 +08:00
|
|
|
collID := ibNode.replica.getCollectionID()
|
|
|
|
sch, err := ibNode.replica.getCollectionSchema(collID, ts)
|
2021-05-18 19:45:00 +08:00
|
|
|
if err != nil {
|
2021-08-30 10:03:58 +08:00
|
|
|
return nil, err
|
2021-05-18 19:45:00 +08:00
|
|
|
}
|
2021-06-08 19:25:37 +08:00
|
|
|
|
|
|
|
meta = &etcdpb.CollectionMeta{
|
2021-06-16 19:03:57 +08:00
|
|
|
ID: collID,
|
|
|
|
Schema: sch,
|
2021-06-08 19:25:37 +08:00
|
|
|
}
|
2021-05-18 19:45:00 +08:00
|
|
|
return
|
|
|
|
}
|
|
|
|
|
|
|
|
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
|
|
|
}
|
|
|
|
|
2021-06-04 16:31:34 +08:00
|
|
|
func newInsertBufferNode(
|
|
|
|
ctx context.Context,
|
|
|
|
replica Replica,
|
|
|
|
factory msgstream.Factory,
|
|
|
|
idAllocator allocatorInterface,
|
|
|
|
flushCh <-chan *flushMsg,
|
2021-06-06 13:21:37 +08:00
|
|
|
saveBinlog func(*segmentFlushUnit) error,
|
2021-06-08 19:25:37 +08:00
|
|
|
channelName string,
|
2021-09-23 16:03:54 +08:00
|
|
|
flushingSegCache *Cache,
|
2021-08-30 10:03:58 +08:00
|
|
|
) (*insertBufferNode, error) {
|
2021-06-04 16:31:34 +08:00
|
|
|
|
2021-01-19 11:37:16 +08:00
|
|
|
maxQueueLength := Params.FlowGraphMaxQueueLength
|
|
|
|
maxParallelism := Params.FlowGraphMaxParallelism
|
|
|
|
|
|
|
|
baseNode := BaseNode{}
|
|
|
|
baseNode.SetMaxQueueLength(maxQueueLength)
|
|
|
|
baseNode.SetMaxParallelism(maxParallelism)
|
|
|
|
|
|
|
|
// MinIO
|
|
|
|
option := &miniokv.Option{
|
|
|
|
Address: Params.MinioAddress,
|
|
|
|
AccessKeyID: Params.MinioAccessKeyID,
|
|
|
|
SecretAccessKeyID: Params.MinioSecretAccessKey,
|
|
|
|
UseSSL: Params.MinioUseSSL,
|
|
|
|
CreateBucket: true,
|
|
|
|
BucketName: Params.MinioBucketName,
|
|
|
|
}
|
|
|
|
|
|
|
|
minIOKV, err := miniokv.NewMinIOKV(ctx, option)
|
|
|
|
if err != nil {
|
2021-08-30 10:03:58 +08:00
|
|
|
return nil, err
|
2021-01-19 11:37:16 +08:00
|
|
|
}
|
|
|
|
|
2021-01-21 09:55:25 +08:00
|
|
|
//input stream, data node time tick
|
2021-08-30 10:03:58 +08:00
|
|
|
wTt, err := factory.NewMsgStream(ctx)
|
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
2021-02-04 14:37:12 +08:00
|
|
|
wTt.AsProducer([]string{Params.TimeTickChannelName})
|
2021-08-30 10:03:58 +08:00
|
|
|
log.Debug("datanode AsProducer", zap.String("TimeTickChannelName", Params.TimeTickChannelName))
|
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
|
|
|
|
|
|
|
// update statistics channel
|
2021-08-30 10:03:58 +08:00
|
|
|
segS, err := factory.NewMsgStream(ctx)
|
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
2021-02-04 14:37:12 +08:00
|
|
|
segS.AsProducer([]string{Params.SegmentStatisticsChannelName})
|
2021-08-30 10:03:58 +08:00
|
|
|
log.Debug("datanode AsProducer", zap.String("SegmentStatisChannelName", Params.SegmentStatisticsChannelName))
|
2021-01-21 09:55:25 +08:00
|
|
|
var segStatisticsMsgStream msgstream.MsgStream = segS
|
2021-01-24 21:20:11 +08:00
|
|
|
segStatisticsMsgStream.Start()
|
2021-01-22 19:36:09 +08:00
|
|
|
|
2021-01-19 11:37:16 +08:00
|
|
|
return &insertBufferNode{
|
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
|
|
|
minIOKV: minIOKV,
|
2021-06-08 19:25:37 +08:00
|
|
|
channelName: channelName,
|
2021-06-02 15:58:33 +08:00
|
|
|
|
2021-01-22 09:36:40 +08:00
|
|
|
timeTickStream: wTtMsgStream,
|
|
|
|
segmentStatisticsStream: segStatisticsMsgStream,
|
2021-06-02 15:58:33 +08:00
|
|
|
|
2021-09-23 16:03:54 +08:00
|
|
|
replica: replica,
|
|
|
|
flushMap: sync.Map{},
|
|
|
|
flushChan: flushCh,
|
|
|
|
idAllocator: idAllocator,
|
|
|
|
dsSaveBinlog: saveBinlog,
|
|
|
|
flushingSegCache: flushingSegCache,
|
2021-08-30 10:03:58 +08:00
|
|
|
}, nil
|
2021-01-19 11:37:16 +08:00
|
|
|
}
|