[ 
https://issues.apache.org/jira/browse/FLINK-6988?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16120818#comment-16120818
 ] 

ASF GitHub Bot commented on FLINK-6988:
---------------------------------------

Github user rangadi commented on the issue:

    https://github.com/apache/flink/pull/4239
  
    > Hmmm, are you sure about this thing? That would mean that Kafka doesn't 
support transactional parallel writes from two different process, which would 
be very strange. Could you point to a source of this information?
    
    It does not prohibit parallel transactions. Just restricts what an EOS 
consumer, which reads only the committed messages can see.
    
    See 'Reading Transactional Messages' section in JavaDoc for KafkaConsumer : 
https://github.com/apache/kafka/blob/0.11.0/clients/src/main/java/org/apache/kafka/clients/consumer/KafkaConsumer.java#L421
 : 
    
    > In read_committed mode, the consumer will read only those transactional 
messages which have been successfully committed. It will continue to read 
non-transactional messages as before. There is no client-side buffering in 
read_committed mode. Instead, the end offset of a partition for a 
read_committed consumer would be the offset of the first message in the 
partition belonging to an open transaction. This offset is known as the 'Last 
Stable Offset'(LSO).
    
    If there is an open transaction, the EOS consumers don't read past it.



> Add Apache Kafka 0.11 connector
> -------------------------------
>
>                 Key: FLINK-6988
>                 URL: https://issues.apache.org/jira/browse/FLINK-6988
>             Project: Flink
>          Issue Type: Improvement
>          Components: Kafka Connector
>    Affects Versions: 1.3.1
>            Reporter: Piotr Nowojski
>            Assignee: Piotr Nowojski
>
> Kafka 0.11 (it will be released very soon) add supports for transactions. 
> Thanks to that, Flink might be able to implement Kafka sink supporting 
> "exactly-once" semantic. API changes and whole transactions support is 
> described in 
> [KIP-98|https://cwiki.apache.org/confluence/display/KAFKA/KIP-98+-+Exactly+Once+Delivery+and+Transactional+Messaging].
> The goal is to mimic implementation of existing BucketingSink. New 
> FlinkKafkaProducer011 would 
> * upon creation begin transaction, store transaction identifiers into the 
> state and would write all incoming data to an output Kafka topic using that 
> transaction
> * on `snapshotState` call, it would flush the data and write in state 
> information that current transaction is pending to be committed
> * on `notifyCheckpointComplete` we would commit this pending transaction
> * in case of crash between `snapshotState` and `notifyCheckpointComplete` we 
> either abort this pending transaction (if not every participant successfully 
> saved the snapshot) or restore and commit it. 



--
This message was sent by Atlassian JIRA
(v6.4.14#64029)

Reply via email to