RpcHeader.proto 7.0 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184
  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. /**
  19. * These .proto interfaces are private and stable.
  20. * Please see http://wiki.apache.org/hadoop/Compatibility
  21. * for what changes are allowed for a *stable* .proto interface.
  22. */
  23. syntax = "proto2";
  24. option java_package = "org.apache.hadoop.ipc.protobuf";
  25. option java_outer_classname = "RpcHeaderProtos";
  26. option java_generate_equals_and_hash = true;
  27. package hadoop.common;
  28. /**
  29. * This is the rpc request header. It is sent with every rpc call.
  30. *
  31. * The format of RPC call is as follows:
  32. * +--------------------------------------------------------------+
  33. * | Rpc length in bytes (4 bytes int) sum of next two parts |
  34. * +--------------------------------------------------------------+
  35. * | RpcRequestHeaderProto - serialized delimited ie has len |
  36. * +--------------------------------------------------------------+
  37. * | RpcRequest The actual rpc request |
  38. * | This request is serialized based on RpcKindProto |
  39. * +--------------------------------------------------------------+
  40. *
  41. */
  42. /**
  43. * RpcKind determine the rpcEngine and the serialization of the rpc request
  44. */
  45. enum RpcKindProto {
  46. RPC_BUILTIN = 0; // Used for built in calls by tests
  47. RPC_WRITABLE = 1; // Use WritableRpcEngine
  48. RPC_PROTOCOL_BUFFER = 2; // Use ProtobufRpcEngine
  49. }
  50. /**
  51. * Used to pass through the information necessary to continue
  52. * a trace after an RPC is made. All we need is the traceid
  53. * (so we know the overarching trace this message is a part of), and
  54. * the id of the current span when this message was sent, so we know
  55. * what span caused the new span we will create when this message is received.
  56. */
  57. message RPCTraceInfoProto {
  58. optional int64 traceId = 1; // parentIdHigh
  59. optional int64 parentId = 2; // parentIdLow
  60. optional bytes spanContext = 3; // Trace SpanContext
  61. }
  62. /**
  63. * Used to pass through the call context entry after an RPC is made.
  64. */
  65. message RPCCallerContextProto {
  66. required string context = 1;
  67. optional bytes signature = 2;
  68. }
  69. message RpcRequestHeaderProto { // the header for the RpcRequest
  70. enum OperationProto {
  71. RPC_FINAL_PACKET = 0; // The final RPC Packet
  72. RPC_CONTINUATION_PACKET = 1; // not implemented yet
  73. RPC_CLOSE_CONNECTION = 2; // close the rpc connection
  74. }
  75. optional RpcKindProto rpcKind = 1;
  76. optional OperationProto rpcOp = 2;
  77. required sint32 callId = 3; // a sequence number that is sent back in response
  78. required bytes clientId = 4; // Globally unique client ID
  79. // clientId + callId uniquely identifies a request
  80. // retry count, 1 means this is the first retry
  81. optional sint32 retryCount = 5 [default = -1];
  82. optional RPCTraceInfoProto traceInfo = 6; // tracing info
  83. optional RPCCallerContextProto callerContext = 7; // call context
  84. optional int64 stateId = 8; // The last seen Global State ID
  85. }
  86. /**
  87. * Rpc Response Header
  88. * +------------------------------------------------------------------+
  89. * | Rpc total response length in bytes (4 bytes int) |
  90. * | (sum of next two parts) |
  91. * +------------------------------------------------------------------+
  92. * | RpcResponseHeaderProto - serialized delimited ie has len |
  93. * +------------------------------------------------------------------+
  94. * | if request is successful: |
  95. * | - RpcResponse - The actual rpc response bytes follow |
  96. * | the response header |
  97. * | This response is serialized based on RpcKindProto |
  98. * | if request fails : |
  99. * | The rpc response header contains the necessary info |
  100. * +------------------------------------------------------------------+
  101. *
  102. * Note that rpc response header is also used when connection setup fails.
  103. * Ie the response looks like a rpc response with a fake callId.
  104. */
  105. message RpcResponseHeaderProto {
  106. /**
  107. *
  108. * RpcStastus - success or failure
  109. * The reponseHeader's errDetail, exceptionClassName and errMsg contains
  110. * further details on the error
  111. **/
  112. enum RpcStatusProto {
  113. SUCCESS = 0; // RPC succeeded
  114. ERROR = 1; // RPC or error - connection left open for future calls
  115. FATAL = 2; // Fatal error - connection closed
  116. }
  117. enum RpcErrorCodeProto {
  118. // Non-fatal Rpc error - connection left open for future rpc calls
  119. ERROR_APPLICATION = 1; // RPC Failed - rpc app threw exception
  120. ERROR_NO_SUCH_METHOD = 2; // Rpc error - no such method
  121. ERROR_NO_SUCH_PROTOCOL = 3; // Rpc error - no such protocol
  122. ERROR_RPC_SERVER = 4; // Rpc error on server side
  123. ERROR_SERIALIZING_RESPONSE = 5; // error serializign response
  124. ERROR_RPC_VERSION_MISMATCH = 6; // Rpc protocol version mismatch
  125. // Fatal Server side Rpc error - connection closed
  126. FATAL_UNKNOWN = 10; // unknown Fatal error
  127. FATAL_UNSUPPORTED_SERIALIZATION = 11; // IPC layer serilization type invalid
  128. FATAL_INVALID_RPC_HEADER = 12; // fields of RpcHeader are invalid
  129. FATAL_DESERIALIZING_REQUEST = 13; // could not deserilize rpc request
  130. FATAL_VERSION_MISMATCH = 14; // Ipc Layer version mismatch
  131. FATAL_UNAUTHORIZED = 15; // Auth failed
  132. }
  133. required uint32 callId = 1; // callId used in Request
  134. required RpcStatusProto status = 2;
  135. optional uint32 serverIpcVersionNum = 3; // Sent if success or fail
  136. optional string exceptionClassName = 4; // if request fails
  137. optional string errorMsg = 5; // if request fails, often contains strack trace
  138. optional RpcErrorCodeProto errorDetail = 6; // in case of error
  139. optional bytes clientId = 7; // Globally unique client ID
  140. optional sint32 retryCount = 8 [default = -1];
  141. optional int64 stateId = 9; // The last written Global State ID
  142. }
  143. message RpcSaslProto {
  144. enum SaslState {
  145. SUCCESS = 0;
  146. NEGOTIATE = 1;
  147. INITIATE = 2;
  148. CHALLENGE = 3;
  149. RESPONSE = 4;
  150. WRAP = 5;
  151. }
  152. message SaslAuth {
  153. required string method = 1;
  154. required string mechanism = 2;
  155. optional string protocol = 3;
  156. optional string serverId = 4;
  157. optional bytes challenge = 5;
  158. }
  159. optional uint32 version = 1;
  160. required SaslState state = 2;
  161. optional bytes token = 3;
  162. repeated SaslAuth auths = 4;
  163. }