Re: [DISCUSS] KIP-210: Provide for custom error handling when Kafka Streams fails to produce

2017-12-05 Thread Guozhang Wang
Hello Matt, Thanks for writing up the KIP. I made a pass over it and here is a few minor comments. I think you can consider starting a voting thread for this KIP while addressing them. 1) We do not have the "ProductionExceptionHandler" interface defined in the wiki page, thought it is sort of

Re: [DISCUSS] KIP-228 Negative record timestamp support

2017-12-05 Thread Dong Lin
Sounds good. I don't think there is concern with using Long.MIN_VALUE to indicate that timestamp is not available. As Matthias also mentioned, using Long.MIN_VALUE to indicate missing timestamp seems better than overloading -1 semantics. Do you want to update the "NO_TIMESTAMP (−1) problem"

Re: 答复: [VOTE] KIP-223 - Add per-topic min lead and per-partition lead metrics to KafkaConsumer

2017-12-05 Thread Guozhang Wang
Hello Xi, You can see that in o.a.k.common.metrics.Sensor, we allow constructors to pass in one or more "parent" Sensors of the constructed Sensor, behind the scene when a child sensor's metrics have been updated, the updates will be propagated all the way up to its parents and ancestors (you can

Re: [DISCUSS]KIP-216: IQ should throw different exceptions for different errors

2017-12-05 Thread Matthias J. Sax
Thanks for the info. There is no hurry. Was just curious :) -Matthias On 12/5/17 7:34 PM, vito jeng wrote: > Matthias, > > Still in progress. I've been busy in recent weeks with my job. > > But I believe I will update this KIP within few days. > > > > > --- > Vito > > On Tue, Dec 5, 2017

Re: [DISCUSS] KIP-227: Introduce Incremental FetchRequests to Increase Partition Scalability

2017-12-05 Thread Becket Qin
Hi Jun, That is true, but in reality it seems rare that the fetch size is smaller than index interval. In the worst case, we may need to do another look up. In the future, when we have the mechanism to inform the clients about the broker configurations, the clients may want to configure

Re: [DISCUSS] KIP-228 Negative record timestamp support

2017-12-05 Thread Boerge Svingen
Thank you for the suggestion. We considered this before. It works, but it’s a hack, and we would be providing a bad user experience for our consumers if we had to explain, “if you want to start consuming in 2014, you have to pretend to want 2214”. We would rather solve the underlying problem.

Re: Comparing Pulsar and Kafka: unified queuing and streaming

2017-12-05 Thread Khurrum Nasim
Jason, Comments inline. On Tue, Dec 5, 2017 at 10:59 AM, Jason Gustafson wrote: > > I believe a lot of users are using the kafka high level consumers, it is > > effectively an **unordered** messaging/streaming pattern. People using > high > > level consumers don't actually

Re: [DISCUSS]KIP-216: IQ should throw different exceptions for different errors

2017-12-05 Thread vito jeng
Matthias, Still in progress. I've been busy in recent weeks with my job. But I believe I will update this KIP within few days. --- Vito On Tue, Dec 5, 2017 at 8:05 AM, Matthias J. Sax wrote: > Vito, > > is there any update with regard to this KIP? > > > -Matthias >

Re: [DISCUSS] KIP-228 Negative record timestamp support

2017-12-05 Thread Ted Yu
bq. you can add X to the timestamp before you produce Kafka message This assumes the earliest timestamp (for user application) is known beforehand. However, what if this earliest timestamp shifts even earlier (e.g. due to some discovery) ? Cheers On Tue, Dec 5, 2017 at 6:36 PM, Dong Lin

[jira] [Created] (KAFKA-6313) Kafka Core maven dependencies are missing SLF4J API

2017-12-05 Thread Randall Hauch (JIRA)
Randall Hauch created KAFKA-6313: Summary: Kafka Core maven dependencies are missing SLF4J API Key: KAFKA-6313 URL: https://issues.apache.org/jira/browse/KAFKA-6313 Project: Kafka Issue

Re: [DISCUSS] KIP-228 Negative record timestamp support

2017-12-05 Thread Dong Lin
Hey Boerge, Thanks for the blog link. I will read this blog later. Here is another alternative solution which may be worth thinking. We know that the Unix time 0 corresponds to January 1, 1970. Let's say the earliest time you may want to use as the timestamp of the Kafka message is within X

Re: [DISCUSS] KIP-227: Introduce Incremental FetchRequests to Increase Partition Scalability

2017-12-05 Thread Jun Rao
Hi, Jiangjie, Not sure returning the fetch response at the index boundary is a general solution. The index interval is configurable. If one configures the index interval larger than the per partition fetch size, we probably have to return data not at the index boundary. Thanks, Jun On Tue, Dec

Re: [DISCUSS] KIP-228 Negative record timestamp support

2017-12-05 Thread Boerge Svingen
Yes. To provide a little more detail, we are using Kafka to store everything ever published by The New York Times, and to make this content available to a range of systems and applications. Assets are published to Kafka chronologically, so that consumers can seek to any point in time and start

[jira] [Reopened] (KAFKA-5878) Add sensor for queue size of the controller-event-thread

2017-12-05 Thread Dong Lin (JIRA)
[ https://issues.apache.org/jira/browse/KAFKA-5878?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Dong Lin reopened KAFKA-5878: - > Add sensor for queue size of the controller-event-thread >

Re: [DISCUSS] KIP-228 Negative record timestamp support

2017-12-05 Thread Ted Yu
What if the negative timestamp is stored this way ? Long.MIN_VALUE + delta (where delta is positvie) and calculated this way when used: 1/1/1970 - delta This approach avoids the ambiguity of -1 timestamp since -1 would be stored as Long.MIN_VALUE+1 Log retention can handle such format with

[jira] [Resolved] (KAFKA-5878) Add sensor for queue size of the controller-event-thread

2017-12-05 Thread Dong Lin (JIRA)
[ https://issues.apache.org/jira/browse/KAFKA-5878?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Dong Lin resolved KAFKA-5878. - Resolution: Won't Fix > Add sensor for queue size of the controller-event-thread >

[jira] [Resolved] (KAFKA-6285) OffsetCommitRequest should have read-after-write logic

2017-12-05 Thread Dong Lin (JIRA)
[ https://issues.apache.org/jira/browse/KAFKA-6285?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel ] Dong Lin resolved KAFKA-6285. - Resolution: Duplicate Duplicate of KAFKA-6285 > OffsetCommitRequest should have read-after-write logic >

[GitHub] kafka pull request #4295: KAFKA-6299. Fix AdminClient error handling when me...

2017-12-05 Thread cmccabe
GitHub user cmccabe opened a pull request: https://github.com/apache/kafka/pull/4295 KAFKA-6299. Fix AdminClient error handling when metadata changes AdminClient should only call Metadata#requestUpdate when needed. AdminClient should retry requests for which the controller

Re: [DISCUSS] KIP-228 Negative record timestamp support

2017-12-05 Thread Dong Lin
Hey Konstantin, According to KIP-32 the timestamp is also used for log rolling and log retention. Therefore, unless broker is configured to never delete any message based on time, messages produced with negative timestamp in your use-case will be deleted by the broker anyway. Do you actually plan

Re: [DISCUSS] KIP-227: Introduce Incremental FetchRequests to Increase Partition Scalability

2017-12-05 Thread Becket Qin
Hi Colin, Thinking about this again. I do see the reason that we want to have a epoch to avoid out of order registration of the interested set. But I am wondering if the following semantic would meet what we want better: - Session Id: the id assigned to a single client for life long time. i.e it

Re: [DISCUSS] KIP-228 Negative record timestamp support

2017-12-05 Thread Matthias J. Sax
Thanks for the KIP Konstantin. From my understanding, you propose to just remove the negative timestamp check in KafkaProducer and KafkaStreams. If topics are configured with `CreateTime` brokers also write negative timestamps if they are embedded in the message. However, I am not sure about the

Re: [DISCUSS] KIP-231: Improve the Required ACL of ListGroups API

2017-12-05 Thread Vahid S Hashemian
I forgot to mention that OffsetFetch is another example that is affected by such filtering. Starting from version 2 and when offsets of all topics are requested, only topics to which user has describe access will appear in the result. If user does not have describe access to any topic (assuming

Re: [DISCUSS] KIP-210: Provide for custom error handling when Kafka Streams fails to produce

2017-12-05 Thread Matthias J. Sax
Thanks a lot for the update! Great write-up! Very clearly explained what the change will look like! Looks good to me. No further comments from my side. -Matthias On 12/5/17 9:14 AM, Matt Farmer wrote: > I have updated this KIP accordingly. > > Can you please take a look and let me know if

Re: [DISCUSS] KIP-231: Improve the Required ACL of ListGroups API

2017-12-05 Thread Vahid S Hashemian
Hi Dong, Thanks for your message. My driving point for this KIP was a -seemingly- broken logic rather than a particular use case. I also considered the semantics of the ListGroups API and its expected behavior: To me, a user who can fetch/commit offsets in some consumer groups should be able

Re: [DISCUSS] KIP-231: Improve the Required ACL of ListGroups API

2017-12-05 Thread Vahid S Hashemian
Hi Ismael, Thanks for the pointer. That's a good example of how we already implemented a similar filtering. --Vahid From: Ismael Juma To: dev Date: 12/05/2017 01:24 AM Subject:Re: [DISCUSS] KIP-231: Improve the Required ACL of

Re: [DISCUSS] KIP-227: Introduce Incremental FetchRequests to Increase Partition Scalability

2017-12-05 Thread Colin McCabe
On Tue, Dec 5, 2017, at 13:13, Jan Filipiak wrote: > Hi Colin > > Addressing the topic of how to manage slots from the other thread. > With tcp connections all this comes for free essentially. Hi Jan, I don't think that it's accurate to say that cache management "comes for free" by coupling the

Re: [DISCUSS] KIP-227: Introduce Incremental FetchRequests to Increase Partition Scalability

2017-12-05 Thread Ted Yu
Thanks for responding, Colin. bq. If we have a bunch of small fetch sessions and a bigger client comes in, we might have to evict many small sessions to fit the bigger one. Suppose there were N small fetch sessions and 1 big fetch session comes in. If the plan is to use number of partitions to

Re: [DISCUSS] KIP-227: Introduce Incremental FetchRequests to Increase Partition Scalability

2017-12-05 Thread Colin McCabe
On Tue, Dec 5, 2017, at 11:24, Ted Yu wrote: > bq. We also have a tunable for total number of cache slots. We never > cache > more than this number of incremental fetch sessions. > > Is it possible to manage the cache based on heap consumption instead of > number of slots ? > It seems heap

[jira] [Created] (KAFKA-6312) Add documentation about kafka-consumer-groups.sh's ability to set/change offsets

2017-12-05 Thread James Cheng (JIRA)
James Cheng created KAFKA-6312: -- Summary: Add documentation about kafka-consumer-groups.sh's ability to set/change offsets Key: KAFKA-6312 URL: https://issues.apache.org/jira/browse/KAFKA-6312 Project:

Re: [DISCUSS] KIP-228 Negative record timestamp support

2017-12-05 Thread Konstantin Chukhlomin
Hi Dong, Currently we are storing historical timestamp in the message. What we are trying to achieve is to make it possible to do Kafka lookup by timestamp. Ideally I would do `offsetsForTimes` to find articles published in 1910s (if we are storing articles on the log). So first two

Re: [DISCUSS] KIP-227: Introduce Incremental FetchRequests to Increase Partition Scalability

2017-12-05 Thread Jan Filipiak
Hi Colin Addressing the topic of how to manage slots from the other thread. With tcp connections all this comes for free essentially. I still would argue we disable it by default and make a flag in the broker to ask the leader to maintain the cache while replicating and also only have it

Re: [DISCUSS] KIP-228 Negative record timestamp support

2017-12-05 Thread Dong Lin
Hey Konstantin, Thanks for the KIP. I have a few questions below. Strictly speaking Kafka actually allows you to store historical data. And user are free to encode arbitrary timestamp field in their Kafka message. For example, your Kafka message can currently have Json or Avro format and you can

Re: [DISCUSS] KIP-228 Negative record timestamp support

2017-12-05 Thread Konstantin Chukhlomin
Yes, the point of that new class is to return same timestamp no mater if it's negative or not. And 5000 BC would be a valid timestamp. But I haven't tried to use streams with such historical data yet. > On Dec 5, 2017, at 3:02 PM, Ted Yu wrote: > > In the diff you gave,

Re: [DISCUSS] KIP-228 Negative record timestamp support

2017-12-05 Thread Ted Yu
In the diff you gave, onInvalidTimestamp() doesn't have any check. What if the timestamp corresponds to 5000 BC ? Is that still allowed ? Cheers On Tue, Dec 5, 2017 at 10:29 AM, Konstantin Chukhlomin wrote: > Hi Ted, > > Thank you for the response. > I made a relevant

Re: [DISCUSS] KIP-227: Introduce Incremental FetchRequests to Increase Partition Scalability

2017-12-05 Thread Ted Yu
bq. We also have a tunable for total number of cache slots. We never cache more than this number of incremental fetch sessions. Is it possible to manage the cache based on heap consumption instead of number of slots ? It seems heap estimation can be done by counting PartitionData (along with

Re: [DISCUSS] KIP 226 - Dynamic Broker Configuration

2017-12-05 Thread Colin McCabe
On Tue, Dec 5, 2017, at 06:01, Rajini Sivaram wrote: > Hi Colin, > > KAFKA-5722 already has an owner, so I didn't want to confuse the two > KIPs. They can be done independently of each other. The goal is to try and > validate every config to the minimum validation already in the broker for > the

Re: [DISCUSS] KIP-227: Introduce Incremental FetchRequests to Increase Partition Scalability

2017-12-05 Thread Colin McCabe
On Tue, Dec 5, 2017, at 08:51, Jason Gustafson wrote: > Hi Colin, > > Thanks for the response. A couple replies: > > > > I’m a bit ambivalent about letting the client choose the session > > timeout. What if clients choose timeouts that are too long? Hmm > > I do agree the timeout should be

[jira] [Created] (KAFKA-6311) Expose Kafka cluster ID in Connect REST API

2017-12-05 Thread JIRA
Xavier Léauté created KAFKA-6311: Summary: Expose Kafka cluster ID in Connect REST API Key: KAFKA-6311 URL: https://issues.apache.org/jira/browse/KAFKA-6311 Project: Kafka Issue Type:

Re: Comparing Pulsar and Kafka: unified queuing and streaming

2017-12-05 Thread Jason Gustafson
> I believe a lot of users are using the kafka high level consumers, it is > effectively an **unordered** messaging/streaming pattern. People using high > level consumers don't actually need any ordering guarantees. In this sense, > a *shared* subscription in Apache Pulsar seems to be better than

Re: [DISCUSS] KIP-228 Negative record timestamp support

2017-12-05 Thread Konstantin Chukhlomin
Hi Ted, Thank you for the response. I made a relevant changes to the KIP. > On Dec 5, 2017, at 11:59 AM, Ted Yu wrote: > > In KeepTimestampOnInvalidTimestamp, there should be check that timestamp is > < 0. > This would protect against future change to onInvalidTimestamp()

Re: [DISCUSS] KIP-227: Introduce Incremental FetchRequests to Increase Partition Scalability

2017-12-05 Thread Colin McCabe
On Sun, Dec 3, 2017, at 16:28, Becket Qin wrote: > >The correlation ID is used within a single TCP session, to uniquely > >associate a request with a response. The correlation ID is not unique > >(and has no meaning) outside the context of that single TCP session. > > > >Keep in mind,

Re: [VOTE] KIP-179: Change ReassignPartitionsCommand to use AdminClient

2017-12-05 Thread Tom Bentley
I am withdrawing this KIP. See the [DISCUSS] thread for the reasons why. On 7 November 2017 at 08:15, Tom Bentley wrote: > Hi, > > I would like to start a vote on KIP-179 which would add an AdminClient API > for partition reassignment and interbroker replication

Re: [DISCUSS] KIP-179: Change ReassignPartitionsCommand to use AdminClient

2017-12-05 Thread Tom Bentley
Hi all, I've been thinking about the proposed changes in KIP-179 and, on reflection, I don't think the API presented is really ideal. Some of the limitations it has include: 1. It sticks to the current, batch oriented (i.e. a single set of reassignments at a time), model. 2. It still doesn't

Re: [DISCUSS]KIP-235 DNS alias and secured connections

2017-12-05 Thread Tom Bentley
Hi Jonathan, It might be worth mentioning in the KIP that this is necessary only for *Kerberos* on SASL, and not other SASL mechanisms. Reading the JIRA it makes sensem, but I was confused up until that point. Cheers, Tom On 5 December 2017 at 17:53, Skrzypek, Jonathan

Re: Comparing Pulsar and Kafka: unified queuing and streaming

2017-12-05 Thread Khurrum Nasim
Hi Marina, On Tue, Dec 5, 2017 at 6:58 AM, Marina Popova wrote: > Hi, > I don't think it would be such a great idea to start modifying the very > foundation of Kafka's design to accommodate more and more extra use cases. > Kafka because so widely adopted and popular

[DISCUSS]KIP-235 DNS alias and secured connections

2017-12-05 Thread Skrzypek, Jonathan
Hi, I would like to discuss a KIP I've submitted : https://cwiki.apache.org/confluence/display/KAFKA/KIP-235%3A+Add+DNS+alias+support+for+secured+connection Feedback and suggestions welcome ! Regards, Jonathan Skrzypek Middleware Engineering Messaging Engineering Goldman Sachs International

Re: [DISCUSS] KIP-210: Provide for custom error handling when Kafka Streams fails to produce

2017-12-05 Thread Matt Farmer
I have updated this KIP accordingly. Can you please take a look and let me know if what I wrote looks correct to you? https://cwiki.apache.org/confluence/display/KAFKA/KIP-210+-+Provide+for+custom+error+handling++when+Kafka+Streams+fails+to+produce Thanks! Matt On December 4, 2017 at 9:39:13

Re: [DISCUSS] KIP-228 Negative record timestamp support

2017-12-05 Thread Ted Yu
In KeepTimestampOnInvalidTimestamp, there should be check that timestamp is < 0. This would protect against future change to onInvalidTimestamp() callback. Wednesday, December 31, 1969 11:59:59 PM UTC was in the past. Can you enrich Motivation section on why the proposal is made (writing data

Re: [DISCUSS] KIP-227: Introduce Incremental FetchRequests to Increase Partition Scalability

2017-12-05 Thread Jason Gustafson
Hi Colin, Thanks for the response. A couple replies: > I’m a bit ambivalent about letting the client choose the session > timeout. What if clients choose timeouts that are too long? Hmm > I do agree the timeout should be sized proportional to > max.poll.interval.ms. We have solved this

[DISCUSS] KIP-228 Negative record timestamp support

2017-12-05 Thread Konstantin Chukhlomin
Hi all, I have created a KIP to support negative timestamp: https://cwiki.apache.org/confluence/display/KAFKA/KIP-228+Negative+record+timestamp+support Here are proposed changes:

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

2017-12-05 Thread Bill Bejeck
Matthias, Overall I agree with what you've presented here. Initially, I was hesitant to remove information from the context of the result records (Joins or Aggregations) with the thought that when there are unexpected results, the source information would be useful for tracing back where the

Re: Comparing Pulsar and Kafka: unified queuing and streaming

2017-12-05 Thread Marina Popova
Hi, I don't think it would be such a great idea to start modifying the very foundation of Kafka's design to accommodate more and more extra use cases. Kafka because so widely adopted and popular because its creator made a brilliant decision to make it "dumb broker - smart consumer" type of the

Re: [DISCUSS] KIP-233: Simplify StreamsBuilder#addGlobalStore

2017-12-05 Thread Bill Bejeck
Hi Panuwat, Thanks for the KIP, overall looks good to me. I want to play the devil's advocate for a second and ask do we want to keep the older method with the extra parameters vs. deprecation? Although ATM I can't think of a good reason to keep the old method with the extra parameters.

[GitHub] kafka pull request #4294: MINOR: Include client-id in client authentication ...

2017-12-05 Thread rajinisivaram
Github user rajinisivaram closed the pull request at: https://github.com/apache/kafka/pull/4294 ---

Re: [DISCUSS] KIP 226 - Dynamic Broker Configuration

2017-12-05 Thread Rajini Sivaram
Hi Colin, KAFKA-5722 already has an owner, so I didn't want to confuse the two KIPs. They can be done independently of each other. The goal is to try and validate every config to the minimum validation already in the broker for the static configs, but in some cases to a more restrictive level. So

[GitHub] kafka pull request #4294: MINOR: Include client-id in client authentication ...

2017-12-05 Thread rajinisivaram
GitHub user rajinisivaram opened a pull request: https://github.com/apache/kafka/pull/4294 MINOR: Include client-id in client authentication failure error messages ### Committer Checklist (excluded from commit message) - [ ] Verify design and implementation - [ ] Verify

Re: [VOTE] KIP-225 - Use tags for consumer “records.lag” metrics

2017-12-05 Thread charly molter
Thanks Jun and Becket! I think your point about 1.0 vs 2.0 makes sense I can update the KIP to reflect this. What's the process for 2.0 contributions as I can see that trunk is 1.1 and no 2.x branch? Here's what I can do: - Not write the code change until trunk moves to 2.0. - Write the change

[jira] [Created] (KAFKA-6310) ConcurrentModificationException when reporting requests-in-flight in producer

2017-12-05 Thread Charly Molter (JIRA)
Charly Molter created KAFKA-6310: Summary: ConcurrentModificationException when reporting requests-in-flight in producer Key: KAFKA-6310 URL: https://issues.apache.org/jira/browse/KAFKA-6310 Project:

Re: [DISCUSS] KIP-233: Simplify StreamsBuilder#addGlobalStore

2017-12-05 Thread Ted Yu
Fine by me. On Tue, Dec 5, 2017 at 2:45 AM, Panuwat Anawatmongkhon < panuwat.anawatmongk...@gmail.com> wrote: > Thank you, Matthias. > > Ted, > How about this. > > String globalTopicName = "testGlobalTopic"; > String globalStoreName = "testAddGlobalStore"; > final StreamsBuilder builder = new

Re: [DISCUSS] KIP-233: Simplify StreamsBuilder#addGlobalStore

2017-12-05 Thread Panuwat Anawatmongkhon
Thank you, Matthias. Ted, How about this. String globalTopicName = "testGlobalTopic"; String globalStoreName = "testAddGlobalStore"; final StreamsBuilder builder = new StreamsBuilder(); final KeyValueStoreBuilder globalStoreBuilder = EasyMock.createNiceMock(KeyValueStoreBuilder.class);

[GitHub] kafka pull request #4293: KAFKA-6308: Connect Struct should use deepEquals/d...

2017-12-05 Thread tobiasgies
GitHub user tobiasgies opened a pull request: https://github.com/apache/kafka/pull/4293 KAFKA-6308: Connect Struct should use deepEquals/deepHashCode This changes the Struct's equals and hashCode method to use Arrays#deepEquals and Arrays#deepHashCode, respectively. This

Re: [DISCUSS] KIP-231: Improve the Required ACL of ListGroups API

2017-12-05 Thread Ismael Juma
One comment below. On Mon, Dec 4, 2017 at 11:40 PM, Dong Lin wrote: > In my opinion this changes the semantics of ListGroupsResponse in a > counter-intuitive way. Usually we use the ACL to determine whether the > operation on the specified object can be performed or not.

[GitHub] kafka pull request #4292: Is there something wrong? (for kafka 1.0)

2017-12-05 Thread helowken
GitHub user helowken opened a pull request: https://github.com/apache/kafka/pull/4292 Is there something wrong? (for kafka 1.0) In kafka stream source code: InternalTopologyBuilder.java