Re: Kafka Streams scaling questions

2016-03-22 Thread Kishore Senji
nob to consider. Consumers (actually the > leader consumer) can control which partitions they get assigned. KStreams > already uses this feature to do things like create stand by replicas, but I > don’t think (but I may be wrong) this helps you with your problem directly. > > All the b

Kafka Streams scaling questions

2016-03-19 Thread Kishore Senji
Hi All, I read through the doc on KStreams here: http://www.confluent.io/blog/introducing-kafka-streams-stream-processing-made-simple

Re: java.lang.NoClassDefFoundError: Could not initialize class com.yammer.metrics.Metrics

2015-09-02 Thread Kishore Senji
It is a NoClassDefFoundError so the clinit of Metrics class is failing. Please look for surrounding stack traces, there must be a root cause for why the class initialization failed. On Wed, Sep 2, 2015 at 1:05 PM, Vadim Keylis wrote: > I suddenly started getting this error using the code that wa

Re: 0.8.2 producer and single message requests

2015-09-01 Thread Kishore Senji
ducer had just one more API that took a list of messages > and > > handed me back a callback for that list, things would've been much > simpler. > > > > > > On Mon, Aug 17, 2015 at 10:41 PM, Kishore Senji > wrote: > > > > > If linger.ms is 0, batc

Re: KafkaProducer recovery/restart if broker dies

2015-08-28 Thread Kishore Senji
I faced the exact same problem recently. The JIRA is filed here: https://issues.apache.org/jira/browse/KAFKA-2459 Please have reconnect.backoff.ms to be greater than retry.backoff.ms (like 1sec more). I think the metadata expired and when it is trying to fetch the new metadata for this producer in

Re: Mirror a partition of a topic

2015-08-25 Thread Kishore Senji
Flume could be an option with an Interceptor although the throughput could be less compared to Mirror Maker with compression and shallow iterator enabled. On Tue, Aug 25, 2015 at 10:28 PM tao xiao wrote: > In the trunk code mirror maker provides the ability to filter out messages > on demand by s

Re: bootstrap.servers for the new Producer

2015-08-22 Thread Kishore Senji
; verify in a test environment. > > I've filed https://issues.apache.org/jira/browse/KAFKA-2459 for that > issue. > > -Ewen > > > On Fri, Aug 21, 2015 at 11:42 PM, Kishore Senji wrote: > > > Thank you Ewen. This behavior is something that I'm obse

Re: bootstrap.servers for the new Producer

2015-08-21 Thread Kishore Senji
for a > couple of servers. If anything happens to those servers too quickly (within > the metadata refresh period) you might potentially get stuck with only > references to dead nodes. > > -Ewen > > On Fri, Aug 21, 2015 at 6:56 PM, Kishore Senji wrote: > > > If one of the br

bootstrap.servers for the new Producer

2015-08-21 Thread Kishore Senji
If one of the broker we specify in the bootstrap servers list is down, there is a chance that the Producer (a brand new instance with no prior metadata) will never be able to publish anything to Kafka until that broker is up. Because the logic for getting the initial metadata is based on some rando

Re: Possible DEAD LOCK for one day at broker controller?

2015-08-20 Thread Kishore Senji
Hi Zhao, Do you see any other errors regarding checkpoint file? Is this reproducible by you and if you can you enable debug log level to get more info.​ On Thu, Aug 20, 2015 at 7:44 AM, Zhao Weinan wrote: > Hi Kishore Senji, > > I've been busy recovering some data these two day

Re: Error handling in New AsyncProducer

2015-08-18 Thread Kishore Senji
e > ?, could you please explain > > On Tue, Aug 18, 2015 at 11:47 AM, Kishore Senji wrote: > > > But this will reduce the throughput in a good scenario. May be we need to > > enhance the Callback interface appropriately. > > > > On Mon, Aug 17, 2015 at 7:15 PM, sunil

Re: Possible DEAD LOCK for one day at broker controller?

2015-08-18 Thread Kishore Senji
brokers are fine. On Tue, Aug 18, 2015 at 12:48 AM Zhao Weinan wrote: > Hi Kishore Senji, > > Did you constantly send messages to your test topic? Or just one time send? > I've just did some test, the log.lastModified is updated with every message > received (or every flush t

Re: Possible DEAD LOCK for one day at broker controller?

2015-08-17 Thread Kishore Senji
the delete thread runs (every five min), it deletes even the latest Segment because that Segment age is older than retention.ms On Mon, Aug 17, 2015 at 11:30 PM, Zhao Weinan wrote: > Hi Kishore Senji, > > The size of segement file is default 1GB. > > According to th

Re: Error handling in New AsyncProducer

2015-08-17 Thread Kishore Senji
But this will reduce the throughput in a good scenario. May be we need to enhance the Callback interface appropriately. On Mon, Aug 17, 2015 at 7:15 PM, sunil kalva wrote: > tx jeff, > Actually we need to set "buffer.memory" to minimum (default is ~35 MB) and > "block.on.buffer.full" to "true"

Re: 0.8.2 producer and single message requests

2015-08-17 Thread Kishore Senji
ce in front of Kafka more involved than the 0.8.1 API. > > On Sun, Aug 16, 2015 at 12:02 AM, Kishore Senji wrote: > > > Adding to what Gwen already mentioned - > > > > The programming model for the Producer is send() with an optional > callback > > an

Re: Reduce latency

2015-08-17 Thread Kishore Senji
cantly. > > > > But is there only one thread io thread sending the batches? Can I > increase > > the number of threads sending the batches so more than one batch could be > > sent at the same time? > > > > Thanks. > > > > > > > > O

Re: Possible DEAD LOCK for one day at broker controller?

2015-08-17 Thread Kishore Senji
17, 2015 at 6:59 AM Zhao Weinan wrote: > Hi Kishore Senji, > > Thanks for the reply. > > Do you have some suggestions before the fix came up? Try not to modify the > retention.ms? Or disable the auto rebalance? Cause this problem is 100% > reproduceable in my scenario (two t

Re: Possible DEAD LOCK for one day at broker controller?

2015-08-16 Thread Kishore Senji
Interesting problem you ran in to. It seems like this broker was chosen as the Controller and onControllerFailure() method was called. This will schedule the checkAndTriggerPartitionRebalance method to execute after 5 seconds (when auto rebalance enabled). In the mean time this broker lost zookeepe

Re: 0.8.2 producer and single message requests

2015-08-16 Thread Kishore Senji
Adding to what Gwen already mentioned - The programming model for the Producer is send() with an optional callback and we get a Future. This model does not change whether behind the scenes batching is done or not. So your fault tolerance logic really should not depend on whether batching is done o

Re: use page cache as much as possiblee

2015-08-15 Thread Kishore Senji
e usage should not be > reflected when using "watch df -h" command, am I correct? > > Thanks. > > On Fri, Aug 14, 2015 at 10:12 PM, Kishore Senji wrote: > > > Actually in 0.8.2, flush.ms & flush.messages are recommended to be left > > defaults (Long.MAX

Re: use page cache as much as possiblee

2015-08-14 Thread Kishore Senji
> producer and consumer? > > Thanks. > > > On Fri, Aug 14, 2015 at 11:57 AM, Kishore Senji wrote: > > > Thank you Gwen for correcting me. This document ( > > https://cwiki.apache.org/confluence/display/KAFKA/Kafka+Replication) in > > "Writes" section also

Re: use page cache as much as possiblee

2015-08-14 Thread Kishore Senji
ng correct here? On Thu, Aug 13, 2015 at 10:50 PM, Gwen Shapira wrote: > On Thu, Aug 13, 2015 at 4:10 PM, Kishore Senji wrote: > > > Consumers can only fetch data up to the committed offset and the reason > is > > reliability and durability on a broker crash (some consumers m

Re: use page cache as much as possiblee

2015-08-13 Thread Kishore Senji
Consumers can only fetch data up to the committed offset and the reason is reliability and durability on a broker crash (some consumers might get the new data and some may not as the data is not yet committed and lost). Data will be committed when it is flushed. So if you delay the flushing, consum

Re: Reduce latency

2015-08-13 Thread Kishore Senji
Your batch.size is 8196 and your buffer.memory is 67108864. This means 67108864/8196 ~ 8188 batches are in memory ready to the sent. There is only one thread io thread sending them. I would guess that the io thread ( kafka-producer-network-thread) would be busy. Please check it in visual vm. In st