Class RpcEndpoint

  • All Implemented Interfaces:
    AutoCloseable, RpcGateway, AutoCloseableAsync
    Direct Known Subclasses:
    FencedRpcEndpoint, MetricQueryService, TaskExecutor

    public abstract class RpcEndpoint
    extends Object
    implements RpcGateway, AutoCloseableAsync
    Base class for RPC endpoints. Distributed components which offer remote procedure calls have to extend the RPC endpoint base class. An RPC endpoint is backed by an RpcService.

    Endpoint and Gateway

    To be done...

    Single Threaded Endpoint Execution

    All RPC calls on the same endpoint are called by the same thread (referred to as the endpoint's main thread). Thus, by executing all state changing operations within the main thread, we don't have to reason about concurrent accesses, in the same way in the Actor Model of Erlang or Akka.

    The RPC endpoint provides runAsync(Runnable), callAsync(Callable, Duration) and the getMainThreadExecutor() to execute code in the RPC endpoint's main thread.

    Lifecycle

    The RPC endpoint has the following stages:

    • The RPC endpoint is created in a non-running state and does not serve any RPC requests.
    • Calling the start() method triggers the start of the RPC endpoint and schedules overridable onStart() method call to the main thread.
    • When the start operation ends the RPC endpoint is moved to the running state and starts to serve and complete RPC requests.
    • Calling the closeAsync() method triggers the termination of the RPC endpoint and schedules overridable onStop() method call to the main thread.
    • When onStop() method is called, it triggers an asynchronous stop operation. The RPC endpoint is not in the running state anymore but it continues to serve RPC requests.
    • When the asynchronous stop operation ends, the RPC endpoint terminates completely and does not serve RPC requests anymore.

    The running state can be queried in a RPC method handler or in the main thread by calling isRunning() method.

    • Field Detail

      • log

        protected final org.slf4j.Logger log
      • rpcServer

        protected final RpcServer rpcServer
        Interface to access the underlying rpc server.
    • Constructor Detail

      • RpcEndpoint

        protected RpcEndpoint​(RpcService rpcService,
                              String endpointId,
                              Map<String,​String> loggingContext)
        Initializes the RPC endpoint.
        Parameters:
        rpcService - The RPC server that dispatches calls to this RPC endpoint.
        endpointId - Unique identifier for this endpoint
      • RpcEndpoint

        protected RpcEndpoint​(RpcService rpcService,
                              String endpointId)
        Initializes the RPC endpoint.
        Parameters:
        rpcService - The RPC server that dispatches calls to this RPC endpoint.
        endpointId - Unique identifier for this endpoint
      • RpcEndpoint

        protected RpcEndpoint​(RpcService rpcService)
        Initializes the RPC endpoint with a random endpoint id.
        Parameters:
        rpcService - The RPC server that dispatches calls to this RPC endpoint.
    • Method Detail

      • getEndpointId

        public String getEndpointId()
        Returns the rpc endpoint's identifier.
        Returns:
        Rpc endpoint's identifier.
      • isRunning

        protected boolean isRunning()
        Returns whether the RPC endpoint is started and not stopped or being stopped.
        Returns:
        whether the RPC endpoint is started and not stopped or being stopped.
      • start

        public final void start()
        Triggers start of the rpc endpoint. This tells the underlying rpc server that the rpc endpoint is ready to process remote procedure calls.
      • internalCallOnStart

        public final void internalCallOnStart()
                                       throws Exception
        Internal method which is called by the RpcService implementation to start the RpcEndpoint.
        Throws:
        Exception - indicating that the rpc endpoint could not be started. If an exception occurs, then the rpc endpoint will automatically terminate.
      • onStart

        protected void onStart()
                        throws Exception
        User overridable callback which is called from internalCallOnStart().

        This method is called when the RpcEndpoint is being started. The method is guaranteed to be executed in the main thread context and can be used to start the rpc endpoint in the context of the rpc endpoint's main thread.

        IMPORTANT: This method should never be called directly by the user.

        Throws:
        Exception - indicating that the rpc endpoint could not be started. If an exception occurs, then the rpc endpoint will automatically terminate.
      • stop

        protected final void stop()
        Triggers stop of the rpc endpoint. This tells the underlying rpc server that the rpc endpoint is no longer ready to process remote procedure calls.
      • internalCallOnStop

        public final CompletableFuture<Void> internalCallOnStop()
        Internal method which is called by the RpcService implementation to stop the RpcEndpoint.
        Returns:
        Future which is completed once all post stop actions are completed. If an error occurs this future is completed exceptionally
      • registerResource

        protected void registerResource​(Closeable closeableResource)
        Register the given closeable resource to CloseableRegistry.
        Parameters:
        closeableResource - the given closeable resource
      • unregisterResource

        protected boolean unregisterResource​(Closeable closeableResource)
        Unregister the given closeable resource from CloseableRegistry.
        Parameters:
        closeableResource - the given closeable resource
        Returns:
        true if the given resource unregister successful, otherwise false
      • onStop

        protected CompletableFuture<Void> onStop()
        User overridable callback which is called from internalCallOnStop().

        This method is called when the RpcEndpoint is being shut down. The method is guaranteed to be executed in the main thread context and can be used to clean up internal state.

        IMPORTANT: This method should never be called directly by the user.

        Returns:
        Future which is completed once all post stop actions are completed. If an error occurs this future is completed exceptionally
      • closeAsync

        public final CompletableFuture<Void> closeAsync()
        Triggers the shut down of the rpc endpoint. The shut down is executed asynchronously.

        In order to wait on the completion of the shut down, obtain the termination future via getTerminationFuture()} and wait on its completion.

        Specified by:
        closeAsync in interface AutoCloseableAsync
        Returns:
        Future which is completed once the resource has been closed
      • getSelfGateway

        public <C extends RpcGateway> C getSelfGateway​(Class<C> selfGatewayType)
        Returns a self gateway of the specified type which can be used to issue asynchronous calls against the RpcEndpoint.

        IMPORTANT: The self gateway type must be implemented by the RpcEndpoint. Otherwise the method will fail.

        Type Parameters:
        C - type of the self gateway to create
        Parameters:
        selfGatewayType - class of the self gateway type
        Returns:
        Self gateway of the specified type which can be used to issue asynchronous rpcs
      • getAddress

        public String getAddress()
        Gets the address of the underlying RPC endpoint. The address should be fully qualified so that a remote system can connect to this RPC endpoint via this address.
        Specified by:
        getAddress in interface RpcGateway
        Returns:
        Fully qualified address of the underlying RPC endpoint
      • getHostname

        public String getHostname()
        Gets the hostname of the underlying RPC endpoint.
        Specified by:
        getHostname in interface RpcGateway
        Returns:
        Hostname on which the RPC endpoint is running
      • getMainThreadExecutor

        protected RpcEndpoint.MainThreadExecutor getMainThreadExecutor()
        Gets the main thread execution context. The main thread execution context can be used to execute tasks in the main thread of the underlying RPC endpoint.
        Returns:
        Main thread execution context
      • getMainThreadExecutor

        protected Executor getMainThreadExecutor​(JobID jobID)
        Gets the main thread execution context. The main thread execution context can be used to execute tasks in the main thread of the underlying RPC endpoint.
        Parameters:
        jobID - the JobID to scope the returned ComponentMainThreadExecutor to, i.e. add/remove before/after the invocations using the returned executor
        Returns:
        Main thread execution context
      • getRpcService

        public RpcService getRpcService()
        Gets the endpoint's RPC service.
        Returns:
        The endpoint's RPC service
      • getTerminationFuture

        public CompletableFuture<Void> getTerminationFuture()
        Return a future which is completed with true when the rpc endpoint has been terminated. In case of a failure, this future is completed with the occurring exception.
        Returns:
        Future which is completed when the rpc endpoint has been terminated.
      • runAsync

        protected void runAsync​(Runnable runnable)
        Execute the runnable in the main thread of the underlying RPC endpoint.
        Parameters:
        runnable - Runnable to be executed in the main thread of the underlying RPC endpoint
      • scheduleRunAsync

        protected void scheduleRunAsync​(Runnable runnable,
                                        Duration delay)
        Execute the runnable in the main thread of the underlying RPC endpoint, with a delay of the given number of milliseconds.
        Parameters:
        runnable - Runnable to be executed
        delay - The delay after which the runnable will be executed
      • scheduleRunAsync

        protected void scheduleRunAsync​(Runnable runnable,
                                        long delay,
                                        TimeUnit unit)
        Execute the runnable in the main thread of the underlying RPC endpoint, with a delay of the given number of milliseconds.
        Parameters:
        runnable - Runnable to be executed
        delay - The delay after which the runnable will be executed
      • callAsync

        protected <V> CompletableFuture<V> callAsync​(Callable<V> callable,
                                                     Duration timeout)
        Execute the callable in the main thread of the underlying RPC service, returning a future for the result of the callable. If the callable is not completed within the given timeout, then the future will be failed with a TimeoutException.
        Type Parameters:
        V - Return type of the callable
        Parameters:
        callable - Callable to be executed in the main thread of the underlying rpc server
        timeout - Timeout for the callable to be completed
        Returns:
        Future for the result of the callable.
      • validateRunsInMainThread

        public void validateRunsInMainThread()
        Validates that the method call happens in the RPC endpoint's main thread.

        IMPORTANT: This check only happens when assertions are enabled, such as when running tests.

        This can be used for additional checks, like

        
         protected void concurrencyCriticalMethod() {
             validateRunsInMainThread();
        
             // some critical stuff
         }