Versions Compared

Key

  • This line was added.
  • This line was removed.
  • Formatting was changed.

Status

...

Page properties


Discussion thread

...

...

)
Vote threadhttp://apache-flink-mailing-list-archive.1008284.n3.nabble.com/VOTE-Watermark-propagation-with-Sink-API-td51029.html
JIRA

Jira
serverASF JIRA
serverId5aa69414-a9e9-3523-82ec-879b028fb15b
keyFLINK-22700

...

Release1.14


Please keep the discussion on the mailing list rather than commenting on the wiki (wiki discussions get unwieldy fast).

...

package org.apache.flink.api.connector.sink;
import org.apache.flink.api.common.eventtime.Watermark;

@Experimental
public interface SinkWriter<InputT, CommT, WriterStateT> extends AutoCloseable {

/**
* Add a watermark to the writer.
*
* @param watermark The watermark.
* @throws IOException if fail to add a watermark.
*/
default void writeWatermark(Watermark watermark) throws IOException {}

void write(InputT element, Context context) throws IOException;
}

...

  1. Combine a process function and a sink function to achieve similar effects with increased complexity.
  2. Introduce event-time timers to sink functions.  This approach seems too element-centric for the use cases.   Also, event-time timers are generally keyed.

Future Enhancements

  1. Add support for idleness.  A sink operator may become idled due to upstream status changes.  A future enhancement would be to inform the sink function in this situation.