2021-09-26 21:05:58 +08:00
## 7. Query Coordinator
2020-11-19 10:59:10 +08:00
2021-09-26 21:05:58 +08:00
#### 7.1 Overview
2020-11-19 10:59:10 +08:00
2021-06-21 19:22:15 +08:00
< img src = "./figs/query_coord.png" width = 500 >
2021-01-04 14:16:43 +08:00
2021-09-26 21:05:58 +08:00
#### 7.2 Query Coordinator Interface
2020-12-27 09:05:24 +08:00
```go
2021-06-22 16:44:09 +08:00
type QueryCoord interface {
2021-04-12 12:45:38 +08:00
Component
TimeTickProvider
2021-10-27 14:51:08 +08:00
// ShowCollections notifies RootCoord to list all collection names and other info in database at specified timestamp
2021-04-12 12:45:38 +08:00
ShowCollections(ctx context.Context, req *querypb.ShowCollectionsRequest) (*querypb.ShowCollectionsResponse, error)
2021-10-29 09:38:41 +08:00
// LoadCollection notifies Proxy to load a collection's data
2021-04-12 12:45:38 +08:00
LoadCollection(ctx context.Context, req *querypb.LoadCollectionRequest) (*commonpb.Status, error)
2021-11-01 16:28:14 +08:00
// ReleaseCollection notifies Proxy to release a collection's data
2021-04-12 12:45:38 +08:00
ReleaseCollection(ctx context.Context, req *querypb.ReleaseCollectionRequest) (*commonpb.Status, error)
2021-11-02 21:22:57 +08:00
// ShowPartitions notifies RootCoord to list all partition names and other info in the collection
2021-04-12 12:45:38 +08:00
ShowPartitions(ctx context.Context, req *querypb.ShowPartitionsRequest) (*querypb.ShowPartitionsResponse, error)
2021-11-03 10:42:34 +08:00
// LoadPartitions notifies Proxy to load partition's data
2021-04-12 12:45:38 +08:00
LoadPartitions(ctx context.Context, req *querypb.LoadPartitionsRequest) (*commonpb.Status, error)
2021-11-04 11:02:32 +08:00
// ReleasePartitions notifies Proxy to release collection's data
2021-04-12 12:45:38 +08:00
ReleasePartitions(ctx context.Context, req *querypb.ReleasePartitionsRequest) (*commonpb.Status, error)
2021-11-09 19:27:06 +08:00
// CreateQueryChannel creates the channels for querying in QueryCoord.
2021-04-12 12:45:38 +08:00
CreateQueryChannel(ctx context.Context) (*querypb.CreateQueryChannelResponse, error)
GetPartitionStates(ctx context.Context, req *querypb.GetPartitionStatesRequest) (*querypb.GetPartitionStatesResponse, error)
2021-11-05 13:17:59 +08:00
// GetSegmentInfo requests segment info
2021-04-12 12:45:38 +08:00
GetSegmentInfo(ctx context.Context, req *querypb.GetSegmentInfoRequest) (*querypb.GetSegmentInfoResponse, error)
2021-11-08 13:03:55 +08:00
// GetMetrics gets the metrics about QueryCoord.
2021-09-22 16:08:07 +08:00
GetMetrics(ctx context.Context, req *milvuspb.GetMetricsRequest) (*milvuspb.GetMetricsResponse, error)
2020-12-27 09:05:24 +08:00
}
```
2021-10-08 15:07:05 +08:00
- _MsgBase_
2020-12-27 09:05:24 +08:00
```go
2021-01-15 14:38:36 +08:00
type MsgBase struct {
2021-04-12 12:45:38 +08:00
MsgType MsgType
MsgID UniqueID
Timestamp Timestamp
SourceID UniqueID
2021-01-13 11:08:03 +08:00
}
2021-01-11 18:35:54 +08:00
```
2021-10-08 15:07:05 +08:00
- _ShowCollections_
2020-12-27 09:05:24 +08:00
2021-01-12 18:03:24 +08:00
```go
type ShowCollectionRequest struct {
2021-09-22 16:11:59 +08:00
Base *commonpb.MsgBase
DbID UniqueID
CollectionIDs []int64
2021-01-11 18:35:54 +08:00
}
2020-12-27 09:05:24 +08:00
2021-01-12 18:03:24 +08:00
type ShowCollectionResponse struct {
2021-09-22 16:11:59 +08:00
Status *commonpb.Status
CollectionIDs []UniqueID
InMemoryPercentages []int64
2021-01-11 18:35:54 +08:00
}
```
2021-10-08 15:07:05 +08:00
- _LoadCollection_
2021-01-13 11:08:03 +08:00
```go
type LoadCollectionRequest struct {
2021-04-12 12:45:38 +08:00
Base *commonpb.MsgBase
DbID UniqueID
CollectionID UniqueID
schema *schemapb.CollectionSchema
2021-01-13 11:08:03 +08:00
}
```
2021-10-08 15:07:05 +08:00
- _ReleaseCollection_
2021-01-13 11:08:03 +08:00
```go
type ReleaseCollectionRequest struct {
2021-04-12 12:45:38 +08:00
Base *commonpb.MsgBase
DbID UniqueID
CollectionID UniqueID
2021-01-13 11:08:03 +08:00
}
```
2021-10-08 15:07:05 +08:00
- _ShowPartitions_
2020-12-29 18:02:44 +08:00
2020-12-27 09:05:24 +08:00
```go
2021-01-12 18:03:24 +08:00
type ShowPartitionRequest struct {
2021-04-12 12:45:38 +08:00
Base *commonpb.MsgBase
DbID UniqueID
CollectionID UniqueID
2021-09-22 16:11:59 +08:00
PartitionIDs []int64
2020-12-29 18:02:44 +08:00
}
2021-01-12 18:03:24 +08:00
type ShowPartitionResponse struct {
2021-09-22 16:11:59 +08:00
Status *commonpb.Status
PartitionIDs []UniqueID
InMemoryPercentages []int64
2021-01-12 18:03:24 +08:00
}
```
2021-10-08 15:07:05 +08:00
- _GetPartitionStates_
2021-01-12 18:03:24 +08:00
```go
2020-12-27 09:05:24 +08:00
type PartitionState = int
const (
2021-04-12 12:45:38 +08:00
PartitionState_NotExist PartitionState = 0
PartitionState_NotPresent PartitionState = 1
PartitionState_OnDisk PartitionState = 2
PartitionState_PartialInMemory PartitionState = 3
PartitionState_InMemory PartitionState = 4
PartitionState_PartialInGPU PartitionState = 5
PartitionState_InGPU PartitionState = 6
2020-12-27 09:05:24 +08:00
)
2021-01-12 18:03:24 +08:00
type PartitionStatesRequest struct {
2021-04-12 12:45:38 +08:00
Base *commonpb.MsgBase
DbID UniqueID
CollectionID UniqueID
PartitionIDs []UniqueID
2020-12-29 18:02:44 +08:00
}
2021-01-12 18:03:24 +08:00
type PartitionStates struct {
2021-04-12 12:45:38 +08:00
PartitionID UniqueID
State PartitionState
2020-12-27 09:05:24 +08:00
}
2020-12-29 18:02:44 +08:00
2021-01-12 18:03:24 +08:00
type PartitionStatesResponse struct {
2021-04-12 12:45:38 +08:00
Status *commonpb.Status
PartitionDescriptions []*PartitionStates
2020-12-29 18:02:44 +08:00
}
```
2021-10-08 15:07:05 +08:00
- _LoadPartitions_
2020-12-29 18:02:44 +08:00
2021-01-13 11:08:03 +08:00
```go
type LoadPartitonRequest struct {
2021-04-12 12:45:38 +08:00
Base *commonpb.MsgBase
DbID UniqueID
CollectionID UniqueID
PartitionIDs []UniqueID
Schema *schemapb.CollectionSchema
2021-01-13 11:08:03 +08:00
}
```
2021-10-08 15:07:05 +08:00
- _ReleasePartitions_
2021-01-13 11:08:03 +08:00
```go
type ReleasePartitionRequest struct {
2021-04-12 12:45:38 +08:00
Base *commonpb.MsgBase
DbID UniqueID
CollectionID UniqueID
PartitionIDs []UniqueID
2021-01-13 11:08:03 +08:00
}
```
2020-12-29 18:02:44 +08:00
2021-10-08 15:07:05 +08:00
- _CreateQueryChannel_
2020-12-29 18:02:44 +08:00
```go
2021-01-13 11:08:03 +08:00
type CreateQueryChannelResponse struct {
2021-04-12 12:45:38 +08:00
Status *commonpb.Status
RequestChannelName string
ResultChannelName string
2020-12-29 18:02:44 +08:00
}
```
2021-10-08 15:07:05 +08:00
- _GetSegmentInfo_ \*
2020-12-29 18:02:44 +08:00
2021-02-04 11:40:14 +08:00
```go
2021-04-12 12:45:38 +08:00
type GetSegmentInfoRequest struct {
Base *commonpb.MsgBase
SegmentIDs []UniqueID
2021-02-04 11:40:14 +08:00
}
2021-03-04 10:35:28 +08:00
type SegmentInfo struct {
2021-04-12 12:45:38 +08:00
SegmentID UniqueID
CollectionID UniqueID
PartitionID UniqueID
MemSize UniqueID
NumRows UniqueID
IndexName string
IndexID UniqueID
2021-03-04 10:35:28 +08:00
}
2021-04-12 12:45:38 +08:00
type GetSegmentInfoResponse struct {
Status *commonpb.Status
Infos []*SegmentInfo
2021-02-04 11:40:14 +08:00
}
```
2020-12-29 18:02:44 +08:00
2021-09-26 21:05:58 +08:00
#### 7.3 Query Channel
2021-01-15 14:38:36 +08:00
2021-10-08 15:07:05 +08:00
- _SearchMsg_
2021-04-12 12:45:38 +08:00
2021-01-15 14:38:36 +08:00
```go
type SearchRequest struct {
2021-09-22 16:11:59 +08:00
Base *commonpb.MsgBase
ResultChannelID string
DbID int64
CollectionID int64
PartitionIDs []int64
Dsl string
PlaceholderGroup []byte
DslType commonpb.DslType
SerializedExprPlan []byte
OutputFieldsId []int64
TravelTimestamp uint64
GuaranteeTimestamp uint64
2021-04-12 12:45:38 +08:00
}
type SearchMsg struct {
BaseMsg
SearchRequest
2021-01-15 14:38:36 +08:00
}
```
2021-11-10 20:05:56 +08:00
- _RetrieveMsg_
2021-10-08 15:07:05 +08:00
2021-09-22 16:11:59 +08:00
```go
2021-11-10 20:05:56 +08:00
type RetrieveRequest struct {
2021-09-22 16:11:59 +08:00
Base *commonpb.MsgBase
ResultChannelID string
DbID int64
CollectionID int64
PartitionIDs []int64
SerializedExprPlan []byte
OutputFieldsId []int64
TravelTimestamp uint64
GuaranteeTimestamp uint64
}
2021-11-10 20:05:56 +08:00
type RetrieveMsg struct {
2021-09-22 16:11:59 +08:00
BaseMsg
RetrieveRequest
}
```
2021-01-15 14:38:36 +08:00
2021-09-26 21:05:58 +08:00
#### 7.4 Query Node Interface
2020-12-29 18:02:44 +08:00
```go
2021-01-13 11:08:03 +08:00
type QueryNode interface {
2021-04-12 12:45:38 +08:00
Component
TimeTickProvider
2021-10-24 08:57:19 +08:00
// AddQueryChannel notifies QueryNode to subscribe a query channel and be a producer of a query result channel.
2021-04-12 12:45:38 +08:00
AddQueryChannel(ctx context.Context, req *querypb.AddQueryChannelRequest) (*commonpb.Status, error)
2021-11-17 16:03:42 +08:00
// RemoveQueryChannel removes the query channel for QueryNode component.
2021-04-12 12:45:38 +08:00
RemoveQueryChannel(ctx context.Context, req *querypb.RemoveQueryChannelRequest) (*commonpb.Status, error)
2021-11-18 22:17:57 +08:00
// WatchDmChannels watches the channels about data manipulation.
2021-04-12 12:45:38 +08:00
WatchDmChannels(ctx context.Context, req *querypb.WatchDmChannelsRequest) (*commonpb.Status, error)
2021-10-30 18:43:51 +08:00
// LoadSegments notifies QueryNode to load the sealed segments from storage. The load tasks are sync to this
// rpc, QueryNode will return after all the sealed segments are loaded.
2021-04-12 12:45:38 +08:00
LoadSegments(ctx context.Context, req *querypb.LoadSegmentsRequest) (*commonpb.Status, error)
2021-11-06 12:15:31 +08:00
// ReleaseCollection notifies Proxy to release a collection's data
2021-04-12 12:45:38 +08:00
ReleaseCollection(ctx context.Context, req *querypb.ReleaseCollectionRequest) (*commonpb.Status, error)
2021-11-08 09:33:09 +08:00
// ReleasePartitions notifies Proxy to release partitions' data
2021-04-12 12:45:38 +08:00
ReleasePartitions(ctx context.Context, req *querypb.ReleasePartitionsRequest) (*commonpb.Status, error)
2021-11-10 23:45:49 +08:00
// ReleaseSegments releases the data of the specified segments in QueryNode.
2021-04-12 12:45:38 +08:00
ReleaseSegments(ctx context.Context, req *querypb.ReleaseSegmentsRequest) (*commonpb.Status, error)
2021-11-12 13:23:46 +08:00
// GetSegmentInfo requests segment info
2021-04-12 12:45:38 +08:00
GetSegmentInfo(ctx context.Context, req *querypb.GetSegmentInfoRequest) (*querypb.GetSegmentInfoResponse, error)
2021-11-15 19:03:46 +08:00
// GetMetrics gets the metrics about QueryNode.
2021-09-23 21:07:55 +08:00
GetMetrics(ctx context.Context, in *milvuspb.GetMetricsRequest, opts ...grpc.CallOption) (*milvuspb.GetMetricsResponse, error)
2020-12-29 18:02:44 +08:00
}
```
2021-10-08 15:07:05 +08:00
- _AddQueryChannel_
2020-12-29 18:02:44 +08:00
```go
2021-01-13 11:08:03 +08:00
type AddQueryChannelRequest struct {
2021-04-12 12:45:38 +08:00
Base *commonpb.MsgBase
2021-09-23 21:07:55 +08:00
NodeID int64
CollectionID int64
2021-04-12 12:45:38 +08:00
RequestChannelID string
ResultChannelID string
2021-01-13 11:08:03 +08:00
}
```
2021-10-08 15:07:05 +08:00
- _RemoveQueryChannel_
2021-01-13 11:08:03 +08:00
```go
type RemoveQueryChannelRequest struct {
2021-04-12 12:45:38 +08:00
Base *commonpb.MsgBase
2021-09-23 21:07:55 +08:00
NodeID int64
CollectionID int64
2021-04-12 12:45:38 +08:00
RequestChannelID string
ResultChannelID string
2020-12-29 18:02:44 +08:00
}
2020-12-27 09:05:24 +08:00
```
2021-10-08 15:07:05 +08:00
- _WatchDmChannels_
2020-12-27 09:05:24 +08:00
2021-01-13 11:08:03 +08:00
```go
2021-03-11 10:08:46 +08:00
type WatchDmChannelsRequest struct {
2021-04-12 12:45:38 +08:00
Base *commonpb.MsgBase
2021-09-23 21:07:55 +08:00
NodeID int64
2021-04-12 12:45:38 +08:00
CollectionID int64
2021-09-23 21:07:55 +08:00
PartitionID int64
Infos []*datapb.VchannelInfo
Schema *schemapb.CollectionSchema
ExcludeInfos []*datapb.SegmentInfo
2021-01-13 11:08:03 +08:00
}
```
2020-12-27 09:05:24 +08:00
2021-10-08 15:07:05 +08:00
- _LoadSegments_
2021-01-11 18:35:54 +08:00
```go
2021-04-12 12:45:38 +08:00
type LoadSegmentsRequest struct {
Base *commonpb.MsgBase
2021-09-23 21:07:55 +08:00
NodeID int64
Infos []*SegmentLoadInfo
2021-04-12 12:45:38 +08:00
Schema *schemapb.CollectionSchema
2021-10-08 15:07:05 +08:00
LoadCondition TriggerCondition
2021-03-04 10:35:28 +08:00
}
```
2021-10-08 15:07:05 +08:00
- _ReleaseCollection_
2021-03-04 10:35:28 +08:00
```go
type ReleaseCollectionRequest struct {
2021-04-12 12:45:38 +08:00
Base *commonpb.MsgBase
DbID UniqueID
CollectionID UniqueID
2021-09-23 21:07:55 +08:00
NodeID int64
2021-03-04 10:35:28 +08:00
}
```
2021-10-08 15:07:05 +08:00
- _ReleasePartitions_
2021-03-04 10:35:28 +08:00
```go
2021-04-12 12:45:38 +08:00
type ReleasePartitionsRequest struct {
Base *commonpb.MsgBase
DbID UniqueID
CollectionID UniqueID
PartitionIDs []UniqueID
2021-09-23 21:07:55 +08:00
NodeID int64
2021-01-11 18:35:54 +08:00
}
```
2021-10-08 15:07:05 +08:00
- _ReleaseSegments_
2021-01-11 18:35:54 +08:00
```go
2021-04-12 12:45:38 +08:00
type ReleaseSegmentsRequest struct {
Base *commonpb.MsgBase
2021-09-23 21:07:55 +08:00
NodeID int64
2021-04-12 12:45:38 +08:00
DbID UniqueID
CollectionID UniqueID
PartitionIDs []UniqueID
SegmentIDs []UniqueID
2021-01-11 18:35:54 +08:00
}
```
2021-10-08 15:07:05 +08:00
- _GetSegmentInfo_
2021-03-04 10:35:28 +08:00
```go
2021-04-12 12:45:38 +08:00
type GetSegmentInfoRequest struct {
Base *commonpb.MsgBase
SegmentIDs []Unique
2021-03-04 10:35:28 +08:00
}
2021-04-12 12:45:38 +08:00
type GetSegmentInfoResponse struct {
Status *commonpb.Status
Infos []*SegmentInfo
2021-03-04 10:35:28 +08:00
}
```
2021-01-11 18:35:54 +08:00
2021-03-04 10:35:28 +08:00
//TODO
2021-10-08 15:07:05 +08:00
2021-09-26 21:05:58 +08:00
#### 7.5 Collection Replica
2020-11-19 10:59:10 +08:00
2021-11-04 13:38:21 +08:00
$collectionReplica$ contains an in-memory local copy of persistent collections. In common cases, the system has multiple query nodes. Data of a collection will be distributed across all the available query nodes, and each query node's $collectionReplica$ will maintain its own share (only part of the collection).
2020-11-19 10:59:10 +08:00
Every replica tracks a value called tSafe which is the maximum timestamp that the replica is up-to-date.
2021-10-08 15:07:05 +08:00
- _Collection_
2020-11-19 10:59:10 +08:00
2021-10-08 15:07:05 +08:00
```go
2021-04-12 12:45:38 +08:00
type collectionReplica struct {
tSafes map[UniqueID]tSafer // map[collectionID]tSafer
mu sync.RWMutex // guards all
collections map[UniqueID]*Collection
partitions map[UniqueID]*Partition
segments map[UniqueID]*Segment
2021-09-23 21:07:55 +08:00
excludedSegments map[UniqueID][]*datapb.SegmentInfo // map[collectionID]segmentIDs
2020-11-19 10:59:10 +08:00
}
```
2021-10-08 15:07:05 +08:00
- _Collection_
2021-04-12 12:45:38 +08:00
```go
type FieldSchema struct {
FieldID int64
Name string
IsPrimaryKey bool
Description string
DataType DataType
TypeParams []*commonpb.KeyValuePair
IndexParams []*commonpb.KeyValuePair
}
type CollectionSchema struct {
Name string
Description string
AutoID bool
Fields []*FieldSchema
}
type Collection struct {
collectionPtr C.CCollection
id UniqueID
partitionIDs []UniqueID
schema *schemapb.CollectionSchema
2021-09-23 21:07:55 +08:00
vChannels []Channel
pChannels []Channel
loadType loadType
releaseMu sync.RWMutex
releasedPartitions map[UniqueID]struct{}
releaseTime Timestamp
2021-04-12 12:45:38 +08:00
}
```
2021-10-08 15:07:05 +08:00
- _Partition_
2020-11-19 10:59:10 +08:00
```go
2021-04-12 12:45:38 +08:00
type Partition struct {
collectionID UniqueID
partitionID UniqueID
segmentIDs []UniqueID
2020-11-19 10:59:10 +08:00
}
```
2021-10-08 15:07:05 +08:00
- _Segment_
2020-11-19 10:59:10 +08:00
2021-10-08 15:07:05 +08:00
```go
2021-04-12 12:45:38 +08:00
type segmentType int32
const (
segmentTypeInvalid segmentType = iota
segmentTypeGrowing
segmentTypeSealed
segmentTypeIndexing
)
type indexParam = map[string]string
2020-11-19 10:59:10 +08:00
type Segment struct {
2021-04-12 12:45:38 +08:00
segmentPtr C.CSegmentInterface
segmentID UniqueID
partitionID UniqueID
collectionID UniqueID
2021-09-23 21:07:55 +08:00
onService bool
vChannelID Channel
2021-04-12 12:45:38 +08:00
lastMemSize int64
lastRowCount int64
once sync.Once // guards enableIndex
enableIndex bool
rmMutex sync.Mutex // guards recentlyModified
recentlyModified bool
typeMu sync.Mutex // guards builtIndex
segmentType segmentType
paramMutex sync.RWMutex // guards index
2021-09-23 21:07:55 +08:00
indexInfos map[FieldID]*indexInfo
idBinlogRowSizes []int64
vectorFieldMutex sync.RWMutex // guards vectorFieldInfos
vectorFieldInfos map[UniqueID]*VectorFieldInfo
2021-11-11 13:15:27 +08:00
pkFilter *bloom.BloomFilter // bloom filter of pk inside a segment
2020-11-19 10:59:10 +08:00
}
```
2021-10-08 15:07:05 +08:00
- _Data Sync Service_
2020-11-19 10:59:10 +08:00
```go
2021-01-13 11:08:03 +08:00
type dataSyncService struct {
2021-04-12 12:45:38 +08:00
ctx context.Context
2021-09-23 21:07:55 +08:00
mu sync.Mutex // guards FlowGraphs
collectionFlowGraphs map[UniqueID]map[Channel]*queryNodeFlowGraph // map[collectionID]flowGraphs
partitionFlowGraphs map[UniqueID]map[Channel]*queryNodeFlowGraph // map[partitionID]flowGraphs
2021-04-12 12:45:38 +08:00
2021-09-23 21:07:55 +08:00
streamingReplica ReplicaInterface
tSafeReplica TSafeReplicaInterface
msFactory msgstream.Factory
2020-11-19 10:59:10 +08:00
}
```