Re: [DISCUSS]: KIP-149: Enabling key access in ValueTransformer, ValueMapper, and ValueJoiner

2017-06-04 Thread Matthias J. Sax
I guess I missunderstood you. Your are right that overloading the method would also work. As both interfaces will be independent from each other, there should be no problem. The initial proposal was to use > interface ValueMapperWithKey extends ValueMapper and this would break Lambda. We totally

[jira] [Updated] (KAFKA-5054) ChangeLoggingKeyValueByteStore delete and putIfAbsent should be synchronized

2017-06-04 Thread Guozhang Wang (JIRA)
[ https://issues.apache.org/jira/browse/KAFKA-5054?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Guozhang Wang updated KAFKA-5054: - Fix Version/s: (was: 0.11.0.0) 0.11.0.1 > ChangeLoggingKeyValueByteStore de

Re: KIP-160 - Augment KStream.print() to allow users pass in extra parameters in the printed string

2017-06-04 Thread Matthias J. Sax
Not with the scope of the current discussion. So far, we discuss to add `RecordContext`, but the context object we use could also provide some more metadata. I see no reason why not to expose the node name there. We already expose TaskId vie `ProcessorContext`. We could also add thread name. IMHO,

[GitHub] kafka pull request #3232: MINOR: update docs with regard to KIP-123

2017-06-04 Thread asfgit
Github user asfgit closed the pull request at: https://github.com/apache/kafka/pull/3232 --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enab

Re: [DISCUSS]: KIP-149: Enabling key access in ValueTransformer, ValueMapper, and ValueJoiner

2017-06-04 Thread Guozhang Wang
On Sun, Jun 4, 2017 at 8:41 PM, Matthias J. Sax wrote: > We started with this proposal but it does not allow for Lambdas (in case > you want key access). Do you think preserving Lambdas is not important > enough for this case -- I agree that there is some price to be paid to > keep Lambdas. > No

[jira] [Commented] (KAFKA-5296) Unable to write to some partitions of newly created topic in 10.2

2017-06-04 Thread Abhisek Saikia (JIRA)
[ https://issues.apache.org/jira/browse/KAFKA-5296?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16036542#comment-16036542 ] Abhisek Saikia commented on KAFKA-5296: --- The issue gets resolved automatically after

[jira] [Commented] (KAFKA-4325) Improve processing of late records for window operations

2017-06-04 Thread Guozhang Wang (JIRA)
[ https://issues.apache.org/jira/browse/KAFKA-4325?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16036512#comment-16036512 ] Guozhang Wang commented on KAFKA-4325: -- I think the timestamp computation needs some

[jira] [Commented] (KAFKA-5070) org.apache.kafka.streams.errors.LockException: task [0_18] Failed to lock the state directory: /opt/rocksdb/pulse10/0_18

2017-06-04 Thread Matthias J. Sax (JIRA)
[ https://issues.apache.org/jira/browse/KAFKA-5070?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16036506#comment-16036506 ] Matthias J. Sax commented on KAFKA-5070: As Streams library controls the thread, t

[jira] [Updated] (KAFKA-4913) creating a window store with one segment throws division by zero error

2017-06-04 Thread Guozhang Wang (JIRA)
[ https://issues.apache.org/jira/browse/KAFKA-4913?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Guozhang Wang updated KAFKA-4913: - Fix Version/s: (was: 0.11.0.0) 0.11.0.1 > creating a window store with one

Re: KIP-160 - Augment KStream.print() to allow users pass in extra parameters in the printed string

2017-06-04 Thread Guozhang Wang
Matthias, I think even with KIP-159 users would not be able to access the processor node name right? Guozhang On Thu, Jun 1, 2017 at 10:28 PM, Matthias J. Sax wrote: > Thanks for the KIP. > > Two comments: > - I think we should include #writeAsText() > - I am not sure if we should use > > >

Re: [DISCUSS]: KIP-149: Enabling key access in ValueTransformer, ValueMapper, and ValueJoiner

2017-06-04 Thread Matthias J. Sax
We started with this proposal but it does not allow for Lambdas (in case you want key access). Do you think preserving Lambdas is not important enough for this case -- I agree that there is some price to be paid to keep Lambdas. About API consistency: I can't remember a concrete user request to ha

[GitHub] kafka pull request #3235: MINOR: Running the ConsumerIteratorTest unit tests...

2017-06-04 Thread 10110346
GitHub user 10110346 opened a pull request: https://github.com/apache/kafka/pull/3235 MINOR: Running the ConsumerIteratorTest unit tests,TMP directory has not been deleted when an exception occurs OS:Windows 7 After the running ConsumerIteratorTest unit tests, the TMP d

Re: [DISCUSS]: KIP-159: Introducing Rich functions to Streams

2017-06-04 Thread Matthias J. Sax
Yes, we did consider this, and there is no consensus yet what the best alternative is. @Jeyhun: the email thread got pretty long. Maybe you can give a quick summary of the current state of the discussion? -Matthias On 6/4/17 6:04 PM, Guozhang Wang wrote: > Thanks for the explanation Jeyhun and

Re: [DISCUSS]: KIP-149: Enabling key access in ValueTransformer, ValueMapper, and ValueJoiner

2017-06-04 Thread Guozhang Wang
With KIP-159, the added "valueMapperWithKey" and "valueTransformerWithKey" along seem to be just adding a few syntax-sugars? E.g. we can simply implement: mapValues(ValueMapperWithKey mapperWithKey) as map((k, v) -> (k, mapperWithKey(k, v)) -- I'm not sure how many users wo

Re: [DISCUSS]: KIP-159: Introducing Rich functions to Streams

2017-06-04 Thread Guozhang Wang
Thanks for the explanation Jeyhun and Matthias. I have just read through both KIP-149 and KIP-159 and am wondering if you guys have considered a slight different approach for rich function, that is to add the `RecordContext` into the apply functions as an additional parameter. For example: --

[jira] [Updated] (KAFKA-5364) Producer attempts to send transactional messages before adding partitions to transaction

2017-06-04 Thread Ismael Juma (JIRA)
[ https://issues.apache.org/jira/browse/KAFKA-5364?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Ismael Juma updated KAFKA-5364: --- Status: Patch Available (was: Reopened) > Producer attempts to send transactional messages before add

[jira] [Commented] (KAFKA-5378) Last Stable Offset not returned in Fetch request

2017-06-04 Thread Ismael Juma (JIRA)
[ https://issues.apache.org/jira/browse/KAFKA-5378?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16036438#comment-16036438 ] Ismael Juma commented on KAFKA-5378: Since this is a case of Kafka not complying with

[jira] [Updated] (KAFKA-5378) Last Stable Offset not returned in Fetch request

2017-06-04 Thread Ismael Juma (JIRA)
[ https://issues.apache.org/jira/browse/KAFKA-5378?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Ismael Juma updated KAFKA-5378: --- Priority: Critical (was: Major) > Last Stable Offset not returned in Fetch request >

[jira] [Updated] (KAFKA-5376) Transactions: Concurrent transactional consumer reads aborted messages

2017-06-04 Thread Apurva Mehta (JIRA)
[ https://issues.apache.org/jira/browse/KAFKA-5376?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Apurva Mehta updated KAFKA-5376: Attachment: KAFKA-5376.tar.gz [~hachikuji] I attached the logs for a failed run. These include all d

[jira] [Resolved] (KAFKA-5355) Broker returns messages beyond "latest stable offset" to transactional consumer in read_committed mode

2017-06-04 Thread Ismael Juma (JIRA)
[ https://issues.apache.org/jira/browse/KAFKA-5355?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Ismael Juma resolved KAFKA-5355. Resolution: Fixed > Broker returns messages beyond "latest stable offset" to transactional > consum

[jira] [Commented] (KAFKA-5355) Broker returns messages beyond "latest stable offset" to transactional consumer in read_committed mode

2017-06-04 Thread ASF GitHub Bot (JIRA)
[ https://issues.apache.org/jira/browse/KAFKA-5355?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16036426#comment-16036426 ] ASF GitHub Bot commented on KAFKA-5355: --- Github user asfgit closed the pull request

[GitHub] kafka pull request #3230: KAFKA-5355: Test cases to ensure isolation level p...

2017-06-04 Thread asfgit
Github user asfgit closed the pull request at: https://github.com/apache/kafka/pull/3230 --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enab

[jira] [Commented] (KAFKA-5070) org.apache.kafka.streams.errors.LockException: task [0_18] Failed to lock the state directory: /opt/rocksdb/pulse10/0_18

2017-06-04 Thread Kevin Chen (JIRA)
[ https://issues.apache.org/jira/browse/KAFKA-5070?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16036404#comment-16036404 ] Kevin Chen commented on KAFKA-5070: --- correct, that is why I increased. I had configured

[GitHub] kafka pull request #3233: MINOR: Remove redundant volatile write in RecordHe...

2017-06-04 Thread asfgit
Github user asfgit closed the pull request at: https://github.com/apache/kafka/pull/3233 --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enab

[jira] [Comment Edited] (KAFKA-5098) KafkaProducer.send() blocks and generates TimeoutException if topic name has illegal char

2017-06-04 Thread Ismael Juma (JIRA)
[ https://issues.apache.org/jira/browse/KAFKA-5098?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16036327#comment-16036327 ] Ismael Juma edited comment on KAFKA-5098 at 6/4/17 4:36 PM: [~

[jira] [Updated] (KAFKA-5098) KafkaProducer.send() blocks and generates TimeoutException if topic name has illegal char

2017-06-04 Thread Ismael Juma (JIRA)
[ https://issues.apache.org/jira/browse/KAFKA-5098?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Ismael Juma updated KAFKA-5098: --- Fix Version/s: (was: 0.11.0.0) 0.11.1.0 > KafkaProducer.send() blocks and gener

[jira] [Reopened] (KAFKA-5098) KafkaProducer.send() blocks and generates TimeoutException if topic name has illegal char

2017-06-04 Thread Ismael Juma (JIRA)
[ https://issues.apache.org/jira/browse/KAFKA-5098?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Ismael Juma reopened KAFKA-5098: [~onurkaraman] correctly asked about the potential performance impact of doing this check for each prod

[GitHub] kafka pull request #3234: MINOR: Optimise performance of `Topic.validate()`

2017-06-04 Thread ijuma
GitHub user ijuma opened a pull request: https://github.com/apache/kafka/pull/3234 MINOR: Optimise performance of `Topic.validate()` I included a JMH benchmark and the results follow. The implementation in this PR takes no more than 1/10th of the time when compared to trunk.

[jira] [Updated] (KAFKA-5371) SyncProducerTest.testReachableServer has become flaky

2017-06-04 Thread Ismael Juma (JIRA)
[ https://issues.apache.org/jira/browse/KAFKA-5371?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Ismael Juma updated KAFKA-5371: --- Resolution: Fixed Status: Resolved (was: Patch Available) > SyncProducerTest.testReachableSer

[jira] [Commented] (KAFKA-5371) SyncProducerTest.testReachableServer has become flaky

2017-06-04 Thread ASF GitHub Bot (JIRA)
[ https://issues.apache.org/jira/browse/KAFKA-5371?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16036294#comment-16036294 ] ASF GitHub Bot commented on KAFKA-5371: --- Github user asfgit closed the pull request

[GitHub] kafka pull request #3225: KAFKA-5371: Increase request timeout for producer ...

2017-06-04 Thread asfgit
Github user asfgit closed the pull request at: https://github.com/apache/kafka/pull/3225 --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enab

[GitHub] kafka pull request #3233: MINOR: Remove volatile write in RecordHeaders

2017-06-04 Thread ijuma
GitHub user ijuma opened a pull request: https://github.com/apache/kafka/pull/3233 MINOR: Remove volatile write in RecordHeaders The JMH benchmark included shows that the redundant volatile write causes the constructor of `ProducerRecord` to take more than 50% longer:

Re?? [DISCUSS] KIP-148: Add a connect timeout for client

2017-06-04 Thread ????????
>I guess one obvious question is, how does this interact with retries? >Does it result in a failure getting delivered to the end user more >quickly if connecting is impossible the first few times we try? Does >exponential backoff still apply? Yes, for the retries it will make the end user more