Re: Partition size for topic

2016-04-01 Thread Ewen Cheslack-Postava
Oleg, Normally the number of partitions doesn't change (or infrequently, at least) so regardless of how you got the number of partitions there shouldn't be an inconsistency. Are you actually seeing an inconsistency causing this exception? And is the number of partitions not changing? Is it

Kafka constant shrinking and expanding after deleting a topic

2016-04-01 Thread Yifan Ying
Hi All, We deleted a deprecated topic on Kafka cluster(0.8) and started observing constant 'Expanding ISR for partition' and 'Shrinking ISR for partition' for other topics. As a result we saw a huge number of under replicated partitions and very high request latency from Kafka. And it doesn't

Log Retention: What gets deleted

2016-04-01 Thread Heath Ivie
Hi, I have some questions about the log retention and specifically what gets deleted. I have a test app where I am writing 10 logs to the topic every second. What I would expect is a lag in a group would be somewhere around 10 if I have retention.ms at 1000. What I am seeing that the lag

Re: Listener for Lead broker change for topic partition in the 0.9.* consumer

2016-04-01 Thread Ryan Phillips
I believe the way this works is that you will get a UnknownTopicOrPartitionException on a poll. On Fri, Apr 1, 2016 at 1:36 PM, Wang, Howard wrote: > Hi, > > I have a use case where I need to be notified about the change of lead > broker for my topic partitions. I’m

Listener for Lead broker change for topic partition in the 0.9.* consumer

2016-04-01 Thread Wang, Howard
Hi, I have a use case where I need to be notified about the change of lead broker for my topic partitions. I’m using the new API 0.9.0. Is there any way of doing this in 0.9* API? Thanks. Howard

Re: Kafka Streams: context.forward() with downstream name

2016-04-01 Thread Guozhang Wang
Hi Yuto, That is a good suggestion, the child index is not very intuitive from programmer's view and we can even consider replacing it with the processor name instead of overloading it. Could you file a JIRA? Also I am wondering if you have looked at the higher-level Streams DSL, and if yes

Re: KStream-KTable join with the KTable given a "head start"

2016-04-01 Thread Guozhang Wang
Hi Jeff, Yes, the record timestamps are extracted once they are polled by the consumer, and are used in multiple places including stream synchronization. For details you can read this part of the docs: http://docs.confluent.io/2.1.0-alpha1/streams/architecture.html#flow-control-with-timestamps

Re: Kafka connect to database to push consumed data

2016-04-01 Thread Jay Kreps
There isn't a jdbc sink yet, though that is actually a very useful and easy connector to write. I think it can be pretty efficient as long as it uses prepared statements and batch insert... -Jay On Fri, Apr 1, 2016 at 6:06 AM, Kavitha Veluri wrote: > Thank you for

Re: Multiple streaming jobs on the same topic

2016-04-01 Thread R Krishna
Then, can you specify a size/percentage of cache per consumer group? On Apr 1, 2016 9:09 AM, "Cees de Groot" wrote: > One of Kafka's design ideas is to keep data in the JVM to a minimum, > offloading caching to the OS. So on the Kafka level, there's pretty much > not much you

Re: New async producer NetworkClient endlessly trying to fetch metadata for deleted topics

2016-04-01 Thread Stevo Slavić
Thanks for sharing Ismael! On Fri, Apr 1, 2016 at 6:24 PM, Ismael Juma wrote: > See: > > https://github.com/apache/kafka/pull/645 > https://github.com/apache/kafka/pull/735 > > Ismael > > On Fri, Apr 1, 2016 at 5:04 PM, Stevo Slavić wrote: > > >

Deadlock when committing offsets

2016-04-01 Thread Oleg Zhurakousky
Hi Was wondering what could be the causes of what appears to be an intermittent deadlock when calling Consumer.commitOffsets(..) Below are the few relevant thread dump segments: "StandardProcessScheduler Thread-7" Id=115 BLOCKED on java.lang.Object@2baae51 at

Re: Wait for 0.10 or go to 0.9 first

2016-04-01 Thread Ismael Juma
Hi Cees, On Fri, Apr 1, 2016 at 4:50 PM, Cees de Groot wrote: > FWIW we're using the new consumer in production. 0.9.0.0 had some glitches, > but we haven't had issues with 0.9.0.1. YMMV, of course. > That's good to hear. We believe 0.9.0.1 to be a solid release indeed.

Re: New async producer NetworkClient endlessly trying to fetch metadata for deleted topics

2016-04-01 Thread Ismael Juma
See: https://github.com/apache/kafka/pull/645 https://github.com/apache/kafka/pull/735 Ismael On Fri, Apr 1, 2016 at 5:04 PM, Stevo Slavić wrote: > Correction, it's much more often logging these warnings - every 100ms. > Current workaround is to regularly deploy fresh

Re: Multiple streaming jobs on the same topic

2016-04-01 Thread Cees de Groot
One of Kafka's design ideas is to keep data in the JVM to a minimum, offloading caching to the OS. So on the Kafka level, there's pretty much not much you can do - the old data is buffered by the system (has to be to be read into userspace) and thus this reduces the amount of cache available to

Re: New async producer NetworkClient endlessly trying to fetch metadata for deleted topics

2016-04-01 Thread Stevo Slavić
Correction, it's much more often logging these warnings - every 100ms. Current workaround is to regularly deploy fresh instance of app using KafkaProducer, or recreate KafkaProducer instances, so that cache of topics/partitions in it gets invalidated, cleared from deleted topics. On Fri, Apr 1,

New async producer NetworkClient endlessly trying to fetch metadata for deleted topics

2016-04-01 Thread Stevo Slavić
Hello Apache Kafka community, With kafka-clients 0.9.0.1, async producer doesn't seem to handle deleted topics well. If I create a topic, publish few messages to it, and delete it, KafkaProducer instance and underlying NetworkClient which was used to publish to that topic, will endlessly every

Re: Wait for 0.10 or go to 0.9 first

2016-04-01 Thread Cees de Groot
FWIW we're using the new consumer in production. 0.9.0.0 had some glitches, but we haven't had issues with 0.9.0.1. YMMV, of course. With respect to upgrading, as far as I understand, the Big Thing in 0.9 was the new consumer code, and in 0.10 will be Kafka streams. So the big upgrade for a

Re: ConsumerRebalanceFailedException with the kafka-console-consumer (bug?)

2016-04-01 Thread Cees de Groot
Is that a single node, local host only installation? For starters, I'd advice you to use the --new-consumer option - it will use what I think is a way less hairy method of doing consumer groups. W.r.t. Zookeeper, I've found that in test installations, I often end up wiping its data and

RE: Java API for kafka-acls.sh

2016-04-01 Thread Kalpesh Jadhav
Hi Gerard, I am using kerberised environment. By providing below jass file: KafkaClient { com.sun.security.auth.module.Krb5LoginModule required doNotPrompt=true useTicketCache=true principal="ctad...@hscale.com" useKeyTab=true serviceName="kafka"

Re: Kafka connect to database to push consumed data

2016-04-01 Thread Kavitha Veluri
Thank you for the guidance . Does Kafka connect JDBC also supports writing data into database. I understand it efficiently reads data from any relational database, but I'm looking for writing the consumed topics into database . Thanks, Kavitha veluri On Mar 31, 2016, at 10:51 PM, Surendra,

Kafka Streams: context.forward() with downstream name

2016-04-01 Thread Yuto KAWAMURA
When I tried to implement a task which does kinda dispatching to downstream processors or sinks, looks like relying on context.forward(K, V, int childIndex) is the only way now. I have a question why this method implemented using childIndex(which is just an index of children "List" that based on

Re: dumping JMX data

2016-04-01 Thread Achanta Vamsi Subhash
Why not use tools like jmxtrans and send your metrics to Graphite/OpenTsdb.etc? Why do serialization/de-serialization twice? On Fri, Apr 1, 2016 at 9:51 AM, Gerard Klijs wrote: > Don't know if adding it to Kafka is a good

Multiple streaming jobs on the same topic

2016-04-01 Thread Mayur Mohite
Hi, We have a kafka cluster running in production and there are two spark streaming job (J1 and J2) that fetches the data from the same topic. We noticed that if one of the two jobs (say J1) starts reading data from old offset (that job failed for 2 hours and when we started the job after fixing

RE: About AMQP connector and Kafka Connect framework

2016-04-01 Thread Paolo Patierno
Hi Ewen, thanks for your reply. My objective here is to access Kafka through AMQP protocol (now I'm working on a bridge from scratch without using Kafka Connect). Consider the following scenario ... Producer side : An AMQP client connects to the SourceConnector which is listening on AMQP