Re: [DISCUSS] KIP-291: Have separate queues for control requests and data requests

2018-07-18 Thread Lucas Wang
Hi Dong,

Sure. Regarding the 2nd case you mentioned
"- If the controller has not received response for R1 before it is
disconnected, it will re-send R1 followed by R2 after it is re-connected to
the broker."

with the max inflight request set to 1, after the connection is
re-established, the controller won't send R2
before it gets a response for R1, right? Plus the controller is using
blocking calls for each request, i.e.
NetworkClientUtils.sendAndReceive, with infinite retries for each request
within the same instance of RequestSendThread.
So within the same instance of RequestSendThread, sending out multiple
different requests seems impossible.

However, based on the comments in the source code, it seems multiple
requests can happen if
the broker loses its zk session, and then reconnects with zookeeper,
multiple generations of RequestSendThreads can trigger multiple different
requests.
In that case, we cannot prevent out-of-order processing even with the queue
since those multiple requests are from different connections.
Broker generations can help in those cases, but I won't dive into that
discussion.
Is that right?

Lucas

On Wed, Jul 18, 2018 at 9:08 PM, Dong Lin  wrote:

> Hey Lucas,
>
> I think for now we can probably discuss based on the existing Kafka's
> design where controller to a broker is hard coded to be 1. It looks like
> Becket has provided a good example in which requests from the same
> controller can be processed out of order.
>
> Thanks,
> Dong
>
> On Wed, Jul 18, 2018 at 8:35 PM, Lucas Wang  wrote:
>
> > @Becket and Dong,
> > I think currently the ordering guarantee is achieved because
> > the max inflight request from the controller to a broker is hard coded to
> > be 1.
> >
> > If let's hypothetically say the max inflight requests is > 1, then I
> think
> > Dong
> > is right to say that even the separate queue cannot guarantee ordered
> > processing,
> > For example, Req1 and Req2 are sent to a broker, and after a connection
> > reconnection,
> > both requests are sent again, causing the broker to have 4 requests in
> the
> > following order
> > Req2 > Req1 > Req2 > Req1.
> >
> > In summary, it seems using the dequeue should not cause problems with
> > out-of-order processing.
> > Is that right?
> >
> > Lucas
> >
> > On Wed, Jul 18, 2018 at 6:24 PM, Dong Lin  wrote:
> >
> > > Hey Becket,
> > >
> > > It seems that the requests from the old controller will be discarded
> due
> > to
> > > old controller epoch. It is not clear whether this is a problem.
> > >
> > > And if this out-of-order processing of controller requests is a
> problem,
> > it
> > > seems like an existing problem which also applies to the multi-queue
> > based
> > > design. So it is probably not a concern specific to the use of deque.
> > Does
> > > that sound reasonable?
> > >
> > > Thanks,
> > > Dong
> > >
> > >
> > > On Wed, 18 Jul 2018 at 6:17 PM Becket Qin 
> wrote:
> > >
> > > > Hi Mayuresh/Joel,
> > > >
> > > > Using the request channel as a dequeue was bright up some time ago
> when
> > > we
> > > > initially thinking of prioritizing the request. The concern was that
> > the
> > > > controller requests are supposed to be processed in order. If we can
> > > ensure
> > > > that there is one controller request in the request channel, the
> order
> > is
> > > > not a concern. But in cases that there are more than one controller
> > > request
> > > > inserted into the queue, the controller request order may change and
> > > cause
> > > > problem. For example, think about the following sequence:
> > > > 1. Controller successfully sent a request R1 to broker
> > > > 2. Broker receives R1 and put the request to the head of the request
> > > queue.
> > > > 3. Controller to broker connection failed and the controller
> > reconnected
> > > to
> > > > the broker.
> > > > 4. Controller sends a request R2 to the broker
> > > > 5. Broker receives R2 and add it to the head of the request queue.
> > > > Now on the broker side, R2 will be processed before R1 is processed,
> > > which
> > > > may cause problem.
> > > >
> > > > Thanks,
> > > >
> > > > Jiangjie (Becket) Qin
> > > >
> > > >
> > > >
> > > > On Thu, Jul 19, 2018 at 3:23 AM, Joel Koshy 
> > wrote:
> > > >
> > > > > @Mayuresh - I like your idea. It appears to be a simpler less
> > invasive
> > > > > alternative and it should work. Jun/Becket/others, do you see any
> > > > pitfalls
> > > > > with this approach?
> > > > >
> > > > > On Wed, Jul 18, 2018 at 12:03 PM, Lucas Wang <
> lucasatu...@gmail.com>
> > > > > wrote:
> > > > >
> > > > > > @Mayuresh,
> > > > > > That's a very interesting idea that I haven't thought before.
> > > > > > It seems to solve our problem at hand pretty well, and also
> > > > > > avoids the need to have a new size metric and capacity config
> > > > > > for the controller request queue. In fact, if we were to adopt
> > > > > > this design, there is no public interface change, and we
> > > > > > probably don't need a KIP.
> > > > > > Also 

Re: [DISCUSS] KIP-291: Have separate queues for control requests and data requests

2018-07-18 Thread Dong Lin
Hey Lucas,

I think for now we can probably discuss based on the existing Kafka's
design where controller to a broker is hard coded to be 1. It looks like
Becket has provided a good example in which requests from the same
controller can be processed out of order.

Thanks,
Dong

On Wed, Jul 18, 2018 at 8:35 PM, Lucas Wang  wrote:

> @Becket and Dong,
> I think currently the ordering guarantee is achieved because
> the max inflight request from the controller to a broker is hard coded to
> be 1.
>
> If let's hypothetically say the max inflight requests is > 1, then I think
> Dong
> is right to say that even the separate queue cannot guarantee ordered
> processing,
> For example, Req1 and Req2 are sent to a broker, and after a connection
> reconnection,
> both requests are sent again, causing the broker to have 4 requests in the
> following order
> Req2 > Req1 > Req2 > Req1.
>
> In summary, it seems using the dequeue should not cause problems with
> out-of-order processing.
> Is that right?
>
> Lucas
>
> On Wed, Jul 18, 2018 at 6:24 PM, Dong Lin  wrote:
>
> > Hey Becket,
> >
> > It seems that the requests from the old controller will be discarded due
> to
> > old controller epoch. It is not clear whether this is a problem.
> >
> > And if this out-of-order processing of controller requests is a problem,
> it
> > seems like an existing problem which also applies to the multi-queue
> based
> > design. So it is probably not a concern specific to the use of deque.
> Does
> > that sound reasonable?
> >
> > Thanks,
> > Dong
> >
> >
> > On Wed, 18 Jul 2018 at 6:17 PM Becket Qin  wrote:
> >
> > > Hi Mayuresh/Joel,
> > >
> > > Using the request channel as a dequeue was bright up some time ago when
> > we
> > > initially thinking of prioritizing the request. The concern was that
> the
> > > controller requests are supposed to be processed in order. If we can
> > ensure
> > > that there is one controller request in the request channel, the order
> is
> > > not a concern. But in cases that there are more than one controller
> > request
> > > inserted into the queue, the controller request order may change and
> > cause
> > > problem. For example, think about the following sequence:
> > > 1. Controller successfully sent a request R1 to broker
> > > 2. Broker receives R1 and put the request to the head of the request
> > queue.
> > > 3. Controller to broker connection failed and the controller
> reconnected
> > to
> > > the broker.
> > > 4. Controller sends a request R2 to the broker
> > > 5. Broker receives R2 and add it to the head of the request queue.
> > > Now on the broker side, R2 will be processed before R1 is processed,
> > which
> > > may cause problem.
> > >
> > > Thanks,
> > >
> > > Jiangjie (Becket) Qin
> > >
> > >
> > >
> > > On Thu, Jul 19, 2018 at 3:23 AM, Joel Koshy 
> wrote:
> > >
> > > > @Mayuresh - I like your idea. It appears to be a simpler less
> invasive
> > > > alternative and it should work. Jun/Becket/others, do you see any
> > > pitfalls
> > > > with this approach?
> > > >
> > > > On Wed, Jul 18, 2018 at 12:03 PM, Lucas Wang 
> > > > wrote:
> > > >
> > > > > @Mayuresh,
> > > > > That's a very interesting idea that I haven't thought before.
> > > > > It seems to solve our problem at hand pretty well, and also
> > > > > avoids the need to have a new size metric and capacity config
> > > > > for the controller request queue. In fact, if we were to adopt
> > > > > this design, there is no public interface change, and we
> > > > > probably don't need a KIP.
> > > > > Also implementation wise, it seems
> > > > > the java class LinkedBlockingQueue can readily satisfy the
> > requirement
> > > > > by supporting a capacity, and also allowing inserting at both ends.
> > > > >
> > > > > My only concern is that this design is tied to the coincidence that
> > > > > we have two request priorities and there are two ends to a deque.
> > > > > Hence by using the proposed design, it seems the network layer is
> > > > > more tightly coupled with upper layer logic, e.g. if we were to add
> > > > > an extra priority level in the future for some reason, we would
> > > probably
> > > > > need to go back to the design of separate queues, one for each
> > priority
> > > > > level.
> > > > >
> > > > > In summary, I'm ok with both designs and lean toward your suggested
> > > > > approach.
> > > > > Let's hear what others think.
> > > > >
> > > > > @Becket,
> > > > > In light of Mayuresh's suggested new design, I'm answering your
> > > question
> > > > > only in the context
> > > > > of the current KIP design: I think your suggestion makes sense, and
> > I'm
> > > > ok
> > > > > with removing the capacity config and
> > > > > just relying on the default value of 20 being sufficient enough.
> > > > >
> > > > > Thanks,
> > > > > Lucas
> > > > >
> > > > >
> > > > >
> > > > >
> > > > >
> > > > >
> > > > >
> > > > >
> > > > >
> > > > >
> > > > > On Wed, Jul 18, 2018 at 9:57 AM, Mayuresh Gharat <
> > > > > 

Re: [DISCUSS] KIP-291: Have separate queues for control requests and data requests

2018-07-18 Thread Dong Lin
Hey Becket,

Sorry I misunderstood your example. I thought you mean requests from
different controller are re-ordered.

I think you have provided a very good example and it should be safer to
still use two queues. Let me clarify the example a bit more below.

- If the controller has received response for R1 before it is disconnected
from the broker, it will send a new request R2 after it is re-connected to
the broker. There is no issue in this case because R1 will be processed
before R2.

- If the controller has not received response for R1 before it is
disconnected, it will re-send R1 followed by R2 after it is re-connected to
the broker. With high probability the order of processing should be R1, R1
and R2. This is because we have multiple request handler threads and the
first two R1 will typically both be processed before R2. With low
probability the order of processing will be R1, R2, R1, which can
potentially be a problem.

Thanks,
Dong

On Wed, Jul 18, 2018 at 6:24 PM, Dong Lin  wrote:

> Hey Becket,
>
> It seems that the requests from the old controller will be discarded due
> to old controller epoch. It is not clear whether this is a problem.
>
> And if this out-of-order processing of controller requests is a problem,
> it seems like an existing problem which also applies to the multi-queue
> based design. So it is probably not a concern specific to the use of deque.
> Does that sound reasonable?
>
> Thanks,
> Dong
>
>
> On Wed, 18 Jul 2018 at 6:17 PM Becket Qin  wrote:
>
>> Hi Mayuresh/Joel,
>>
>> Using the request channel as a dequeue was bright up some time ago when we
>> initially thinking of prioritizing the request. The concern was that the
>> controller requests are supposed to be processed in order. If we can
>> ensure
>> that there is one controller request in the request channel, the order is
>> not a concern. But in cases that there are more than one controller
>> request
>> inserted into the queue, the controller request order may change and cause
>> problem. For example, think about the following sequence:
>> 1. Controller successfully sent a request R1 to broker
>> 2. Broker receives R1 and put the request to the head of the request
>> queue.
>> 3. Controller to broker connection failed and the controller reconnected
>> to
>> the broker.
>> 4. Controller sends a request R2 to the broker
>> 5. Broker receives R2 and add it to the head of the request queue.
>> Now on the broker side, R2 will be processed before R1 is processed, which
>> may cause problem.
>>
>> Thanks,
>>
>> Jiangjie (Becket) Qin
>>
>>
>>
>> On Thu, Jul 19, 2018 at 3:23 AM, Joel Koshy  wrote:
>>
>> > @Mayuresh - I like your idea. It appears to be a simpler less invasive
>> > alternative and it should work. Jun/Becket/others, do you see any
>> pitfalls
>> > with this approach?
>> >
>> > On Wed, Jul 18, 2018 at 12:03 PM, Lucas Wang 
>> > wrote:
>> >
>> > > @Mayuresh,
>> > > That's a very interesting idea that I haven't thought before.
>> > > It seems to solve our problem at hand pretty well, and also
>> > > avoids the need to have a new size metric and capacity config
>> > > for the controller request queue. In fact, if we were to adopt
>> > > this design, there is no public interface change, and we
>> > > probably don't need a KIP.
>> > > Also implementation wise, it seems
>> > > the java class LinkedBlockingQueue can readily satisfy the requirement
>> > > by supporting a capacity, and also allowing inserting at both ends.
>> > >
>> > > My only concern is that this design is tied to the coincidence that
>> > > we have two request priorities and there are two ends to a deque.
>> > > Hence by using the proposed design, it seems the network layer is
>> > > more tightly coupled with upper layer logic, e.g. if we were to add
>> > > an extra priority level in the future for some reason, we would
>> probably
>> > > need to go back to the design of separate queues, one for each
>> priority
>> > > level.
>> > >
>> > > In summary, I'm ok with both designs and lean toward your suggested
>> > > approach.
>> > > Let's hear what others think.
>> > >
>> > > @Becket,
>> > > In light of Mayuresh's suggested new design, I'm answering your
>> question
>> > > only in the context
>> > > of the current KIP design: I think your suggestion makes sense, and
>> I'm
>> > ok
>> > > with removing the capacity config and
>> > > just relying on the default value of 20 being sufficient enough.
>> > >
>> > > Thanks,
>> > > Lucas
>> > >
>> > >
>> > >
>> > >
>> > >
>> > >
>> > >
>> > >
>> > >
>> > >
>> > > On Wed, Jul 18, 2018 at 9:57 AM, Mayuresh Gharat <
>> > > gharatmayures...@gmail.com
>> > > > wrote:
>> > >
>> > > > Hi Lucas,
>> > > >
>> > > > Seems like the main intent here is to prioritize the controller
>> request
>> > > > over any other requests.
>> > > > In that case, we can change the request queue to a dequeue, where
>> you
>> > > > always insert the normal requests (produce, consume,..etc) to the
>> end
>> > of
>> > > > the dequeue, but if 

Re: [DISCUSS] KIP-291: Have separate queues for control requests and data requests

2018-07-18 Thread Lucas Wang
@Becket and Dong,
I think currently the ordering guarantee is achieved because
the max inflight request from the controller to a broker is hard coded to
be 1.

If let's hypothetically say the max inflight requests is > 1, then I think
Dong
is right to say that even the separate queue cannot guarantee ordered
processing,
For example, Req1 and Req2 are sent to a broker, and after a connection
reconnection,
both requests are sent again, causing the broker to have 4 requests in the
following order
Req2 > Req1 > Req2 > Req1.

In summary, it seems using the dequeue should not cause problems with
out-of-order processing.
Is that right?

Lucas

On Wed, Jul 18, 2018 at 6:24 PM, Dong Lin  wrote:

> Hey Becket,
>
> It seems that the requests from the old controller will be discarded due to
> old controller epoch. It is not clear whether this is a problem.
>
> And if this out-of-order processing of controller requests is a problem, it
> seems like an existing problem which also applies to the multi-queue based
> design. So it is probably not a concern specific to the use of deque. Does
> that sound reasonable?
>
> Thanks,
> Dong
>
>
> On Wed, 18 Jul 2018 at 6:17 PM Becket Qin  wrote:
>
> > Hi Mayuresh/Joel,
> >
> > Using the request channel as a dequeue was bright up some time ago when
> we
> > initially thinking of prioritizing the request. The concern was that the
> > controller requests are supposed to be processed in order. If we can
> ensure
> > that there is one controller request in the request channel, the order is
> > not a concern. But in cases that there are more than one controller
> request
> > inserted into the queue, the controller request order may change and
> cause
> > problem. For example, think about the following sequence:
> > 1. Controller successfully sent a request R1 to broker
> > 2. Broker receives R1 and put the request to the head of the request
> queue.
> > 3. Controller to broker connection failed and the controller reconnected
> to
> > the broker.
> > 4. Controller sends a request R2 to the broker
> > 5. Broker receives R2 and add it to the head of the request queue.
> > Now on the broker side, R2 will be processed before R1 is processed,
> which
> > may cause problem.
> >
> > Thanks,
> >
> > Jiangjie (Becket) Qin
> >
> >
> >
> > On Thu, Jul 19, 2018 at 3:23 AM, Joel Koshy  wrote:
> >
> > > @Mayuresh - I like your idea. It appears to be a simpler less invasive
> > > alternative and it should work. Jun/Becket/others, do you see any
> > pitfalls
> > > with this approach?
> > >
> > > On Wed, Jul 18, 2018 at 12:03 PM, Lucas Wang 
> > > wrote:
> > >
> > > > @Mayuresh,
> > > > That's a very interesting idea that I haven't thought before.
> > > > It seems to solve our problem at hand pretty well, and also
> > > > avoids the need to have a new size metric and capacity config
> > > > for the controller request queue. In fact, if we were to adopt
> > > > this design, there is no public interface change, and we
> > > > probably don't need a KIP.
> > > > Also implementation wise, it seems
> > > > the java class LinkedBlockingQueue can readily satisfy the
> requirement
> > > > by supporting a capacity, and also allowing inserting at both ends.
> > > >
> > > > My only concern is that this design is tied to the coincidence that
> > > > we have two request priorities and there are two ends to a deque.
> > > > Hence by using the proposed design, it seems the network layer is
> > > > more tightly coupled with upper layer logic, e.g. if we were to add
> > > > an extra priority level in the future for some reason, we would
> > probably
> > > > need to go back to the design of separate queues, one for each
> priority
> > > > level.
> > > >
> > > > In summary, I'm ok with both designs and lean toward your suggested
> > > > approach.
> > > > Let's hear what others think.
> > > >
> > > > @Becket,
> > > > In light of Mayuresh's suggested new design, I'm answering your
> > question
> > > > only in the context
> > > > of the current KIP design: I think your suggestion makes sense, and
> I'm
> > > ok
> > > > with removing the capacity config and
> > > > just relying on the default value of 20 being sufficient enough.
> > > >
> > > > Thanks,
> > > > Lucas
> > > >
> > > >
> > > >
> > > >
> > > >
> > > >
> > > >
> > > >
> > > >
> > > >
> > > > On Wed, Jul 18, 2018 at 9:57 AM, Mayuresh Gharat <
> > > > gharatmayures...@gmail.com
> > > > > wrote:
> > > >
> > > > > Hi Lucas,
> > > > >
> > > > > Seems like the main intent here is to prioritize the controller
> > request
> > > > > over any other requests.
> > > > > In that case, we can change the request queue to a dequeue, where
> you
> > > > > always insert the normal requests (produce, consume,..etc) to the
> end
> > > of
> > > > > the dequeue, but if its a controller request, you insert it to the
> > head
> > > > of
> > > > > the queue. This ensures that the controller request will be given
> > > higher
> > > > > priority over other requests.
> > > > >
> > > > > Also 

Re: [Discuss] KIP-321: Add method to get TopicNameExtractor in TopologyDescription

2018-07-18 Thread Nishanth Pradeep
I have revised the KIP
.
Here is a summary of the changes.

   1. Changed return type from String to Set for Source#topics().

   Set Source#topics()

   2. Added method in TopologyDescription#Source to return the Pattern used
   by the Source node.

   Pattern Source#topicPattern()

   3. Changed return type of Sink#topicNameExtractor from Class to just TopicNameExtractor.

   TopicNameExtractor Sink#topicNameExtractor()

Best,
Nishanth Pradeep

On Sun, Jul 15, 2018 at 11:24 PM Guozhang Wang  wrote:

> Hi Nishanth,
>
> Since even combining these two the scope is still relatively small I'd
> suggest just do it in one KIP if you're willing to work on them. If you do
> not then pleas feel free to create the JIRA for the second step so that
> others can pick it up.
>
>
> Guozhang
>
> On Sun, Jul 15, 2018 at 6:14 PM, Matthias J. Sax 
> wrote:
>
> > There is no general protocol. We can include the changes in the current
> > KIP or do a second KIP.
> >
> > If you don't want to include the change in this KIP, please create a new
> > JIRA to track the other changes. You can assign the JIRA to yourself and
> > start a second KIP if you want. You can also "link" both JIRAs as
> > related to each other.
> >
> >
> > -Matthias
> >
> > On 7/15/18 12:50 PM, Nishanth Pradeep wrote:
> > > Thank you for the comments! I agree with these changes.
> > >
> > > So is the general protocol to update the same KIP, or is to scrap the
> > > current KIP and create a new one since it's beyond the scope of the
> > > original KIP? I am happy to do either.
> > >
> > > On Wed, Jul 4, 2018 at 1:48 PM Matthias J. Sax 
> > > wrote:
> > >
> > >> Sounds good to me.
> > >>
> > >> -Matthias
> > >>
> > >> On 7/4/18 10:53 AM, Guozhang Wang wrote:
> > >>> After looked through the current TopologyDescription I think I'd want
> > to
> > >>> combine the suggestions from John and Matthias on the API proposal.
> The
> > >>> motivations is that we have two relatively different functionalities
> > >>> provided from the APIs today:
> > >>>
> > >>> 1. Each interface's public functions, like
> > >>> SourceNode#topics(), GlobalStore#source(), which returns non-String
> > typed
> > >>> data. The hope was to let users programmatically leverage on those
> APIs
> > >> for
> > >>> runtime checking.
> > >>> 2. Each interface's impl class also have an implicit toString()
> > >> overridden
> > >>> to print the necessary information. This was designed for debugging
> > >>> purposes only during development cycles.
> > >>>
> > >>> What we've observed so far, though, is that users leverage 2) much
> more
> > >>> than 1) in practice, since it is more convienent to parse strings
> than
> > >>> recursively calling the APIs to get non-string fields. On the other
> > hand,
> > >>> the discussion controversy is more around 1), not 2). As for 2)
> people
> > >> seem
> > >>> to be on the right page anyways: print the topic lists if it is not
> > >>> dynamic, or print extractor string format otherwise. For 1) above we
> > >> should
> > >>> probably have all three `Set topics()`, `Pattern
> > topicPattern()`
> > >>> and `TopicNameExtractor topicExtractor()`; while for 2) I feel
> > >> comfortable
> > >>> relying on the TopicNameExtractor#toString() in `Source#toString()`
> > impl
> > >>> since even if users do not override this function, the default value
> > >>> `className@hashcode` still looks fine to me.
> > >>>
> > >>>
> > >>> Guozhang
> > >>>
> > >>>
> > >>>
> > >>> On Tue, Jul 3, 2018 at 11:22 PM, Matthias J. Sax <
> > matth...@confluent.io>
> > >>> wrote:
> > >>>
> >  I just double checked the discussion thread of KIP-120 that
> introduced
> >  `TopologyDescription`. Back than the argument was, that using the
> >  simplest option might be sufficient because the description is
> mostly
> >  used for debugging.
> > 
> >  Not sure if this argument holds. It seem that people built first
> more
> >  sophisticated tools using TopologyDescription.
> > 
> >  Final note: if we really want to add `topicPattern()` we might want
> to
> >  deprecate `topic()` and replace with `Set topics()`,
> because a
> >  source node can take a multiple topics, too.
> > 
> >  Just adding this for completeness of context to the discussion.
> > 
> > 
> >  -Matthias
> > 
> >  On 7/3/18 11:09 PM, Matthias J. Sax wrote:
> > > John,
> > >
> > > I am a little bit on the fence. In retrospective, it might have
> been
> > > better to add `topic()` and `topicPattern()` to source node and
> > return
> > >> a
> > > proper `Pattern` object instead of the pattern as a String.
> > >
> > > All other "payload" is just names and thus String naturally. From
> my
> > > point of view `TopologyDescription` should represent the `Topology`
> > in
> > >> a
> > > "machine readable" form plus 

Build failed in Jenkins: kafka-2.0-jdk8 #75

2018-07-18 Thread Apache Jenkins Server
See 


Changes:

[rajinisivaram] MINOR: Remove references to version 1.2 in docs (#5386)

[matthias] MINOR: improve docs version numbers (#5372)

--
[...truncated 2.47 MB...]
org.apache.kafka.streams.integration.ResetIntegrationTest > 
testReprocessingByDurationAfterResetWithoutIntermediateUserTopic STARTED

org.apache.kafka.streams.integration.ResetIntegrationTest > 
testReprocessingByDurationAfterResetWithoutIntermediateUserTopic FAILED
kafka.zookeeper.ZooKeeperClientTimeoutException: Timed out waiting for 
connection while in state: CONNECTING
at 
kafka.zookeeper.ZooKeeperClient$$anonfun$kafka$zookeeper$ZooKeeperClient$$waitUntilConnected$1.apply$mcV$sp(ZooKeeperClient.scala:225)
at 
kafka.zookeeper.ZooKeeperClient$$anonfun$kafka$zookeeper$ZooKeeperClient$$waitUntilConnected$1.apply(ZooKeeperClient.scala:221)
at 
kafka.zookeeper.ZooKeeperClient$$anonfun$kafka$zookeeper$ZooKeeperClient$$waitUntilConnected$1.apply(ZooKeeperClient.scala:221)
at kafka.utils.CoreUtils$.inLock(CoreUtils.scala:251)
at 
kafka.zookeeper.ZooKeeperClient.kafka$zookeeper$ZooKeeperClient$$waitUntilConnected(ZooKeeperClient.scala:221)
at kafka.zookeeper.ZooKeeperClient.(ZooKeeperClient.scala:95)
at kafka.zk.KafkaZkClient$.apply(KafkaZkClient.scala:1580)
at kafka.zk.KafkaZkClient.apply(KafkaZkClient.scala)
at 
org.apache.kafka.streams.integration.utils.KafkaEmbedded.createZkClient(KafkaEmbedded.java:181)
at 
org.apache.kafka.streams.integration.utils.KafkaEmbedded.deleteTopic(KafkaEmbedded.java:188)
at 
org.apache.kafka.streams.integration.utils.EmbeddedKafkaCluster.deleteTopicsAndWait(EmbeddedKafkaCluster.java:268)
at 
org.apache.kafka.streams.integration.utils.EmbeddedKafkaCluster.deleteAndRecreateTopics(EmbeddedKafkaCluster.java:278)
at 
org.apache.kafka.streams.integration.AbstractResetIntegrationTest.prepareTest(AbstractResetIntegrationTest.java:195)
at 
org.apache.kafka.streams.integration.ResetIntegrationTest.before(ResetIntegrationTest.java:62)

org.apache.kafka.streams.integration.ResetIntegrationTest > 
testReprocessingFromScratchAfterResetWithIntermediateUserTopic STARTED

org.apache.kafka.streams.integration.ResetIntegrationTest > 
testReprocessingFromScratchAfterResetWithIntermediateUserTopic PASSED

org.apache.kafka.streams.integration.ResetIntegrationTest > 
testReprocessingFromScratchAfterResetWithoutIntermediateUserTopic STARTED

org.apache.kafka.streams.integration.ResetIntegrationTest > 
testReprocessingFromScratchAfterResetWithoutIntermediateUserTopic PASSED

org.apache.kafka.streams.integration.ResetIntegrationTest > 
testReprocessingFromFileAfterResetWithoutIntermediateUserTopic STARTED

org.apache.kafka.streams.integration.ResetIntegrationTest > 
testReprocessingFromFileAfterResetWithoutIntermediateUserTopic PASSED

org.apache.kafka.streams.integration.ResetIntegrationTest > 
testReprocessingFromDateTimeAfterResetWithoutIntermediateUserTopic STARTED

org.apache.kafka.streams.integration.ResetIntegrationTest > 
testReprocessingFromDateTimeAfterResetWithoutIntermediateUserTopic PASSED

org.apache.kafka.streams.integration.ResetIntegrationTest > 
shouldNotAllowToResetWhileStreamsRunning STARTED

org.apache.kafka.streams.integration.ResetIntegrationTest > 
shouldNotAllowToResetWhileStreamsRunning PASSED

org.apache.kafka.streams.integration.TableTableJoinIntegrationTest > 
testLeftInner[caching enabled = true] STARTED

org.apache.kafka.streams.integration.TableTableJoinIntegrationTest > 
testLeftInner[caching enabled = true] PASSED

org.apache.kafka.streams.integration.TableTableJoinIntegrationTest > 
testLeftOuter[caching enabled = true] STARTED

org.apache.kafka.streams.integration.TableTableJoinIntegrationTest > 
testLeftOuter[caching enabled = true] PASSED

org.apache.kafka.streams.integration.TableTableJoinIntegrationTest > 
testLeftLeft[caching enabled = true] STARTED

org.apache.kafka.streams.integration.TableTableJoinIntegrationTest > 
testLeftLeft[caching enabled = true] PASSED

org.apache.kafka.streams.integration.TableTableJoinIntegrationTest > 
testOuterLeft[caching enabled = true] STARTED

org.apache.kafka.streams.integration.TableTableJoinIntegrationTest > 
testOuterLeft[caching enabled = true] PASSED

org.apache.kafka.streams.integration.TableTableJoinIntegrationTest > 
testInner[caching enabled = true] STARTED

org.apache.kafka.streams.integration.TableTableJoinIntegrationTest > 
testInner[caching enabled = true] PASSED

org.apache.kafka.streams.integration.TableTableJoinIntegrationTest > 
testOuter[caching enabled = true] STARTED

org.apache.kafka.streams.integration.TableTableJoinIntegrationTest > 
testOuter[caching enabled = true] PASSED

org.apache.kafka.streams.integration.TableTableJoinIntegrationTest > 
testLeft[caching enabled = true] STARTED


Re: [DISCUSS] KIP-291: Have separate queues for control requests and data requests

2018-07-18 Thread Dong Lin
Hey Becket,

It seems that the requests from the old controller will be discarded due to
old controller epoch. It is not clear whether this is a problem.

And if this out-of-order processing of controller requests is a problem, it
seems like an existing problem which also applies to the multi-queue based
design. So it is probably not a concern specific to the use of deque. Does
that sound reasonable?

Thanks,
Dong


On Wed, 18 Jul 2018 at 6:17 PM Becket Qin  wrote:

> Hi Mayuresh/Joel,
>
> Using the request channel as a dequeue was bright up some time ago when we
> initially thinking of prioritizing the request. The concern was that the
> controller requests are supposed to be processed in order. If we can ensure
> that there is one controller request in the request channel, the order is
> not a concern. But in cases that there are more than one controller request
> inserted into the queue, the controller request order may change and cause
> problem. For example, think about the following sequence:
> 1. Controller successfully sent a request R1 to broker
> 2. Broker receives R1 and put the request to the head of the request queue.
> 3. Controller to broker connection failed and the controller reconnected to
> the broker.
> 4. Controller sends a request R2 to the broker
> 5. Broker receives R2 and add it to the head of the request queue.
> Now on the broker side, R2 will be processed before R1 is processed, which
> may cause problem.
>
> Thanks,
>
> Jiangjie (Becket) Qin
>
>
>
> On Thu, Jul 19, 2018 at 3:23 AM, Joel Koshy  wrote:
>
> > @Mayuresh - I like your idea. It appears to be a simpler less invasive
> > alternative and it should work. Jun/Becket/others, do you see any
> pitfalls
> > with this approach?
> >
> > On Wed, Jul 18, 2018 at 12:03 PM, Lucas Wang 
> > wrote:
> >
> > > @Mayuresh,
> > > That's a very interesting idea that I haven't thought before.
> > > It seems to solve our problem at hand pretty well, and also
> > > avoids the need to have a new size metric and capacity config
> > > for the controller request queue. In fact, if we were to adopt
> > > this design, there is no public interface change, and we
> > > probably don't need a KIP.
> > > Also implementation wise, it seems
> > > the java class LinkedBlockingQueue can readily satisfy the requirement
> > > by supporting a capacity, and also allowing inserting at both ends.
> > >
> > > My only concern is that this design is tied to the coincidence that
> > > we have two request priorities and there are two ends to a deque.
> > > Hence by using the proposed design, it seems the network layer is
> > > more tightly coupled with upper layer logic, e.g. if we were to add
> > > an extra priority level in the future for some reason, we would
> probably
> > > need to go back to the design of separate queues, one for each priority
> > > level.
> > >
> > > In summary, I'm ok with both designs and lean toward your suggested
> > > approach.
> > > Let's hear what others think.
> > >
> > > @Becket,
> > > In light of Mayuresh's suggested new design, I'm answering your
> question
> > > only in the context
> > > of the current KIP design: I think your suggestion makes sense, and I'm
> > ok
> > > with removing the capacity config and
> > > just relying on the default value of 20 being sufficient enough.
> > >
> > > Thanks,
> > > Lucas
> > >
> > >
> > >
> > >
> > >
> > >
> > >
> > >
> > >
> > >
> > > On Wed, Jul 18, 2018 at 9:57 AM, Mayuresh Gharat <
> > > gharatmayures...@gmail.com
> > > > wrote:
> > >
> > > > Hi Lucas,
> > > >
> > > > Seems like the main intent here is to prioritize the controller
> request
> > > > over any other requests.
> > > > In that case, we can change the request queue to a dequeue, where you
> > > > always insert the normal requests (produce, consume,..etc) to the end
> > of
> > > > the dequeue, but if its a controller request, you insert it to the
> head
> > > of
> > > > the queue. This ensures that the controller request will be given
> > higher
> > > > priority over other requests.
> > > >
> > > > Also since we only read one request from the socket and mute it and
> > only
> > > > unmute it after handling the request, this would ensure that we don't
> > > > handle controller requests out of order.
> > > >
> > > > With this approach we can avoid the second queue and the additional
> > > config
> > > > for the size of the queue.
> > > >
> > > > What do you think ?
> > > >
> > > > Thanks,
> > > >
> > > > Mayuresh
> > > >
> > > >
> > > > On Wed, Jul 18, 2018 at 3:05 AM Becket Qin 
> > wrote:
> > > >
> > > > > Hey Joel,
> > > > >
> > > > > Thank for the detail explanation. I agree the current design makes
> > > sense.
> > > > > My confusion is about whether the new config for the controller
> queue
> > > > > capacity is necessary. I cannot think of a case in which users
> would
> > > > change
> > > > > it.
> > > > >
> > > > > Thanks,
> > > > >
> > > > > Jiangjie (Becket) Qin
> > > > >
> > > > > On Wed, Jul 18, 2018 at 6:00 PM, Becket 

Re: [DISCUSS] KIP-291: Have separate queues for control requests and data requests

2018-07-18 Thread Becket Qin
Hi Mayuresh/Joel,

Using the request channel as a dequeue was bright up some time ago when we
initially thinking of prioritizing the request. The concern was that the
controller requests are supposed to be processed in order. If we can ensure
that there is one controller request in the request channel, the order is
not a concern. But in cases that there are more than one controller request
inserted into the queue, the controller request order may change and cause
problem. For example, think about the following sequence:
1. Controller successfully sent a request R1 to broker
2. Broker receives R1 and put the request to the head of the request queue.
3. Controller to broker connection failed and the controller reconnected to
the broker.
4. Controller sends a request R2 to the broker
5. Broker receives R2 and add it to the head of the request queue.
Now on the broker side, R2 will be processed before R1 is processed, which
may cause problem.

Thanks,

Jiangjie (Becket) Qin



On Thu, Jul 19, 2018 at 3:23 AM, Joel Koshy  wrote:

> @Mayuresh - I like your idea. It appears to be a simpler less invasive
> alternative and it should work. Jun/Becket/others, do you see any pitfalls
> with this approach?
>
> On Wed, Jul 18, 2018 at 12:03 PM, Lucas Wang 
> wrote:
>
> > @Mayuresh,
> > That's a very interesting idea that I haven't thought before.
> > It seems to solve our problem at hand pretty well, and also
> > avoids the need to have a new size metric and capacity config
> > for the controller request queue. In fact, if we were to adopt
> > this design, there is no public interface change, and we
> > probably don't need a KIP.
> > Also implementation wise, it seems
> > the java class LinkedBlockingQueue can readily satisfy the requirement
> > by supporting a capacity, and also allowing inserting at both ends.
> >
> > My only concern is that this design is tied to the coincidence that
> > we have two request priorities and there are two ends to a deque.
> > Hence by using the proposed design, it seems the network layer is
> > more tightly coupled with upper layer logic, e.g. if we were to add
> > an extra priority level in the future for some reason, we would probably
> > need to go back to the design of separate queues, one for each priority
> > level.
> >
> > In summary, I'm ok with both designs and lean toward your suggested
> > approach.
> > Let's hear what others think.
> >
> > @Becket,
> > In light of Mayuresh's suggested new design, I'm answering your question
> > only in the context
> > of the current KIP design: I think your suggestion makes sense, and I'm
> ok
> > with removing the capacity config and
> > just relying on the default value of 20 being sufficient enough.
> >
> > Thanks,
> > Lucas
> >
> >
> >
> >
> >
> >
> >
> >
> >
> >
> > On Wed, Jul 18, 2018 at 9:57 AM, Mayuresh Gharat <
> > gharatmayures...@gmail.com
> > > wrote:
> >
> > > Hi Lucas,
> > >
> > > Seems like the main intent here is to prioritize the controller request
> > > over any other requests.
> > > In that case, we can change the request queue to a dequeue, where you
> > > always insert the normal requests (produce, consume,..etc) to the end
> of
> > > the dequeue, but if its a controller request, you insert it to the head
> > of
> > > the queue. This ensures that the controller request will be given
> higher
> > > priority over other requests.
> > >
> > > Also since we only read one request from the socket and mute it and
> only
> > > unmute it after handling the request, this would ensure that we don't
> > > handle controller requests out of order.
> > >
> > > With this approach we can avoid the second queue and the additional
> > config
> > > for the size of the queue.
> > >
> > > What do you think ?
> > >
> > > Thanks,
> > >
> > > Mayuresh
> > >
> > >
> > > On Wed, Jul 18, 2018 at 3:05 AM Becket Qin 
> wrote:
> > >
> > > > Hey Joel,
> > > >
> > > > Thank for the detail explanation. I agree the current design makes
> > sense.
> > > > My confusion is about whether the new config for the controller queue
> > > > capacity is necessary. I cannot think of a case in which users would
> > > change
> > > > it.
> > > >
> > > > Thanks,
> > > >
> > > > Jiangjie (Becket) Qin
> > > >
> > > > On Wed, Jul 18, 2018 at 6:00 PM, Becket Qin 
> > > wrote:
> > > >
> > > > > Hi Lucas,
> > > > >
> > > > > I guess my question can be rephrased to "do we expect user to ever
> > > change
> > > > > the controller request queue capacity"? If we agree that 20 is
> > already
> > > a
> > > > > very generous default number and we do not expect user to change
> it,
> > is
> > > > it
> > > > > still necessary to expose this as a config?
> > > > >
> > > > > Thanks,
> > > > >
> > > > > Jiangjie (Becket) Qin
> > > > >
> > > > > On Wed, Jul 18, 2018 at 2:29 AM, Lucas Wang  >
> > > > wrote:
> > > > >
> > > > >> @Becket
> > > > >> 1. Thanks for the comment. You are right that normally there
> should
> > be
> > > > >> just
> > > > >> one controller request because of muting,

Build failed in Jenkins: kafka-1.0-jdk7 #221

2018-07-18 Thread Apache Jenkins Server
See 


Changes:

[github] MINOR: improve docs version numbers (#5372)

--
[...truncated 379.25 KB...]

kafka.admin.AdminTest > testPartitionReassignmentWithLeaderInNewReplicas STARTED

kafka.admin.AdminTest > testPartitionReassignmentWithLeaderInNewReplicas PASSED

kafka.admin.AdminTest > shouldPropagateDynamicBrokerConfigs STARTED

kafka.admin.AdminTest > shouldPropagateDynamicBrokerConfigs PASSED

kafka.admin.AdminTest > testShutdownBroker STARTED

kafka.admin.AdminTest > testShutdownBroker PASSED

kafka.admin.AdminTest > testTopicCreationWithCollision STARTED

kafka.admin.AdminTest > testTopicCreationWithCollision PASSED

kafka.admin.AdminTest > testTopicCreationInZK STARTED

kafka.admin.AdminTest > testTopicCreationInZK PASSED

kafka.admin.AclCommandTest > testInvalidAuthorizerProperty STARTED

kafka.admin.AclCommandTest > testInvalidAuthorizerProperty PASSED

kafka.admin.AclCommandTest > testAclCli STARTED

kafka.admin.AclCommandTest > testAclCli PASSED

kafka.admin.AclCommandTest > testProducerConsumerCli STARTED

kafka.admin.AclCommandTest > testProducerConsumerCli PASSED

kafka.admin.ConfigCommandTest > testScramCredentials STARTED

kafka.admin.ConfigCommandTest > testScramCredentials PASSED

kafka.admin.ConfigCommandTest > shouldParseArgumentsForTopicsEntityType STARTED

kafka.admin.ConfigCommandTest > shouldParseArgumentsForTopicsEntityType PASSED

kafka.admin.ConfigCommandTest > shouldExitWithNonZeroStatusOnZkCommandError 
STARTED

kafka.admin.ConfigCommandTest > shouldExitWithNonZeroStatusOnZkCommandError 
PASSED

kafka.admin.ConfigCommandTest > testUserClientQuotaOpts STARTED

kafka.admin.ConfigCommandTest > testUserClientQuotaOpts PASSED

kafka.admin.ConfigCommandTest > shouldAddTopicConfig STARTED

kafka.admin.ConfigCommandTest > shouldAddTopicConfig PASSED

kafka.admin.ConfigCommandTest > shouldAddClientConfig STARTED

kafka.admin.ConfigCommandTest > shouldAddClientConfig PASSED

kafka.admin.ConfigCommandTest > shouldDeleteBrokerConfig STARTED

kafka.admin.ConfigCommandTest > shouldDeleteBrokerConfig PASSED

kafka.admin.ConfigCommandTest > testQuotaConfigEntity STARTED

kafka.admin.ConfigCommandTest > testQuotaConfigEntity PASSED

kafka.admin.ConfigCommandTest > 
shouldNotUpdateBrokerConfigIfMalformedBracketConfig STARTED

kafka.admin.ConfigCommandTest > 
shouldNotUpdateBrokerConfigIfMalformedBracketConfig PASSED

kafka.admin.ConfigCommandTest > shouldFailIfUnrecognisedEntityType STARTED

kafka.admin.ConfigCommandTest > shouldFailIfUnrecognisedEntityType PASSED

kafka.admin.ConfigCommandTest > 
shouldNotUpdateBrokerConfigIfNonExistingConfigIsDeleted STARTED

kafka.admin.ConfigCommandTest > 
shouldNotUpdateBrokerConfigIfNonExistingConfigIsDeleted PASSED

kafka.admin.ConfigCommandTest > 
shouldNotUpdateBrokerConfigIfMalformedEntityName STARTED

kafka.admin.ConfigCommandTest > 
shouldNotUpdateBrokerConfigIfMalformedEntityName PASSED

kafka.admin.ConfigCommandTest > shouldSupportCommaSeparatedValues STARTED

kafka.admin.ConfigCommandTest > shouldSupportCommaSeparatedValues PASSED

kafka.admin.ConfigCommandTest > shouldNotUpdateBrokerConfigIfMalformedConfig 
STARTED

kafka.admin.ConfigCommandTest > shouldNotUpdateBrokerConfigIfMalformedConfig 
PASSED

kafka.admin.ConfigCommandTest > shouldParseArgumentsForBrokersEntityType STARTED

kafka.admin.ConfigCommandTest > shouldParseArgumentsForBrokersEntityType PASSED

kafka.admin.ConfigCommandTest > shouldAddBrokerConfig STARTED

kafka.admin.ConfigCommandTest > shouldAddBrokerConfig PASSED

kafka.admin.ConfigCommandTest > testQuotaDescribeEntities STARTED

kafka.admin.ConfigCommandTest > testQuotaDescribeEntities PASSED

kafka.admin.ConfigCommandTest > shouldParseArgumentsForClientsEntityType STARTED

kafka.admin.ConfigCommandTest > shouldParseArgumentsForClientsEntityType PASSED

kafka.admin.ConfigCommandTest > shouldExitWithNonZeroStatusOnArgError STARTED

kafka.admin.ConfigCommandTest > shouldExitWithNonZeroStatusOnArgError PASSED

kafka.producer.ProducerTest > testSendToNewTopic STARTED

kafka.producer.ProducerTest > testSendToNewTopic PASSED

kafka.producer.ProducerTest > testAsyncSendCanCorrectlyFailWithTimeout STARTED

kafka.producer.ProducerTest > testAsyncSendCanCorrectlyFailWithTimeout PASSED

kafka.producer.ProducerTest > testSendNullMessage STARTED

kafka.producer.ProducerTest > testSendNullMessage PASSED

kafka.producer.ProducerTest > testUpdateBrokerPartitionInfo STARTED

kafka.producer.ProducerTest > testUpdateBrokerPartitionInfo PASSED

kafka.producer.ProducerTest > testSendWithDeadBroker STARTED

kafka.producer.ProducerTest > testSendWithDeadBroker PASSED

kafka.producer.SyncProducerTest > testReachableServer STARTED

kafka.producer.SyncProducerTest > testReachableServer PASSED

kafka.producer.SyncProducerTest > testMessageSizeTooLarge STARTED

kafka.producer.SyncProducerTest > testMessageSizeTooLarge PASSED


KAFKA-7137 - max time guarantees of compaction for GDPR et al

2018-07-18 Thread Brett Rann
re: https://issues.apache.org/jira/browse/KAFKA-7137

My team is investigating what would be involved in code changes to give
some finer control over when compaction runs.  Detail in the ticket but
essentially the current way is that dirty.ratio is hijacked to be set to 0
to give guarantees that a tombstone record will result in previous record
being deleted within a set time.

This results in a lot of unnecessary compaction, and we would like to add
the ability to provide a max delay for compaction to be used in combination
with a "healthy" dirty.ratio setting to meet this requirement instead.

And to consider an API to trigger compaction (but that is looking difficult
because compaction isn't really triggered, it's evaluated by the log
cleaner thread).

We'd like to get some discussion going around this.  Should we draft up a
KIP first then kick off a discussion?  (if yes, can I get KIP access for:
brettr...@gmail.com )

Thanks!

-- 

Brett Rann

Senior DevOps Engineer


Zendesk International Ltd

395 Collins Street, Melbourne VIC 3000 Australia

Mobile: +61 (0) 418 826 017


Build failed in Jenkins: kafka-trunk-jdk8 #2804

2018-07-18 Thread Apache Jenkins Server
See 


Changes:

[github] MINOR: remove unused MeteredKeyValueStore (#5380)

[github] MINOR: Remove references to version 1.2 in docs (#5386)

[matthias] MINOR: improve docs version numbers (#5372)

--
[...truncated 878.89 KB...]
kafka.utils.ShutdownableThreadTest > testShutdownWhenCalledAfterThreadStart 
PASSED

kafka.utils.SchedulerTest > testMockSchedulerNonPeriodicTask STARTED

kafka.utils.SchedulerTest > testMockSchedulerNonPeriodicTask PASSED

kafka.utils.SchedulerTest > testMockSchedulerPeriodicTask STARTED

kafka.utils.SchedulerTest > testMockSchedulerPeriodicTask PASSED

kafka.utils.SchedulerTest > testNonPeriodicTask STARTED

kafka.utils.SchedulerTest > testNonPeriodicTask PASSED

kafka.utils.SchedulerTest > testRestart STARTED

kafka.utils.SchedulerTest > testRestart PASSED

kafka.utils.SchedulerTest > testReentrantTaskInMockScheduler STARTED

kafka.utils.SchedulerTest > testReentrantTaskInMockScheduler PASSED

kafka.utils.SchedulerTest > testPeriodicTask STARTED

kafka.utils.SchedulerTest > testPeriodicTask PASSED

kafka.utils.LoggingTest > testLog4jControllerIsRegistered STARTED

kafka.utils.LoggingTest > testLog4jControllerIsRegistered PASSED

kafka.utils.LoggingTest > testLogName STARTED

kafka.utils.LoggingTest > testLogName PASSED

kafka.utils.LoggingTest > testLogNameOverride STARTED

kafka.utils.LoggingTest > testLogNameOverride PASSED

kafka.utils.ZkUtilsTest > testGetSequenceIdMethod STARTED

kafka.utils.ZkUtilsTest > testGetSequenceIdMethod PASSED

kafka.utils.ZkUtilsTest > testAbortedConditionalDeletePath STARTED

kafka.utils.ZkUtilsTest > testAbortedConditionalDeletePath PASSED

kafka.utils.ZkUtilsTest > testGetAllPartitionsTopicWithoutPartitions STARTED

kafka.utils.ZkUtilsTest > testGetAllPartitionsTopicWithoutPartitions PASSED

kafka.utils.ZkUtilsTest > testSuccessfulConditionalDeletePath STARTED

kafka.utils.ZkUtilsTest > testSuccessfulConditionalDeletePath PASSED

kafka.utils.ZkUtilsTest > testPersistentSequentialPath STARTED

kafka.utils.ZkUtilsTest > testPersistentSequentialPath PASSED

kafka.utils.ZkUtilsTest > testClusterIdentifierJsonParsing STARTED

kafka.utils.ZkUtilsTest > testClusterIdentifierJsonParsing PASSED

kafka.utils.ZkUtilsTest > testGetLeaderIsrAndEpochForPartition STARTED

kafka.utils.ZkUtilsTest > testGetLeaderIsrAndEpochForPartition PASSED

kafka.utils.CoreUtilsTest > testGenerateUuidAsBase64 STARTED

kafka.utils.CoreUtilsTest > testGenerateUuidAsBase64 PASSED

kafka.utils.CoreUtilsTest > testAbs STARTED

kafka.utils.CoreUtilsTest > testAbs PASSED

kafka.utils.CoreUtilsTest > testReplaceSuffix STARTED

kafka.utils.CoreUtilsTest > testReplaceSuffix PASSED

kafka.utils.CoreUtilsTest > testCircularIterator STARTED

kafka.utils.CoreUtilsTest > testCircularIterator PASSED

kafka.utils.CoreUtilsTest > testReadBytes STARTED

kafka.utils.CoreUtilsTest > testReadBytes PASSED

kafka.utils.CoreUtilsTest > testCsvList STARTED

kafka.utils.CoreUtilsTest > testCsvList PASSED

kafka.utils.CoreUtilsTest > testReadInt STARTED

kafka.utils.CoreUtilsTest > testReadInt PASSED

kafka.utils.CoreUtilsTest > testAtomicGetOrUpdate STARTED

kafka.utils.CoreUtilsTest > testAtomicGetOrUpdate PASSED

kafka.utils.CoreUtilsTest > testUrlSafeBase64EncodeUUID STARTED

kafka.utils.CoreUtilsTest > testUrlSafeBase64EncodeUUID PASSED

kafka.utils.CoreUtilsTest > testCsvMap STARTED

kafka.utils.CoreUtilsTest > testCsvMap PASSED

kafka.utils.CoreUtilsTest > testInLock STARTED

kafka.utils.CoreUtilsTest > testInLock PASSED

kafka.utils.CoreUtilsTest > testTryAll STARTED

kafka.utils.CoreUtilsTest > testTryAll PASSED

kafka.utils.CoreUtilsTest > testSwallow STARTED

kafka.utils.CoreUtilsTest > testSwallow PASSED

kafka.utils.IteratorTemplateTest > testIterator STARTED

kafka.utils.IteratorTemplateTest > testIterator PASSED

kafka.utils.json.JsonValueTest > testJsonObjectIterator STARTED

kafka.utils.json.JsonValueTest > testJsonObjectIterator PASSED

kafka.utils.json.JsonValueTest > testDecodeLong STARTED

kafka.utils.json.JsonValueTest > testDecodeLong PASSED

kafka.utils.json.JsonValueTest > testAsJsonObject STARTED

kafka.utils.json.JsonValueTest > testAsJsonObject PASSED

kafka.utils.json.JsonValueTest > testDecodeDouble STARTED

kafka.utils.json.JsonValueTest > testDecodeDouble PASSED

kafka.utils.json.JsonValueTest > testDecodeOption STARTED

kafka.utils.json.JsonValueTest > testDecodeOption PASSED

kafka.utils.json.JsonValueTest > testDecodeString STARTED

kafka.utils.json.JsonValueTest > testDecodeString PASSED

kafka.utils.json.JsonValueTest > testJsonValueToString STARTED

kafka.utils.json.JsonValueTest > testJsonValueToString PASSED

kafka.utils.json.JsonValueTest > testAsJsonObjectOption STARTED

kafka.utils.json.JsonValueTest > testAsJsonObjectOption PASSED

kafka.utils.json.JsonValueTest > testAsJsonArrayOption STARTED

kafka.utils.json.JsonValueTest > 

Re: [VOTE] KIP-338 Support to exclude the internal topics in kafka-topics.sh command

2018-07-18 Thread Jason Gustafson
>
> (just a double check) the option name used in kip-338 is
> "--exclude-internal" rather than "--exclude-internal-topics". Do you prefer
> "--exclude-internal-topics"?


Nah, just a typo on my part. I think --exclude-internal is clear enough
given the name of the tool. Sorry for the confusion.

-Jason

On Tue, Jul 17, 2018 at 6:51 PM, Gwen Shapira  wrote:

> +1 (binding)
>
> Thank you for the improvement!
>
> On Mon, Jul 16, 2018 at 4:10 AM, Chia-Ping Tsai 
> wrote:
>
> > hi folks,
> >
> > The discussion[1] of KIP-338[2] did not get any objection for last 6 days
> > so it is time to start the voting thread.
> >
> > Thanks for your time!
> >
> > [1] https://lists.apache.org/thread.html/9bd4e61b73c901b51132ada49743b9
> > b703d40b85fc4eeaa5c9099900@%3Cdev.kafka.apache.org%3E
> >
> > [2] https://cwiki.apache.org/confluence/display/KAFKA/KIP-
> > 338+Support+to+exclude+the+internal+topics+in+kafka-topics.sh+command
> >
> > Cheers,
> > chia-ping
> >
>
>
>
> --
> *Gwen Shapira*
> Product Manager | Confluent
> 650.450.2760 | @gwenshap
> Follow us: Twitter  | blog
> 
>


Re: [DISCUSS] KIP-341: Update Sticky Assignor's User Data Protocol

2018-07-18 Thread Vahid S Hashemian
The KIP is updated to follow the suggestion of using consumer group 
generation.

--Vahid



From:   "Vahid S Hashemian" 
To: dev@kafka.apache.org
Date:   07/17/2018 02:32 PM
Subject:Re: [DISCUSS] KIP-341: Update Sticky Assignor's User Data 
Protocol



Hi Jason,

Thanks for the feedback. That sounds good to me.
I'll update the KIP accordingly and move the local generation option to 
"Rejected Alternatives".
I'll send a note when the revised KIP is ready.

--Vahid



From:   Jason Gustafson 
To: dev 
Date:   07/17/2018 02:22 PM
Subject:Re: [DISCUSS] KIP-341: Update Sticky Assignor's User Data 
Protocol



Hey Vahid,

I'm with Mike that it seems simpler to just use the consumer group
generation. Even if you can figure out how to reason about the local
generation, it still seems confusing to have something called "generation"
which is not the consumer group generation. There doesn't seem to be much
downside to using the actual generation as far as I can tell and it is 
much
easier to reason about. I was thinking we could expose it in
PartitionAssignor.Assignment as an additional field. I think this wouldn't
impact existing assignors, though it would allow for other stateful
assignors to use the field in a similar way.

It's a pity we didn't add a version to this schema, by the way. There's 
not
much we can do if we can only add fields at the end of the schema, but I
guess we're hoping this is the last change ;-)

Thanks,
Jason

On Fri, Jul 13, 2018 at 10:07 PM, Vahid S Hashemian <
vahidhashem...@us.ibm.com> wrote:

> Hi Mike,
>
>
>
> Thanks a lot for reviewing the KIP and sharing your feedback.
>
> I agree that such an issue could surface with option 1, but the existing
>
> PR (that currently implements this option) checks for such duplicate
>
> assignments and ignores one in favor of the other. So at the end there
>
> will be valid (non-duplicate) assignments to consumers, but they might
>
> deviate a bit from the ideal assignment.
>
> If rare scenarios like this are deemed troublesome and we want to avoid
>
> them, option 2 would probably be the way to go. In that case and in my
>
> opinion, option 2 would a better solution compared to introducing 
another
>
> field (e.g. timestamp).
>
>
>
> Regards.
>
> --Vahid
>
>
>
>
>
>
>
> From:   Mike Freyberger 
>
> To: "dev@kafka.apache.org" 
>
> Date:   07/13/2018 08:42 PM
>
> Subject:Re: [DISCUSS] KIP-341: Update Sticky Assignor's User 
Data
>
> Protocol
>
>
>
>
>
>
>
> This is great!
>
>
>
> For the client side implementation, I think it’s still possible for 
there
>
> to be a duplication. I’ll try to walk through the example here.
>
>
>
> Let’s says there are 2 consumers, 1 topic with 2 partitions.
>
>
>
> After the initial rebalance, generation 0:
>
> Consumer A has partition 0
>
> Consumer B has partition 1
>
>
>
> Let’s say consumer B leaves the group (long debug, GC pause). This leads
>
> to another rebalance. This rebalance will be considered generation 1 and
>
> will result in:
>
>
>
> Generation 1, Consumer A owns partition 0,1
>
>
>
> Now let’s say Consumer B is still out of the group and then Consumer A
>
> leaves as well. While Consumer A is out of the group, Consumer B rejoins
>
> the group. During this rebalance, the only previous state would be the
>
> initial generation 0 assignment. So this assignment would be considered
>
> generation 1 as well and would result in:
>
>
>
> Generation 1, Consumer B owns partition 0,1
>
>
>
> When A rejoins the group, both consumers would claim ownership of both
>
> partitions and they would report the assignment was from generation 1.
>
> This gets us back into the same issue as before because the generation
>
> number cannot help at all. You could add a timestamp in addition to the
>
> generation marker, but that’d still be vulnerable to clock skew.
>
>
>
> Would hooking into the existing generation marker protect the assignor 
for
>
> this kind of situation? We need to make sure the selected implantation 
is
>
> protected against the kind of failure mentioned above.
>
>
>
> Also, I have been working on KIP-315, which is another Sticky Assignor,
>
> which also requires some kind of epoch/generation marker to be protected
>
> against zombies. So, I’d be in favor of a generic solution here that 
other
>
> assignors can leverage.
>
>
>
> Best,
>
>
>
> Mike Freyberger
>
>
>
> > On Jul 13, 2018, at 6:15 PM, Vahid S Hashemian
>
>  wrote:
>
> >
>
> > Hi all,
>
> >
>
> > I create a short KIP to address an issue in Sticky Assignor assignment
>
> > logic:
>
> >
>
> 
https://cwiki.apache.org/confluence/display/KAFKA/KIP-


> 341%3A+Update+Sticky+Assignor%27s+User+Data+Protocol
>
>
>
> > Please take a look and share your feedback / comments.
>
> >
>
> > In particular, there is a Generation Marker section (
>
> >
>
> 
https://cwiki.apache.org/confluence/display/KAFKA/KIP-


> 341%3A+Update+Sticky+Assignor%27s+User+Data+Protocol#KIP-
> 

Re: [DISCUSS] KIP-342 Add Customizable SASL extensions to OAuthBearer authentication

2018-07-18 Thread Stanislav Kozlovski
Hey Ron,

You brought up some great points. I did my best to address them and updated
the KIP.

I should mention that I used commas to separate extensions in the protocol,
because we did not use the recommended Control-A character for separators
in the OAuth message and I figured I would not change it.
Now that I saw your PR about implementing the proper separators in OAUTH
 and will change my
implementation once yours gets merged, meaning commas will be a supported
value for extensions.

About the implementation: yes you're right, you should define `
sasl.client.callback.handler.class` which has the same functionality as `
OAuthBearerSaslClientCallbackHandler` plus the additional functionality of
handling the `SaslExtensionsCallback` by attaching extensions to it.
The only reason you'd populate the `Subject` object with the extensions is
if you used the default `SaslClientCallbackHandler` (which handles the
extensions callback by adding whatever's in the subject), as the SCRAM
authentication does.
https://github.com/stanislavkozlovski/kafka/blob/KAFKA-7169-custom-sasl-extensions/clients/src/main/java/org/apache/kafka/common/security/oauthbearer/internals/OAuthBearerSaslClient.java#L92
And yes, in that case you would need a custom `LoginModule` which populates
the Subject in that case, although I'm not sure if Kafka supports pluggable
LoginModules. Does it?

Best,
Stanislav

On Wed, Jul 18, 2018 at 9:50 AM Ron Dagostino  wrote:

> Hi Stanislav.  Could you add something to the KIP about the security
> implications related to the CSV name/value pairs sent in the extension?
> For example, the OAuth access token may have a digital signature, but the
> extensions generally will not (unless one of the values is a JWS compact
> serialization, but I doubt anyone would go that far), so the server
> generally cannot trust the extensions to be accurate for anything
> critical.  You mentioned the "better tracing and troubleshooting" use case,
> which I think is fine despite the lack of security; given that lack of
> security, though, I believe it is important to also state what the
> extensions should *not* be used for.
>
> Also, could you indicate in the KIP how the extensions might actually be
> added?  My take on that would be to extend OAuthBearerLoginModule to
> override the initialize() and commit() methods so that the derived class
> would have access to the Subject instance and could add a map to the
> subject's public or private credentials when the commit succeeds; then I
> think the sasl.client.callback.handler.class would have to be explicitly
> set to a class that extends the default implementation
> (OAuthBearerSaslClientCallbackHandler) and retrieves the map when handling
> the SaslExtensionsCallback.  But maybe you are thinking about it
> differently?  Some guidance on how to actually take advantage of the
> feature via an implementation would be a useful addition to the KIP.
>
> Finally, I note that the extension parsing does not support a comma in keys
> or values.  This should be addressed somehow -- either by supporting via an
> escaping mechanism or by explicitly acknowledging that it is unsupported.
>
> Thanks for the KIP and the simultaneous PR -- having both at the same time
> really helped.
>
> Ron
>
> On Tue, Jul 17, 2018 at 6:22 PM Stanislav Kozlovski <
> stanis...@confluent.io>
> wrote:
>
> > Hey group,
> >
> > I just created a new KIP about adding customizable SASL extensions to the
> > OAuthBearer authentication mechanism. More details in the proposal
> >
> > KIP:
> >
> >
> https://cwiki.apache.org/confluence/display/KAFKA/KIP-342%3A+Add+support+for+Custom+SASL+extensions+in+OAuthBearer+authentication
> > JIRA: KAFKA-7169 
> > PR: Pull request 
> >
> > --
> > Best,
> > Stanislav
> >
>


-- 
Best,
Stanislav


[jira] [Created] (KAFKA-7183) Add a trogdor test that creates many connections to brokers

2018-07-18 Thread Colin P. McCabe (JIRA)
Colin P. McCabe created KAFKA-7183:
--

 Summary: Add a trogdor test that creates many connections to 
brokers
 Key: KAFKA-7183
 URL: https://issues.apache.org/jira/browse/KAFKA-7183
 Project: Kafka
  Issue Type: Test
  Components: system tests
Reporter: Colin P. McCabe
Assignee: Colin P. McCabe


Add a trogdor test that creates many connections to brokers



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)


Build failed in Jenkins: kafka-trunk-jdk10 #289

2018-07-18 Thread Apache Jenkins Server
See 


Changes:

[github] KAFKA-7168: Treat connection close during SSL handshake as retriable

[github] MINOR: remove unused MeteredKeyValueStore (#5380)

[github] MINOR: Remove references to version 1.2 in docs (#5386)

[matthias] MINOR: improve docs version numbers (#5372)

--
[...truncated 1.56 MB...]

kafka.controller.PartitionStateMachineTest > 
testInvalidNonexistentPartitionToOnlinePartitionTransition PASSED

kafka.controller.PartitionStateMachineTest > 
testInvalidNonexistentPartitionToOfflinePartitionTransition STARTED

kafka.controller.PartitionStateMachineTest > 
testInvalidNonexistentPartitionToOfflinePartitionTransition PASSED

kafka.controller.PartitionStateMachineTest > 
testOnlinePartitionToOnlineTransition STARTED

kafka.controller.PartitionStateMachineTest > 
testOnlinePartitionToOnlineTransition PASSED

kafka.controller.PartitionStateMachineTest > 
testNewPartitionToOnlinePartitionTransitionZkUtilsExceptionFromCreateStates 
STARTED

kafka.controller.PartitionStateMachineTest > 
testNewPartitionToOnlinePartitionTransitionZkUtilsExceptionFromCreateStates 
PASSED

kafka.controller.PartitionStateMachineTest > 
testInvalidNewPartitionToNonexistentPartitionTransition STARTED

kafka.controller.PartitionStateMachineTest > 
testInvalidNewPartitionToNonexistentPartitionTransition PASSED

kafka.controller.PartitionStateMachineTest > 
testNewPartitionToOnlinePartitionTransition STARTED

kafka.controller.PartitionStateMachineTest > 
testNewPartitionToOnlinePartitionTransition PASSED

kafka.controller.PartitionStateMachineTest > 
testInvalidOnlinePartitionToNewPartitionTransition STARTED

kafka.controller.PartitionStateMachineTest > 
testInvalidOnlinePartitionToNewPartitionTransition PASSED

kafka.controller.PartitionStateMachineTest > 
testOfflinePartitionToOnlinePartitionTransitionErrorCodeFromStateLookup STARTED

kafka.controller.PartitionStateMachineTest > 
testOfflinePartitionToOnlinePartitionTransitionErrorCodeFromStateLookup PASSED

kafka.controller.PartitionStateMachineTest > 
testOnlinePartitionToOnlineTransitionForControlledShutdown STARTED

kafka.controller.PartitionStateMachineTest > 
testOnlinePartitionToOnlineTransitionForControlledShutdown PASSED

kafka.controller.PartitionStateMachineTest > 
testOfflinePartitionToOnlinePartitionTransitionZkUtilsExceptionFromStateLookup 
STARTED

kafka.controller.PartitionStateMachineTest > 
testOfflinePartitionToOnlinePartitionTransitionZkUtilsExceptionFromStateLookup 
PASSED

kafka.controller.PartitionStateMachineTest > 
testInvalidOnlinePartitionToNonexistentPartitionTransition STARTED

kafka.controller.PartitionStateMachineTest > 
testInvalidOnlinePartitionToNonexistentPartitionTransition PASSED

kafka.controller.PartitionStateMachineTest > 
testInvalidOfflinePartitionToNewPartitionTransition STARTED

kafka.controller.PartitionStateMachineTest > 
testInvalidOfflinePartitionToNewPartitionTransition PASSED

kafka.controller.PartitionStateMachineTest > 
testOfflinePartitionToOnlinePartitionTransition STARTED

kafka.controller.PartitionStateMachineTest > 
testOfflinePartitionToOnlinePartitionTransition PASSED

kafka.controller.ControllerEventManagerTest > testEventThatThrowsException 
STARTED

kafka.controller.ControllerEventManagerTest > testEventThatThrowsException 
PASSED

kafka.controller.ControllerEventManagerTest > testSuccessfulEvent STARTED

kafka.controller.ControllerEventManagerTest > testSuccessfulEvent PASSED

kafka.controller.ControllerFailoverTest > classMethod STARTED

kafka.controller.ControllerFailoverTest > classMethod FAILED
java.lang.AssertionError: Found unexpected threads during @BeforeClass, 
allThreads=Set(Reference Handler, ExpirationReaper-0-Produce, 
ReplicaFetcherThread-0-0, scala-execution-context-global-222, 
kafka-request-handler-2, ThrottledChannelReaper-Request, Test worker, Test 
worker-SendThread(some.invalid.hostname.foo.bar.local:65535), 
scala-execution-context-global-225, /0:0:0:0:0:0:0:1:56736 to 
/0:0:0:0:0:0:0:1:47167 workers Thread 2, /0:0:0:0:0:0:0:1:56736 to 
/0:0:0:0:0:0:0:1:47167 workers Thread 3, metrics-meter-tick-thread-1, 
metrics-meter-tick-thread-2, Thread-1303-EventThread, Signal Dispatcher, main, 
ExpirationReaper-0-DeleteRecords, ThrottledChannelReaper-Produce, Test 
worker-SendThread(localhost:41342), Test 
worker-SendThread(some.invalid.hostname.foo.bar.local:2181), Test 
worker-SendThread(localhost:50992), Thread-1303-SendThread(localhost:51283), 
Test worker-EventThread, shutdownable-thread-test, ExpirationReaper-0-Fetch, 
Common-Cleaner, Finalizer, ThrottledChannelReaper-Fetch, 
kafka-admin-client-thread | adminclient-135), 
unexpected=Set(kafka-admin-client-thread | adminclient-135)

kafka.controller.ControllerFailoverTest > classMethod STARTED

kafka.controller.ControllerFailoverTest > classMethod FAILED
java.lang.AssertionError: Found unexpected threads during 

[jira] [Created] (KAFKA-7182) SASL/OAUTHBEARER client response is missing %x01 separators

2018-07-18 Thread Ron Dagostino (JIRA)
Ron Dagostino created KAFKA-7182:


 Summary: SASL/OAUTHBEARER client response is missing %x01 
separators
 Key: KAFKA-7182
 URL: https://issues.apache.org/jira/browse/KAFKA-7182
 Project: Kafka
  Issue Type: Bug
  Components: clients
Affects Versions: 2.0.0
Reporter: Ron Dagostino
Assignee: Ron Dagostino


The format of the SASL/OAUTHBEARER client response is defined in [RFC 7628 
Section 3.1|https://tools.ietf.org/html/rfc7628#section-3.1] as follows:

{noformat}
 kvsep  = %x01
 key= 1*(ALPHA)
 value  = *(VCHAR / SP / HTAB / CR / LF )
 kvpair = key "=" value kvsep
 client-resp= (gs2-header kvsep *kvpair kvsep) / kvsep
{noformat}

;;gs2-header = See [RFC 5801 (Section 
4)|https://tools.ietf.org/html/rfc5801#section-4]

The SASL/OAUTHBEARER client response as currently implemented in 
OAuthBearerSaslClient sends the valid gs2-header "n,," but then sends the 
"auth" key and value immediately after it, like this:

{code:java}
String.format("n,,auth=Bearer %s", callback.token().value())
{code}

This does not conform to the specification because there is no %x01 after the 
gs2-header, no %x01 after the auth value, and no terminating %x01.  The code 
should instead be as follows:

{code:java}
String.format("n,,\u0001auth=Bearer %s\u0001\u0001", callback.token().value())
{code}

Similarly, the parsing of the client response in OAuthBearerSaslServer, which 
currently allows the malformed text, must also change.

*This should be fixed prior to the initial release of the SASL/OAUTHBEARER code 
in 2.0.0 to prevent compatibility problems.*



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)


[jira] [Created] (KAFKA-7181) Kafka Streams State stuck in rebalancing after one of the StreamThread encounters IllegalStateException

2018-07-18 Thread Romil Kumar Vasani (JIRA)
Romil Kumar Vasani created KAFKA-7181:
-

 Summary: Kafka Streams State stuck in rebalancing after one of the 
StreamThread encounters IllegalStateException
 Key: KAFKA-7181
 URL: https://issues.apache.org/jira/browse/KAFKA-7181
 Project: Kafka
  Issue Type: Bug
  Components: streams
Affects Versions: 1.1.0
Reporter: Romil Kumar Vasani


One the StreamThread encounters an IllegalStateException and is marked DEAD, 
shut down.

The application doesn't spawn a new thread in it's place, the partitions of 
that thread are assigned to a different thread and it synchronizes. But the 
application is stuck in REBALANCING state, as not all StreamThreads are in 
RUNNING state.

Excepted: New thread should come up and after synchronization/rebalancing it 
the KafkaStream.State should be RUNNING

Since all the active threads (that are not marked DEAD) are in RUNNING state, 
the KafkaStreams.State should be RUNNING

P.S. I am reporting an issue for the first time. If there is more information 
needed I can provide.

Below are the logs from the IllegalStateException: 

2018-07-18 03:02:27.510 ERROR 1 --- [-StreamThread-2] 
o.a.k.s.p.internals.StreamThread : stream-thread 
[prd1565.prod.nuke.ops.v1-StreamThread-2] Encountered the following error 
during processing:

java.lang.IllegalStateException: No current assignment for partition 
consumerGroup-stateStore-changelog-10
 at 
org.apache.kafka.clients.consumer.internals.SubscriptionState.assignedState(SubscriptionState.java:259)
 at 
org.apache.kafka.clients.consumer.internals.SubscriptionState.resetFailed(SubscriptionState.java:413)
 at 
org.apache.kafka.clients.consumer.internals.Fetcher$2.onFailure(Fetcher.java:595)
 at 
org.apache.kafka.clients.consumer.internals.RequestFuture.fireFailure(RequestFuture.java:177)
 at 
org.apache.kafka.clients.consumer.internals.RequestFuture.raise(RequestFuture.java:147)
 at 
org.apache.kafka.clients.consumer.internals.RequestFutureAdapter.onFailure(RequestFutureAdapter.java:30)
 at 
org.apache.kafka.clients.consumer.internals.RequestFuture$1.onFailure(RequestFuture.java:209)
 at 
org.apache.kafka.clients.consumer.internals.RequestFuture.fireFailure(RequestFuture.java:177)
 at 
org.apache.kafka.clients.consumer.internals.RequestFuture.raise(RequestFuture.java:147)
 at 
org.apache.kafka.clients.consumer.internals.ConsumerNetworkClient$RequestFutureCompletionHandler.fireCompletion(ConsumerNetworkClient.java:553)
 at 
org.apache.kafka.clients.consumer.internals.ConsumerNetworkClient.firePendingCompletedRequests(ConsumerNetworkClient.java:390)
 at 
org.apache.kafka.clients.consumer.internals.ConsumerNetworkClient.poll(ConsumerNetworkClient.java:293)
 at 
org.apache.kafka.clients.consumer.internals.ConsumerNetworkClient.poll(ConsumerNetworkClient.java:233)
 at 
org.apache.kafka.clients.consumer.KafkaConsumer.pollOnce(KafkaConsumer.java:1171)
 at 
org.apache.kafka.clients.consumer.KafkaConsumer.poll(KafkaConsumer.java:1115)
 at 
org.apache.kafka.streams.processor.internals.StreamThread.maybeUpdateStandbyTasks(StreamThread.java:1040)
 at 
org.apache.kafka.streams.processor.internals.StreamThread.runOnce(StreamThread.java:812)
 at 
org.apache.kafka.streams.processor.internals.StreamThread.runLoop(StreamThread.java:750)
 at 
org.apache.kafka.streams.processor.internals.StreamThread.run(StreamThread.java:720)

2018-07-18 03:02:27.511 INFO 1 --- [-StreamThread-2] 
o.a.k.s.p.internals.StreamThread : stream-thread 
[prd1565.prod.nuke.ops.v1-StreamThread-2] State transition from RUNNING to 
PENDING_SHUTDOWN
2018-07-18 03:02:27.511 INFO 1 --- [-StreamThread-2] 
o.a.k.s.p.internals.StreamThread : stream-thread 
[prd1565.prod.nuke.ops.v1-StreamThread-2] Shutting down
2018-07-18 03:02:27.571 INFO 1 --- [-StreamThread-2] 
o.a.k.clients.producer.KafkaProducer : [Producer 
clientId=prd1565.prod.nuke.ops.v1-StreamThread-2-producer] Closing the Kafka 
producer with timeoutMillis = 9223372036854775807 ms.
2018-07-18 03:02:27.579 INFO 1 --- [-StreamThread-2] 
o.a.k.s.p.internals.StreamThread : stream-thread 
[prd1565.prod.nuke.ops.v1-StreamThread-2] State transition from 
PENDING_SHUTDOWN to DEAD
2018-07-18 03:02:27.579 INFO 1 --- [-StreamThread-2] 
o.a.k.s.p.internals.StreamThread : stream-thread [consumerGroup-StreamThread-2] 
Shutdown complete
2018-07-18 03:02:27.579 ERROR 1 --- [-StreamThread-2] xxx.xxx.xxx.AppRunner : 
Unhandled exception in thread: 43:consumerGroup-StreamThread-2

java.lang.IllegalStateException: No current assignment for partition 
consumerGroup-inventoryStore-changelog-10
 at 
org.apache.kafka.clients.consumer.internals.SubscriptionState.assignedState(SubscriptionState.java:259)
 at 
org.apache.kafka.clients.consumer.internals.SubscriptionState.resetFailed(SubscriptionState.java:413)
 at 
org.apache.kafka.clients.consumer.internals.Fetcher$2.onFailure(Fetcher.java:595)
 at 

Re: [DISCUSS] KIP-291: Have separate queues for control requests and data requests

2018-07-18 Thread Joel Koshy
@Mayuresh - I like your idea. It appears to be a simpler less invasive
alternative and it should work. Jun/Becket/others, do you see any pitfalls
with this approach?

On Wed, Jul 18, 2018 at 12:03 PM, Lucas Wang  wrote:

> @Mayuresh,
> That's a very interesting idea that I haven't thought before.
> It seems to solve our problem at hand pretty well, and also
> avoids the need to have a new size metric and capacity config
> for the controller request queue. In fact, if we were to adopt
> this design, there is no public interface change, and we
> probably don't need a KIP.
> Also implementation wise, it seems
> the java class LinkedBlockingQueue can readily satisfy the requirement
> by supporting a capacity, and also allowing inserting at both ends.
>
> My only concern is that this design is tied to the coincidence that
> we have two request priorities and there are two ends to a deque.
> Hence by using the proposed design, it seems the network layer is
> more tightly coupled with upper layer logic, e.g. if we were to add
> an extra priority level in the future for some reason, we would probably
> need to go back to the design of separate queues, one for each priority
> level.
>
> In summary, I'm ok with both designs and lean toward your suggested
> approach.
> Let's hear what others think.
>
> @Becket,
> In light of Mayuresh's suggested new design, I'm answering your question
> only in the context
> of the current KIP design: I think your suggestion makes sense, and I'm ok
> with removing the capacity config and
> just relying on the default value of 20 being sufficient enough.
>
> Thanks,
> Lucas
>
>
>
>
>
>
>
>
>
>
> On Wed, Jul 18, 2018 at 9:57 AM, Mayuresh Gharat <
> gharatmayures...@gmail.com
> > wrote:
>
> > Hi Lucas,
> >
> > Seems like the main intent here is to prioritize the controller request
> > over any other requests.
> > In that case, we can change the request queue to a dequeue, where you
> > always insert the normal requests (produce, consume,..etc) to the end of
> > the dequeue, but if its a controller request, you insert it to the head
> of
> > the queue. This ensures that the controller request will be given higher
> > priority over other requests.
> >
> > Also since we only read one request from the socket and mute it and only
> > unmute it after handling the request, this would ensure that we don't
> > handle controller requests out of order.
> >
> > With this approach we can avoid the second queue and the additional
> config
> > for the size of the queue.
> >
> > What do you think ?
> >
> > Thanks,
> >
> > Mayuresh
> >
> >
> > On Wed, Jul 18, 2018 at 3:05 AM Becket Qin  wrote:
> >
> > > Hey Joel,
> > >
> > > Thank for the detail explanation. I agree the current design makes
> sense.
> > > My confusion is about whether the new config for the controller queue
> > > capacity is necessary. I cannot think of a case in which users would
> > change
> > > it.
> > >
> > > Thanks,
> > >
> > > Jiangjie (Becket) Qin
> > >
> > > On Wed, Jul 18, 2018 at 6:00 PM, Becket Qin 
> > wrote:
> > >
> > > > Hi Lucas,
> > > >
> > > > I guess my question can be rephrased to "do we expect user to ever
> > change
> > > > the controller request queue capacity"? If we agree that 20 is
> already
> > a
> > > > very generous default number and we do not expect user to change it,
> is
> > > it
> > > > still necessary to expose this as a config?
> > > >
> > > > Thanks,
> > > >
> > > > Jiangjie (Becket) Qin
> > > >
> > > > On Wed, Jul 18, 2018 at 2:29 AM, Lucas Wang 
> > > wrote:
> > > >
> > > >> @Becket
> > > >> 1. Thanks for the comment. You are right that normally there should
> be
> > > >> just
> > > >> one controller request because of muting,
> > > >> and I had NOT intended to say there would be many enqueued
> controller
> > > >> requests.
> > > >> I went through the KIP again, and I'm not sure which part conveys
> that
> > > >> info.
> > > >> I'd be happy to revise if you point it out the section.
> > > >>
> > > >> 2. Though it should not happen in normal conditions, the current
> > design
> > > >> does not preclude multiple controllers running
> > > >> at the same time, hence if we don't have the controller queue
> capacity
> > > >> config and simply make its capacity to be 1,
> > > >> network threads handling requests from different controllers will be
> > > >> blocked during those troublesome times,
> > > >> which is probably not what we want. On the other hand, adding the
> > extra
> > > >> config with a default value, say 20, guards us from issues in those
> > > >> troublesome times, and IMO there isn't much downside of adding the
> > extra
> > > >> config.
> > > >>
> > > >> @Mayuresh
> > > >> Good catch, this sentence is an obsolete statement based on a
> previous
> > > >> design. I've revised the wording in the KIP.
> > > >>
> > > >> Thanks,
> > > >> Lucas
> > > >>
> > > >> On Tue, Jul 17, 2018 at 10:33 AM, Mayuresh Gharat <
> > > >> gharatmayures...@gmail.com> wrote:
> > > >>
> > > >> > Hi Lucas,
> > > 

Re: [DISCUSS] KIP-291: Have separate queues for control requests and data requests

2018-07-18 Thread Lucas Wang
@Mayuresh,
That's a very interesting idea that I haven't thought before.
It seems to solve our problem at hand pretty well, and also
avoids the need to have a new size metric and capacity config
for the controller request queue. In fact, if we were to adopt
this design, there is no public interface change, and we
probably don't need a KIP.
Also implementation wise, it seems
the java class LinkedBlockingQueue can readily satisfy the requirement
by supporting a capacity, and also allowing inserting at both ends.

My only concern is that this design is tied to the coincidence that
we have two request priorities and there are two ends to a deque.
Hence by using the proposed design, it seems the network layer is
more tightly coupled with upper layer logic, e.g. if we were to add
an extra priority level in the future for some reason, we would probably
need to go back to the design of separate queues, one for each priority
level.

In summary, I'm ok with both designs and lean toward your suggested
approach.
Let's hear what others think.

@Becket,
In light of Mayuresh's suggested new design, I'm answering your question
only in the context
of the current KIP design: I think your suggestion makes sense, and I'm ok
with removing the capacity config and
just relying on the default value of 20 being sufficient enough.

Thanks,
Lucas










On Wed, Jul 18, 2018 at 9:57 AM, Mayuresh Gharat  wrote:

> Hi Lucas,
>
> Seems like the main intent here is to prioritize the controller request
> over any other requests.
> In that case, we can change the request queue to a dequeue, where you
> always insert the normal requests (produce, consume,..etc) to the end of
> the dequeue, but if its a controller request, you insert it to the head of
> the queue. This ensures that the controller request will be given higher
> priority over other requests.
>
> Also since we only read one request from the socket and mute it and only
> unmute it after handling the request, this would ensure that we don't
> handle controller requests out of order.
>
> With this approach we can avoid the second queue and the additional config
> for the size of the queue.
>
> What do you think ?
>
> Thanks,
>
> Mayuresh
>
>
> On Wed, Jul 18, 2018 at 3:05 AM Becket Qin  wrote:
>
> > Hey Joel,
> >
> > Thank for the detail explanation. I agree the current design makes sense.
> > My confusion is about whether the new config for the controller queue
> > capacity is necessary. I cannot think of a case in which users would
> change
> > it.
> >
> > Thanks,
> >
> > Jiangjie (Becket) Qin
> >
> > On Wed, Jul 18, 2018 at 6:00 PM, Becket Qin 
> wrote:
> >
> > > Hi Lucas,
> > >
> > > I guess my question can be rephrased to "do we expect user to ever
> change
> > > the controller request queue capacity"? If we agree that 20 is already
> a
> > > very generous default number and we do not expect user to change it, is
> > it
> > > still necessary to expose this as a config?
> > >
> > > Thanks,
> > >
> > > Jiangjie (Becket) Qin
> > >
> > > On Wed, Jul 18, 2018 at 2:29 AM, Lucas Wang 
> > wrote:
> > >
> > >> @Becket
> > >> 1. Thanks for the comment. You are right that normally there should be
> > >> just
> > >> one controller request because of muting,
> > >> and I had NOT intended to say there would be many enqueued controller
> > >> requests.
> > >> I went through the KIP again, and I'm not sure which part conveys that
> > >> info.
> > >> I'd be happy to revise if you point it out the section.
> > >>
> > >> 2. Though it should not happen in normal conditions, the current
> design
> > >> does not preclude multiple controllers running
> > >> at the same time, hence if we don't have the controller queue capacity
> > >> config and simply make its capacity to be 1,
> > >> network threads handling requests from different controllers will be
> > >> blocked during those troublesome times,
> > >> which is probably not what we want. On the other hand, adding the
> extra
> > >> config with a default value, say 20, guards us from issues in those
> > >> troublesome times, and IMO there isn't much downside of adding the
> extra
> > >> config.
> > >>
> > >> @Mayuresh
> > >> Good catch, this sentence is an obsolete statement based on a previous
> > >> design. I've revised the wording in the KIP.
> > >>
> > >> Thanks,
> > >> Lucas
> > >>
> > >> On Tue, Jul 17, 2018 at 10:33 AM, Mayuresh Gharat <
> > >> gharatmayures...@gmail.com> wrote:
> > >>
> > >> > Hi Lucas,
> > >> >
> > >> > Thanks for the KIP.
> > >> > I am trying to understand why you think "The memory consumption can
> > rise
> > >> > given the total number of queued requests can go up to 2x" in the
> > impact
> > >> > section. Normally the requests from controller to a Broker are not
> > high
> > >> > volume, right ?
> > >> >
> > >> >
> > >> > Thanks,
> > >> >
> > >> > Mayuresh
> > >> >
> > >> > On Tue, Jul 17, 2018 at 5:06 AM Becket Qin 
> > >> wrote:
> > >> >
> > >> > > Thanks for the KIP, Lucas. Separating the control plane from 

Re: [kafka-clients] Re: [VOTE] 1.1.1 RC3

2018-07-18 Thread Matthias J. Sax
Thanks Dong.

I am a little late, but +1, too.

 - verified signatures
 - build from sources
 - run unit test suite
 - run streams quickstart


Thanks for running the release!

-Matthias

On 7/18/18 10:24 AM, Dong Lin wrote:
> Thank you all for taking time to certify and vote for the release!
> 
> This vote has passed with 10 +1 votes (3 bindings) and no 0 or -1 votes.
> 
> 
> +1 vote from PMC Members:
> - Jason Gustafson
> - Rajini Sivaram
> - Ismael Juma
> 
> 
> +1 vote from Committers:
> - Sriharsha Chintalapani
> - Dong Lin
> 
> +1 vote from Community:
> - Ted Yu
> - Satish Duggana
> - Jakub Scholz
> - Brett Rann
> - ManiKumar Reddy
> 
> 0 vote: none
> 
> -1 vote: none
> 
> Vote thread: the thread is not found on http://markmail.org/message yet.
> Will pose it when it is available.
> 
> I'll continue with the release process and the release announcement will
> follow.
> 
> 
> Cheers,
> Dong
> 
> 
> On Wed, Jul 18, 2018 at 7:46 AM, Ismael Juma  > wrote:
> 
> +1 (binding)
> 
> Verified signature of source artifact, ran tests and verified
> quickstart on source artifact with Java 8, verified quickstart on
> binary artifact (Scala 2.12) with Java 8, sanity checked release
> notes and Maven staging repository.
> 
> Thanks for managing the release Dong!
> 
> Ismael
> 
> On Sun, Jul 8, 2018 at 3:36 PM Dong Lin  > wrote:
> 
> Hello Kafka users, developers and client-developers,
> 
> This is the fourth candidate for release of Apache Kafka 1.1.1.
> 
> Apache Kafka 1.1.1 is a bug-fix release for the 1.1 branch that
> was first
> released with 1.1.0 about 3 months ago. We have fixed about 25
> issues since
> that release. A few of the more significant fixes include:
> 
> KAFKA-6925  > - Fix memory
> leak in StreamsMetricsThreadImpl
> KAFKA-6937  > - In-sync
> replica delayed during fetch if replica throttle is exceeded
> KAFKA-6917  > - Process txn
> completion asynchronously to avoid deadlock
> KAFKA-6893  > - Create
> processors before starting acceptor to avoid ArithmeticException
> KAFKA-6870  > -
> Fix ConcurrentModificationException in SampledStat
> KAFKA-6878  > - Fix
> NullPointerException when querying global state store
> KAFKA-6879  > - Invoke
> session init callbacks outside lock to avoid Controller deadlock
> KAFKA-6857  > - Prevent
> follower from truncating to the wrong offset if undefined leader
> epoch is
> requested
> KAFKA-6854  > - Log cleaner
> fails with transaction markers that are deleted during clean
> KAFKA-6747  > - Check
> whether there is in-flight transaction before aborting transaction
> KAFKA-6748  > - Double
> check before scheduling a new task after the punctuate call
> KAFKA-6739  > -
> Fix IllegalArgumentException when down-converting from V2 to V0/V1
> KAFKA-6728  > -
> Fix NullPointerException when instantiating the HeaderConverter
> 
> Kafka 1.1.1 release plan:
> https://cwiki.apache.org/confluence/display/KAFKA/Release+Plan+1.1.1
> 
> 
> Release notes for the 1.1.1 release:
> http://home.apache.org/~lindong/kafka-1.1.1-rc3/RELEASE_NOTES.html
> 

Re: [kafka-clients] Re: [VOTE] 1.1.1 RC3

2018-07-18 Thread Dong Lin
Thank you all for taking time to certify and vote for the release!

This vote has passed with 10 +1 votes (3 bindings) and no 0 or -1 votes.


+1 vote from PMC Members:
- Jason Gustafson
- Rajini Sivaram
- Ismael Juma


+1 vote from Committers:
- Sriharsha Chintalapani
- Dong Lin

+1 vote from Community:
- Ted Yu
- Satish Duggana
- Jakub Scholz
- Brett Rann
- ManiKumar Reddy

0 vote: none

-1 vote: none

Vote thread: the thread is not found on http://markmail.org/message yet.
Will pose it when it is available.

I'll continue with the release process and the release announcement will
follow.


Cheers,
Dong


On Wed, Jul 18, 2018 at 7:46 AM, Ismael Juma  wrote:

> +1 (binding)
>
> Verified signature of source artifact, ran tests and verified quickstart
> on source artifact with Java 8, verified quickstart on binary artifact (Scala
> 2.12) with Java 8, sanity checked release notes and Maven staging
> repository.
>
> Thanks for managing the release Dong!
>
> Ismael
>
> On Sun, Jul 8, 2018 at 3:36 PM Dong Lin  wrote:
>
>> Hello Kafka users, developers and client-developers,
>>
>> This is the fourth candidate for release of Apache Kafka 1.1.1.
>>
>> Apache Kafka 1.1.1 is a bug-fix release for the 1.1 branch that was first
>> released with 1.1.0 about 3 months ago. We have fixed about 25 issues
>> since
>> that release. A few of the more significant fixes include:
>>
>> KAFKA-6925  - Fix
>> memory
>> leak in StreamsMetricsThreadImpl
>> KAFKA-6937  - In-sync
>> replica delayed during fetch if replica throttle is exceeded
>> KAFKA-6917  - Process
>> txn
>> completion asynchronously to avoid deadlock
>> KAFKA-6893  - Create
>> processors before starting acceptor to avoid ArithmeticException
>> KAFKA-6870  -
>> Fix ConcurrentModificationException in SampledStat
>> KAFKA-6878  - Fix
>> NullPointerException when querying global state store
>> KAFKA-6879  - Invoke
>> session init callbacks outside lock to avoid Controller deadlock
>> KAFKA-6857  - Prevent
>> follower from truncating to the wrong offset if undefined leader epoch is
>> requested
>> KAFKA-6854  - Log
>> cleaner
>> fails with transaction markers that are deleted during clean
>> KAFKA-6747  - Check
>> whether there is in-flight transaction before aborting transaction
>> KAFKA-6748  - Double
>> check before scheduling a new task after the punctuate call
>> KAFKA-6739  -
>> Fix IllegalArgumentException when down-converting from V2 to V0/V1
>> KAFKA-6728  -
>> Fix NullPointerException when instantiating the HeaderConverter
>>
>> Kafka 1.1.1 release plan:
>> https://cwiki.apache.org/confluence/display/KAFKA/Release+Plan+1.1.1
>>
>> Release notes for the 1.1.1 release:
>> http://home.apache.org/~lindong/kafka-1.1.1-rc3/RELEASE_NOTES.html
>>
>> *** Please download, test and vote by Thursday, July 12, 12pm PT ***
>>
>> Kafka's KEYS file containing PGP keys we use to sign the release:
>> http://kafka.apache.org/KEYS
>>
>> * Release artifacts to be voted upon (source and binary):
>> http://home.apache.org/~lindong/kafka-1.1.1-rc3/
>>
>> * Maven artifacts to be voted upon:
>> https://repository.apache.org/content/groups/staging/
>>
>> * Javadoc:
>> http://home.apache.org/~lindong/kafka-1.1.1-rc3/javadoc/
>>
>> * Tag to be voted upon (off 1.1 branch) is the 1.1.1-rc3 tag:
>> https://github.com/apache/kafka/tree/1.1.1-rc3
>>
>> * Documentation:
>> http://kafka.apache.org/11/documentation.html
>>
>> * Protocol:
>> http://kafka.apache.org/11/protocol.html
>>
>> * Successful Jenkins builds for the 1.1 branch:
>> Unit/integration tests: *https://builds.apache.org/job/kafka-1.1-jdk7/162
>> *
>> System tests:
>> https://jenkins.confluent.io/job/system-test-kafka/job/1.1/156/
>>
>> Please test and verify the release artifacts and submit a vote for this
>> RC,
>> or report any issues so we can fix them and get a new RC out ASAP.
>> Although
>> this release vote requires PMC votes to pass, testing, votes, and bug
>> reports are valuable and appreciated from everyone.
>>
>>
>> Regards,
>> Dong
>>
> --
> You received this message because you are subscribed to the Google Groups
> "kafka-clients" group.
> To unsubscribe from this group and stop receiving emails from it, send an
> email to kafka-clients+unsubscr...@googlegroups.com.
> To post to this group, send email to 

[jira] [Created] (KAFKA-7180) In testHWCheckpointWithFailuresSingleLogSegment, wait until server1 has joined the ISR before shutting down server2

2018-07-18 Thread Lucas Wang (JIRA)
Lucas Wang created KAFKA-7180:
-

 Summary: In testHWCheckpointWithFailuresSingleLogSegment, wait 
until server1 has joined the ISR before shutting down server2
 Key: KAFKA-7180
 URL: https://issues.apache.org/jira/browse/KAFKA-7180
 Project: Kafka
  Issue Type: Bug
Reporter: Lucas Wang
Assignee: Lucas Wang


In the testHWCheckpointWithFailuresSingleLogSegment method, the test logic is 
1. shutdown server1 and then capture the leadership of a partition in the 
variable "leader", which should be server2
2. shutdown server2 and wait until the leadership has changed to a broker other 
than server2
through the line 
waitUntilLeaderIsElectedOrChanged(zkClient, topic, partitionId, oldLeaderOpt = 
Some(leader))

However when we execute step 2 and shutdown server2, it's possible that server1 
has not caught up with the partition, and has not joined the ISR. With unclean 
leader election turned off, the leadership cannot be transferred to server1, 
causing the waited condition in step 2 to be never met. 

The obvious fix is to wait until server1 has joined the ISR before shutting 
down server2.



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)


Re: [DISCUSS] KIP-291: Have separate queues for control requests and data requests

2018-07-18 Thread Mayuresh Gharat
Hi Lucas,

Seems like the main intent here is to prioritize the controller request
over any other requests.
In that case, we can change the request queue to a dequeue, where you
always insert the normal requests (produce, consume,..etc) to the end of
the dequeue, but if its a controller request, you insert it to the head of
the queue. This ensures that the controller request will be given higher
priority over other requests.

Also since we only read one request from the socket and mute it and only
unmute it after handling the request, this would ensure that we don't
handle controller requests out of order.

With this approach we can avoid the second queue and the additional config
for the size of the queue.

What do you think ?

Thanks,

Mayuresh


On Wed, Jul 18, 2018 at 3:05 AM Becket Qin  wrote:

> Hey Joel,
>
> Thank for the detail explanation. I agree the current design makes sense.
> My confusion is about whether the new config for the controller queue
> capacity is necessary. I cannot think of a case in which users would change
> it.
>
> Thanks,
>
> Jiangjie (Becket) Qin
>
> On Wed, Jul 18, 2018 at 6:00 PM, Becket Qin  wrote:
>
> > Hi Lucas,
> >
> > I guess my question can be rephrased to "do we expect user to ever change
> > the controller request queue capacity"? If we agree that 20 is already a
> > very generous default number and we do not expect user to change it, is
> it
> > still necessary to expose this as a config?
> >
> > Thanks,
> >
> > Jiangjie (Becket) Qin
> >
> > On Wed, Jul 18, 2018 at 2:29 AM, Lucas Wang 
> wrote:
> >
> >> @Becket
> >> 1. Thanks for the comment. You are right that normally there should be
> >> just
> >> one controller request because of muting,
> >> and I had NOT intended to say there would be many enqueued controller
> >> requests.
> >> I went through the KIP again, and I'm not sure which part conveys that
> >> info.
> >> I'd be happy to revise if you point it out the section.
> >>
> >> 2. Though it should not happen in normal conditions, the current design
> >> does not preclude multiple controllers running
> >> at the same time, hence if we don't have the controller queue capacity
> >> config and simply make its capacity to be 1,
> >> network threads handling requests from different controllers will be
> >> blocked during those troublesome times,
> >> which is probably not what we want. On the other hand, adding the extra
> >> config with a default value, say 20, guards us from issues in those
> >> troublesome times, and IMO there isn't much downside of adding the extra
> >> config.
> >>
> >> @Mayuresh
> >> Good catch, this sentence is an obsolete statement based on a previous
> >> design. I've revised the wording in the KIP.
> >>
> >> Thanks,
> >> Lucas
> >>
> >> On Tue, Jul 17, 2018 at 10:33 AM, Mayuresh Gharat <
> >> gharatmayures...@gmail.com> wrote:
> >>
> >> > Hi Lucas,
> >> >
> >> > Thanks for the KIP.
> >> > I am trying to understand why you think "The memory consumption can
> rise
> >> > given the total number of queued requests can go up to 2x" in the
> impact
> >> > section. Normally the requests from controller to a Broker are not
> high
> >> > volume, right ?
> >> >
> >> >
> >> > Thanks,
> >> >
> >> > Mayuresh
> >> >
> >> > On Tue, Jul 17, 2018 at 5:06 AM Becket Qin 
> >> wrote:
> >> >
> >> > > Thanks for the KIP, Lucas. Separating the control plane from the
> data
> >> > plane
> >> > > makes a lot of sense.
> >> > >
> >> > > In the KIP you mentioned that the controller request queue may have
> >> many
> >> > > requests in it. Will this be a common case? The controller requests
> >> still
> >> > > goes through the SocketServer. The SocketServer will mute the
> channel
> >> > once
> >> > > a request is read and put into the request channel. So assuming
> there
> >> is
> >> > > only one connection between controller and each broker, on the
> broker
> >> > side,
> >> > > there should be only one controller request in the controller
> request
> >> > queue
> >> > > at any given time. If that is the case, do we need a separate
> >> controller
> >> > > request queue capacity config? The default value 20 means that we
> >> expect
> >> > > there are 20 controller switches to happen in a short period of
> time.
> >> I
> >> > am
> >> > > not sure whether someone should increase the controller request
> queue
> >> > > capacity to handle such case, as it seems indicating something very
> >> wrong
> >> > > has happened.
> >> > >
> >> > > Thanks,
> >> > >
> >> > > Jiangjie (Becket) Qin
> >> > >
> >> > >
> >> > > On Fri, Jul 13, 2018 at 1:10 PM, Dong Lin 
> >> wrote:
> >> > >
> >> > > > Thanks for the update Lucas.
> >> > > >
> >> > > > I think the motivation section is intuitive. It will be good to
> >> learn
> >> > > more
> >> > > > about the comments from other reviewers.
> >> > > >
> >> > > > On Thu, Jul 12, 2018 at 9:48 PM, Lucas Wang <
> lucasatu...@gmail.com>
> >> > > wrote:
> >> > > >
> >> > > > > Hi Dong,
> >> > > > >
> >> > > > > I've updated the 

Re: [DISCUSS] KIP-342 Add Customizable SASL extensions to OAuthBearer authentication

2018-07-18 Thread Ron Dagostino
Hi Stanislav.  Could you add something to the KIP about the security
implications related to the CSV name/value pairs sent in the extension?
For example, the OAuth access token may have a digital signature, but the
extensions generally will not (unless one of the values is a JWS compact
serialization, but I doubt anyone would go that far), so the server
generally cannot trust the extensions to be accurate for anything
critical.  You mentioned the "better tracing and troubleshooting" use case,
which I think is fine despite the lack of security; given that lack of
security, though, I believe it is important to also state what the
extensions should *not* be used for.

Also, could you indicate in the KIP how the extensions might actually be
added?  My take on that would be to extend OAuthBearerLoginModule to
override the initialize() and commit() methods so that the derived class
would have access to the Subject instance and could add a map to the
subject's public or private credentials when the commit succeeds; then I
think the sasl.client.callback.handler.class would have to be explicitly
set to a class that extends the default implementation
(OAuthBearerSaslClientCallbackHandler) and retrieves the map when handling
the SaslExtensionsCallback.  But maybe you are thinking about it
differently?  Some guidance on how to actually take advantage of the
feature via an implementation would be a useful addition to the KIP.

Finally, I note that the extension parsing does not support a comma in keys
or values.  This should be addressed somehow -- either by supporting via an
escaping mechanism or by explicitly acknowledging that it is unsupported.

Thanks for the KIP and the simultaneous PR -- having both at the same time
really helped.

Ron

On Tue, Jul 17, 2018 at 6:22 PM Stanislav Kozlovski 
wrote:

> Hey group,
>
> I just created a new KIP about adding customizable SASL extensions to the
> OAuthBearer authentication mechanism. More details in the proposal
>
> KIP:
>
> https://cwiki.apache.org/confluence/display/KAFKA/KIP-342%3A+Add+support+for+Custom+SASL+extensions+in+OAuthBearer+authentication
> JIRA: KAFKA-7169 
> PR: Pull request 
>
> --
> Best,
> Stanislav
>


Re: KAFKA-6690 Priorities for Source Topics

2018-07-18 Thread Stanislav Kozlovski
Hey Nick,

This seems like a non-trivial change and more importantly - a change in the
public interface. As such, following our contributing guidelines
, it warrants a KIP

(Kafka Improvement Proposal).
Please look into those, write an extensive one and start a [DISCUSS] thread
in the mailing group. You might not have permissions to create one and
you'll need to ask in a separate e-mail for somebody to give you (I can't).

This seems like an interesting change, I'm looking forward to it!

Best,
Stanislav

On Wed, Jul 18, 2018 at 5:07 AM Nick  wrote:

> Hi all,
>
> I picked up KAFKA-6690 Priorities for Source Topics.
>
>https://issues.apache.org/jira/browse/KAFKA-6690
>
> In the ticket I made some notes about extending the API to incorporate
> topic priorities.
>
> For implementation, I was thinking of filtering out lower priority
> topics in method Fetcher.fetchablePartitions.
>
> Thanks for any input.
>
> Regards,
> --
>  Nick
>


-- 
Best,
Stanislav


Build failed in Jenkins: kafka-trunk-jdk8 #2803

2018-07-18 Thread Apache Jenkins Server
See 


Changes:

[harsha] MINOR: Updated Quickstart to mention log.dirs (#5361)

[github] KAFKA-7168: Treat connection close during SSL handshake as retriable

--
[...truncated 2.48 MB...]

org.apache.kafka.streams.TopologyTest > shouldNotAddNullStateStoreSupplier 
PASSED

org.apache.kafka.streams.TopologyTest > 
shouldNotAllowToAddStateStoreToNonExistingProcessor STARTED

org.apache.kafka.streams.TopologyTest > 
shouldNotAllowToAddStateStoreToNonExistingProcessor PASSED

org.apache.kafka.streams.TopologyTest > 
sourceAndProcessorShouldHaveSingleSubtopology STARTED

org.apache.kafka.streams.TopologyTest > 
sourceAndProcessorShouldHaveSingleSubtopology PASSED

org.apache.kafka.streams.TopologyTest > 
tableNamedMaterializedCountShouldPreserveTopologyStructure STARTED

org.apache.kafka.streams.TopologyTest > 
tableNamedMaterializedCountShouldPreserveTopologyStructure PASSED

org.apache.kafka.streams.TopologyTest > 
shouldNotAllowNullStoreNameWhenConnectingProcessorAndStateStores STARTED

org.apache.kafka.streams.TopologyTest > 
shouldNotAllowNullStoreNameWhenConnectingProcessorAndStateStores PASSED

org.apache.kafka.streams.TopologyTest > 
shouldNotAllowNullNameWhenAddingSourceWithTopic STARTED

org.apache.kafka.streams.TopologyTest > 
shouldNotAllowNullNameWhenAddingSourceWithTopic PASSED

org.apache.kafka.streams.TopologyTest > 
sourceAndProcessorWithStateShouldHaveSingleSubtopology STARTED

org.apache.kafka.streams.TopologyTest > 
sourceAndProcessorWithStateShouldHaveSingleSubtopology PASSED

org.apache.kafka.streams.TopologyTest > shouldNotAllowNullTopicWhenAddingSink 
STARTED

org.apache.kafka.streams.TopologyTest > shouldNotAllowNullTopicWhenAddingSink 
PASSED

org.apache.kafka.streams.TopologyTest > 
shouldNotAllowToAddGlobalStoreWithSourceNameEqualsProcessorName STARTED

org.apache.kafka.streams.TopologyTest > 
shouldNotAllowToAddGlobalStoreWithSourceNameEqualsProcessorName PASSED

org.apache.kafka.streams.TopologyTest > 
singleSourceWithListOfTopicsShouldHaveSingleSubtopology STARTED

org.apache.kafka.streams.TopologyTest > 
singleSourceWithListOfTopicsShouldHaveSingleSubtopology PASSED

org.apache.kafka.streams.TopologyTest > 
sourceWithMultipleProcessorsShouldHaveSingleSubtopology STARTED

org.apache.kafka.streams.TopologyTest > 
sourceWithMultipleProcessorsShouldHaveSingleSubtopology PASSED

org.apache.kafka.streams.TopologyTest > 
shouldNotAllowZeroStoreNameWhenConnectingProcessorAndStateStores STARTED

org.apache.kafka.streams.TopologyTest > 
shouldNotAllowZeroStoreNameWhenConnectingProcessorAndStateStores PASSED

org.apache.kafka.streams.TopologyTest > 
kTableNamedMaterializedMapValuesShouldPreserveTopologyStructure STARTED

org.apache.kafka.streams.TopologyTest > 
kTableNamedMaterializedMapValuesShouldPreserveTopologyStructure PASSED

org.apache.kafka.streams.TopologyTest > 
timeWindowZeroArgCountShouldPreserveTopologyStructure STARTED

org.apache.kafka.streams.TopologyTest > 
timeWindowZeroArgCountShouldPreserveTopologyStructure PASSED

org.apache.kafka.streams.TopologyTest > 
multipleSourcesWithProcessorsShouldHaveDistinctSubtopologies STARTED

org.apache.kafka.streams.TopologyTest > 
multipleSourcesWithProcessorsShouldHaveDistinctSubtopologies PASSED

org.apache.kafka.streams.TopologyTest > shouldThrowOnUnassignedStateStoreAccess 
STARTED

org.apache.kafka.streams.TopologyTest > shouldThrowOnUnassignedStateStoreAccess 
PASSED

org.apache.kafka.streams.TopologyTest > 
shouldNotAllowNullProcessorNameWhenConnectingProcessorAndStateStores STARTED

org.apache.kafka.streams.TopologyTest > 
shouldNotAllowNullProcessorNameWhenConnectingProcessorAndStateStores PASSED

org.apache.kafka.streams.TopologyTest > 
kTableAnonymousMaterializedMapValuesShouldPreserveTopologyStructure STARTED

org.apache.kafka.streams.TopologyTest > 
kTableAnonymousMaterializedMapValuesShouldPreserveTopologyStructure PASSED

org.apache.kafka.streams.TopologyTest > 
sessionWindowZeroArgCountShouldPreserveTopologyStructure STARTED

org.apache.kafka.streams.TopologyTest > 
sessionWindowZeroArgCountShouldPreserveTopologyStructure PASSED

org.apache.kafka.streams.TopologyTest > 
tableAnonymousMaterializedCountShouldPreserveTopologyStructure STARTED

org.apache.kafka.streams.TopologyTest > 
tableAnonymousMaterializedCountShouldPreserveTopologyStructure PASSED

org.apache.kafka.streams.TopologyTest > shouldDescribeGlobalStoreTopology 
STARTED

org.apache.kafka.streams.TopologyTest > shouldDescribeGlobalStoreTopology PASSED

org.apache.kafka.streams.TopologyTest > 
kTableNonMaterializedMapValuesShouldPreserveTopologyStructure STARTED

org.apache.kafka.streams.TopologyTest > 
kTableNonMaterializedMapValuesShouldPreserveTopologyStructure PASSED

org.apache.kafka.streams.TopologyTest > 
kGroupedStreamAnonymousMaterializedCountShouldPreserveTopologyStructure STARTED

org.apache.kafka.streams.TopologyTest > 

Build failed in Jenkins: kafka-2.0-jdk8 #74

2018-07-18 Thread Apache Jenkins Server
See 


Changes:

[wangguoz] MINOR: web docs fixes on message header (#5381)

[rajinisivaram] MINOR: Updated Quickstart to mention log.dirs (#5361)

[rajinisivaram] KAFKA-7168: Treat connection close during SSL handshake as 
retriable

--
[...truncated 2.46 MB...]

org.apache.kafka.streams.processor.internals.ProcessorStateManagerTest > 
testRegisterPersistentStore PASSED

org.apache.kafka.streams.processor.internals.ProcessorStateManagerTest > 
shouldThrowIllegalArgumentExceptionOnRegisterWhenStoreHasAlreadyBeenRegistered 
STARTED

org.apache.kafka.streams.processor.internals.ProcessorStateManagerTest > 
shouldThrowIllegalArgumentExceptionOnRegisterWhenStoreHasAlreadyBeenRegistered 
PASSED

org.apache.kafka.streams.processor.internals.ProcessorStateManagerTest > 
shouldRestoreStoreWithSinglePutRestoreSpecification STARTED

org.apache.kafka.streams.processor.internals.ProcessorStateManagerTest > 
shouldRestoreStoreWithSinglePutRestoreSpecification PASSED

org.apache.kafka.streams.processor.internals.ProcessorStateManagerTest > 
shouldNotChangeOffsetsIfAckedOffsetsIsNull STARTED

org.apache.kafka.streams.processor.internals.ProcessorStateManagerTest > 
shouldNotChangeOffsetsIfAckedOffsetsIsNull PASSED

org.apache.kafka.streams.processor.internals.ProcessorStateManagerTest > 
shouldThrowIllegalArgumentExceptionIfStoreNameIsSameAsCheckpointFileName STARTED

org.apache.kafka.streams.processor.internals.ProcessorStateManagerTest > 
shouldThrowIllegalArgumentExceptionIfStoreNameIsSameAsCheckpointFileName PASSED

org.apache.kafka.streams.processor.internals.GlobalStateTaskTest > 
shouldCloseStateManagerWithOffsets STARTED

org.apache.kafka.streams.processor.internals.GlobalStateTaskTest > 
shouldCloseStateManagerWithOffsets PASSED

org.apache.kafka.streams.processor.internals.GlobalStateTaskTest > 
shouldProcessRecordsForTopic STARTED

org.apache.kafka.streams.processor.internals.GlobalStateTaskTest > 
shouldProcessRecordsForTopic PASSED

org.apache.kafka.streams.processor.internals.GlobalStateTaskTest > 
shouldInitializeProcessorTopology STARTED

org.apache.kafka.streams.processor.internals.GlobalStateTaskTest > 
shouldInitializeProcessorTopology PASSED

org.apache.kafka.streams.processor.internals.GlobalStateTaskTest > 
shouldInitializeContext STARTED

org.apache.kafka.streams.processor.internals.GlobalStateTaskTest > 
shouldInitializeContext PASSED

org.apache.kafka.streams.processor.internals.GlobalStateTaskTest > 
shouldCheckpointOffsetsWhenStateIsFlushed STARTED

org.apache.kafka.streams.processor.internals.GlobalStateTaskTest > 
shouldCheckpointOffsetsWhenStateIsFlushed PASSED

org.apache.kafka.streams.processor.internals.GlobalStateTaskTest > 
shouldThrowStreamsExceptionWhenKeyDeserializationFails STARTED

org.apache.kafka.streams.processor.internals.GlobalStateTaskTest > 
shouldThrowStreamsExceptionWhenKeyDeserializationFails PASSED

org.apache.kafka.streams.processor.internals.GlobalStateTaskTest > 
shouldInitializeStateManager STARTED

org.apache.kafka.streams.processor.internals.GlobalStateTaskTest > 
shouldInitializeStateManager PASSED

org.apache.kafka.streams.processor.internals.GlobalStateTaskTest > 
shouldProcessRecordsForOtherTopic STARTED

org.apache.kafka.streams.processor.internals.GlobalStateTaskTest > 
shouldProcessRecordsForOtherTopic PASSED

org.apache.kafka.streams.processor.internals.GlobalStateTaskTest > 
shouldNotThrowStreamsExceptionWhenValueDeserializationFails STARTED

org.apache.kafka.streams.processor.internals.GlobalStateTaskTest > 
shouldNotThrowStreamsExceptionWhenValueDeserializationFails PASSED

org.apache.kafka.streams.processor.internals.GlobalStateTaskTest > 
shouldThrowStreamsExceptionWhenValueDeserializationFails STARTED

org.apache.kafka.streams.processor.internals.GlobalStateTaskTest > 
shouldThrowStreamsExceptionWhenValueDeserializationFails PASSED

org.apache.kafka.streams.processor.internals.GlobalStateTaskTest > 
shouldNotThrowStreamsExceptionWhenKeyDeserializationFailsWithSkipHandler STARTED

org.apache.kafka.streams.processor.internals.GlobalStateTaskTest > 
shouldNotThrowStreamsExceptionWhenKeyDeserializationFailsWithSkipHandler PASSED

org.apache.kafka.streams.processor.internals.StreamsMetricsImplTest > 
testThroughputMetrics STARTED

org.apache.kafka.streams.processor.internals.StreamsMetricsImplTest > 
testThroughputMetrics PASSED

org.apache.kafka.streams.processor.internals.StreamsMetricsImplTest > 
testLatencyMetrics STARTED

org.apache.kafka.streams.processor.internals.StreamsMetricsImplTest > 
testLatencyMetrics PASSED

org.apache.kafka.streams.processor.internals.StreamsMetricsImplTest > 
testRemoveSensor STARTED

org.apache.kafka.streams.processor.internals.StreamsMetricsImplTest > 
testRemoveSensor PASSED

org.apache.kafka.streams.processor.internals.StreamsMetricsImplTest > 
testRemoveNullSensor STARTED


Re: [VOTE] 2.0.0 RC2

2018-07-18 Thread Ismael Juma
Thanks Rajini! A documentation issue that we must fix before the release
(but does not require another RC), 1.2 (which became 2.0) is mentioned in
the upgrade notes:

http://kafka.apache.org/20/documentation.html#upgrade

Ismael

On Sun, Jul 15, 2018 at 9:25 AM Rajini Sivaram 
wrote:

> Hi Ismael,
>
> Thank you for pointing that out. I have re-uploaded the RC2 artifacts to
> maven including streams-scala_2.12. Also submitted a PR to update build &
> release scripts to include this.
>
> Thank you,
>
> Rajini
>
>
>
> On Fri, Jul 13, 2018 at 7:19 AM, Ismael Juma  wrote:
>
> > Hi Rajini,
> >
> > Thanks for generating the RC. It seems like the kafka-streams-scala 2.12
> > artifact is missing from the Maven repository:
> >
> > https://repository.apache.org/content/groups/staging/org/apache/kafka/
> >
> > Since this is the first time we are publishing this artifact, it is
> > possible that this never worked properly.
> >
> > Ismael
> >
> > On Tue, Jul 10, 2018 at 10:17 AM Rajini Sivaram  >
> > wrote:
> >
> > > Hello Kafka users, developers and client-developers,
> > >
> > >
> > > This is the third candidate for release of Apache Kafka 2.0.0.
> > >
> > >
> > > This is a major version release of Apache Kafka. It includes 40 new
> KIPs
> > > and
> > >
> > > several critical bug fixes. Please see the 2.0.0 release plan for more
> > > details:
> > >
> > > https://cwiki.apache.org/confluence/pages/viewpage.
> > action?pageId=80448820
> > >
> > >
> > > A few notable highlights:
> > >
> > >- Prefixed wildcard ACLs (KIP-290), Fine grained ACLs for
> CreateTopics
> > >(KIP-277)
> > >- SASL/OAUTHBEARER implementation (KIP-255)
> > >- Improved quota communication and customization of quotas (KIP-219,
> > >KIP-257)
> > >- Efficient memory usage for down conversion (KIP-283)
> > >- Fix log divergence between leader and follower during fast leader
> > >failover (KIP-279)
> > >- Drop support for Java 7 and remove deprecated code including old
> > scala
> > >clients
> > >- Connect REST extension plugin, support for externalizing secrets
> and
> > >improved error handling (KIP-285, KIP-297, KIP-298 etc.)
> > >- Scala API for Kafka Streams and other Streams API improvements
> > >(KIP-270, KIP-150, KIP-245, KIP-251 etc.)
> > >
> > >
> > > Release notes for the 2.0.0 release:
> > >
> > > http://home.apache.org/~rsivaram/kafka-2.0.0-rc2/RELEASE_NOTES.html
> > >
> > >
> > > *** Please download, test and vote by Friday, July 13, 4pm PT
> > >
> > >
> > > Kafka's KEYS file containing PGP keys we use to sign the release:
> > >
> > > http://kafka.apache.org/KEYS
> > >
> > >
> > > * Release artifacts to be voted upon (source and binary):
> > >
> > > http://home.apache.org/~rsivaram/kafka-2.0.0-rc2/
> > >
> > >
> > > * Maven artifacts to be voted upon:
> > >
> > > https://repository.apache.org/content/groups/staging/
> > >
> > >
> > > * Javadoc:
> > >
> > > http://home.apache.org/~rsivaram/kafka-2.0.0-rc2/javadoc/
> > >
> > >
> > > * Tag to be voted upon (off 2.0 branch) is the 2.0.0 tag:
> > >
> > > https://github.com/apache/kafka/tree/2.0.0-rc2
> > >
> > >
> > >
> > > * Documentation:
> > >
> > > http://kafka.apache.org/20/documentation.html
> > >
> > >
> > > * Protocol:
> > >
> > > http://kafka.apache.org/20/protocol.html
> > >
> > >
> > > * Successful Jenkins builds for the 2.0 branch:
> > >
> > > Unit/integration tests:
> https://builds.apache.org/job/kafka-2.0-jdk8/72/
> > >
> > > System tests:
> > > https://jenkins.confluent.io/job/system-test-kafka/job/2.0/27/
> > >
> > >
> > > /**
> > >
> > >
> > > Thanks,
> > >
> > >
> > > Rajini
> > >
> >
>


Re: [VOTE] 1.1.1 RC3

2018-07-18 Thread Ismael Juma
+1 (binding)

Verified signature of source artifact, ran tests and verified quickstart on
source artifact with Java 8, verified quickstart on binary artifact (Scala
2.12) with Java 8, sanity checked release notes and Maven staging
repository.

Thanks for managing the release Dong!

Ismael

On Sun, Jul 8, 2018 at 3:36 PM Dong Lin  wrote:

> Hello Kafka users, developers and client-developers,
>
> This is the fourth candidate for release of Apache Kafka 1.1.1.
>
> Apache Kafka 1.1.1 is a bug-fix release for the 1.1 branch that was first
> released with 1.1.0 about 3 months ago. We have fixed about 25 issues since
> that release. A few of the more significant fixes include:
>
> KAFKA-6925  - Fix memory
> leak in StreamsMetricsThreadImpl
> KAFKA-6937  - In-sync
> replica delayed during fetch if replica throttle is exceeded
> KAFKA-6917  - Process
> txn
> completion asynchronously to avoid deadlock
> KAFKA-6893  - Create
> processors before starting acceptor to avoid ArithmeticException
> KAFKA-6870  -
> Fix ConcurrentModificationException in SampledStat
> KAFKA-6878  - Fix
> NullPointerException when querying global state store
> KAFKA-6879  - Invoke
> session init callbacks outside lock to avoid Controller deadlock
> KAFKA-6857  - Prevent
> follower from truncating to the wrong offset if undefined leader epoch is
> requested
> KAFKA-6854  - Log
> cleaner
> fails with transaction markers that are deleted during clean
> KAFKA-6747  - Check
> whether there is in-flight transaction before aborting transaction
> KAFKA-6748  - Double
> check before scheduling a new task after the punctuate call
> KAFKA-6739  -
> Fix IllegalArgumentException when down-converting from V2 to V0/V1
> KAFKA-6728  -
> Fix NullPointerException when instantiating the HeaderConverter
>
> Kafka 1.1.1 release plan:
> https://cwiki.apache.org/confluence/display/KAFKA/Release+Plan+1.1.1
>
> Release notes for the 1.1.1 release:
> http://home.apache.org/~lindong/kafka-1.1.1-rc3/RELEASE_NOTES.html
>
> *** Please download, test and vote by Thursday, July 12, 12pm PT ***
>
> Kafka's KEYS file containing PGP keys we use to sign the release:
> http://kafka.apache.org/KEYS
>
> * Release artifacts to be voted upon (source and binary):
> http://home.apache.org/~lindong/kafka-1.1.1-rc3/
>
> * Maven artifacts to be voted upon:
> https://repository.apache.org/content/groups/staging/
>
> * Javadoc:
> http://home.apache.org/~lindong/kafka-1.1.1-rc3/javadoc/
>
> * Tag to be voted upon (off 1.1 branch) is the 1.1.1-rc3 tag:
> https://github.com/apache/kafka/tree/1.1.1-rc3
>
> * Documentation:
> http://kafka.apache.org/11/documentation.html
>
> * Protocol:
> http://kafka.apache.org/11/protocol.html
>
> * Successful Jenkins builds for the 1.1 branch:
> Unit/integration tests: *https://builds.apache.org/job/kafka-1.1-jdk7/162
> *
> System tests:
> https://jenkins.confluent.io/job/system-test-kafka/job/1.1/156/
>
> Please test and verify the release artifacts and submit a vote for this RC,
> or report any issues so we can fix them and get a new RC out ASAP. Although
> this release vote requires PMC votes to pass, testing, votes, and bug
> reports are valuable and appreciated from everyone.
>
>
> Regards,
> Dong
>


[jira] [Created] (KAFKA-7179) No response to INIT_PRODUCER_ID request on handler exception

2018-07-18 Thread Bastien Bouclet (JIRA)
Bastien Bouclet created KAFKA-7179:
--

 Summary: No response to INIT_PRODUCER_ID request on handler 
exception
 Key: KAFKA-7179
 URL: https://issues.apache.org/jira/browse/KAFKA-7179
 Project: Kafka
  Issue Type: Bug
  Components: core
Affects Versions: 1.1.0
Reporter: Bastien Bouclet


We were recently hit by an issue similar to KAFKA-6446 where transactional java 
producers would indefinitely block when calling KafkaProducer.initTransactions. 
We were not able to identify the root cause of the issue we encountered. 
Restarting our brokers caused the issue to disappear.

However when reading the broker source code for the INIT_PRODUCER_ID request we 
noticed that there was no code to send an error response if an exception occurs 
in TransactionCoordinator.handleInitProducerId. If no response is sent to the 
INIT_PRODUCER_ID request, the java producer waits indefinitely in versions < 
2.0.0, and waits until a timeout in newer versions.

Is that expected behavior or is that an oversight?



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)


KAFKA-6690 Priorities for Source Topics

2018-07-18 Thread Nick

Hi all,

I picked up KAFKA-6690 Priorities for Source Topics.

  https://issues.apache.org/jira/browse/KAFKA-6690

In the ticket I made some notes about extending the API to incorporate 
topic priorities.


For implementation, I was thinking of filtering out lower priority 
topics in method Fetcher.fetchablePartitions.


Thanks for any input.

Regards,
--
Nick


[jira] [Resolved] (KAFKA-7168) Broker shutdown during SSL handshake may be handled as handshake failure

2018-07-18 Thread Rajini Sivaram (JIRA)


 [ 
https://issues.apache.org/jira/browse/KAFKA-7168?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Rajini Sivaram resolved KAFKA-7168.
---
   Resolution: Fixed
Fix Version/s: 2.1.0
   2.0.1

> Broker shutdown during SSL handshake may be handled as handshake failure
> 
>
> Key: KAFKA-7168
> URL: https://issues.apache.org/jira/browse/KAFKA-7168
> Project: Kafka
>  Issue Type: Bug
>  Components: security
>Affects Versions: 1.0.2, 1.1.1, 2.0.0
>Reporter: Rajini Sivaram
>Assignee: Rajini Sivaram
>Priority: Major
> Fix For: 2.0.1, 2.1.0
>
>
> If broker is shutdown while SSL handshake of a client connection is in 
> progress, the client may process the resulting SSLException as a 
> non-retriable handshake failure rather than a retriable I/O exception. This 
> can cause streams applications to fail during rolling restarts.
> Exception stack trace:
> {quote}
> org.apache.kafka.common.errors.SslAuthenticationException: SSL handshake 
> failed
> Caused by: javax.net.ssl.SSLException: Received close_notify during handshake
>     at sun.security.ssl.Alerts.getSSLException(Alerts.java:208)
>     at sun.security.ssl.SSLEngineImpl.fatal(SSLEngineImpl.java:1639)
>     at sun.security.ssl.SSLEngineImpl.fatal(SSLEngineImpl.java:1607)
>     at sun.security.ssl.SSLEngineImpl.recvAlert(SSLEngineImpl.java:1752)
>     at sun.security.ssl.SSLEngineImpl.readRecord(SSLEngineImpl.java:1068)
>     at 
> sun.security.ssl.SSLEngineImpl.readNetRecord(SSLEngineImpl.java:890)
>     at sun.security.ssl.SSLEngineImpl.unwrap(SSLEngineImpl.java:764)
>     at javax.net.ssl.SSLEngine.unwrap(SSLEngine.java:624)
>     at 
> org.apache.kafka.common.network.SslTransportLayer.handshakeUnwrap(SslTransportLayer.java:465)
>     at 
> org.apache.kafka.common.network.SslTransportLayer.handshake(SslTransportLayer.java:266)
>     at 
> org.apache.kafka.common.network.KafkaChannel.prepare(KafkaChannel.java:88)
>     at 
> org.apache.kafka.common.network.Selector.pollSelectionKeys(Selector.java:474)
>     at org.apache.kafka.common.network.Selector.poll(Selector.java:412)
>     at org.apache.kafka.clients.NetworkClient.poll(NetworkClient.java:460)
>     at 
> org.apache.kafka.clients.consumer.internals.ConsumerNetworkClient.poll(ConsumerNetworkClient.java:258)
>     at 
> org.apache.kafka.clients.consumer.internals.ConsumerNetworkClient.poll(ConsumerNetworkClient.java:230)
>     at 
> org.apache.kafka.clients.consumer.internals.ConsumerNetworkClient.poll(ConsumerNetworkClient.java:206)
>     at 
> org.apache.kafka.clients.consumer.internals.AbstractCoordinator.ensureCoordinatorReady(AbstractCoordinator.java:219)
>     at 
> org.apache.kafka.clients.consumer.internals.AbstractCoordinator.ensureCoordinatorReady(AbstractCoordinator.java:205)
>     at 
> org.apache.kafka.clients.consumer.internals.ConsumerCoordinator.poll(ConsumerCoordinator.java:284)
>     at 
> org.apache.kafka.clients.consumer.KafkaConsumer.pollOnce(KafkaConsumer.java:1146)
>     at 
> org.apache.kafka.clients.consumer.KafkaConsumer.poll(KafkaConsumer.java:)
>     at 
> org.apache.kafka.streams.processor.internals.StreamThread.pollRequests(StreamThread.java:848)
>     at 
> org.apache.kafka.streams.processor.internals.StreamThread.runOnce(StreamThread.java:805)
>     at 
> org.apache.kafka.streams.processor.internals.StreamThread.runLoop(StreamThread.java:771)
>     at 
> org.apache.kafka.streams.processor.internals.StreamThread.run(StreamThread.java:741)
> {quote}
>  



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)


Request for review of pull request

2018-07-18 Thread David Nies
Dear Kafka developers,

my company is using Kafka for quite some time now. Lately we’ve come across an 
issue that caused some headache. I’ve tried to mitigate this problem with the 
changed in pull request #5365  which 
I’m kindly asking you to have a look at and tell me what you think.

This is my first pull request in the Kafka project, so I’m probably not abiding 
some conventions by accident. I would really like your feedback.

Thank you very much, yours
-David Nies

Re: [DISCUSS] KIP-291: Have separate queues for control requests and data requests

2018-07-18 Thread Becket Qin
Hey Joel,

Thank for the detail explanation. I agree the current design makes sense.
My confusion is about whether the new config for the controller queue
capacity is necessary. I cannot think of a case in which users would change
it.

Thanks,

Jiangjie (Becket) Qin

On Wed, Jul 18, 2018 at 6:00 PM, Becket Qin  wrote:

> Hi Lucas,
>
> I guess my question can be rephrased to "do we expect user to ever change
> the controller request queue capacity"? If we agree that 20 is already a
> very generous default number and we do not expect user to change it, is it
> still necessary to expose this as a config?
>
> Thanks,
>
> Jiangjie (Becket) Qin
>
> On Wed, Jul 18, 2018 at 2:29 AM, Lucas Wang  wrote:
>
>> @Becket
>> 1. Thanks for the comment. You are right that normally there should be
>> just
>> one controller request because of muting,
>> and I had NOT intended to say there would be many enqueued controller
>> requests.
>> I went through the KIP again, and I'm not sure which part conveys that
>> info.
>> I'd be happy to revise if you point it out the section.
>>
>> 2. Though it should not happen in normal conditions, the current design
>> does not preclude multiple controllers running
>> at the same time, hence if we don't have the controller queue capacity
>> config and simply make its capacity to be 1,
>> network threads handling requests from different controllers will be
>> blocked during those troublesome times,
>> which is probably not what we want. On the other hand, adding the extra
>> config with a default value, say 20, guards us from issues in those
>> troublesome times, and IMO there isn't much downside of adding the extra
>> config.
>>
>> @Mayuresh
>> Good catch, this sentence is an obsolete statement based on a previous
>> design. I've revised the wording in the KIP.
>>
>> Thanks,
>> Lucas
>>
>> On Tue, Jul 17, 2018 at 10:33 AM, Mayuresh Gharat <
>> gharatmayures...@gmail.com> wrote:
>>
>> > Hi Lucas,
>> >
>> > Thanks for the KIP.
>> > I am trying to understand why you think "The memory consumption can rise
>> > given the total number of queued requests can go up to 2x" in the impact
>> > section. Normally the requests from controller to a Broker are not high
>> > volume, right ?
>> >
>> >
>> > Thanks,
>> >
>> > Mayuresh
>> >
>> > On Tue, Jul 17, 2018 at 5:06 AM Becket Qin 
>> wrote:
>> >
>> > > Thanks for the KIP, Lucas. Separating the control plane from the data
>> > plane
>> > > makes a lot of sense.
>> > >
>> > > In the KIP you mentioned that the controller request queue may have
>> many
>> > > requests in it. Will this be a common case? The controller requests
>> still
>> > > goes through the SocketServer. The SocketServer will mute the channel
>> > once
>> > > a request is read and put into the request channel. So assuming there
>> is
>> > > only one connection between controller and each broker, on the broker
>> > side,
>> > > there should be only one controller request in the controller request
>> > queue
>> > > at any given time. If that is the case, do we need a separate
>> controller
>> > > request queue capacity config? The default value 20 means that we
>> expect
>> > > there are 20 controller switches to happen in a short period of time.
>> I
>> > am
>> > > not sure whether someone should increase the controller request queue
>> > > capacity to handle such case, as it seems indicating something very
>> wrong
>> > > has happened.
>> > >
>> > > Thanks,
>> > >
>> > > Jiangjie (Becket) Qin
>> > >
>> > >
>> > > On Fri, Jul 13, 2018 at 1:10 PM, Dong Lin 
>> wrote:
>> > >
>> > > > Thanks for the update Lucas.
>> > > >
>> > > > I think the motivation section is intuitive. It will be good to
>> learn
>> > > more
>> > > > about the comments from other reviewers.
>> > > >
>> > > > On Thu, Jul 12, 2018 at 9:48 PM, Lucas Wang 
>> > > wrote:
>> > > >
>> > > > > Hi Dong,
>> > > > >
>> > > > > I've updated the motivation section of the KIP by explaining the
>> > cases
>> > > > that
>> > > > > would have user impacts.
>> > > > > Please take a look at let me know your comments.
>> > > > >
>> > > > > Thanks,
>> > > > > Lucas
>> > > > >
>> > > > > On Mon, Jul 9, 2018 at 5:53 PM, Lucas Wang > >
>> > > > wrote:
>> > > > >
>> > > > > > Hi Dong,
>> > > > > >
>> > > > > > The simulation of disk being slow is merely for me to easily
>> > > construct
>> > > > a
>> > > > > > testing scenario
>> > > > > > with a backlog of produce requests. In production, other than
>> the
>> > > disk
>> > > > > > being slow, a backlog of
>> > > > > > produce requests may also be caused by high produce QPS.
>> > > > > > In that case, we may not want to kill the broker and that's when
>> > this
>> > > > KIP
>> > > > > > can be useful, both for JBOD
>> > > > > > and non-JBOD setup.
>> > > > > >
>> > > > > > Going back to your previous question about each ProduceRequest
>> > > covering
>> > > > > 20
>> > > > > > partitions that are randomly
>> > > > > > distributed, let's say a LeaderAndIsr request is enqueued that
>> > tries
>> > 

Re: [DISCUSS] KIP-291: Have separate queues for control requests and data requests

2018-07-18 Thread Becket Qin
Hi Lucas,

I guess my question can be rephrased to "do we expect user to ever change
the controller request queue capacity"? If we agree that 20 is already a
very generous default number and we do not expect user to change it, is it
still necessary to expose this as a config?

Thanks,

Jiangjie (Becket) Qin

On Wed, Jul 18, 2018 at 2:29 AM, Lucas Wang  wrote:

> @Becket
> 1. Thanks for the comment. You are right that normally there should be just
> one controller request because of muting,
> and I had NOT intended to say there would be many enqueued controller
> requests.
> I went through the KIP again, and I'm not sure which part conveys that
> info.
> I'd be happy to revise if you point it out the section.
>
> 2. Though it should not happen in normal conditions, the current design
> does not preclude multiple controllers running
> at the same time, hence if we don't have the controller queue capacity
> config and simply make its capacity to be 1,
> network threads handling requests from different controllers will be
> blocked during those troublesome times,
> which is probably not what we want. On the other hand, adding the extra
> config with a default value, say 20, guards us from issues in those
> troublesome times, and IMO there isn't much downside of adding the extra
> config.
>
> @Mayuresh
> Good catch, this sentence is an obsolete statement based on a previous
> design. I've revised the wording in the KIP.
>
> Thanks,
> Lucas
>
> On Tue, Jul 17, 2018 at 10:33 AM, Mayuresh Gharat <
> gharatmayures...@gmail.com> wrote:
>
> > Hi Lucas,
> >
> > Thanks for the KIP.
> > I am trying to understand why you think "The memory consumption can rise
> > given the total number of queued requests can go up to 2x" in the impact
> > section. Normally the requests from controller to a Broker are not high
> > volume, right ?
> >
> >
> > Thanks,
> >
> > Mayuresh
> >
> > On Tue, Jul 17, 2018 at 5:06 AM Becket Qin  wrote:
> >
> > > Thanks for the KIP, Lucas. Separating the control plane from the data
> > plane
> > > makes a lot of sense.
> > >
> > > In the KIP you mentioned that the controller request queue may have
> many
> > > requests in it. Will this be a common case? The controller requests
> still
> > > goes through the SocketServer. The SocketServer will mute the channel
> > once
> > > a request is read and put into the request channel. So assuming there
> is
> > > only one connection between controller and each broker, on the broker
> > side,
> > > there should be only one controller request in the controller request
> > queue
> > > at any given time. If that is the case, do we need a separate
> controller
> > > request queue capacity config? The default value 20 means that we
> expect
> > > there are 20 controller switches to happen in a short period of time. I
> > am
> > > not sure whether someone should increase the controller request queue
> > > capacity to handle such case, as it seems indicating something very
> wrong
> > > has happened.
> > >
> > > Thanks,
> > >
> > > Jiangjie (Becket) Qin
> > >
> > >
> > > On Fri, Jul 13, 2018 at 1:10 PM, Dong Lin  wrote:
> > >
> > > > Thanks for the update Lucas.
> > > >
> > > > I think the motivation section is intuitive. It will be good to learn
> > > more
> > > > about the comments from other reviewers.
> > > >
> > > > On Thu, Jul 12, 2018 at 9:48 PM, Lucas Wang 
> > > wrote:
> > > >
> > > > > Hi Dong,
> > > > >
> > > > > I've updated the motivation section of the KIP by explaining the
> > cases
> > > > that
> > > > > would have user impacts.
> > > > > Please take a look at let me know your comments.
> > > > >
> > > > > Thanks,
> > > > > Lucas
> > > > >
> > > > > On Mon, Jul 9, 2018 at 5:53 PM, Lucas Wang 
> > > > wrote:
> > > > >
> > > > > > Hi Dong,
> > > > > >
> > > > > > The simulation of disk being slow is merely for me to easily
> > > construct
> > > > a
> > > > > > testing scenario
> > > > > > with a backlog of produce requests. In production, other than the
> > > disk
> > > > > > being slow, a backlog of
> > > > > > produce requests may also be caused by high produce QPS.
> > > > > > In that case, we may not want to kill the broker and that's when
> > this
> > > > KIP
> > > > > > can be useful, both for JBOD
> > > > > > and non-JBOD setup.
> > > > > >
> > > > > > Going back to your previous question about each ProduceRequest
> > > covering
> > > > > 20
> > > > > > partitions that are randomly
> > > > > > distributed, let's say a LeaderAndIsr request is enqueued that
> > tries
> > > to
> > > > > > switch the current broker, say broker0, from leader to follower
> > > > > > *for one of the partitions*, say *test-0*. For the sake of
> > argument,
> > > > > > let's also assume the other brokers, say broker1, have *stopped*
> > > > fetching
> > > > > > from
> > > > > > the current broker, i.e. broker0.
> > > > > > 1. If the enqueued produce requests have acks =  -1 (ALL)
> > > > > >   1.1 without this KIP, the ProduceRequests ahead of LeaderAndISR
> > > will
> > > 

Build failed in Jenkins: kafka-trunk-jdk10 #288

2018-07-18 Thread Apache Jenkins Server
See 


Changes:

[jason] MINOR: Fix broken Javadoc on [AbstractIndex|OffsetIndex] (#5370)

[github] MINOR: web docs fixes on message header (#5381)

[harsha] MINOR: Updated Quickstart to mention log.dirs (#5361)

--
[...truncated 1.99 MB...]
org.apache.kafka.streams.kstream.internals.KTableImplTest > 
shouldNotAllowNullJoinerOnOuterJoin STARTED

org.apache.kafka.streams.kstream.internals.KTableImplTest > 
shouldNotAllowNullJoinerOnOuterJoin PASSED

org.apache.kafka.streams.kstream.internals.KTableImplTest > 
shouldCreateSourceAndSinkNodesForRepartitioningTopic STARTED

org.apache.kafka.streams.kstream.internals.KTableImplTest > 
shouldCreateSourceAndSinkNodesForRepartitioningTopic PASSED

org.apache.kafka.streams.kstream.internals.KTableImplTest > 
shouldNotAllowNullJoinerJoin STARTED

org.apache.kafka.streams.kstream.internals.KTableImplTest > 
shouldNotAllowNullJoinerJoin PASSED

org.apache.kafka.streams.kstream.internals.KTableImplTest > 
shouldAllowNullStoreInJoin STARTED

org.apache.kafka.streams.kstream.internals.KTableImplTest > 
shouldAllowNullStoreInJoin PASSED

org.apache.kafka.streams.kstream.internals.KTableImplTest > 
shouldThrowNullPointerOnTransformValuesWithKeyWhenTransformerSupplierIsNull 
STARTED

org.apache.kafka.streams.kstream.internals.KTableImplTest > 
shouldThrowNullPointerOnTransformValuesWithKeyWhenTransformerSupplierIsNull 
PASSED

org.apache.kafka.streams.kstream.internals.KTableImplTest > 
testStateStoreLazyEval STARTED

org.apache.kafka.streams.kstream.internals.KTableImplTest > 
testStateStoreLazyEval PASSED

org.apache.kafka.streams.kstream.internals.KTableImplTest > 
shouldThrowNullPointerOnLeftJoinWhenMaterializedIsNull STARTED

org.apache.kafka.streams.kstream.internals.KTableImplTest > 
shouldThrowNullPointerOnLeftJoinWhenMaterializedIsNull PASSED

org.apache.kafka.streams.kstream.internals.KTableImplTest > 
shouldThrowNullPointerOnFilterWhenMaterializedIsNull STARTED

org.apache.kafka.streams.kstream.internals.KTableImplTest > 
shouldThrowNullPointerOnFilterWhenMaterializedIsNull PASSED

org.apache.kafka.streams.kstream.internals.KTableImplTest > testKTable STARTED

org.apache.kafka.streams.kstream.internals.KTableImplTest > testKTable PASSED

org.apache.kafka.streams.kstream.internals.KTableImplTest > 
shouldNotAllowNullPredicateOnFilterNot STARTED

org.apache.kafka.streams.kstream.internals.KTableImplTest > 
shouldNotAllowNullPredicateOnFilterNot PASSED

org.apache.kafka.streams.kstream.internals.KTableImplTest > 
shouldThrowNullPointerOnTransformValuesWithKeyWhenMaterializedIsNull STARTED

org.apache.kafka.streams.kstream.internals.KTableImplTest > 
shouldThrowNullPointerOnTransformValuesWithKeyWhenMaterializedIsNull PASSED

org.apache.kafka.streams.kstream.internals.KTableImplTest > testValueGetter 
STARTED

org.apache.kafka.streams.kstream.internals.KTableImplTest > testValueGetter 
PASSED

org.apache.kafka.streams.kstream.internals.KTableKTableLeftJoinTest > 
testSendingOldValue STARTED

org.apache.kafka.streams.kstream.internals.KTableKTableLeftJoinTest > 
testSendingOldValue PASSED

org.apache.kafka.streams.kstream.internals.KTableKTableLeftJoinTest > testJoin 
STARTED

org.apache.kafka.streams.kstream.internals.KTableKTableLeftJoinTest > testJoin 
PASSED

org.apache.kafka.streams.kstream.internals.KTableKTableLeftJoinTest > 
testNotSendingOldValue STARTED

org.apache.kafka.streams.kstream.internals.KTableKTableLeftJoinTest > 
testNotSendingOldValue PASSED

org.apache.kafka.streams.kstream.internals.KTableKTableLeftJoinTest > 
shouldNotThrowIllegalStateExceptionWhenMultiCacheEvictions STARTED

org.apache.kafka.streams.kstream.internals.KTableKTableLeftJoinTest > 
shouldNotThrowIllegalStateExceptionWhenMultiCacheEvictions PASSED

org.apache.kafka.streams.kstream.internals.KTableKTableLeftJoinTest > 
shouldLogAndMeterSkippedRecordsDueToNullLeftKey STARTED

org.apache.kafka.streams.kstream.internals.KTableKTableLeftJoinTest > 
shouldLogAndMeterSkippedRecordsDueToNullLeftKey PASSED

org.apache.kafka.streams.kstream.internals.AbstractStreamTest > 
testShouldBeExtensible STARTED

org.apache.kafka.streams.kstream.internals.AbstractStreamTest > 
testShouldBeExtensible PASSED

org.apache.kafka.streams.kstream.internals.AbstractStreamTest > 
testToInternalValueTransformerSupplierSuppliesNewTransformers STARTED

org.apache.kafka.streams.kstream.internals.AbstractStreamTest > 
testToInternalValueTransformerSupplierSuppliesNewTransformers PASSED

org.apache.kafka.streams.kstream.internals.AbstractStreamTest > 
testToInternlValueTransformerSupplierSuppliesNewTransformers STARTED

org.apache.kafka.streams.kstream.internals.AbstractStreamTest > 
testToInternlValueTransformerSupplierSuppliesNewTransformers PASSED

org.apache.kafka.streams.kstream.internals.KStreamKTableLeftJoinTest > 
shouldJoinRegardlessIfMatchFoundOnStreamUpdates STARTED


Re: [VOTE] 1.1.1 RC3

2018-07-18 Thread Rajini Sivaram
Hi Dong,

+1 (binding)

Verified binary using quick start, ran tests from source, checked release
notes.

Thanks for running the release!

Regards,

Rajini


On Tue, Jul 17, 2018 at 9:19 PM, Jason Gustafson  wrote:

> +1
>
> - Did basic quickstarts on the 2.11 and 2.12 artifacts
> - Checked upgrade notes and documentation
>
> Thanks for managing the release, Dong!
>
>
> -Jason
>
> On Thu, Jul 12, 2018 at 2:36 PM, Harsha  wrote:
>
> > +1.
> > 1. Ran unit tests
> > 2. Ran 3 node cluster to run few tests.
> >
> > Thanks,
> > Harsha
> >
> > On Thu, Jul 12, 2018, at 7:29 AM, Manikumar wrote:
> > > +1 (non-binding)  Ran tests,  Verified quick start,  producer/consumer
> > perf
> > > tests
> > >
> > >
> > >
> > > On Thu, Jul 12, 2018 at 11:06 AM Brett Rann  >
> > > wrote:
> > >
> > > > +1 (non binding)
> > > > rolling upgrade of shared staging multitenacy (200+ consumer groups)
> > > > cluster from 1.1.0 to 1.1.1-rc3 using the kafka_2.11-1.1.1.tgz
> > artifact.
> > > > cluster looks healthy after upgrade. Lack of burrow lag suggests
> > consumers
> > > > are still happy, and incoming messages remains the same.
> > > >
> > > > On Mon, Jul 9, 2018 at 8:36 AM Dong Lin  wrote:
> > > >
> > > > > Hello Kafka users, developers and client-developers,
> > > > >
> > > > > This is the fourth candidate for release of Apache Kafka 1.1.1.
> > > > >
> > > > > Apache Kafka 1.1.1 is a bug-fix release for the 1.1 branch that was
> > first
> > > > > released with 1.1.0 about 3 months ago. We have fixed about 25
> issues
> > > > since
> > > > > that release. A few of the more significant fixes include:
> > > > >
> > > > > KAFKA-6925  > > > > > - Fix memory
> > > > > leak in StreamsMetricsThreadImpl
> > > > > KAFKA-6937  > > > > > - In-sync
> > > > > replica delayed during fetch if replica throttle is exceeded
> > > > > KAFKA-6917  > > > > > - Process txn
> > > > > completion asynchronously to avoid deadlock
> > > > > KAFKA-6893  > > > > > - Create
> > > > > processors before starting acceptor to avoid ArithmeticException
> > > > > KAFKA-6870  > > > > > -
> > > > > Fix ConcurrentModificationException in SampledStat
> > > > > KAFKA-6878  > > > > > - Fix
> > > > > NullPointerException when querying global state store
> > > > > KAFKA-6879  > > > > > - Invoke
> > > > > session init callbacks outside lock to avoid Controller deadlock
> > > > > KAFKA-6857  > > > > > - Prevent
> > > > > follower from truncating to the wrong offset if undefined leader
> > epoch is
> > > > > requested
> > > > > KAFKA-6854  > > > > > - Log cleaner
> > > > > fails with transaction markers that are deleted during clean
> > > > > KAFKA-6747  > > > > > - Check
> > > > > whether there is in-flight transaction before aborting transaction
> > > > > KAFKA-6748  > > > > > - Double
> > > > > check before scheduling a new task after the punctuate call
> > > > > KAFKA-6739  > > > > > -
> > > > > Fix IllegalArgumentException when down-converting from V2 to V0/V1
> > > > > KAFKA-6728  > > > > > -
> > > > > Fix NullPointerException when instantiating the HeaderConverter
> > > > >
> > > > > Kafka 1.1.1 release plan:
> > > > > https://cwiki.apache.org/confluence/display/KAFKA/Release+
> Plan+1.1.1
> > > > >  > Release+Plan+1.1.1>
> > > > >
> > > > > Release notes for the 1.1.1 release:
> > > > > http://home.apache.org/~lindong/kafka-1.1.1-rc3/RELEASE_NOTES.html
> > > > >  NOTES.html>
> > > > >
> > > > > *** Please download, test and vote by Thursday, July 12, 12pm PT
> ***
> > > > >
> > > > > Kafka's KEYS file containing PGP keys we use to sign the release:
> > > > > 

[jira] [Created] (KAFKA-7178) Is kafka compatible with zookeeper 3.5.x

2018-07-18 Thread fwq (JIRA)
fwq created KAFKA-7178:
--

 Summary: Is kafka compatible with zookeeper 3.5.x
 Key: KAFKA-7178
 URL: https://issues.apache.org/jira/browse/KAFKA-7178
 Project: Kafka
  Issue Type: Improvement
Reporter: fwq


Hi, all

I want to know is kafka versions (0.8.x, 0.9.x, 0.10.x 0.11.x 1.x) compatible 
with zookeeper 3.5.x with dynamic reconfiguration feature?

some refs on here: 
https://serverfault.com/questions/854650/kafka-compatible-with-zookeeper-3-5-feature-rebalancing-client-connections



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)


Re: [DISCUSS] KIP-333 Consider a faster form of rebalancing

2018-07-18 Thread Richard Yu
 Hi Becket,
I made some changes and clarified the motivation for this KIP. :)It should be 
easier to understand now since I included a diagram.
Thanks,Richard Yu
On Tuesday, July 17, 2018, 4:38:11 PM GMT+8, Richard Yu 
 wrote:  
 
  Hi Becket,
Thanks for reviewing this KIP. :)
I probably did not explicitly state what we were trying to avoid by introducing 
this mode. As mentioned in the KIP, there is a offset lag which could result 
after a crash. Our main goal is to avoid this lag (i.e. the latency in terms of 
time that results from the crash, not to reduce the number of records 
reprocessed).
I could provide a couple of diagrams with what I am envisioning because some 
points in my KIP might otherwise be hard to grasp (I will also include some 
diagrams to give you a better idea of an use case). As for your questions, I 
could provide a couple of answers:
1. Yes, the two consumers will in fact be processing in parallel. We do this 
because we want to accelerate the processing speed of the records to make up 
for the latency caused by the crash.
2. After the recovery point, records will not be processed twice. Let me 
describe the scenario I was envisioning: we would let the consumer that crashed 
seek to the end of the log using KafkaConsumer#seekToEnd. Meanwhile, a 
secondary consumer will start processing from the latest checkpointed offset 
and continue until it  has hit the place where the first consumer that crashed 
began processing after seekToEnd was first called. Since the consumer that 
crashed skipped from the recovery point to the end of the log, the intermediate 
offsets will be processed only by the secondary consumer. So it is important to 
note that the offset ranges which the two threads process will not overlap. 
(This is important as it prevents offsets from being processed more than once)

3. As for the committed offsets, the possibility of rewinding is not likely. If 
my understanding is correct, you are probably worried that after the crash, 
offsets that has already been previously committed will be committed again. The 
current design prevents that from happening, as the policy of where to start 
processing after a crash is universal across all Consumer instances -- we will 
begin processing from the latest offset committed. 

I hope that you at least got some of your questions answered. I will update the 
KIP soon, so please stay tuned.  

Thanks,Richard Yu
    On Tuesday, July 17, 2018, 2:14:07 PM GMT+8, Becket Qin 
 wrote:  
 
 Hi Richard,

Thanks for the KIP. I am a little confused on what is proposed. The KIP
suggests that after recovery from a consumer crash, there will be two
consumers consuming from the same partition. One consumes starting from the
log end offset at the point of recovery, and another consumes starting from
the last committed offset and keeping consuming with the first consumer in
parallel? Does that mean the messages after the recovery point will be
consumed twice? If those two consumer commits offsets, does that mean the
committed offsets may rewind?

The proposal sounds a little hacky and introduce some non-deterministic
behavior. It would be useful to have a concrete use case example to explain
what is actually needed. If the goal is to reduce the number of records
that are reprocessed when consume crashes, maybe we can have an auto commit
interval based on number of messages. If the application just wants to read
from the end of the log after recovery from crash, would calling seekToEnd
explicitly work?

Thanks,

Jiangjie (Becket) Qin



On Thu, Jul 5, 2018 at 6:46 PM, Richard Yu 
wrote:

> Hi all,
>
> I would like to discuss KIP-333 (which proposes a faster mode of
> rebalancing).
> Here is the link for the KIP:
>
> https://cwiki.apache.org/confluence/display/KAFKA/KIP-
> 333%3A+Add+faster+mode+of+rebalancing
>
> Thanks,
> Richard Yu
>