gensrc/proto/olap_file.proto (595 lines of code) (raw):
// Licensed to the Apache Software Foundation (ASF) 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.
// Define file format struct, like data header, index header.
syntax="proto2";
package doris;
option java_package = "org.apache.doris.proto";
import "olap_common.proto";
import "types.proto";
import "segment_v2.proto";
message ZoneMap {
required bytes min = 1;
required bytes max = 2;
optional bool null_flag = 3;
}
enum RowsetTypePB {
ALPHA_ROWSET = 0; // doris original column storage format
BETA_ROWSET = 1; // new column storage format
}
enum RowsetStatePB {
// Rowset is being written
PREPARED = 0;
// Rowset writting is completed, but the user is not yet visible
// For rowset in this state, BE cannot determine whether to delete it by itself, it must be instructed by FE
COMMITTED = 1;
// Rowset is already visible to the user
VISIBLE = 2;
// Rowset is partial updated, the recycler should use list+delete to recycle segments.
BEGIN_PARTIAL_UPDATE = 10000;
}
// indicate whether date between segments of a rowset is overlappinng
enum SegmentsOverlapPB {
OVERLAP_UNKNOWN = 0; // this enum is added since Doris v0.11, so previous rowset's segment is unknown
OVERLAPPING = 1;
NONOVERLAPPING = 2;
}
message KeyBoundsPB {
required bytes min_key = 1;
required bytes max_key = 2;
}
// ATTN: When adding or deleting fields, please update `message RowsetMetaCloudPB`
// simultaneously and modify the conversion function in the be/src/cloud/pb_convert.{h,cpp}.
message RowsetMetaPB {
required int64 rowset_id = 1; // Deprecated. Use rowset_id_v2 instead.
optional int64 partition_id = 2;
optional int64 tablet_id = 3;
// only for pending rowset
optional int64 txn_id = 4;
// tablet id and schema hash could find tablet
optional int32 tablet_schema_hash = 5;
optional RowsetTypePB rowset_type = 6;
optional RowsetStatePB rowset_state = 7;
// only for visible rowset. For alpha rowset, it's the same as PDelta.start_version
optional int64 start_version = 8;
// only for visible rowset. For alpha rowset, it's the same as PDelta.end_version
optional int64 end_version = 9;
// only for visible rowset. For alpha rowset, it's the same as PDelta.version_hash
optional int64 version_hash = 10;
// total number of rows. For alpha rowset, it's calculated from segment group
optional int64 num_rows = 11;
// calculated by index + data
optional int64 total_disk_size = 12;
// calculated sum(segmentgroup.data_size)
optional int64 data_disk_size = 13;
// calculated sum(segmentgroup.index_size)
optional int64 index_disk_size = 14;
// rowset level column min/max/null statistics, not not used currently
repeated ZoneMap zone_maps = 15;
optional DeletePredicatePB delete_predicate = 16;
// calculated from segment group
optional bool empty = 17;
// only for pending rowset
optional PUniqueId load_id = 18;
// not set, get from olap index header msg?
optional bool delete_flag = 19;
// For alpha rowset, it's the same as PDelta.creation_time
optional int64 creation_time = 20;
optional PUniqueId tablet_uid = 21;
// total number of segments
optional int64 num_segments = 22;
// rowset id definition, it will replace required rowset id
optional string rowset_id_v2 = 23;
// resource id
optional string resource_id = 24;
// used to be oldest write time: earliest write time
reserved 25;
// latest write time
optional int64 newest_write_timestamp = 26 [default = -1];
// the encoded segment min/max key of segments in this rowset,
// only used in unique key data model with primary_key_index support.
repeated KeyBoundsPB segments_key_bounds = 27;
// tablet meta pb, for compaction
optional TabletSchemaPB tablet_schema = 28;
// alpha_rowset_extra_meta_pb is deleted
reserved 50;
// to indicate whether the data between the segments overlap
optional SegmentsOverlapPB segments_overlap_pb = 51 [default = OVERLAP_UNKNOWN];
optional int64 compaction_level = 52 [default = 0];
// For backup/restore, record the tablet id and rowset id of the source cluster.
optional int64 source_tablet_id = 53;
optional string source_rowset_id = 54;
// For cloud
// for data recycling
optional int64 txn_expiration = 1000;
// the field is a vector, rename it
repeated int64 segments_file_size = 1001;
// index_id, schema_version -> schema
optional int64 index_id = 1002;
optional int32 schema_version = 1003;
// If enable_segments_file_size is false,
// the segments_file_size maybe is empty or error
optional bool enable_segments_file_size = 1004;
optional bool has_variant_type_in_schema = 1005;
optional bool enable_inverted_index_file_info = 1006;
repeated InvertedIndexFileInfo inverted_index_file_info = 1007;
optional SchemaDictKeyList schema_dict_key_list = 1008; // align to cloud rowset
}
message SchemaDictKeyList {
// save the dict keys for column pb info
repeated int32 column_dict_key_list = 1;
// save the dict keys for tablet index pb info
repeated int32 index_info_dict_key_list = 2;
// save the dict keys for sparse column pb info
repeated int32 sparse_column_dict_key_list = 3;
};
message RowsetMetaCloudPB {
required int64 rowset_id = 1; // Deprecated. Use rowset_id_v2 instead.
optional int64 partition_id = 2;
optional int64 tablet_id = 3;
// only for pending rowset
optional int64 txn_id = 4;
// tablet id and schema hash could find tablet
optional int32 tablet_schema_hash = 5;
optional RowsetTypePB rowset_type = 6;
optional RowsetStatePB rowset_state = 7;
// only for visible rowset. For alpha rowset, it's the same as PDelta.start_version
optional int64 start_version = 8;
// only for visible rowset. For alpha rowset, it's the same as PDelta.end_version
optional int64 end_version = 9;
// only for visible rowset. For alpha rowset, it's the same as PDelta.version_hash
optional int64 version_hash = 10;
// total number of rows. For alpha rowset, it's calculated from segment group
optional int64 num_rows = 11;
// calculated by index + data
optional int64 total_disk_size = 12;
// calculated sum(segmentgroup.data_size)
optional int64 data_disk_size = 13;
// calculated sum(segmentgroup.index_size)
optional int64 index_disk_size = 14;
// rowset level column min/max/null statistics, not not used currently
repeated ZoneMap zone_maps = 15;
optional DeletePredicatePB delete_predicate = 16;
// calculated from segment group
optional bool empty = 17;
// only for pending rowset
optional PUniqueId load_id = 18;
// not set, get from olap index header msg?
optional bool delete_flag = 19;
// For alpha rowset, it's the same as PDelta.creation_time
optional int64 creation_time = 20;
optional PUniqueId tablet_uid = 21;
// total number of segments
optional int64 num_segments = 22;
// rowset id definition, it will replace required rowset id
optional string rowset_id_v2 = 23;
// resource id
optional string resource_id = 24;
// used to be oldest write time: earliest write time
reserved 25;
// latest write time
optional int64 newest_write_timestamp = 26 [default = -1];
// the encoded segment min/max key of segments in this rowset,
// only used in unique key data model with primary_key_index support.
repeated KeyBoundsPB segments_key_bounds = 27;
// tablet meta pb, for compaction
optional TabletSchemaCloudPB tablet_schema = 28;
// for data recycling in CLOUD_MODE
optional int64 txn_expiration = 29;
// alpha_rowset_extra_meta_pb is deleted
reserved 50;
// to indicate whether the data between the segments overlap
optional SegmentsOverlapPB segments_overlap_pb = 51 [default = OVERLAP_UNKNOWN];
optional int64 compaction_level = 52 [default = 0];
// For backup/restore, record the tablet id and rowset id of the source cluster.
optional int64 source_tablet_id = 53;
optional string source_rowset_id = 54;
// cloud
// the field is a vector, rename it
repeated int64 segments_file_size = 100;
// index_id, schema_version -> schema
optional int64 index_id = 101;
optional int32 schema_version = 102;
// If enable_segments_file_size is false,
// the segments_file_size maybe is empty or error
optional bool enable_segments_file_size = 103;
// extra info for variants
optional bool has_variant_type_in_schema = 104;
// dict key lists for compress schema info
optional SchemaDictKeyList schema_dict_key_list = 105;
optional bool enable_inverted_index_file_info = 106;
repeated InvertedIndexFileInfo inverted_index_file_info = 107;
}
message SegmentStatisticsPB {
optional int64 row_num = 1;
optional int64 data_size = 2;
optional int64 index_size = 3;
optional KeyBoundsPB key_bounds = 4;
}
// kv value for reclaiming remote rowset
message RemoteRowsetGcPB {
required string resource_id = 1;
required int64 tablet_id = 2;
required int64 num_segments = 3;
}
// kv value for reclaiming all remote rowsets of tablet
message RemoteTabletGcPB {
repeated string resource_ids = 1;
}
enum DataFileType {
OLAP_DATA_FILE = 0; //Deprecated. Only columnar-wise format is supported.
COLUMN_ORIENTED_FILE = 1;
}
enum KeysType {
DUP_KEYS = 0;
UNIQUE_KEYS = 1;
AGG_KEYS = 2;
}
message DeletePredicatePB {
required int32 version = 1; // This field is useless, but could not removed, not depend on it
repeated string sub_predicates = 2;
repeated InPredicatePB in_predicates = 3;
repeated DeleteSubPredicatePB sub_predicates_v2 = 4;
}
message DeleteSubPredicatePB {
optional int32 column_unique_id = 1;
optional string column_name = 2;
optional string op = 3;
optional string cond_value = 4;
}
message InPredicatePB {
optional string column_name = 1;
optional bool is_not_in = 2;
repeated string values = 3;
optional int32 column_unique_id = 4;
}
enum AlterTabletState {
ALTER_PREPARED = 0;
ALTER_RUNNING = 1;
ALTER_FINISHED = 2;
ALTER_FAILED = 3;
}
enum AlterTabletType {
SCHEMA_CHANGE = 1;
ROLLUP = 2;
MIGRATION = 3;
}
message AlterTabletPB {
optional AlterTabletState alter_state = 1;
required int64 related_tablet_id = 2;
optional int32 related_schema_hash = 3;
optional AlterTabletType alter_type = 4;
}
message ColumnPB {
required int32 unique_id = 1; // ColumnMessage.unique_id
optional string name = 2; // ColumnMessage.name
required string type = 3; // ColumnMessage.type
optional bool is_key = 4; // ColumnMessage.is_key
optional string aggregation = 5; // ColumnMessage.aggregation
optional bool is_nullable = 6; // ColumnMessage.is_allow_null
optional bytes default_value = 7; // ColumnMessage.default_value ?
optional int32 precision = 8; // ColumnMessage.precision
optional int32 frac = 9; // ColumnMessage.frac
optional int32 length = 10; // ColumnMessage.length
optional int32 index_length = 11; // ColumnMessage.index_length
optional bool is_bf_column = 12; // ColumnMessage.is_bf_column
optional int32 referenced_column_id = 13; // deprecated
optional string referenced_column = 14; // deprecated
optional bool has_bitmap_index = 15 [default=false]; // ColumnMessage.has_bitmap_index
optional bool visible = 16 [default=true];
repeated ColumnPB children_columns = 17;
repeated string children_column_names = 18;
optional bool result_is_nullable = 19;
// persist info for PathInData that represents path in document, e.g. JSON.
optional segment_v2.ColumnPathInfo column_path_info = 20;
// sparse column within a variant column
repeated ColumnPB sparse_columns = 21;
optional bool is_auto_increment = 22;
// only reference by variant sparse columns
optional int32 parent_unique_id = 23;
optional int32 be_exec_version = 24;
}
// Dictionary of Schema info, to reduce TabletSchemaCloudPB fdb kv size
message SchemaCloudDictionary {
map<int32, ColumnPB> column_dict= 1;
// monotonic increasing
optional int64 current_column_dict_id = 2;
map<int32, TabletIndexPB> index_dict = 3;
// monotonic increasing
optional int64 current_index_dict_id = 4;
}
enum IndexType {
BITMAP = 0;
INVERTED = 1;
BLOOMFILTER = 2;
NGRAM_BF = 3;
}
enum InvertedIndexStorageFormatPB {
V1 = 0;
V2 = 1;
V3 = 2;
}
message TabletIndexPB {
optional int64 index_id = 1;
optional string index_name = 2;
optional IndexType index_type = 3;
repeated int32 col_unique_id = 4;
map<string, string> properties = 5;
optional string index_suffix_name = 6;
}
enum SortType {
LEXICAL = 0;
ZORDER = 1;
}
enum UniqueKeyUpdateModePB {
UPSERT = 0;
UPDATE_FIXED_COLUMNS = 1;
UPDATE_FLEXIBLE_COLUMNS = 2;
}
// ATTN: When adding or deleting fields, please update `message TabletSchemaCloudPB`
// simultaneously and modify the conversion function in the be/src/cloud/pb_convert.{h,cpp}.
message TabletSchemaPB {
optional KeysType keys_type = 1; // OLAPHeaderMessage.keys_type
repeated ColumnPB column = 2; // OLAPHeaderMessage.column
optional int32 num_short_key_columns = 3; // OLAPHeaderMessage.num_short_key_fields
optional int32 num_rows_per_row_block = 4; // OLAPHeaderMessage.num_rows_per_data_block
optional CompressKind compress_kind = 5; // OLAPHeaderMessage.compress_kind
optional double bf_fpp = 6; // OLAPHeaderMessage.bf_fpp
optional uint32 next_column_unique_id = 7; // OLAPHeaderMessage.next_column_unique_id
// FIXME(gavin): deprecate and remove in the future
optional bool is_in_memory = 8 [default=false];
optional int32 delete_sign_idx = 9 [default = -1];
optional int32 sequence_col_idx = 10 [default= -1];
optional SortType sort_type = 11;
optional int32 sort_col_num = 12;
optional segment_v2.CompressionTypePB compression_type = 13 [default=LZ4F];
optional int32 schema_version = 14;
optional bool disable_auto_compaction = 15 [default=false];
repeated TabletIndexPB index = 16;
optional int32 version_col_idx = 17 [default = -1];
optional bool store_row_column = 18 [default=false]; // store tuplerow oriented column
optional bool is_dynamic_schema = 19 [default=false]; // deprecated
reserved 20; // deprecated is_partial_update
reserved 21; // deprecated partial_update_input_columns
optional bool enable_single_replica_compaction = 22 [default=false];
optional bool skip_write_index_on_load = 23 [default=false];
repeated int32 cluster_key_uids = 24;
optional InvertedIndexStorageFormatPB inverted_index_storage_format = 25 [default=V1];
// column unique ids for row store columns
repeated int32 row_store_column_unique_ids = 26;
optional int64 row_store_page_size = 27 [default=16384];
optional bool enable_variant_flatten_nested = 28 [default=false];
optional int32 skip_bitmap_col_idx = 29 [default = -1];
optional int64 storage_page_size = 30 [default=65536];
}
message TabletSchemaCloudPB {
optional KeysType keys_type = 1; // OLAPHeaderMessage.keys_type
repeated ColumnPB column = 2; // OLAPHeaderMessage.column
optional int32 num_short_key_columns = 3; // OLAPHeaderMessage.num_short_key_fields
optional int32 num_rows_per_row_block = 4; // OLAPHeaderMessage.num_rows_per_data_block
optional CompressKind compress_kind = 5; // OLAPHeaderMessage.compress_kind
optional double bf_fpp = 6; // OLAPHeaderMessage.bf_fpp
optional uint32 next_column_unique_id = 7; // OLAPHeaderMessage.next_column_unique_id
optional int32 delete_sign_idx = 8 [default = -1];
optional int32 sequence_col_idx = 9 [default= -1];
optional SortType sort_type = 10;
optional int32 sort_col_num = 11;
optional segment_v2.CompressionTypePB compression_type = 12 [default=LZ4F];
optional int32 schema_version = 13;
optional bool disable_auto_compaction = 14 [default=false];
repeated TabletIndexPB index = 15;
optional int32 version_col_idx = 16 [default = -1];
optional bool store_row_column = 18 [default=false]; // store tuplerow oriented column
reserved 20; // deprecated is_partial_update
reserved 21; // deprecated partial_update_input_columns
optional bool enable_single_replica_compaction = 22 [default=false];
optional bool skip_write_index_on_load = 23 [default=false];
repeated int32 cluster_key_uids = 24;
optional InvertedIndexStorageFormatPB inverted_index_storage_format = 25 [default=V1];
// column unique ids for row store columns
repeated int32 row_store_column_unique_ids = 26;
optional int64 row_store_page_size = 27 [default=16384];
reserved 28; // deprecated enable_mow_light_delete
optional bool enable_variant_flatten_nested = 29 [default=false];
optional int32 skip_bitmap_col_idx = 30 [default = -1];
optional int64 storage_page_size = 31 [default=65536];
optional bool is_dynamic_schema = 100 [default=false];
// FIXME(gavin): deprecate and remove in the future
optional bool is_in_memory = 200 [default=false]; // unused, just keep align to TabletSchemaPB
}
enum TabletStatePB {
PB_NOTREADY = 0; // under alter table, rollup, clone
PB_RUNNING = 1;
PB_TOMBSTONED = 2;
PB_STOPPED = 3;
PB_SHUTDOWN = 4;
}
enum TabletTypePB {
TABLET_TYPE_DISK = 0;
TABLET_TYPE_MEMORY = 1;
}
enum StorageMediumPB {
HDD = 0;
SSD = 1;
S3 = 2;
REMOTE_CACHE = 99;
}
message S3StorageParamPB {
optional string s3_endpoint = 1;
optional string s3_region = 2;
optional string s3_ak = 3;
optional string s3_sk = 4;
optional int32 s3_max_conn = 5 [default = 50];
optional int32 s3_request_timeout_ms = 6 [default = 3000];
optional int32 s3_conn_timeout_ms = 7 [default = 1000];
optional string root_path = 8;
}
message BinlogConfigPB {
optional bool enable = 1;
optional int64 ttl_seconds = 2;
optional int64 max_bytes = 3;
optional int64 max_history_nums = 4;
}
// ATTN: When adding or deleting fields, please update `message TabletMetaCloudPB`
// simultaneously and modify the conversion function in the be/src/cloud/pb_convert.{h,cpp}.
message TabletMetaPB {
optional int64 table_id = 1; // ?
optional int64 partition_id = 2; // ?
optional int64 tablet_id = 3; // OlapHeaderMessage.tablet_id
optional int32 schema_hash = 4; // OlapHeaderMessage.schema_hash
optional int32 shard_id = 5; // OlapHeaderMessage.shard
optional int64 creation_time = 6; // OlapHeaderMessage.creation_time
optional int64 cumulative_layer_point = 7; // OlapHeaderMessage.cumulative_layer_point
optional TabletStatePB tablet_state = 8;
optional TabletSchemaPB schema = 9;
repeated RowsetMetaPB rs_metas = 10;
// @Deprecated
repeated RowsetMetaPB inc_rs_metas = 11;
optional AlterTabletPB alter_task = 12;
// if true, this tablet will not do compaction,
// and does not create init version
optional bool in_restore_mode = 13 [default = false]; // OlapHeaderMessage.in_restore_mode
// a uniqued id to identified tablet with same tablet_id and schema hash
optional PUniqueId tablet_uid = 14;
optional int64 end_rowset_id = 15;
optional RowsetTypePB preferred_rowset_type = 16;
optional TabletTypePB tablet_type = 17;
repeated RowsetMetaPB stale_rs_metas = 18;
// optional StorageMediumPB storage_medium = 19 [default = HDD];
// optional string remote_storage_name = 20;
optional int64 replica_id = 21 [default = 0];
// 22 used to be string storage_policy = 22;
reserved 22;
optional DeleteBitmapPB delete_bitmap = 23;
// Use primary key index to speed up tabel unique key model
optional bool enable_unique_key_merge_on_write = 24 [default = false];
optional int64 storage_policy_id = 25;
optional PUniqueId cooldown_meta_id = 26;
optional BinlogConfigPB binlog_config = 27;
optional string compaction_policy = 28 [default = "size_based"];
optional int64 time_series_compaction_goal_size_mbytes = 29 [default = 1024];
optional int64 time_series_compaction_file_count_threshold = 30 [default = 2000];
optional int64 time_series_compaction_time_threshold_seconds = 31 [default = 3600];
optional int64 time_series_compaction_empty_rowsets_threshold = 32 [default = 5];
optional int64 time_series_compaction_level_threshold = 33 [default = 1];
// For cloud
optional int64 index_id = 1000;
optional bool is_in_memory = 1001;
optional bool is_persistent = 1002;
optional string table_name = 1003;
optional int64 ttl_seconds = 1004;
optional int32 schema_version = 1005; // index_id, schema_version -> schema
}
message TabletMetaCloudPB {
optional int64 table_id = 1; // ?
optional int64 partition_id = 2; // ?
optional int64 tablet_id = 3; // OlapHeaderMessage.tablet_id
optional int32 schema_hash = 4; // OlapHeaderMessage.schema_hash
optional int32 shard_id = 5; // OlapHeaderMessage.shard
optional int64 creation_time = 6; // OlapHeaderMessage.creation_time
optional int64 cumulative_layer_point = 7; // OlapHeaderMessage.cumulative_layer_point
optional TabletStatePB tablet_state = 8;
optional TabletSchemaCloudPB schema = 9;
repeated RowsetMetaCloudPB rs_metas = 10;
// @Deprecated
repeated RowsetMetaCloudPB inc_rs_metas = 11;
optional AlterTabletPB alter_task = 12;
// if true, this tablet will not do compaction,
// and does not create init version
optional bool in_restore_mode = 13 [default = false]; // OlapHeaderMessage.in_restore_mode
// a uniqued id to identified tablet with same tablet_id and schema hash
optional PUniqueId tablet_uid = 14;
optional int64 end_rowset_id = 15;
optional RowsetTypePB preferred_rowset_type = 16;
optional TabletTypePB tablet_type = 17;
repeated RowsetMetaCloudPB stale_rs_metas = 18;
// optional StorageMediumPB storage_medium = 19 [default = HDD];
// optional string remote_storage_name = 20;
optional int64 replica_id = 21 [default = 0];
// 22 used to be string storage_policy = 22;
reserved 22;
optional DeleteBitmapPB delete_bitmap = 23;
// Use primary key index to speed up tabel unique key model
optional bool enable_unique_key_merge_on_write = 24 [default = false];
optional int64 index_id = 25;
optional bool is_in_memory = 26 [default=false];
optional bool is_persistent = 27 [default=false];
optional int64 storage_policy_id = 28;
optional PUniqueId cooldown_meta_id = 29;
optional BinlogConfigPB binlog_config = 30;
optional string compaction_policy = 31 [default = "size_based"];
optional int64 time_series_compaction_goal_size_mbytes = 32 [default = 1024];
optional int64 time_series_compaction_file_count_threshold = 33 [default = 2000];
optional int64 time_series_compaction_time_threshold_seconds = 34 [default = 3600];
reserved 35; // deprecated group_commit_interval_ms
reserved 36; // deprecated group_commit_data_bytes
optional int64 time_series_compaction_empty_rowsets_threshold = 37 [default = 5];
optional int64 time_series_compaction_level_threshold = 38 [default = 1];
// Use for selectdb-cloud
optional string table_name = 101;
optional int64 ttl_seconds = 102;
optional int32 schema_version = 103; // index_id, schema_version -> schema
}
message OLAPRawDeltaHeaderMessage {
required int32 schema_hash = 2;
}
message DeleteBitmapPB {
repeated string rowset_ids = 1;
repeated uint32 segment_ids = 2;
repeated int64 versions = 3;
// Serialized roaring bitmaps indexed with {rowset_id, segment_id, version}
repeated bytes segment_delete_bitmaps = 4;
}
message BinlogMetaEntryPB {
optional int64 version = 1;
optional int64 tablet_id = 2;
optional int64 rowset_id = 3; // Deprecated use rowset_id_v2 instead
optional int64 num_segments = 4;
optional int64 creation_time = 5;
optional string rowset_id_v2 = 6;
}
message PendingPublishInfoPB {
optional int64 partition_id = 1;
optional int64 transaction_id = 2;
}
message RowsetBinlogMetasPB {
message RowsetBinlogMetaPB {
optional string rowset_id = 1;
optional int64 version = 2;
optional int64 num_segments = 3;
optional string meta_key = 4;
optional bytes meta = 5;
optional string data_key = 6;
optional bytes data = 7;
}
repeated RowsetBinlogMetaPB rowset_binlog_metas = 1;
}
message PartialUpdateInfoPB {
optional bool is_partial_update = 1 [default = false]; // deprecated
repeated string partial_update_input_columns = 2;
repeated uint32 missing_cids = 3;
repeated uint32 update_cids = 4;
optional bool can_insert_new_rows_in_partial_update = 5 [default = false];
optional bool is_strict_mode = 6 [default = false];
optional int64 timestamp_ms = 7 [default = 0];
optional string timezone = 8;
optional bool is_input_columns_contains_auto_inc_column = 9 [default = false];
optional bool is_schema_contains_auto_inc_column = 10 [default = false];
repeated string default_values = 11;
optional int64 max_version_in_flush_phase = 12 [default = -1];
optional int32 nano_seconds = 13 [default = 0];
optional UniqueKeyUpdateModePB partial_update_mode = 14 [default = UPSERT];
}