blob: d4e9699ea716e388b76da8d607a329a15e92c9fb [file] [log] [blame]
// 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.
syntax = "proto2";
package kudu.master;
option java_package = "org.apache.kudu.master";
import "kudu/common/common.proto";
import "kudu/common/row_operations.proto";
import "kudu/common/wire_protocol.proto";
import "kudu/consensus/metadata.proto";
import "kudu/consensus/replica_management.proto";
import "kudu/rpc/rpc_header.proto";
import "kudu/security/token.proto";
import "kudu/tablet/metadata.proto";
import "kudu/util/pb_util.proto";
////////////////////////////////////////////////////////////
// Common data structures
////////////////////////////////////////////////////////////
// Master specific errors use this protobuf.
message MasterErrorPB {
enum Code {
// An error which has no more specific error code.
// The code and message in 'status' may reveal more details.
//
// RPCs should avoid returning this, since callers will not be
// able to easily parse the error.
UNKNOWN_ERROR = 1;
// The schema provided for a request was not well-formed.
INVALID_SCHEMA = 2;
// The requested table does not exist
TABLE_NOT_FOUND = 3;
// The name requested for the table is already in use
TABLE_ALREADY_PRESENT = 4;
// The number of tablets requested for a new table is over the per TS limit.
TOO_MANY_TABLETS = 5;
// Catalog manager is not yet initialized.
CATALOG_MANAGER_NOT_INITIALIZED = 6;
// The operation attempted can only be invoked against either the
// leader or a single non-distributed master, which this node
// isn't.
NOT_THE_LEADER = 7;
// The number of replicas requested is greater than the number of live servers
// in the cluster or the configured maximum.
REPLICATION_FACTOR_TOO_HIGH = 8;
// The request or response involved a tablet which is not yet running.
TABLET_NOT_RUNNING = 9;
// The number of replicas requested is even.
EVEN_REPLICATION_FACTOR = 10;
// The number of replicas requested is illegal, i.e. either non-positive
// or not allowed per system policies.
ILLEGAL_REPLICATION_FACTOR = 11;
// The callee detected that its replica management scheme is incompatible
// with the caller's scheme.
INCOMPATIBLE_REPLICA_MANAGEMENT = 12;
// An operation involving the Hive Metastore failed.
HIVE_METASTORE_ERROR = 13;
// The caller is not authorized to perform the attempted operation.
NOT_AUTHORIZED = 14;
// Master is already part of the Raft configuration.
MASTER_ALREADY_PRESENT = 15;
// The requested table is in soft_deleted state.
TABLE_SOFT_DELETED = 16;
}
// The error code.
required Code code = 1;
// The Status object for the error. This will include a textual
// message that may be more useful to present in log messages, etc,
// though its error code is less specific.
required AppStatusPB status = 2;
}
// Common information sent with every request from the tablet server
// to the master.
message TSToMasterCommonPB {
// The instance of the tablet server sending the heartbeat.
required NodeInstancePB ts_instance = 1;
}
message TableIdentifierPB {
// The table ID to fetch info.
optional bytes table_id = 1;
// The table name to fetch info.
optional string table_name = 2;
}
////////////////////////////////////////////////////////////
// Sys Tables Metadata
////////////////////////////////////////////////////////////
// The on-disk entry in the sys.catalog table ("metadata" column) for
// tablets entries.
message SysTabletsEntryPB {
enum State {
UNKNOWN = 999;
PREPARING = 0;
CREATING = 1;
RUNNING = 2;
REPLACED = 3;
DELETED = 4;
SOFT_DELETED = 5;
}
// DEPRECATED. Replaced by 'partition'.
optional bytes DEPRECATED_start_key = 1;
optional bytes DEPRECATED_end_key = 2;
// Tablet partition.
optional PartitionPB partition = 7;
// The latest consensus state reported to the Master.
optional consensus.ConsensusStatePB consensus_state = 3;
// Debug state for the tablet.
optional State state = 4 [ default = UNKNOWN ];
optional bytes state_msg = 5;
// The table id for the tablet.
required bytes table_id = 6;
// The dimension label for the tablet. Used for dimension-specific
// placement of the tablet's replicas.
optional string dimension_label = 8;
// The delete time of the tablet, in seconds since the epoch.
optional int64 delete_timestamp = 9;
}
// The on-disk entry in the sys.catalog table ("metadata" column) for
// tables entries.
message SysTablesEntryPB {
enum State {
UNKNOWN = 0;
PREPARING = 1;
RUNNING = 2;
ALTERING = 3;
REMOVED = 4;
SOFT_DELETED = 5;
}
// Table name
required bytes name = 1;
// sequence-id for the table metadata.
// Used on tablet-report to avoid sending "alter-table" notifications.
required uint32 version = 2;
// Newest table schema (every TS will eventually have it).
required SchemaPB schema = 3;
// Last table schema that is guaranteed to have reached every TS, though
// not necessarily the newest schema.
//
// This is the schema provided to the user on client->GetSchema(tableName).
optional SchemaPB fully_applied_schema = 4;
// The table's partition schema.
optional PartitionSchemaPB partition_schema = 9;
// The next column ID to assign to newly added columns in this table.
// This prevents column ID reuse.
optional int32 next_column_id = 8;
// Number of TS replicas
required int32 num_replicas = 5;
// Debug state for the table.
optional State state = 6 [ default = UNKNOWN ];
optional bytes state_msg = 7;
// The create time of the table, in seconds since the epoch.
optional int64 create_timestamp = 10;
// The last alter time of the table, in seconds since the epoch.
optional int64 alter_timestamp = 11;
// The table's extra configuration properties.
optional TableExtraConfigPB extra_config = 12;
// The user that owns the table.
optional string owner = 13;
// The table type. If not set, it is assumed this table is a user-defined
// table, rather than a system table.
optional TableTypePB table_type = 14;
// Table disk size limit.
optional int64 table_disk_size_limit = 15;
// Table row count limit.
optional int64 table_row_count_limit = 16;
// The comment on the table.
optional string comment = 17;
// The delete time of the table, in seconds since the epoch.
optional int64 delete_timestamp = 18;
// The reservation time interval (in seconds) between soft delete and delete.
optional uint32 soft_deleted_reserved_seconds = 19;
}
// The on-disk entry in the sys.catalog table ("metadata" column) to represent
// certificate authority (CA) information. Not more than one entry of this type
// should exist in the sys.catalog table at any time.
message SysCertAuthorityEntryPB {
// Private key body in DER format. This information is sensitive,
// that's why the REDACT attribute is present.
required bytes private_key = 1 [(kudu.REDACT) = true];
// Certificate body in DER format. This information is not sensitive,
// but we are redacting most of security-related info anyway
// (handshake/SASL tokens, etc.).
required bytes certificate = 2 [(kudu.REDACT) = true];
}
// The on-disk entry in the sys.catalog table ("metadata" column) to represent
// a Token Signing Key (TSK) object. Multiple entries of this type
// can simultaneously co-exist in the sys.catalog table.
message SysTskEntryPB {
// TokenSigningPrivateKeyPB message representing a TSK.
required security.TokenSigningPrivateKeyPB tsk = 1;
}
// The on-disk entry in the sys.catalog table ("metadata" column) to represent
// the latest processed Hive Metastore notification log event ID.
message SysNotificationLogEventIdPB {
optional int64 latest_notification_log_event_id = 1;
}
// The on-disk entry in the sys.catalog table ("metadata" column) to represent
// the cluster ID.
message SysClusterIdEntryPB {
optional string cluster_id = 1;
}
// The on-disk entry in the sys.catalog table to represent the existence of
// on-going tserver state (e.g. maintenance mode).
message SysTServerStateEntryPB {
optional TServerStatePB state = 1;
// Timestamp, in seconds since the epoch, at which this state was set.
optional int64 timestamp_secs = 2;
}
////////////////////////////////////////////////////////////
// RPCs
////////////////////////////////////////////////////////////
message PingRequestPB {
}
message PingResponsePB {
}
message ReportedTabletPB {
required bytes tablet_id = 1;
optional tablet.TabletStatePB state = 2 [ default = UNKNOWN ];
optional tablet.TabletDataState tablet_data_state = 6 [ default = TABLET_DATA_UNKNOWN ];
// The latest consensus state.
// This will be missing if the tablet is not in a RUNNING state
// (i.e. if it is BOOTSTRAPPING).
optional consensus.ConsensusStatePB consensus_state = 3;
optional AppStatusPB error = 4;
optional uint32 schema_version = 5;
// Tablet statistics.
// This is only included in the report if the replica is a LEADER.
optional tablet.ReportedTabletStatsPB stats = 7;
}
// Sent by the tablet server to report the set of tablets hosted by that TS.
message TabletReportPB {
// If false, then this is a full report, and any prior information about
// tablets hosted by this server should be dropped.
required bool is_incremental = 1;
// Tablets for which to update information. If 'is_incremental' is false,
// then this is the full set of tablets on the server, and any tablets
// which the master is aware of but not listed in this protobuf should
// be assumed to have been removed from this server.
repeated ReportedTabletPB updated_tablets = 2;
// Tablet IDs which the tablet server has removed and should no longer be
// considered hosted here. This will always be empty in a non-incremental
// report.
repeated bytes removed_tablet_ids = 3;
// Every time the TS generates a tablet report, it creates a sequence
// number. This can be useful in debugging, and also determining which
// changes have not yet been reported to the master.
// The first tablet report (non-incremental) is sequence number 0.
required int32 sequence_number = 4;
}
message ReportedTabletUpdatesPB {
required bytes tablet_id = 1;
optional string state_msg = 2;
}
// Sent by the Master in response to the TS tablet report (part of the heartbeats)
message TabletReportUpdatesPB {
repeated ReportedTabletUpdatesPB tablets = 1;
}
// Heartbeat sent from the tablet-server to the master
// to establish liveness and report back any status changes.
message TSHeartbeatRequestPB {
required TSToMasterCommonPB common = 1;
// Sent upon start-up of the TS, or in response to 'needs_reregister' on a heartbeat
// response.
optional ServerRegistrationPB registration = 2;
// Sent when the tablet information has changed, or in response to
// 'needs_full_tablet_report'.
optional TabletReportPB tablet_report = 3;
// TODO; add a heartbeat sequence number?
// TODO: perhaps add some very basic metrics reporting here, like
// free space, reqs/sec, etc?
// The number of tablets that are BOOTSTRAPPING or RUNNING.
// Used by the master to determine load when creating new tablet replicas.
optional int32 num_live_tablets = 4;
// If the tablet server needs its certificate signed, the CSR
// in DER format.
optional bytes csr_der = 5;
// The most recently known TSK sequence number. Allows the master to
// selectively notify the tablet server of more recent TSKs.
optional int64 latest_tsk_seq_num = 6;
// Replica management parameters that the tablet server is running with.
// This field is set only if the registration field is present.
optional consensus.ReplicaManagementInfoPB replica_management_info = 7;
// The number of tablets that are BOOTSTRAPPING or RUNNING in each dimension.
// Used by the master to determine load when creating new tablet replicas
// based on dimension.
map<string, int32> num_live_tablets_by_dimension = 8;
}
message TSHeartbeatResponsePB {
optional MasterErrorPB error = 1;
// As with most other master RPC responses (e.g.,
// ListTablesResponsePB), all fields below are optional as they may
// not be set if there is an error.
optional NodeInstancePB master_instance = 2;
// Indicates that the server which heartbeated needs to re-register
// with the master -- i.e send a heartbeat with the 'registration'
// filled in.
optional bool needs_reregister = 3 [ default = false ];
optional bool needs_full_tablet_report = 4 [ default = false ];
// Sent when the master receives a TabletReport
optional TabletReportUpdatesPB tablet_report = 5;
// Specify whether or not the node is the leader master.
optional bool leader_master = 6;
// If the heartbeat request had a CSR, then the successfully
// signed certificate will be returned in DER format.
optional bytes signed_cert_der = 7;
// Any CA certs used by the cluster, currently only included when
// 'signed_cert_der' is also sent (however the tablet server will always check
// for all heartbeat responses). Currently the master only uses one cert, but
// we may support rolling this cert in the future, so tablet servers should
// add all returned certs to their trusted CA list.
//
// NOTE: this is not necessarily a "certificate chain" but rather a set of
// independent certs to be trusted. They may or may not have any signing
// relationship between them.
repeated bytes ca_cert_der = 8;
// Token signing keys which the tablet server should begin trusting.
repeated security.TokenSigningPublicKeyPB tsks = 9;
}
//////////////////////////////
// GetTabletLocations
//////////////////////////////
message TabletLocationsPB {
// DEPRECATED.
// TODO: new clients should prefer the 'Interned' type below.
// Remove 'ReplicaPB' when we stop using it internally.
message DEPRECATED_ReplicaPB {
required TSInfoPB ts_info = 1;
required consensus.RaftPeerPB.Role role = 2;
optional string dimension_label = 3;
}
message InternedReplicaPB {
// Index into the 'ts_infos' list in the top-level RPC response.
required uint32 ts_info_idx = 1;
required consensus.RaftPeerPB.Role role = 2;
optional string dimension_label = 3;
}
required bytes tablet_id = 1;
// DEPRECATED.
optional bytes start_key = 2;
optional bytes end_key = 3;
optional PartitionPB partition = 6;
// DEPRECATED.
// Used only if interned replicas are not supported by client.
repeated DEPRECATED_ReplicaPB DEPRECATED_replicas = 4;
// More efficient representation of replicas: instead of duplicating the TSInfoPB
// in each tablet location, instead we just encode indexes into a list of TSInfoPB
// which is serialized in the top-level response.
//
// Used when supported by client.
repeated InternedReplicaPB interned_replicas = 7;
// DEPRECATED. Still set by servers, but should be ignored by clients.
optional bool DEPRECATED_stale = 5;
}
// Info about a single tablet server, returned to the client as part
// of the GetTabletLocations response. This can be used on the client
// to update the local cache of where each TS UUID is located. In
// the future we may also want to transmit software version info,
// load info, topology, etc.
message TSInfoPB {
required bytes permanent_uuid = 1;
repeated HostPortPB rpc_addresses = 2;
optional string location = 3;
// The path of a UNIX domain socket where the server is listening.
// An '@' prefix indicates the abstract namespace. May be missing
// if this feature is not enabled.
optional string unix_domain_socket_path = 4;
}
// Selector to specify policy for listing tablet replicas in
// GetTableLocationResponsePB and GetTabletLocationsResponsePB.
enum ReplicaTypeFilter {
UNKNOWN = 999;
ANY_REPLICA = 0;
VOTER_REPLICA = 1;
}
message GetTabletLocationsRequestPB {
// The tablet IDs about which to fetch info.
repeated bytes tablet_ids = 1;
// What type of tablet replicas to include in the response.
optional ReplicaTypeFilter replica_type_filter = 2 [ default = VOTER_REPLICA ];
// Whether the response should use the 'interned_replicas' field.
optional bool intern_ts_infos_in_response = 3 [ default = false ];
}
message GetTabletLocationsResponsePB {
optional MasterErrorPB error = 1;
repeated TabletLocationsPB tablet_locations = 2;
// Used if 'intern_ts_infos_in_response' was requested.
// See InternedReplicaPB above.
repeated TSInfoPB ts_infos = 4;
message Error {
required bytes tablet_id = 1;
required AppStatusPB status = 2;
}
repeated Error errors = 3;
}
// ============================================================================
// Catalog
// ============================================================================
message CreateTableRequestPB {
required string name = 1;
required SchemaPB schema = 2;
// repeated bytes pre_split_keys = 3;
// repeated PartialRowPB split_rows = 5;
// Holds either the split rows or the range bounds (or both) of the table.
optional RowOperationsPB split_rows_range_bounds = 6;
// Holds the table's partition schema, table-wide hash bucket schema, and,
// optionally, custom hash bucket schemas for each range. The latter is
// populated only when the 'split_rows_range_bounds' field above specifies
// range bounds, and must not be present if any split rows are specified.
optional PartitionSchemaPB partition_schema = 7;
// repeated PartitionSchemaPB.PerRangeHashBucketSchemasPB range_hash_schemas = 12;
// Number of replicas for a partition/tablet, a.k.a. table's replication
// factor. All tablets of the same table has same replication factor.
optional int32 num_replicas = 4;
// If set, uses the provided value as the table owner when creating the table.
// Otherwise, the owner is defaulted to the user that created the table.
optional string owner = 8;
// The table's extra configuration properties.
map<string, string> extra_configs = 9;
// The dimension label for tablets that were created during table creation. Used for
// dimension-specific placement of tablet replicas corresponding to the partitions of
// the newly created table.
optional string dimension_label = 10;
// The table type. If not set, it is assumed this table is a user-defined
// table, rather than a system table.
optional TableTypePB table_type = 11;
// The comment on the table.
optional string comment = 13;
}
message CreateTableResponsePB {
// The error, if an error occurred with this request.
optional MasterErrorPB error = 1;
optional bytes table_id = 2;
}
message IsCreateTableDoneRequestPB {
required TableIdentifierPB table = 1;
}
message IsCreateTableDoneResponsePB {
// The error, if an error occurred with this request.
optional MasterErrorPB error = 1;
// true if the create operation is completed, false otherwise
optional bool done = 3;
}
message DeleteTableRequestPB {
required TableIdentifierPB table = 1;
// Whether to apply the deletion to external catalogs, such as the Hive Metastore,
// which the Kudu master has been configured to integrate with.
optional bool modify_external_catalogs = 2 [default = true];
// Reserve seconds after the table has been deleted.
optional uint32 reserve_seconds = 3;
}
message DeleteTableResponsePB {
// The error, if an error occurred with this request.
optional MasterErrorPB error = 1;
}
message RecallDeletedTableRequestPB {
required TableIdentifierPB table = 1;
// If this field is set, that's the name for the recalled table.
// Otherwise, the recalled table will use the original table name.
optional string new_table_name = 2;
}
message RecallDeletedTableResponsePB {
// The error, if an error occurred with this request.
optional MasterErrorPB error = 1;
}
message ListTablesRequestPB {
// When used, only returns tables that satisfy a substring match on name_filter.
optional string name_filter = 1;
// The filter for the table types to return. If not set or empty, it is
// interpreted as if it were set to [TableTypePB::DEFAULT_TABLE], meaning
// to include only user-defined tables.
repeated TableTypePB type_filter = 2;
// Set this field 'true' to include information on the partition backed by
// each tablet in the result list.
optional bool list_tablet_with_partition = 3 [default = false];
// Use to select the tables type for display.
// Only show regular tables if false.
// Only show soft_deleted tables if true.
optional bool show_soft_deleted = 4;
}
message ListTablesResponsePB {
// The error, if an error occurred with this request.
optional MasterErrorPB error = 1;
message TabletWithPartition {
optional string tablet_id = 1;
optional PartitionPB partition = 2;
}
message TableInfo {
required bytes id = 1;
required string name = 2;
optional uint64 live_row_count = 3;
optional int32 num_tablets = 4;
optional int32 num_replicas = 5;
repeated TabletWithPartition tablet_with_partition = 6;
}
repeated TableInfo tables = 2;
}
message GetTableStatisticsRequestPB {
required TableIdentifierPB table = 1;
}
message GetTableStatisticsResponsePB {
// The error, if an error occurred with this request.
optional MasterErrorPB error = 1;
// The table statistics from table metrics.
optional int64 on_disk_size = 2;
optional int64 live_row_count = 3;
// The table limit
optional int64 disk_size_limit = 4;
optional int64 row_count_limit = 5;
}
// This data structure is used to specify a table's partition key.
// That's used to find partitions (i.e. tablets) that cover the key range span
// between two partition keys.
message PartitionKeyPB {
// The range part of the encoded key.
optional bytes range_key = 1 [(kudu.REDACT) = true];
// The hash part of the encoded key.
optional bytes hash_key = 2;
}
message GetTableLocationsRequestPB {
required TableIdentifierPB table = 1;
// These two fields are used to specify key range span if the table
// has a table-wide hash schema. Otherwise, neither of these two fields
// should not be present: use 'key_start' and 'key_end' fields below
// to specify key range information instead.
optional bytes partition_key_start = 3 [(kudu.REDACT) = true];
optional bytes partition_key_end = 4 [(kudu.REDACT) = true];
optional uint32 max_returned_locations = 5 [ default = 10 ];
// What type of tablet replicas to include in the
// 'GetTableLocationsResponsePB::tablet_locations' response field.
optional ReplicaTypeFilter replica_type_filter = 6 [ default = VOTER_REPLICA ];
// Whether the response should use the 'interned_replicas' field.
optional bool intern_ts_infos_in_response = 7 [ default = false ];
// Partition key range, start and end: these fields make the
// 'partition_key_start' and 'partition_key_end' fields above obsolete.
// It's crucial to specify these (or at least one of these if the range is
// unbounded) when the table has custom hash schemas per range, otherwise
// the catalog manager responds back with an error. Even in case of
// table-wide hash schema the clients compiled with this proto specification
// are encouraged to use these fields instead of legacy 'partition_key_start'
// and 'partition_key_end'.
optional PartitionKeyPB key_start = 8;
optional PartitionKeyPB key_end = 9;
}
// The response to a GetTableLocations RPC. The master guarantees that:
//
// * The response contains a location for all tablets in the requested range,
// limited by the request's 'max_returned_locations'.
// * The tablet locations are returned in sorted order by the partition key range.
// * If *any* tablet in the response is not running, then the entire response
// will fail with MasterErrorPB::TABLET_NOT_RUNNING, and the tablet_locations
// field will be empty.
// * A gap between the partition key ranges of consecutive tablets indicates a
// non-covered partition range.
// * If the request's start partition key falls in a non-covered partition
// range, the response will contain the tablet immediately before the
// non-covered range, if it exists.
message GetTableLocationsResponsePB {
// The error, if an error occurred with this request.
optional MasterErrorPB error = 1;
repeated TabletLocationsPB tablet_locations = 2;
// Used if 'intern_ts_infos_in_response' was requested.
// See InternedReplicaPB above.
repeated TSInfoPB ts_infos = 4;
// If the client caches table locations, the entries should not live longer
// than this timeout. Defaults to one hour.
optional uint32 ttl_millis = 3 [default = 3600000];
}
message AlterTableRequestPB {
enum StepType {
UNKNOWN = 0;
ADD_COLUMN = 1;
DROP_COLUMN = 2;
RENAME_COLUMN = 3;
ALTER_COLUMN = 4;
ADD_RANGE_PARTITION = 5;
DROP_RANGE_PARTITION = 6;
}
message AddColumn {
// The schema to add.
// NOTE: the 'id' field of the schema should not be provided here --
// the server will assign an ID.
required ColumnSchemaPB schema = 1;
}
message DropColumn {
// Name of the column to drop.
required string name = 1;
}
// This should eventually be deprecated in favor of AlterColumn.
message RenameColumn {
// Name of the column to rename;
required string old_name = 1;
required string new_name = 2;
}
message AlterColumn {
optional ColumnSchemaDeltaPB delta = 1;
}
message AddRangePartition {
// A structure to define range-specific hash schema. This separate type
// exists to distinguish from an empty hash schema (i.e. no hash bucketing)
// and the absence of range-specific hash schema when a range partition
// uses the table-wide hash schema instead. Otherwise, using a field of
// repeated HashBucketSchemaPB wouldn't allow to tell between those cases.
message CustomHashSchema {
repeated PartitionSchemaPB.HashBucketSchemaPB hash_schema = 1;
}
// A set of row operations containing the lower and upper range bound for
// the range partition to add or drop.
optional RowOperationsPB range_bounds = 1;
// The dimension label for the tablet. Used for dimension-specific placement
// of the tablet's replicas.
optional string dimension_label = 2;
// The custom hash partition schema for the range, if specified. If absent,
// the range uses table-wide hash schema.
optional CustomHashSchema custom_hash_schema = 3;
}
message DropRangePartition {
// A set of row operations containing the lower and upper range bound for
// the range partition to add or drop.
optional RowOperationsPB range_bounds = 1;
}
message Step {
optional StepType type = 1 [ default = UNKNOWN ];
// Exactly one of the following must be set, based on 'type'
optional AddColumn add_column = 2;
optional DropColumn drop_column = 3;
optional RenameColumn rename_column = 4;
optional AddRangePartition add_range_partition = 5;
optional DropRangePartition drop_range_partition = 6;
optional AlterColumn alter_column = 7;
}
required TableIdentifierPB table = 1;
repeated Step alter_schema_steps = 2;
optional string new_table_name = 3;
// The table schema to use when decoding the range bound row operations. Only
// necessary when partitions are being added or dropped.
optional SchemaPB schema = 4;
// Whether to apply the alteration to external catalogs, such as the Hive Metastore,
// which the Kudu master has been configured to integrate with.
optional bool modify_external_catalogs = 5 [default = true];
map<string, string> new_extra_configs = 6;
optional string new_table_owner = 7;
optional int64 disk_size_limit = 8;
optional int64 row_count_limit = 9;
optional string new_table_comment = 10;
optional int32 num_replicas = 11;
}
message AlterTableResponsePB {
// The error, if an error occurred with this request.
optional MasterErrorPB error = 1;
// The schema version of the altered table.
optional uint32 schema_version = 2;
// The table ID of the altered table.
optional bytes table_id = 3;
}
message IsAlterTableDoneRequestPB {
required TableIdentifierPB table = 1;
}
message IsAlterTableDoneResponsePB {
// The error, if an error occurred with this request.
optional MasterErrorPB error = 1;
// this is the current schema, or the 'new' schema version if an alter is in progress
optional uint32 schema_version = 2;
// true if the alter operation is completed, false otherwise
optional bool done = 3;
}
message GetTableSchemaRequestPB {
required TableIdentifierPB table = 1;
}
message GetTableSchemaResponsePB {
// create_table_done (field number 6) was removed in Kudu 1.6.
reserved 6;
// The error, if an error occurred with this request.
optional MasterErrorPB error = 1;
// This is the schema that every TS should be able to understand
// if your alter is keeping the schema compatible.
// In case of an alter table in progress, this is the previous schema;
// otherwise it is the latest schema.
optional SchemaPB schema = 2;
// The table's partition schema.
optional PartitionSchemaPB partition_schema = 5;
optional int32 num_replicas = 3;
// The ID of the table.
optional bytes table_id = 4;
// The table name.
optional string table_name = 7;
// A token that the client can use to authorize further access to the table.
// A token can always be expected with this response, unless the response is
// from an older version of Kudu or unless the request resulted in an error.
optional security.SignedTokenPB authz_token = 8;
// The table's extra configuration properties.
map<string, string> extra_configs = 9;
// The user that owns the table.
optional string owner = 10;
// The comment on the table.
optional string comment = 11;
}
message ConnectToMasterRequestPB {
}
message ConnectToMasterResponsePB {
// Set if there is an error.
optional MasterErrorPB error = 1;
// The current role of the master.
optional consensus.RaftPeerPB.Role role = 2;
// Any CA certs used by the cluster. Currently the master only uses
// one cert, but we may support rolling this cert in the future, so
// clients should add all returned certs to their trusted CA list.
//
// NOTE: this is not necessarily a "certificate chain" but rather a set of
// independent certs to be trusted. They may or may not have any signing
// relationship between them.
repeated bytes ca_cert_der = 3;
// If the client requested an authentication token, and security is
// enabled on the cluster, the master returns a signed authn token.
optional security.SignedTokenPB authn_token = 4;
// The hosts and ports of the masters in this cluster.
//
// NOTE: Added in Kudu 1.6.
//
// NOTE: it is likely, but not guaranteed that the hostnames advertised here
// will be usable by all clients. Client implementations should not use this
// field for "discovery" of other masters, since then it's likely that
// users will configure applications to only talk to one the masters in
// an HA setup. If that master then fails, the applications would go
// offline.
repeated HostPortPB master_addrs = 5;
// If the master is configured with the Hive Metastore integration enabled,
// this field will include the configuration options.
optional HiveMetastoreConfig hms_config = 6;
// The location of the client assigned by the master.
optional string client_location = 7;
// The cluster ID of this master.
optional string cluster_id = 8;
}
// Hive Metastore integration options and configuration.
message HiveMetastoreConfig {
// Address(es) of the Hive Metastore instance(s).
//
// For more info see the Kudu master --hive_metastore_uris flag for more info,
// or the Hive Metastore hive.metastore.uris configuration.
optional string hms_uris = 1;
// Whether the Hive Metastore instance uses SASL (Kerberos) security.
//
// For more info see the Kudu master --hive_metastore_sasl_enabled flag, or
// the Hive Metastore hive.metastore.sasl.enabled configuration.
optional bool hms_sasl_enabled = 2;
// An ID which uniquely identifies the Hive Metastore instance.
//
// NOTE: this is provided on a best-effort basis, as not all Hive Metastore
// versions which Kudu is compatible with include the necessary APIs. See
// HIVE-16452 for more info.
optional string hms_uuid = 3;
}
// ============================================================================
// Administration/monitoring
// ============================================================================
message ListTabletServersRequestPB {
// Whether or not to include the tserver states.
// Note: this may include states of tservers that haven't been registered and
// thus don't have a complete response Entries. In such cases, the Entries
// will be returned with bogus info w.r.t registration and heartbeating.
optional bool include_states = 1;
}
message ListTabletServersResponsePB {
optional MasterErrorPB error = 1;
message Entry {
required NodeInstancePB instance_id = 1;
optional ServerRegistrationPB registration = 2;
optional int32 millis_since_heartbeat = 3;
optional string location = 4;
optional TServerStatePB state = 5;
}
repeated Entry servers = 2;
}
// Representation of the state of a tablet server.
// TODO(KUDU-1827): add state for decommissioning.
enum TServerStatePB {
// Default value for backwards compatibility.
UNKNOWN_STATE = 0;
// No state for the tserver.
NONE = 1;
// New replicas are not added to the tserver, and failed replicas on the
// tserver are not re-replicated.
MAINTENANCE_MODE = 2;
}
message TServerStateChangePB {
// Representation of a change in tserver state.
enum StateChange {
// Default value for backwards compatibility.
UNKNOWN_STATE_CHANGE = 0;
// Transitions from having no state to being in maintenance mode. If the
// tserver is already in maintenance mode, this is a no-op.
ENTER_MAINTENANCE_MODE = 1;
// Transitions from being in maintenance mode to having no tserver state. If
// the tserver already has no state, this is a no-op.
EXIT_MAINTENANCE_MODE = 2;
}
// The tserver UUID on which to apply the state change.
optional string uuid = 1;
// The change in tserver state.
optional StateChange change = 2;
}
message ChangeTServerStateRequestPB {
// TODO(awong): consider setting tserver state in batches.
optional TServerStateChangePB change = 1;
enum HandleMissingTS {
UNKNOWN_HANDLING = 0;
ALLOW_MISSING_TSERVER = 1;
DONT_ALLOW_MISSING_TSERVER = 2;
}
// If set to ALLOW_MISSING_TSERVER, will allow setting the state of the
// tserver even if it hasn't been registered with the master and there is no
// existing state associated with that server.
optional HandleMissingTS handle_missing_tserver = 2 [default = DONT_ALLOW_MISSING_TSERVER];
}
message ChangeTServerStateResponsePB {
optional MasterErrorPB error = 1;
}
message AddMasterRequestPB {
optional HostPortPB rpc_addr = 1;
}
message AddMasterResponsePB {
optional MasterErrorPB error = 1;
}
message RemoveMasterRequestPB {
// HostPort of the master to be removed
optional HostPortPB rpc_addr = 1;
// Optional UUID of the master to be removed.
optional string master_uuid = 2;
}
message RemoveMasterResponsePB {
optional MasterErrorPB error = 1;
}
// GetMasterRegistrationRequest/Response: get the instance id and
// HTTP/RPC addresses for this Master server.
message GetMasterRegistrationRequestPB {
}
message GetMasterRegistrationResponsePB {
// Node instance information is always set.
required NodeInstancePB instance_id = 1;
// These fields are optional, as they won't be set if there's an
// error retrieving the host/port information.
optional ServerRegistrationPB registration = 2;
// This server's role in the consensus configuration.
optional consensus.RaftPeerPB.Role role = 3;
// Set if there an error retrieving the registration information.
optional MasterErrorPB error = 4;
// The unique cluster ID of the cluster this server belongs too.
optional string cluster_id = 5;
// This server's membership type in the consensus configuration.
optional consensus.RaftPeerPB.MemberType member_type = 6;
}
// ListMastersRequest/Response: get information about all of the known
// master servers, including this node.
message ListMastersRequestPB {
}
message ListMastersResponsePB {
// An entry for each individual master server.
repeated ServerEntryPB masters = 1;
// DEPRECATED, use 'error' field.
optional AppStatusPB DEPRECATED_error = 2;
// Set if there an error retrieving the registration information.
optional MasterErrorPB error = 3;
}
// ReplaceTabletRequest/Response: replace a tablet with a new, empty one.
// The replaced tablet is deleted.
message ReplaceTabletRequestPB {
optional bytes tablet_id = 1;
}
message ReplaceTabletResponsePB {
// Set if there is an error replacing the tablet.
optional MasterErrorPB error = 1;
// The tablet id of the replacement tablet.
optional bytes replacement_tablet_id = 2;
}
// RefreshAuthzCache{Request/Result}PB: refresh the authz privileges cache,
// download new policies (does not guarantee the old cache is invalidated).
message RefreshAuthzCacheRequestPB {
}
message RefreshAuthzCacheResponsePB {
optional MasterErrorPB error = 1;
}
// RemoveTabletServerRequest/Response: remove a tablet server from master's
// in-memory map and persisted catalog.
message UnregisterTServerRequestPB {
// The tserver UUID to be unregistered.
optional string uuid = 1;
// Whether to return an error in case the tserver is not presumed to be dead,
// per --tserver_unresponsive_timeout_ms.
// Disable this by default to make sure the tserver has been brought down.
optional bool force_unregister_live_tserver = 2 [default = false];
}
message UnregisterTServerResponsePB {
optional MasterErrorPB error = 1;
}
enum MasterFeatures {
UNKNOWN_FEATURE = 0;
// The master supports creating tables with non-covering range partitions.
RANGE_PARTITION_BOUNDS = 1;
// The master supports adding and dropping range partitions.
ADD_DROP_RANGE_PARTITIONS = 2;
// The master supports the 'ConnectToMaster' RPC.
CONNECT_TO_MASTER = 3;
// The catalog manager supports different replica management schemes
// (see "kudu/consensus/replica_management.proto"). At least, both the
// EVICT_FIRST (a.k.a. 3-2-3) and the PREPARE_REPLACEMENT_BEFORE_EVICTION
// (a.k.a. 3-4-3) schemes.
REPLICA_MANAGEMENT = 4;
// The master supports generating and dispensing authz tokens.
GENERATE_AUTHZ_TOKEN = 5;
// The master supports dynamic addition/removal of masters
DYNAMIC_MULTI_MASTER = 6;
// Whether the cluster supports INSERT_IGNORE, DELETE_IGNORE, and UPDATE_IGNORE operations.
// Though this is technically a tserver feature, it's unreasonable to check if every
// tablet server supports this feature. Instead we use the master as a proxy.
IGNORE_OPERATIONS = 7;
// Whether master supports tables with range-specific hash schemas.
RANGE_SPECIFIC_HASH_SCHEMA = 8;
// Similar to IGNORE_OPERATIONS, but this is for UPSERT_IGNORE specifically.
UPSERT_IGNORE = 9;
// Whether master supports immutable attribute on column schema.
IMMUTABLE_COLUMN_ATTRIBUTE = 10;
}
service MasterService {
// Set the default authz method to something invalid, so that if
// we forget to set the option on a new RPC call, we'll get a build
// failure.
option (kudu.rpc.default_authz_method) = "MUST_SET_AUTHZ_PER_RPC";
// TS->Master RPCs
// ------------------------------------------------------------
rpc TSHeartbeat(TSHeartbeatRequestPB) returns (TSHeartbeatResponsePB) {
option (kudu.rpc.authz_method) = "AuthorizeServiceUser";
}
// Client->Master RPCs
// ------------------------------------------------------------
// Used only by Kudu 1.3 and later.
rpc ConnectToMaster(ConnectToMasterRequestPB) returns (ConnectToMasterResponsePB) {
option (kudu.rpc.authz_method) = "AuthorizeClientOrServiceUser";
}
rpc GetTabletLocations(GetTabletLocationsRequestPB) returns (GetTabletLocationsResponsePB) {
option (kudu.rpc.authz_method) = "AuthorizeClientOrServiceUser";
}
rpc CreateTable(CreateTableRequestPB) returns (CreateTableResponsePB) {
option (kudu.rpc.authz_method) = "AuthorizeClientOrServiceUser";
}
rpc IsCreateTableDone(IsCreateTableDoneRequestPB) returns (IsCreateTableDoneResponsePB) {
option (kudu.rpc.authz_method) = "AuthorizeClientOrServiceUser";
}
rpc DeleteTable(DeleteTableRequestPB) returns (DeleteTableResponsePB) {
option (kudu.rpc.authz_method) = "AuthorizeClient";
}
rpc RecallDeletedTable(RecallDeletedTableRequestPB) returns (RecallDeletedTableResponsePB) {
option (kudu.rpc.authz_method) = "AuthorizeClient";
}
rpc AlterTable(AlterTableRequestPB) returns (AlterTableResponsePB) {
option (kudu.rpc.authz_method) = "AuthorizeClientOrServiceUser";
}
rpc IsAlterTableDone(IsAlterTableDoneRequestPB) returns (IsAlterTableDoneResponsePB) {
option (kudu.rpc.authz_method) = "AuthorizeClientOrServiceUser";
}
rpc ListTables(ListTablesRequestPB) returns (ListTablesResponsePB) {
option (kudu.rpc.authz_method) = "AuthorizeClientOrServiceUser";
}
rpc GetTableStatistics(GetTableStatisticsRequestPB) returns (GetTableStatisticsResponsePB) {
option (kudu.rpc.authz_method) = "AuthorizeClientOrServiceUser";
}
rpc GetTableLocations(GetTableLocationsRequestPB) returns (GetTableLocationsResponsePB) {
option (kudu.rpc.authz_method) = "AuthorizeClientOrServiceUser";
}
rpc GetTableSchema(GetTableSchemaRequestPB) returns (GetTableSchemaResponsePB) {
option (kudu.rpc.authz_method) = "AuthorizeClientOrServiceUser";
}
// Administrative/monitoring RPCs
// ------------------------------------------------------------
rpc ListTabletServers(ListTabletServersRequestPB) returns (ListTabletServersResponsePB) {
option (kudu.rpc.authz_method) = "AuthorizeClient";
}
rpc ListMasters(ListMastersRequestPB) returns (ListMastersResponsePB) {
option (kudu.rpc.authz_method) = "AuthorizeClient";
}
rpc Ping(PingRequestPB) returns (PingResponsePB) {
option (kudu.rpc.authz_method) = "AuthorizeClientOrServiceUser";
}
rpc ReplaceTablet(ReplaceTabletRequestPB) returns (ReplaceTabletResponsePB) {
option (kudu.rpc.authz_method) = "AuthorizeSuperUser";
}
rpc RefreshAuthzCache(RefreshAuthzCacheRequestPB) returns (RefreshAuthzCacheResponsePB) {
option (kudu.rpc.authz_method) = "AuthorizeSuperUser";
}
rpc ChangeTServerState(ChangeTServerStateRequestPB) returns
(ChangeTServerStateResponsePB) {
option (kudu.rpc.authz_method) = "AuthorizeSuperUser";
}
// Add a new master to the existing cluster.
rpc AddMaster(AddMasterRequestPB) returns (AddMasterResponsePB) {
// NOTE: the master service may send this RPC as a part of its automated
// addition of masters to existing clusters, hence the inclusion of the
// service user.
option (kudu.rpc.authz_method) = "AuthorizeClientOrServiceUser";
}
// Remove a master from the existing cluster.
rpc RemoveMaster(RemoveMasterRequestPB) returns (RemoveMasterResponsePB) {
option (kudu.rpc.authz_method) = "AuthorizeSuperUser";
}
rpc UnregisterTServer(UnregisterTServerRequestPB) returns (UnregisterTServerResponsePB) {
option (kudu.rpc.authz_method) = "AuthorizeSuperUser";
}
// Master->Master RPCs
// ------------------------------------------------------------
// NOTE: this RPC is also used by Kudu client <= 1.2 when first connecting to the
// cluster.
rpc GetMasterRegistration(GetMasterRegistrationRequestPB) returns
(GetMasterRegistrationResponsePB) {
option (kudu.rpc.authz_method) = "AuthorizeClientOrServiceUser";
}
}