Best way to erase consumer state

2014-12-04 Thread Ben Drees
Hi, What is the best way to erase a stopped, multiprocess consumer's state so that it can be reset (to "smallest offsets", for example) when the consumer is restarted? Deleting /consumers/ from Zookeeper has had the desired effect in the past, but I ran into a situation today where some of the co

Re: How to create a reliable and fast Kafka Producer in a web server?

2014-12-04 Thread Haojin Gui
Thank you, Guozhang! On Thu, Dec 4, 2014 at 9:14 AM, Guozhang Wang wrote: > Haojin, > > With the old "synchronized" API you can also enable asynchrony with this > config "producer.type" in the producer configs: > > http://kafka.apache.org/documentation.html#producerconfigs > > The new "asynchron

How to cleanly shut down ConsumerConnector

2014-12-04 Thread Shannon Lloyd
Hi, I am using the high-level consumer on 0.8.2-beta. I'm attempting to close a ConsumerConnector (actually a handful of connectors), but am not having much luck actually getting it to close cleanly. When I call shutdown on the connector, I see an error in my application's log (these are always IO

Re: Best practice for upgrading Kafka cluster from 0.8.1 to 0.8.1.1

2014-12-04 Thread Yu Yang
Thanks, Guozhang! On Thu, Dec 4, 2014 at 9:08 AM, Guozhang Wang wrote: > You can still do the in-place upgrade, and the logs on the broker will be > then mixed with uncompressed and compressed messages. This is fine also > since the consumers are able to de-compress dynamically based on the > me

Re: [DISCUSSION] adding the serializer api back to the new java producer

2014-12-04 Thread Jay Kreps
Hey Guozhang, These are good points, let me try to address them. 1. Our goal is to be able to provide a best-of-breed serialization package that works out of the box that does most of the magic. This best-of-breed plugin would allow schemas, schema evolution, compatibility checks, etc. We think i

Re: [DISCUSSION] adding the serializer api back to the new java producer

2014-12-04 Thread Jay Kreps
I agree that having the new Producer(KeySerializer, ValueSerializer) interface would be useful. People suggested cases where you want to mix and match serialization types. The ByteArraySerializer is a no-op that would give the current behavior so any odd case where you need to mix and match serial

Re: kafka consumer to write into DB

2014-12-04 Thread Neha Narkhede
This is specific for pentaho but may be useful - https://github.com/RuckusWirelessIL/pentaho-kafka-consumer On Thu, Dec 4, 2014 at 12:58 PM, Sa Li wrote: > Hello, all > > I never developed a kafka consumer, I want to be able to make an advanced > kafka consumer in java to consume the data and co

kafka consumer to write into DB

2014-12-04 Thread Sa Li
Hello, all I never developed a kafka consumer, I want to be able to make an advanced kafka consumer in java to consume the data and continuously write the data into postgresql DB. I am thinking to create a map in memory and getting a predefined number of messages in memory then write into DB in ba

Re: [DISCUSSION] adding the serializer api back to the new java producer

2014-12-04 Thread Jiangjie Qin
I'm just thinking instead of binding serialization with producer, another option is to bind serializer/deserializer with ProducerRecord/ConsumerRecord (please see the detail proposal below.) The arguments for this option is: A. A single producer could send different message type

Re: [DISCUSSION] adding the serializer api back to the new java producer

2014-12-04 Thread Gwen Shapira
Can you elaborate a bit on what an object API wrapper will look like? Since the serialization API already exists today, its very easy to know how I'll use the new producer with serialization - exactly the same way I use the existing one. If we are proposing a change that will require significant c

Re: [DISCUSSION] adding the serializer api back to the new java producer

2014-12-04 Thread Guozhang Wang
I would prefer making the kafka producer as is and wrap the object API on top rather than wiring the serializer configs into producers. Some thoughts: 1. For code sharing, I think it may only be effective for though simple functions such as string serialization, etc. For Avro / Shrift / PB, the se

OutOfMemoryException when starting replacement node.

2014-12-04 Thread Dmitriy Gromov
Hi, We were recently trying to replace a broker instance and were getting an OutOfMemoryException when the new node was coming up. The issue happened during the log replication phase. We were able to circumvent this issue by copying over all of the logs to the new node before starting it. Details

Re: How to create a reliable and fast Kafka Producer in a web server?

2014-12-04 Thread Guozhang Wang
Haojin, With the old "synchronized" API you can also enable asynchrony with this config "producer.type" in the producer configs: http://kafka.apache.org/documentation.html#producerconfigs The new "asynchronized" producer improves performance by using pipelining. Guozhang On Wed, Dec 3, 2014

Re: Best practice for upgrading Kafka cluster from 0.8.1 to 0.8.1.1

2014-12-04 Thread Guozhang Wang
You can still do the in-place upgrade, and the logs on the broker will be then mixed with uncompressed and compressed messages. This is fine also since the consumers are able to de-compress dynamically based on the message type when consuming the data. Guozhang On Wed, Dec 3, 2014 at 11:33 AM, Yu

Re: Rebalance not happening even after increasing max retries causing conflict in ZK

2014-12-04 Thread Mario Lazaro
Hi Mohit Kathuria, We are facing the same issue. We are using same versions of Kafka and ZK. Did you figure out what was happening? Thanks, * * -- *Mario Lazaro* | Software Engineer, Big Data *GumGum* |

Protocol documentation

2014-12-04 Thread Suen, Mathieu
Hi I was reading the protocol documentation on the wiki page: https://cwiki.apache.org/confluence/display/KAFKA/A+Guide+To+The+Kafka+Protocol But it seems that the description is not compete. For example there is no information on the type of the different field (ApiKe, ApiVersion, ClientId, .

Re: KafkaException: Should not set log end offset on partition

2014-12-04 Thread svante karlsson
Thanks, kind of obvious afterwards :-). I used ip addresses in my ansible configuration so everything else worked. 2 -> {"jmx_port":13005,"timestamp":"1417702658359","host":"ubuntu","version":1,"port":9092}... 4 -> {"jmx_port":13005,"timestamp":"1417698861869","host":"ubuntu","version":1,"port":90

Re: [DISCUSSION] adding the serializer api back to the new java producer

2014-12-04 Thread Jun Rao
Jan, Jason, First, within an Kafka cluster, it's unlikely that each topic has a different type serializer. Like Jason mentioned, Square standardizes on protocol. Many other places such as LinkedIn standardize on Avro. Second, dealing with bytes only has limited use cases. Other than copying bytes

How to create a reliable and fast Kafka Producer in a web server?

2014-12-04 Thread Haojin Gui
Hi, I am using Playframework to build a web server that needs to handle very heavy traffic. And it has a Kafka Producer sending messages to Kafka brokers. Currently I am using the synchronized Kafka producer API and seems like it's affecting the performance of the web server. I am considering swi

Re: Partition reassignment reversed

2014-12-04 Thread Andrew Jorgensen
We are currently running 0.8.1.1, I just double checked. One other thing that may be related is I brought up a second kafka cluster today matching the first. I noticed that if I deleted a topic and the re-created it with the same name when I re-created the topic none of the leader elections happ

Re: Failed partition reassignment

2014-12-04 Thread Jun Rao
You can do the following (1) check if there is any error in the controller and the state-change log, (2) use the per partition offset lag JMX in the follower to see if the follower is making good progress. Thanks, Jun On Tue, Dec 2, 2014 at 3:13 PM, Karol Nowak wrote: > I don't have it reprodu

Re: Partition reassignment reversed

2014-12-04 Thread Jun Rao
Not sure exactly what happened there. We did fix a few bugs in reassigning partitions in 0.8.1.1. So, you probably want to upgrade to that one or the upcoming 0.8.2 release. Thanks, Jun On Tue, Dec 2, 2014 at 2:33 PM, Andrew Jorgensen wrote: > I am using kafka 0.8. > Yes I did run —verify, but

Re: Best practice for upgrading Kafka cluster from 0.8.1 to 0.8.1.1

2014-12-04 Thread Yu Yang
Guozhang, We haven't enable message compression yet. In this case, what shall we do when we upgrade to 0.8.2? Must we launch a new cluster, redirect the traffic to the new cluster, and turn off the old one? Thanks! -Yu On Tue, Dec 2, 2014 at 4:33 PM, Guozhang Wang wrote: > Yu, > > Are you e

Re: How many messages does each broker have?

2014-12-04 Thread Palur Sandeep
Thanks Guozhang and others for helping me. I am able to send and receive 10MB size messages now. In case anybody has the same requirement. Please make the following changes *server.properties* message.max.bytes=10485800 replica.fetch.max.bytes=10485800 socket.send.buffer.bytes=104857600 socket.rec

Re: KafkaException: Should not set log end offset on partition

2014-12-04 Thread Jun Rao
Could you look at the broker registration in Zookeeper ( https://cwiki.apache.org/confluence/display/KAFKA/Kafka+data+structures+in+Zookeeper) and make sure the 3 registered hosts are unique? Thanks, Jun On Wed, Dec 3, 2014 at 5:54 AM, svante karlsson wrote: > I've installed (for ansible scrip

Re: [DISCUSSION] adding the serializer api back to the new java producer

2014-12-04 Thread Philippe Laflamme
Sorry for adding noise, but I think Jan has a very good point: applications shouldn't be forced to create multiple producers simply to wire-in the proper Serializer. It's an artificial restriction that wastes resources. It's a common thing for us to create a single producer and slap different "vie

Re: Can Mirroring Preserve Every Topic's Partition?

2014-12-04 Thread Neha Narkhede
As Guozhang mentioned, that will allow exact partitioning but note that this still doesn't ensure identical partitions since your target cluster can have duplicates depending on whether the producer retried sending data or not. On Wed, Nov 26, 2014 at 10:33 AM, Guozhang Wang wrote: > Hello Alex,

Re: isr never update

2014-12-04 Thread Jun Rao
The state-change log does indicate an error. Is the issue easily reproducible? If so, could you try this on the 0.8.2 beta release? We did fix some controller related issues in 0.8.2. Thanks, Jun On Tue, Dec 2, 2014 at 1:46 AM, Shangan Chen wrote: > I checked the max lag and it was 0. > > I gr