Uploaded image for project: 'Kudu'
  1. Kudu
  2. KUDU-2065

Support cancellation for outbound client RPC

    XMLWordPrintableJSON

Details

    • Improvement
    • Status: Resolved
    • Minor
    • Resolution: Fixed
    • None
    • 1.5.0
    • rpc
    • None

    Description

      Currently, there is no way to cancel an outbound client RPC call in Kudu. The following is a proposal to do so. Please feel free to comment on it.

      A new interface void Proxy::Cancel() will be introduced. It enqueues a reactor task which will eventually call void OutboundCall::Cancel(). A new RpcFeature flag CANCELLABLE will be added to indicate whether the server can handle in-flight RPC which is cancelled. More details below. A client can specify whether such functionality is needed by setting a bool flag in RpcController passed to the proxy.

      Depending on the state of an OutboundCall, cancellation will happen as follows:

      • READY
        • it hasn't been scheduled yet. Set the cancellation flag in the OutboundCall object. When it's eventually scheduled by Connection::QueueOutboundCall(), it will check the cancellation flag before assigning a call_id. If it's set, call OutboundCall::SetCancelled() and return.
      • ON_OUTBOUND_QUEUE
        • it's on the outbound transfer queue but transfer hasn't started yet.
          Connection::WriteHandler() will check the cancellation flag before initiating a transfer. If it's set, the transfer will be popped from the queue and deleted. Call OutboundCall::SetCancelled() and return.
      • SENDING
        • some of the payload has already made its way to the other side. To make sure the outbound call doesn't hold on to sidecars_ till the end of the transfer, the outbound call needs to clear sidecars_ and sends the remaining bytes as 0. The entry in CAR map will be removed and OutboundCall::SetCancelled() will be invoked. Please see below on how the server will handle this incomplete RPC message.
      • SENT
        • The payload has been sent. Waiting for a response. Call OutboundCall::SetCancelled(). Incoming response will be dropped on the floor.
      • NEGOTIATION_TIMED_OUT
      • TIMED_OUT
      • CANCELLED
      • FINISHED_NEGOTIATION_ERROR
      • FINISHED_ERROR
      • FINISHED_SUCCESS
        • No-op. Callback has been invoked already.

      OutboundCall::SetCancelled() will mark status_ to Cancelled. Set the state to CANCELLED. In addition, it will clear side_cars_, delete header_buf_ and invoke the callback.

      When an OutboundCall already in the SENDING state is cancelled, sidecars_ will be cleared. This provides guarantee to the RPC client that the RPC subsystem doesn't have any reference left to payload pointed by sidecars_ so the RPC client can safely free them if it's passed in as a SliceSideCar. It's freed immediately if it's passed in as FastStringSideCar. An in-flight RPC will be cancelled by sending the remainder of the payload (encoded in total message length) as 0. An optional bool field is added to the request header. If it's set, it's expected that a single byte at the end which is 1 if the payload has been cancelled when it was in-flight. Potentially, we can do the same treatment for client RPC which times out already.

      +------------------------------------------------+
      | Total message length (4 bytes)                 |
      +------------------------------------------------+
      | RPC Header protobuf length (variable encoding) |
      +------------------------------------------------+
      | RPC Header protobuf                            |
      +------------------------------------------------+
      | Main message length (variable encoding)        |
      +------------------------------------------------+ --- 0
      | Main message protobuf                          |
      +------------------------------------------------+ --- sidecar_offsets(0)
      | Sidecar 0                                      |
      +------------------------------------------------+ --- sidecar_offsets(1)
      | Sidecar 1                                      |
      +------------------------------------------------+ --- sidecar_offsets(2)
      | Sidecar 2                                      |
      +------------------------------------------------+ --- ...
      | ...                                            |
      +------------------------------------------------+
      | cancelled                                      | -- true if RPC has been cancelled mid-flight
      +------------------------------------------------+
      

      Attachments

        Issue Links

          Activity

            People

              kwho Michael Ho
              kwho Michael Ho
              Votes:
              0 Vote for this issue
              Watchers:
              8 Start watching this issue

              Dates

                Created:
                Updated:
                Resolved: