Class KeyedBroadcastProcessFunction<KS,​IN1,​IN2,​OUT>

  • Type Parameters:
    KS - The key type of the input keyed stream.
    IN1 - The input type of the keyed (non-broadcast) side.
    IN2 - The input type of the broadcast side.
    OUT - The output type of the operator.
    All Implemented Interfaces:
    Serializable, Function, RichFunction

    @PublicEvolving
    public abstract class KeyedBroadcastProcessFunction<KS,​IN1,​IN2,​OUT>
    extends BaseBroadcastProcessFunction
    A function to be applied to a BroadcastConnectedStream that connects BroadcastStream, i.e. a stream with broadcast state, with a KeyedStream.

    The stream with the broadcast state can be created using the DataStream.broadcast(MapStateDescriptor[]) keyedStream.broadcast(MapStateDescriptor)} method.

    The user has to implement two methods:

    1. the #processBroadcastElement(Object, Context, Collector) which will be applied to each element in the broadcast side
    2. and the #processElement(Object, ReadOnlyContext, Collector) which will be applied to the non-broadcasted/keyed side.

    The processElementOnBroadcastSide() takes as an argument (among others) a context that allows it to read/write to the broadcast state and also apply a transformation to all (local) keyed states, while the processElement() has read-only access to the broadcast state, but can read/write to the keyed state and register timers.

    See Also:
    Serialized Form
    • Constructor Detail

      • KeyedBroadcastProcessFunction

        public KeyedBroadcastProcessFunction()
    • Method Detail

      • processElement

        public abstract void processElement​(IN1 value,
                                            KeyedBroadcastProcessFunction.ReadOnlyContext ctx,
                                            Collector<OUT> out)
                                     throws Exception
        This method is called for each element in the (non-broadcast) keyed stream.

        It can output zero or more elements using the Collector parameter, query the current processing/event time, and also query and update the local keyed state. In addition, it can get a TimerService for registering timers and querying the time. Finally, it has read-only access to the broadcast state. The context is only valid during the invocation of this method, do not store it.

        Parameters:
        value - The stream element.
        ctx - A KeyedBroadcastProcessFunction.ReadOnlyContext that allows querying the timestamp of the element, querying the current processing/event time and iterating the broadcast state with read-only access. The context is only valid during the invocation of this method, do not store it.
        out - The collector to emit resulting elements to
        Throws:
        Exception - The function may throw exceptions which cause the streaming program to fail and go into recovery.
      • processBroadcastElement

        public abstract void processBroadcastElement​(IN2 value,
                                                     KeyedBroadcastProcessFunction.Context ctx,
                                                     Collector<OUT> out)
                                              throws Exception
        This method is called for each element in the broadcast stream.

        It can output zero or more elements using the Collector parameter, query the current processing/event time, and also query and update the internal broadcast state. In addition, it can register a function to be applied to all keyed states on the local partition. These can be done through the provided KeyedBroadcastProcessFunction.Context. The context is only valid during the invocation of this method, do not store it.

        Parameters:
        value - The stream element.
        ctx - A KeyedBroadcastProcessFunction.Context that allows querying the timestamp of the element, querying the current processing/event time and updating the broadcast state. In addition, it allows the registration of a function to be applied to all keyed state with a given StateDescriptor on the local partition. The context is only valid during the invocation of this method, do not store it.
        out - The collector to emit resulting elements to
        Throws:
        Exception - The function may throw exceptions which cause the streaming program to fail and go into recovery.
      • onTimer

        public void onTimer​(long timestamp,
                            KeyedBroadcastProcessFunction.OnTimerContext ctx,
                            Collector<OUT> out)
                     throws Exception
        Called when a timer set using TimerService fires.
        Parameters:
        timestamp - The timestamp of the firing timer.
        ctx - An KeyedBroadcastProcessFunction.OnTimerContext that allows querying the timestamp of the firing timer, querying the current processing/event time, iterating the broadcast state with read-only access, querying the TimeDomain of the firing timer and getting a TimerService for registering timers and querying the time. The context is only valid during the invocation of this method, do not store it.
        out - The collector for returning result values.
        Throws:
        Exception - This method may throw exceptions. Throwing an exception will cause the operation to fail and may trigger recovery.