Re: ZooKeeper connect/disconnect pattern

2014-03-27 Thread Neha Narkhede
[2014-03-25 12:21:58,848] INFO Client session timed out, have not heard from server in 4001ms f or sessionid 0x54441e4dae0350a, closing socket connection and attempting reconnect (org.apache.zookeeper. ClientCnxn) This is a problem. It indicates a GC pause or something similar on either the Kafka

Re: How are rebalance failures raised to consumers?

2014-03-27 Thread Neha Narkhede
That's correct. To recover, you will have to bounce one of the consumer processes so the group will attempt a rebalance operation. Thanks, Neha On Thu, Mar 27, 2014 at 10:09 AM, Tom Amon wrote: > This means that code waiting on the iterator will continue to wait forever > if rebalance fails? N

Re: Please add Perl client on your wiki

2014-03-27 Thread Neha Narkhede
That's great. Thanks for contributing! Added this to our clients wiki - https://cwiki.apache.org/confluence/display/KAFKA/Clients#Clients-Perl Thanks, Neha On Thu, Mar 27, 2014 at 11:24 AM, Sergiy Zuban wrote: > Kafka 0.8.x > > Pure Perl, Consumer and Producer implementations included. Zookeep

Re: ZK session kill makes high level consumer replay

2014-03-27 Thread Neha Narkhede
When I call consumer.commitOffsets(); before killing session, unit test succeeded. This problem would happen only with autoCommit enabled That seems expected. If you call commitOffsets() explicitly before simulating a GC pause on the consumer, there will be no duplicates since the next consumer in

Please add Perl client on your wiki

2014-03-27 Thread Sergiy Zuban
Kafka 0.8.x Pure Perl, Consumer and Producer implementations included. Zookeeper integration. GZIP and Snappy compression not supported. Maintainer: Sergey Gladkov, License: Artistic License https://github.com/TrackingSoft/Kafka http://search.cpan.org/~sgladkov/Kafka/lib/Kafka.pm -- Sergiy Zu

Re: data loss on replicated topic

2014-03-27 Thread Jun Rao
We don't expect to lose data in that case. So, this sounds like a bug. Do you see any other error/warn in broker log around the time the data is lost? Thanks, Jun On Thu, Mar 27, 2014 at 10:52 AM, Oliver Dain wrote: > Hi Neha, > > Thanks for the reply. I do not see the ³No broker in ISR² messa

Re: ZK session kill makes high level consumer replay

2014-03-27 Thread Bae, Jae Hyeon
When I call consumer.commitOffsets(); before killing session, unit test succeeded. This problem would happen only with autoCommit enabled. Could you fix this problem before releasing 0.8.1.1? Thank you Best, Jae On Thu, Mar 27, 2014 at 3:57 PM, Bae, Jae Hyeon wrote: > Hi > > While testing kaf

ZK session kill makes high level consumer replay

2014-03-27 Thread Bae, Jae Hyeon
Hi While testing kafka 0.8 consumer's zk resilience, I found that on the zk session kill and handleNewSession() is called, high level consumer is replaying messages. Is this know issue? I am attaching unit test source code. package com.netflix.nfkafka.zktest; import com.fasterxml.jackson.core.J

Re: ZooKeeper connect/disconnect pattern

2014-03-27 Thread Tom Amon
From the controller log [2014-03-25 12:22:39,487] INFO [ControllerEpochListener on 2]: Initialized controller epoch to 11 and zk version 10 (kafka.controller.ControllerEpochListener) [2014-03-25 12:22:39,519] INFO [Controller 2]: Controller starting up (kafka.controller.KafkaCo ntrol

Error in fetch name

2014-03-27 Thread Tom Amon
We see the following messages in the broker logs whenever we reboot a broker. These messages filled up 200MB of log files in less than 1 minute. Are these normal? For reference we have enabled controlled shutdown on each broker. [2014-03-25 22:52:45,558] INFO Reconnect due to socket error: null (

Re: data loss on replicated topic

2014-03-27 Thread Oliver Dain
Hi Neha, Thanks for the reply. I do not see the ³No broker in ISR² message. If my original diagnosis was correct (that there were at least 2 replicas alive for the topic at all times) then I believe this is expected, right? I gather this makes it more likely that we¹ve hit KAFKA-1193?? If so, is t

Re: How are rebalance failures raised to consumers?

2014-03-27 Thread Tom Amon
This means that code waiting on the iterator will continue to wait forever if rebalance fails? No exception will be thrown from the iterator? I assume from your message that the only way to tell if a rebalance has failed and consumers have stopped consuming is by monitoring the lag and restarting

Re: New Consumer API discussion

2014-03-27 Thread Neha Narkhede
If people don't have any more thoughts on this, I will go ahead and submit a reviewboard to https://issues.apache.org/jira/browse/KAFKA-1328. Thanks, Neha On Mon, Mar 24, 2014 at 5:39 PM, Neha Narkhede wrote: > I took some time to write some example code using the new consumer APIs to > cover a

Re: Kafka snappy compression

2014-03-27 Thread 小宇
Hi, here is the issue. https://issues.apache.org/jira/browse/KAFKA-1344 2014-03-27 20:31 GMT+08:00 Neha Narkhede : > Currently the console producer only supports GZIP. Could you please file a > JIRA for the snappy support? > > > > > On Thu, Mar 27, 2014 at 5:21 AM, 小宇 wrote: > > > Yes,maybe I m

Re: Kafka snappy compression

2014-03-27 Thread 小宇
Thanks, I will try it. 2014-03-27 20:36 GMT+08:00 Magnus Edenhill : > Some notes on this: > - remember that Messages are wrapped in a compressed MessageSet that is > wrapped in an outer Message which is wrapped in an outer MessageSet (the > mind boggles). > - the Kafka snappy compression has a

Re: Kafka snappy compression

2014-03-27 Thread Magnus Edenhill
Some notes on this: - remember that Messages are wrapped in a compressed MessageSet that is wrapped in an outer Message which is wrapped in an outer MessageSet (the mind boggles). - the Kafka snappy compression has a non-standard framing - snappy-java (prefix "\x82SNAPPY\x0") - this framing is sp

Re: Kafka snappy compression

2014-03-27 Thread Neha Narkhede
Currently the console producer only supports GZIP. Could you please file a JIRA for the snappy support? On Thu, Mar 27, 2014 at 5:21 AM, 小宇 wrote: > Yes,maybe I missed something, and how can I send message in snappy > compression with Kafka-console-producer? > Thanks. > > Neha Narkhede 于2014年

Re: producers limit

2014-03-27 Thread Neha Narkhede
What is the performance overhead if we don't use batching? It can be very significant. I'm not sure if we have published performance numbers for the new producer yet, but you can see the 0.8 producer throughput numbers here - https://cwiki.apache.org/confluence/display/KAFKA/Performance+testing#P

Re: Kafka snappy compression

2014-03-27 Thread 小宇
Yes,maybe I missed something, and how can I send message in snappy compression with Kafka-console-producer? Thanks. Neha Narkhede 于2014年3月27日星期四写道: > Have you followed the wire protocol documented here - > > https://cwiki.apache.org/confluence/display/KAFKA/A+Guide+To+The+Kafka+Protocol > > > > >

Re: How are rebalance failures raised to consumers?

2014-03-27 Thread Neha Narkhede
What Jun described is correct, currently the consumer asynchronously fails rebalance. But you can monitor the consumer to detect when a rebalance operation fails since that will manifest as lag on the consumer. Please see this

Re: data loss on replicated topic

2014-03-27 Thread Neha Narkhede
It is possible that you are hitting KAFKA-1193, but I'm not sure. Do you see the following log line when you observe data loss - "No broker in ISR is alive for ... There's potential data loss." Thanks, Neha On Wed, Mar 26, 2014 at 12:05 PM, Oliver Dain wrote: > I just saw https://issues.apache

Re: Kafka snappy compression

2014-03-27 Thread Neha Narkhede
Have you followed the wire protocol documented here - https://cwiki.apache.org/confluence/display/KAFKA/A+Guide+To+The+Kafka+Protocol On Thu, Mar 27, 2014 at 3:22 AM, 小宇 wrote: > Hi, all > I'm working on snappy compression in our Node.js client ( > https://github.com/SOHU-Co/kafka-node),but I

Re: Log retention policy - offset ?

2014-03-27 Thread Neha Narkhede
Suppose if old log messages are getting deleted what will happens to offset ? Offset of a message will never change. Suppose we have 10 messages in which 3 messages are older than 24 hrs. My consumer consumed upto message 5 (offset 4). still 5 more msgs yet to consume In this case, your consumpt

Log retention policy - offset ?

2014-03-27 Thread Ranjith Venkatesan
Hi,We are about to use kafka-0.8 in our production. We have our config to delete messages less than a day i.e 24 hrs. We have our simple consumer which will have its offset in memory and will handle if offsetoutofrange exception occurs.Suppose if old log messages are getting deleted what will happe

Kafka snappy compression

2014-03-27 Thread 小宇
Hi, all I'm working on snappy compression in our Node.js client ( https://github.com/SOHU-Co/kafka-node),but I found the offset of message always 0 after snappy compression, for example: , I try to test it in kafka-cosole-producer.sh, but it seems not support snappy. So I can't figure out what's wr