flink-user mailing list archives

Site index · List index
Message view « Date » · « Thread »
Top « Date » · « Thread »
From Jinhua Luo <luajit...@gmail.com>
Subject Re: does the flink sink only support bio?
Date Fri, 08 Dec 2017 13:37:55 GMT
Thank you very much!

I have two new questions:

1) the async operator must emit some value to the async collector
(even it acts as a sink), right?

2) How could I use CheckpointListener with async operator? Could you
give a simple example or doc page?


2017-12-08 18:25 GMT+08:00 Stefan Richter <s.richter@data-artisans.com>:
> Hi,
>
> Flink currently does not offer async sinks out of the box, but there is no fundamental
problem against having them and we will probably offer something is this direction in the
future. In the meantime, you can build something like this by replacing the sink with an async
io operator that acts as sink (i.e. does the writes to the db) followed by a discarding sink
for compliance with the API.
>
> You need to be a bit careful if your sink needs exactly-once semantics. In this case
things should either be idempotent or the db must support rolling back changes between checkpoints,
e.g. via transactions. Commits should be triggered for confirmed checkpoints („notifyCheckpointComplete“).
>
> Your assumptions about the blocking behavior of the non-async sinks is correct.
>
> Best,
> Stefan
>
>> Am 08.12.2017 um 08:11 schrieb Jinhua Luo <luajit.io@gmail.com>:
>>
>> Hi, all.
>>
>> The invoke method of sink seems no way to make async io? e.g. returns Future?
>>
>> For example, the redis connector uses jedis lib to execute redis
>> command synchronously:
>>
>> https://github.com/apache/bahir-flink/blob/master/flink-connector-redis/src/main/java/org/apache/flink/streaming/connectors/redis/RedisSink.java
>>
>> Then it will block the task thread of flink waiting the network
>> response from redis server per command?! Is it possible for other
>> operators running in the same thread with sink? If so, then it would
>> block them too?
>>
>> I know flink has asyncio api, but it seems not for used by sink impl?
>>
>> https://ci.apache.org/projects/flink/flink-docs-release-1.3/dev/stream/asyncio.html
>>
>> Thanks.
>

Mime
View raw message