HBASE-23232 Remove rsgroup profile from pom.xml of hbase-assembly (#779)
[hbase.git] / hbase-protocol-shaded / src / main / protobuf / Client.proto
blob810aaaa393a55aa411de0623fb094246b52147fc
1 /**
2  * Licensed to the Apache Software Foundation (ASF) under one
3  * or more contributor license agreements.  See the NOTICE file
4  * distributed with this work for additional information
5  * regarding copyright ownership.  The ASF licenses this file
6  * to you under the Apache License, Version 2.0 (the
7  * "License"); you may not use this file except in compliance
8  * with the License.  You may obtain a copy of the License at
9  *
10  *     http://www.apache.org/licenses/LICENSE-2.0
11  *
12  * Unless required by applicable law or agreed to in writing, software
13  * distributed under the License is distributed on an "AS IS" BASIS,
14  * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15  * See the License for the specific language governing permissions and
16  * limitations under the License.
17  */
18 syntax = "proto2";
19 // This file contains protocol buffers that are used for Client service.
20 package hbase.pb;
22 option java_package = "org.apache.hadoop.hbase.shaded.protobuf.generated";
23 option java_outer_classname = "ClientProtos";
24 option java_generic_services = true;
25 option java_generate_equals_and_hash = true;
26 option optimize_for = SPEED;
28 import "HBase.proto";
29 import "Filter.proto";
30 import "Cell.proto";
31 import "Comparator.proto";
32 import "MapReduce.proto";
34 /**
35  * The protocol buffer version of Authorizations.
36  */
37 message Authorizations {
38   repeated string label = 1;
41 /**
42  * The protocol buffer version of CellVisibility.
43  */
44 message CellVisibility {
45   required string expression = 1;
48 /**
49  * Container for a list of column qualifier names of a family.
50  */
51 message Column {
52   required bytes family = 1;
53   repeated bytes qualifier = 2;
56 /**
57  * Consistency defines the expected consistency level for an operation.
58  */
59 enum Consistency {
60   STRONG   = 0;
61   TIMELINE = 1;
64 /**
65  * The protocol buffer version of Get.
66  * Unless existence_only is specified, return all the requested data
67  * for the row that matches exactly.
68  */
69 message Get {
70   required bytes row = 1;
71   repeated Column column = 2;
72   repeated NameBytesPair attribute = 3;
73   optional Filter filter = 4;
74   optional TimeRange time_range = 5;
75   optional uint32 max_versions = 6 [default = 1];
76   optional bool cache_blocks = 7 [default = true];
77   optional uint32 store_limit = 8;
78   optional uint32 store_offset = 9;
80   // The result isn't asked for, just check for
81   // the existence.
82   optional bool existence_only = 10 [default = false];
84   // If the row to get doesn't exist, return the
85   // closest row before. Deprecated. No longer used!
86   // Since hbase-2.0.0 but left in place so can test
87   // for Gets with this set and throw Exception.
88   optional bool closest_row_before = 11 [default = false];
90   optional Consistency consistency = 12 [default = STRONG];
91   repeated ColumnFamilyTimeRange cf_time_range = 13;
92   optional bool load_column_families_on_demand = 14; /* DO NOT add defaults to load_column_families_on_demand. */
95 message Result {
96   // Result includes the Cells or else it just has a count of Cells
97   // that are carried otherwise.
98   repeated Cell cell = 1;
99   // The below count is set when the associated cells are
100   // not part of this protobuf message; they are passed alongside
101   // and then this Message is just a placeholder with metadata.
102   // The count is needed to know how many to peel off the block of Cells as
103   // ours.  NOTE: This is different from the pb managed cell_count of the
104   // 'cell' field above which is non-null when the cells are pb'd.
105   optional int32 associated_cell_count = 2;
107   // used for Get to check existence only. Not set if existence_only was not set to true
108   //  in the query.
109   optional bool exists = 3;
111   // Whether or not the results are coming from possibly stale data
112   optional bool stale = 4 [default = false];
114   // Whether or not the entire result could be returned. Results will be split when
115   // the RPC chunk size limit is reached. Partial results contain only a subset of the
116   // cells for a row and must be combined with a result containing the remaining cells
117   // to form a complete result. The equivalent flag in o.a.h.h.client.Result is
118   // mayHaveMoreCellsInRow.
119   optional bool partial = 5 [default = false];
123  * The get request. Perform a single Get operation.
124  */
125 message GetRequest {
126   required RegionSpecifier region = 1;
127   required Get get = 2;
130 message GetResponse {
131   optional Result result = 1;
135  * Condition to check if the value of a given cell (row,
136  * family, qualifier) matches a value via a given comparator.
138  * Condition is used in check and mutate operations.
139  */
140 message Condition {
141   required bytes row = 1;
142   optional bytes family = 2;
143   optional bytes qualifier = 3;
144   optional CompareType compare_type = 4;
145   optional Comparator comparator = 5;
146   optional TimeRange time_range = 6;
147   optional Filter filter = 7;
152  * A specific mutation inside a mutate request.
153  * It can be an append, increment, put or delete based
154  * on the mutation type.  It can be fully filled in or
155  * only metadata present because data is being carried
156  * elsewhere outside of pb.
157  */
158 message MutationProto {
159   optional bytes row = 1;
160   optional MutationType mutate_type = 2;
161   repeated ColumnValue column_value = 3;
162   optional uint64 timestamp = 4;
163   repeated NameBytesPair attribute = 5;
164   optional Durability durability = 6 [default = USE_DEFAULT];
166   // For some mutations, a result may be returned, in which case,
167   // time range can be specified for potential performance gain
168   optional TimeRange time_range = 7;
169   // The below count is set when the associated cells are NOT
170   // part of this protobuf message; they are passed alongside
171   // and then this Message is a placeholder with metadata.  The
172   // count is needed to know how many to peel off the block of Cells as
173   // ours.  NOTE: This is different from the pb managed cell_count of the
174   // 'cell' field above which is non-null when the cells are pb'd.
175   optional int32 associated_cell_count = 8;
177   optional uint64 nonce = 9;
179   enum Durability {
180     USE_DEFAULT  = 0;
181     SKIP_WAL     = 1;
182     ASYNC_WAL    = 2;
183     SYNC_WAL     = 3;
184     FSYNC_WAL    = 4;
185   }
187   enum MutationType {
188     APPEND = 0;
189     INCREMENT = 1;
190     PUT = 2;
191     DELETE = 3;
192   }
194   enum DeleteType {
195     DELETE_ONE_VERSION = 0;
196     DELETE_MULTIPLE_VERSIONS = 1;
197     DELETE_FAMILY = 2;
198     DELETE_FAMILY_VERSION = 3;
199   }
201   message ColumnValue {
202     required bytes family = 1;
203     repeated QualifierValue qualifier_value = 2;
205     message QualifierValue {
206       optional bytes qualifier = 1;
207       optional bytes value = 2;
208       optional uint64 timestamp = 3;
209       optional DeleteType delete_type = 4;
210       optional bytes tags = 5;
211     }
212   }
216  * The mutate request. Perform a single Mutate operation.
218  * Optionally, you can specify a condition. The mutate
219  * will take place only if the condition is met.  Otherwise,
220  * the mutate will be ignored.  In the response result,
221  * parameter processed is used to indicate if the mutate
222  * actually happened.
223  */
224 message MutateRequest {
225   required RegionSpecifier region = 1;
226   required MutationProto mutation = 2;
227   optional Condition condition = 3;
228   optional uint64 nonce_group = 4;
231 message MutateResponse {
232   optional Result result = 1;
234   // used for mutate to indicate processed only
235   optional bool processed = 2;
239  * Instead of get from a table, you can scan it with optional filters.
240  * You can specify the row key range, time range, the columns/families
241  * to scan and so on.
243  * This scan is used the first time in a scan request. The response of
244  * the initial scan will return a scanner id, which should be used to
245  * fetch result batches later on before it is closed.
246  */
247 message Scan {
248   repeated Column column = 1;
249   repeated NameBytesPair attribute = 2;
250   optional bytes start_row = 3;
251   optional bytes stop_row = 4;
252   optional Filter filter = 5;
253   optional TimeRange time_range = 6;
254   optional uint32 max_versions = 7 [default = 1];
255   optional bool cache_blocks = 8 [default = true];
256   optional uint32 batch_size = 9;
257   optional uint64 max_result_size = 10;
258   optional uint32 store_limit = 11;
259   optional uint32 store_offset = 12;
260   optional bool load_column_families_on_demand = 13; /* DO NOT add defaults to load_column_families_on_demand. */
261   optional bool small = 14 [deprecated = true];
262   optional bool reversed = 15 [default = false];
263   optional Consistency consistency = 16 [default = STRONG];
264   optional uint32 caching = 17;
265   optional bool allow_partial_results = 18;
266   repeated ColumnFamilyTimeRange cf_time_range = 19;
267   optional uint64 mvcc_read_point = 20 [default = 0];
268   optional bool include_start_row = 21 [default = true];
269   optional bool include_stop_row = 22 [default = false];
270   enum ReadType {
271     DEFAULT = 0;
272     STREAM = 1;
273     PREAD = 2;
274   }
275   optional ReadType readType = 23 [default = DEFAULT];
276   optional bool need_cursor_result = 24 [default = false];
280  * A scan request. Initially, it should specify a scan. Later on, you
281  * can use the scanner id returned to fetch result batches with a different
282  * scan request.
284  * The scanner will remain open if there are more results, and it's not
285  * asked to be closed explicitly.
287  * You can fetch the results and ask the scanner to be closed to save
288  * a trip if you are not interested in remaining results.
289  */
290 message ScanRequest {
291   optional RegionSpecifier region = 1;
292   optional Scan scan = 2;
293   optional uint64 scanner_id = 3;
294   optional uint32 number_of_rows = 4;
295   optional bool close_scanner = 5;
296   optional uint64 next_call_seq = 6;
297   optional bool client_handles_partials = 7;
298   optional bool client_handles_heartbeats = 8;
299   optional bool track_scan_metrics = 9;
300   optional bool renew = 10 [default = false];
301   // if we have returned limit_of_rows rows to client, then close the scanner.
302   optional uint32 limit_of_rows = 11 [default = 0];
306 * Scan cursor to tell client where we are scanning.
308  */
309 message Cursor {
310   optional bytes row = 1;
314  * The scan response. If there are no more results, more_results will
315  * be false.  If it is not specified, it means there are more.
316  */
317 message ScanResponse {
318   // This field is filled in if we are doing cellblocks.  A cellblock is made up
319   // of all Cells serialized out as one cellblock BUT responses from a server
320   // have their Cells grouped by Result.  So we can reconstitute the
321   // Results on the client-side, this field is a list of counts of Cells
322   // in each Result that makes up the response.  For example, if this field
323   // has 3, 3, 3 in it, then we know that on the client, we are to make
324   // three Results each of three Cells each.
325   repeated uint32 cells_per_result = 1;
327   optional uint64 scanner_id = 2;
328   optional bool more_results = 3;
329   optional uint32 ttl = 4;
330   // If cells are not carried in an accompanying cellblock, then they are pb'd here.
331   // This field is mutually exclusive with cells_per_result (since the Cells will
332   // be inside the pb'd Result)
333   repeated Result results = 5;
334   optional bool stale = 6;
336   // This field is filled in if we are doing cellblocks. In the event that a row
337   // could not fit all of its cells into a single RPC chunk, the results will be
338   // returned as partials, and reconstructed into a complete result on the client
339   // side. This field is a list of flags indicating whether or not the result
340   // that the cells belong to is a partial result. For example, if this field
341   // has false, false, true in it, then we know that on the client side, we need to
342   // make another RPC request since the last result was only a partial.
343   repeated bool partial_flag_per_result = 7;
345   // A server may choose to limit the number of results returned to the client for
346   // reasons such as the size in bytes or quantity of results accumulated. This field
347   // will true when more results exist in the current region.
348   optional bool more_results_in_region = 8;
350   // This field is filled in if the server is sending back a heartbeat message.
351   // Heartbeat messages are sent back to the client to prevent the scanner from
352   // timing out. Seeing a heartbeat message communicates to the Client that the
353   // server would have continued to scan had the time limit not been reached.
354   optional bool heartbeat_message = 9;
356   // This field is filled in if the client has requested that scan metrics be tracked.
357   // The metrics tracked here are sent back to the client to be tracked together with
358   // the existing client side metrics.
359   optional ScanMetrics scan_metrics = 10;
361   // The mvcc read point which is used to open the scanner at server side. Client can
362   // make use of this mvcc_read_point when restarting a scanner to get a consistent view
363   // of a row.
364   optional uint64 mvcc_read_point = 11 [default = 0];
366   // If the Scan need cursor, return the row key we are scanning in heartbeat message.
367   // If the Scan doesn't need a cursor, don't set this field to reduce network IO.
368   optional Cursor cursor = 12;
372  * Atomically bulk load multiple HFiles (say from different column families)
373  * into an open region.
374  */
375 message BulkLoadHFileRequest {
376   required RegionSpecifier region = 1;
377   repeated FamilyPath family_path = 2;
378   optional bool assign_seq_num = 3;
379   optional DelegationToken fs_token = 4;
380   optional string bulk_token = 5;
381   optional bool copy_file = 6 [default = false];
382   repeated string cluster_ids = 7;
383   optional bool replicate = 8 [default = true];
385   message FamilyPath {
386     required bytes family = 1;
387     required string path = 2;
388   }
391 message BulkLoadHFileResponse {
392   required bool loaded = 1;
395 message DelegationToken {
396   optional bytes identifier = 1;
397   optional bytes password = 2;
398   optional string kind = 3;
399   optional string service = 4;
402 message PrepareBulkLoadRequest {
403   required TableName table_name = 1;
404   optional RegionSpecifier region = 2;
407 message PrepareBulkLoadResponse {
408   required string bulk_token = 1;
411 message CleanupBulkLoadRequest {
412   required string bulk_token = 1;
413   optional RegionSpecifier region = 2;
416 message CleanupBulkLoadResponse {
419 message CoprocessorServiceCall {
420   required bytes row = 1;
421   required string service_name = 2;
422   required string method_name = 3;
423   required bytes request = 4;
426 message CoprocessorServiceResult {
427   optional NameBytesPair value = 1;
430 message CoprocessorServiceRequest {
431   required RegionSpecifier region = 1;
432   required CoprocessorServiceCall call = 2;
435 message CoprocessorServiceResponse {
436   required RegionSpecifier region = 1;
437   required NameBytesPair value = 2;
440 // Either a Get or a Mutation
441 message Action {
442   // If part of a multi action, useful aligning
443   // result with what was originally submitted.
444   optional uint32 index = 1;
445   optional MutationProto mutation = 2;
446   optional Get get = 3;
447   optional CoprocessorServiceCall service_call = 4;
451  * Actions to run against a Region.
452  */
453 message RegionAction {
454   required RegionSpecifier region = 1;
455   // When set, run mutations as atomic unit.
456   optional bool atomic = 2;
457   repeated Action action = 3;
461 * Statistics about the current load on the region
463 message RegionLoadStats {
464   // Percent load on the memstore. Guaranteed to be positive, between 0 and 100.
465   optional int32 memStoreLoad = 1 [default = 0];
466   // Percent JVM heap occupancy. Guaranteed to be positive, between 0 and 100.
467   // We can move this to "ServerLoadStats" should we develop them.
468   optional int32 heapOccupancy = 2 [default = 0];
469   // Compaction pressure. Guaranteed to be positive, between 0 and 100.
470   optional int32 compactionPressure = 3 [default = 0];
473 message MultiRegionLoadStats{
474   repeated RegionSpecifier region = 1;
475   repeated RegionLoadStats stat = 2;
479  * Either a Result or an Exception NameBytesPair (keyed by
480  * exception name whose value is the exception stringified)
481  * or maybe empty if no result and no exception.
482  */
483 message ResultOrException {
484   // If part of a multi call, save original index of the list of all
485   // passed so can align this response w/ original request.
486   optional uint32 index = 1;
487   optional Result result = 2;
488   optional NameBytesPair exception = 3;
489   // result if this was a coprocessor service call
490   optional CoprocessorServiceResult service_result = 4;
491   // current load on the region
492   optional RegionLoadStats loadStats = 5 [deprecated=true];
496  * The result of a RegionAction.
497  */
498 message RegionActionResult {
499   repeated ResultOrException resultOrException = 1;
500   // If the operation failed globally for this region, this exception is set
501   optional NameBytesPair exception = 2;
505  * Execute a list of actions on a given region in order.
506  * Nothing prevents a request to contains a set of RegionAction on the same region.
507  * For this reason, the matching between the MultiRequest and the MultiResponse is not
508  *  done by the region specifier but by keeping the order of the RegionActionResult vs.
509  *  the order of the RegionAction.
510  */
511 message MultiRequest {
512   repeated RegionAction regionAction = 1;
513   optional uint64 nonceGroup = 2;
514   optional Condition condition = 3;
517 message MultiResponse {
518   repeated RegionActionResult regionActionResult = 1;
519   // used for mutate to indicate processed only
520   optional bool processed = 2;
521   optional MultiRegionLoadStats regionStatistics = 3;
525 service ClientService {
526   rpc Get(GetRequest)
527     returns(GetResponse);
529   rpc Mutate(MutateRequest)
530     returns(MutateResponse);
532   rpc Scan(ScanRequest)
533     returns(ScanResponse);
535   rpc BulkLoadHFile(BulkLoadHFileRequest)
536     returns(BulkLoadHFileResponse);
538   rpc PrepareBulkLoad(PrepareBulkLoadRequest)
539     returns (PrepareBulkLoadResponse);
541   rpc CleanupBulkLoad(CleanupBulkLoadRequest)
542     returns (CleanupBulkLoadResponse);
544   rpc ExecService(CoprocessorServiceRequest)
545     returns(CoprocessorServiceResponse);
547   rpc ExecRegionServerService(CoprocessorServiceRequest)
548     returns(CoprocessorServiceResponse);
550   rpc Multi(MultiRequest)
551     returns(MultiResponse);