, 2014 at 5:51 PM, Bhavesh Mistry
wrote:
> Hi Jun,
>
> Is there any plug-ability that Developer can customize batching logic or
> inject custom code for this ? Shall I file Jira for this issues.
>
> Thanks,
>
> Bhavesh
>
> On Tue, Sep 9, 2014 at 3:52 PM, Jun Rao wro
We actually don't allow "." in the topic name. Topic name can be
alpha-numeric plus "-" and "_".
Thanks,
Jun
On Tue, Sep 9, 2014 at 6:29 PM, Bhavesh Mistry
wrote:
> Thanks, I was using without JMX. I will go through doc. But how about
> Topic or Metric name Topic Name Convention or Metric Na
*
> message.max.bytes=2MB
>
> Thanks,
> Bhavesh
>
> On Wed, Sep 10, 2014 at 10:10 AM, Jun Rao wrote:
>
> > Actually, with the new producer, you can configure the batch size in
> bytes.
> > If you set the batch size to be smaller than the max message siz
I meant whether you start the broker in service containers like jetty or
tomcat.
Thanks,
Jun
On Wed, Sep 10, 2014 at 12:28 AM, Shlomi Hazan wrote:
> Hi, sorry, what do you mean by 'container'? I use bare EC2 instances...
> Shlomi
>
> On Wed, Sep 10, 2014 at 1:41 AM, Jun
As long as the I/O load is reasonable, this is probably ok.
Thanks,
Jun
On Wed, Sep 10, 2014 at 4:59 AM, Hemanth Yamijala
wrote:
> Hi folks,
>
> In order to meet latency requirements for a system we are building, we
> tested with different values of the above two parameters and found that
> se
Storing consumer offsets in the broker will add a bit overhead to the
broker, but shouldn't be too bad. Compared with traditional messaging
systems, Kafka delivers messages to consumers in order and therefore only
needs to keep track of a single last consumed offset per consumer client
per topic pa
David,
Thanks for the pointer. Added the link to our website.
Jun
On Wed, Sep 10, 2014 at 11:05 AM, David Arthur wrote:
> https://botbot.me/freenode/apache-kafka/
>
> Just FYI, wasn't sure if we had any logging in place
>
> Cheers,
> David
>
>
>
Yes, the older producer should work with the 0.8.1.1 brokers.
Thanks,
Jun
On Wed, Sep 10, 2014 at 11:28 AM, Seshadri, Balaji wrote:
> We are planning for upgrading to 0.8.1.1 from 0.8-beta.
>
> Can you please let us know the impacts of doing it ?.
>
> I understand we have fixes for deadlocks i
ormance, on versions < 0.8.1. As long as you're on 0.8.1 or
> > later, it should mostly be fine. You might want to keep a close tab on
> how
> > your iostat numbers are doing, to be sure.
> >
> > On Wed, Sep 10, 2014 at 5:46 PM, Hemanth Yamijala
> > wrote:
&g
This is controlled by linger.ms in the new producer in trunk.
Thanks,
Jun
On Thu, Sep 11, 2014 at 5:56 PM, Bhavesh Mistry
wrote:
> Hi Kafka team,
>
> How do I configure a max amount a message spend in Queue ? In old
> producer, there is property called queue.buffering.max.ms and it is not
> p
sposal to add reasonable "structure" to
> your Kafka topic names. (https://issues.apache.org/jira/browse/KAFKA-1481)
>
> Do you mind sharing your topic naming convention at LinkedIn?
>
> --Michael
>
>
>
>
>
> On 11.09.2014 00:24, Jun Rao wrote:
> >
head() is a scala method. Calling it from java requires you to figure out
the exact class name in byte code. A simpler way is to use the java
iterable api in KafkaStream. By default, it blocks on the hasNext()
call when there is no message.
Thanks,
Jun
On Sat, Sep 13, 2014 at 1:01 AM, Aarti Gupt
We have that in 0.7 too. It's called zk.connectiontimeout.ms.
Thanks,
Jun
On Mon, Sep 15, 2014 at 12:55 AM, Yonghui Zhao
wrote:
> Hi,
>
> In http://kafka.apache.org/08/configuration.html, there are 2 parameter
> about zk in consumer.
>
> zookeeper.session.timeout.ms 6000 Zookeeper session ti
gt;
> However, haven't you said earlier in this thread that only "-" and "_" are
> allowed as special chars? This would exclude dots, although in the error
> message below they (dots) are said to be fine. Is this a bug or a feature?
> ;-)
>
> --Michael
&g
Bhavesh,
Yes, allowing dot in clientId and topic makes it a bit harder to define the
JMX bean names. I see a couple of solutions here.
1. Disable dot in clientId and topic names. The issue is that dot may
already be used in existing deployment.
2. We can represent the JMX bean name differently i
ch part is the topic).
Thanks,
Jun
On Wed, Sep 17, 2014 at 2:32 PM, Otis Gospodnetic <
otis.gospodne...@gmail.com> wrote:
> Hi Jun,
>
> On Wed, Sep 17, 2014 at 12:35 PM, Jun Rao wrote:
>
> > Bhavesh,
> >
> > Yes, allowing dot in clientId and topic makes it a bi
That's right. The rule is that a log segment is deleted if either the size
or the time limit is reached. Log sizes are per partition.
Thanks,
Jun
On Thu, Sep 18, 2014 at 2:55 PM, Cory Watson wrote:
> Hello all!
>
> I'm curious about the interaction of server and topic level retention
> setting
The new consumer api will also allow you to do what you want in a
SimpleConsumer (e.g., subscribe to a static set of partitions, control
initial offsets, etc), only more conveniently.
Thanks,
Jun
On Mon, Sep 22, 2014 at 8:10 AM, Valentin wrote:
>
> Hello,
>
> I am currently working on a Kafka
We allocate a new BoundedByteBufferReceive for every fetch request. Are you
using SimpleConsumer directly? It seems it's started by the high level
consumer through the FetchFetcher thread.
Thanks,
Jun
On Mon, Sep 22, 2014 at 11:41 AM, Jagbir Hooda wrote:
> Note: Re-posting the older message f
What version of Kafka are you using? Have you increased the max message
size on the broker (default to 1MB)?
Thanks,
Jun
On Mon, Sep 22, 2014 at 3:41 PM, Kyle Banker wrote:
> I have a test data set of 1500 messages (~2.5 MB each) that I'm using to
> test Kafka throughput. I'm pushing this data
Also, don't forget to increase replica.fetch.max.bytes to be larger than
the max message size.
Thanks,
Jun
On Mon, Sep 22, 2014 at 9:35 PM, Jun Rao wrote:
> What version of Kafka are you using? Have you increased the max message
> size on the broker (default to 1MB)?
>
>
r is single
> > threaded, creating this many new consumers is roughly the same cost with
> > the old simple consumer.
> >
> > Guozhang
> >
> > On Tue, Sep 23, 2014 at 2:32 AM, Valentin wrote:
> >
> >>
> >> Hi Jun,
> >>
> >> On
the problem. I increased both
> message.max.bytes and replica.fetch.max.bytes on the broker.
>
> For the benefit of future Kafka users, how hard would it be to build out
> some clearer error messaging for this case?
>
> On Mon, Sep 22, 2014 at 10:38 PM, Jun Rao wrote:
>
&
was busy) and the
> thread was always RUNNABLE at the same location. Intuitively reading
> the sizeBuffer should have been much less CPU bound then reading the
> contentBuffer.
>
> Thanks,
> Jagbir
>
> On Mon, Sep 22, 2014 at 9:26 PM, Jun Rao wrote:
> > We allocat
cessFetchRequest(AbstractFetcherThread.scala:96)
> at
> kafka.server.AbstractFetcherThread.doWork(AbstractFetcherThread.scala:88)
> at kafka.utils.ShutdownableThread.run(ShutdownableThread.scala:51)
> -8<
>
> On Wed, Sep 24, 2014 at 9:39 PM,
nt plans for the Kafka 0.9 consumer correctly, it just doesn't work
> well for my use case. Sure, there are workarounds to make it work in my
> scenario, but I doubt any of them would scale as well as my current
> SimpleConsumer approach :|
> Or am I missing something here?
It could be that broker 1 and 3 can't communicate with broker 2 and the
consumer client. You may want to read
https://cwiki.apache.org/confluence/display/KAFKA/FAQ#FAQ-Whycan'tmyconsumers/producersconnecttothebrokers
?
Thanks,
Jun
On Thu, Sep 25, 2014 at 1:52 PM, florent valdelievre <
florentval
kerid":22,"timestamp":"1407187809296"}
>
> In the case when a controller drops out of the ISR for a few seconds, is it
> possible for this confused broker to drop ACKed messages?
>
>
>
> On Thu, Jul 3, 2014 at 12:48 AM, Jun Rao wrote:
>
> > Are you
Could you check that the broker host registered in ZK is the ip that you
are expecting?
Thanks,
Jun
On Tue, Sep 30, 2014 at 3:03 AM, florent valdelievre <
florentvaldelie...@gmail.com> wrote:
> Hi again,
>
> 192.168.1.180
> Zk: 192.168.1.180:2181
> Kafka: 9092
> Broker.id = 1
> zookeeper.connec
Actually, this is probably the more relevant jira: KAFKA-1367.
Thanks,
Jun
On Tue, Sep 30, 2014 at 10:00 AM, Christofer Hedbrandh <
christo...@knewton.com> wrote:
> Hi Kafka users,
>
> Was there ever a JIRA ticket filed for this?
>
> "Re: Stale TopicMetadata"
>
>
> http://mail-archives.apache.o
The reassign partition process only completes after the new replicas are
fully caught up and the old replicas are deleted. So, if the old replica is
down, the process can never complete, which is what you observed. In your
case, if you just want to replace a broker host with a new one, instead of
u
We already cut an 0.8.2 release branch. The plan is to have the remaining
blockers resolved before releasing it. Hopefully this will just take a
couple of weeks.
https://issues.apache.org/jira/browse/KAFKA-1663?filter=-4&jql=project%20%3D%20KAFKA%20AND%20status%20in%20(Open%2C%20%22In%20Progress%2
Can you follow the example in quickstart (
http://kafka.apache.org/documentation.html#quickstart)?
Thanks,
Jun
On Thu, Oct 2, 2014 at 12:01 PM, Sa Li wrote:
> Hi, all
>
> Here I want to run example code associated with kafka package, I run as
> readme says:
>
> To run the demo using scripts:
>
Hmm, not sure what the issue is. You can also just copy the following files
from the 0.8.1 branch.
gradle/wrapper/
gradle-wrapper.jar gradle-wrapper.properties
Thanks,
Jun
On Thu, Oct 2, 2014 at 2:05 PM, Sa Li wrote:
> I git clone the latest kafka package, why can't I build the packa
In general, only writers should trigger auto topic creation, but not the
readers. So, a topic can be auto created by the producer, but not the
consumer.
Thanks,
Jun
On Thu, Oct 2, 2014 at 2:44 PM, Stevo Slavić wrote:
> Hello Apache Kafka community,
>
> auto.create.topics.enable configuration o
The jmx is only available in the high level consumer. If you are using
SimpleConsumer, you have to calculate this yourself (hw from fetchresponse
- fetch offset).
Thanks,
Jun
On Fri, Oct 3, 2014 at 5:32 AM, Devang Shah wrote:
> Refering to the monitoring secton available on the kafka doc websi
s is
> 0) without having UnknownTopicOrPartitionException at least as cause. Is
> this feature or a bug, and more importantly could it be improved?
>
> Kind regards,
> Stevo Slavic.
> On Oct 3, 2014 6:30 AM, "Jun Rao" wrote:
>
> > In general, only writers should trigger auto top
This is a jmx on the consumer side. The server side metric is for the
internal replication consumer and is only available if you have a
replication factor more than 1.
Thanks,
Jun
On Fri, Oct 3, 2014 at 4:28 AM, Shah, Devang1 wrote:
> I tried hooking up jconsole to Kafka server but could not s
Yes, transient error like LeaderNotAvailableException can happen. If you
configure enough retries, then you shouldn't see the exception in the
normal case.
Thanks,
Jun
On Mon, Oct 6, 2014 at 5:35 AM, Stevo Slavić wrote:
> Hello Apache Kafka community,
>
> When trying to publish (using high lev
gards,
> Stevo Slavic
>
> On Mon, Oct 6, 2014 at 5:17 AM, Jun Rao wrote:
>
> > Yes, the docs can be improved. Could you file a jira?
> >
> > For the 2nd issue, the new java producer handles this better.
> >
> > Thanks,
> >
> > jun
> >
>
hould we provide a new patch that
> uses your approach?
>
> Thanks,
> Otis
> --
> Monitoring * Alerting * Anomaly Detection * Centralized Log Management
> Solr & Elasticsearch Support * http://sematext.com/
>
>
> On Thu, Sep 18, 2014 at 4:49 PM, Jun Rao wrote:
>
You can stop the broker and copy some of the log directories to the new
volumes. You have to be a bit careful when you do that. There are two
metadata files recovery-point-offset-checkpoint and
replication-offset-checkpoint that you have to manually split and copy over.
Ideally, we should be able
Yes, this is typical use case for mirror maker.
Thanks,
Jun
On Tue, Oct 7, 2014 at 4:33 PM, István wrote:
> Hi all,
>
> I have just a quick question. I was wondering if MirrorMaker is the right
> solution for this task, basically I need to move a subset of the data from
> a production cluster
ection * Centralized Log Management
> Solr & Elasticsearch Support * http://sematext.com/
>
>
> On Tue, Oct 7, 2014 at 4:04 PM, Jun Rao wrote:
>
> > Otis,
> >
> > Yes, if you guys can help provide a patch in a few days, we can probably
> > get it to the 0.8.2 rel
It's called request.timeout.ms in 0.8.
Thanks,
Jun
On Wed, Oct 8, 2014 at 8:58 AM, ravi singh wrote:
> Kafka 07 has following property for producer.
>
> connect.timeout.ms5000the maximum time spent
> bykafka.producer.SyncProducer trying
> to connect to the kafka broker. Once it elapses, the pr
Which version of Kafka are you using?
Thanks,
Jun
On Wed, Oct 8, 2014 at 12:17 PM, Pradeep Badiger
wrote:
> Hi,
>
> I am trying to create a topic using TopicCommand and I get an error back.
> I see lot of rebalance attempt logs. Can someone help me with this issue?
>
>
> 12702927406-17ccd847],
You can look at the consumer example at
http://kafka.apache.org/documentation.html#highlevelconsumerapi
Thanks,
Jun
On Wed, Oct 8, 2014 at 7:51 PM, Sa Li wrote:
> Hi, All
>
> I setup a kafka cluster, and plan to publish the messages from Web to
> kafka, the messages are in the form of json, I
If enabled request logging, you can find this out.
Thanks,
Jun
On Wed, Oct 8, 2014 at 8:57 PM, Steven Wu wrote:
> I have seen very high "Fetch-Consumer-RequestsPerSec" (like 15K) per broker
> in a relatively idle cluster. My hypothesis some misbehaving consumer has a
> tight polling loop witho
Yes.
Thanks,
Jun
On Wed, Oct 8, 2014 at 10:53 PM, Steven Wu wrote:
> Jun, you mean trace level logging for requestAppender?
> log4j.logger.kafka.network.Processor=TRACE, requestAppender
>
> if it happens again, I can try to enable it.
>
> On Wed, Oct 8, 2014 at 9:54
This may not be easy since you have to implement things like watcher
callbacks. What's your main concern with the ZK dependency?
Thanks,
Jun
On Thu, Oct 9, 2014 at 8:20 AM, S Ahmed wrote:
> Hi,
>
> I was wondering if the zookeeper library (zkutils.scala etc) was designed
> in a more modular wa
Hi, Everyone,
I just committed KAFKA-1555 (min.isr support) to trunk. I felt that it's
probably useful to include it in the 0.8.2 release. Any objections?
Thanks,
Jun
The encoder is instantiated once when the producer is constructed.
Thanks,
Jun
On Thu, Oct 9, 2014 at 6:45 PM, Abraham Jacob wrote:
> Hi All,
>
> I wanted to get some clarification on Kafka's Encoder/Decoder usage.
>
> Lets say I want to implement a custom Encoder.
>
> public class CustomMessa
Is that error transient or persistent?
Thanks,
Jun
On Mon, Oct 13, 2014 at 5:07 PM, Mohit Anchlia
wrote:
> I am new to Kafka and I just installed Kafka. I am getting the following
> error. Zookeeper seems to be running.
>
> [ec2-user@ip-10-231-154-117 kafka_2.10-0.8.1.1]$
> bin/kafka-console-p
The following is a bit weird. It indicates no leader for partition 4, which
is inconsistent with what describe-topic shows.
2014-10-13 19:02:32,611 WARN [main] kafka.producer.BrokerPartitionInfo:
Error while fetching metadata partition 4 leader: nonereplicas: 3
(tr-pan-hclstr-13.amers1b.
Also, which version of Kafka are you using?
Thanks,
Jun
On Tue, Oct 14, 2014 at 5:31 PM, Jun Rao wrote:
> The following is a bit weird. It indicates no leader for partition 4,
> which is inconsistent with what describe-topic shows.
>
> 2014-10-13 19:02:32,61
,6
> Topic: wordcountPartition: 7Leader: 6 Replicas:
> 6,5,7 Isr: 5,6,7
>
> Since then it is been running fine.
>
> My conclusion is that for some reason (which I don't really understand), 3
> brokers were effectively in a hung state and probabl
One way you can do that is to continually load data from Kafka to Hadoop.
During load, you put data into different HDFS directories based on the
timestamp. The Hadoop admin can decide when to open up those directories
for read based on whether data from all data centers have arrived.
Thanks,
Jun
Kyle,
What you wanted is not supported out of box. You can achieve this using the
new java producer. The new java producer allows you to pick an arbitrary
partition when sending a message. If you receive NotEnoughReplicasException
when sending a message, you can resend it to another partition.
Th
The OffsetRequest can only be answered by the leader of the partition. Did
you connect the SimpleConsumer to the leader broker? If not, you need to
use TopicMetadataRequest to find out the leader broker first.
Thanks,
Jun
On Thu, Oct 16, 2014 at 3:56 AM, Magnus Vojbacke <
magnus.vojba...@digital
Which version of ZK are you using? Also, see
https://cwiki.apache.org/confluence/display/KAFKA/FAQ#FAQ-Whyaretheremanyrebalancesinmyconsumerlog
?
Thanks,
Jun
On Thu, Oct 16, 2014 at 3:29 PM, Hari Gorak
wrote:
> Project: Kafka
>
> Issue Type: Bug
>
> Components: consumer
>
> Affects Versions: 0
for 1482 to be included in 0.8.2, if we have a patch
> submitted in a week. I think we've had this JIRA opened for too long and we
> held people back so it's only fair to release this.
>
> On Wed, Oct 8, 2014 at 9:40 PM, Jun Rao wrote:
>
> > Otis,
> >
>
broker selected by my code with the output from
> "bin/kafka-topics.sh --describe[...]" and as far as I can tell, it’s using
> the broker which is described as leader for the topic.
>
> /Magnus
>
>
> On 17 Oct 2014, at 05:55, Jun Rao wrote:
>
> > The OffsetR
Yes, your understanding is correct. For a long time, the implementation is
buggy though. We fixed it (KAFKA-1430) in trunk a few months ago and the
fix will be included in the 0.8.2 release.
Thanks,
Jun
On Fri, Oct 17, 2014 at 5:44 AM, Magnus Vojbacke <
magnus.vojba...@digitalroute.com> wrote:
@deltaprojects.com> wrote:
>
> > +1 to get rid of quotes, thanks!
> >
> >
> >
> >
> >
> >
> > —
> > Sent from Mailbox
> >
> > On Fri, Oct 17, 2014 at 8:54 PM, Jun Rao wrote:
> >
> > > Hi, everyone,
> > > W
Haven't seen this one before. Are you manually committing offsets yourself?
Thanks,
Jun
On Fri, Oct 17, 2014 at 11:10 PM, Yu Yang wrote:
> Hi,
>
> I have a Kafka 0.8.1 cluster. I used the ConsumerOffsetChecker tool to
> check the lag of consumer groups. I found that for some partition, the too
Did you set the replication factor to be more than 1?
Thanks,
Jun
On Sat, Oct 18, 2014 at 2:32 AM, Yu Yang wrote:
> Hi all,
>
> We have a kafka 0.8.1 cluster. We implemented a consumers for the topics on
> the Kafka 0.8 cluster using high-level consumer api. We observed that if
> the Kafka clu
Perhaps you can ask the storm mailing list about the storm-postgresql code?
Thanks,
Jun
On Sat, Oct 18, 2014 at 10:32 AM, Sa Li wrote:
>
> Hi, all
>
> I've just made a 3-node kafka cluster (9 brokers, 3 for each node), the
> performance test is OK. Now I am using tridentKafkaSpout, and being a
You need to pass in the topic name as well. It's better to
use AdminUtils.createTopic() to create the topic.
Thanks,
Jun
On Fri, Oct 17, 2014 at 4:43 PM, Pradeep Badiger
wrote:
> Hi,
>
>
>
> We have issues in getting the topic metadata created when we use the
> TopicCommand.createTopic() API.
will be reset to the smallest valid offset. You can
set auto.offset.reset to "largest" to avoid re-reading all old messages.
Thanks,
Jun
On Sun, Oct 19, 2014 at 9:58 PM, Yu Yang wrote:
> Thanks, Jun! Yes, I set the topic replication factor to 3.
>
> On Sun, Oct 19, 2014
Yes, what you did is correct. See details in
https://cwiki.apache.org/confluence/display/KAFKA/FAQ#FAQ-Whyisdatanotevenlydistributedamongpartitionswhenapartitioningkeyisnotspecified
?
It seems that it doesn't work all the time. What version of Kafka are you
using?
Thanks,
Jun
On Mon, Oct 20, 20
To balance the leaders, you can run the tool in
http://kafka.apache.org/documentation.html#basic_ops_leader_balancing
In the upcoming 0.8.2 release, we have fixed the auto leader balancing
logic. So leaders will be balanced automatically.
Thanks,
Jun
On Tue, Oct 21, 2014 at 12:19 AM, Shlomi Haz
We are voting an 0.8.2 beta release right now.
Thanks,
Jun
On Tue, Oct 21, 2014 at 11:17 AM, Shlomi Hazan wrote:
> Hi All,
> Will version 0.8.1.2 happen?
> Shlomi
>
Could you also set replica.fetch.wait.max.ms in the broker to sth much
smaller?
Thanks,
Jun
On Tue, Oct 21, 2014 at 2:15 PM, Mohit Anchlia
wrote:
> I set the property to 1 in the consumer code that is passed to
> "createJavaConsumerConnector"
> code, but it didn't seem to help
>
> props.put("f
The new java producer to be released in 0.8.2 supports a callback on each
message sent asynchronously.
Thanks,
Jun
On Thu, Oct 23, 2014 at 3:26 PM, Libo Yu wrote:
> Hi,
>
> If I use async publishing plus message acknowledgement,
> is there any API to tell when all the messages in the queue
>
At this moment, the encoder is called from a single thread per producer
instance.
Thanks,
Jun
On Thu, Oct 23, 2014 at 8:20 PM, Rajiv Kurian wrote:
> Are encoders only ever called from a single thread?
>
> I have a stateful utility class that I use to encode my objects. Is it safe
> to only cre
We measure the max lag of each follower replica (
http://kafka.apache.org/documentation.html#monitoring). Could you see if
the lag is widening? Also, could you enable the request log on the leader
and see if the follower is still issuing fetch requests?
Thanks,
Jun
On Fri, Oct 24, 2014 at 9:17 A
The add partition command essentially writes a new json value in the topic
path in ZK. You can figure out the format and write the json yourself.
Thanks,
Jun
On Fri, Oct 24, 2014 at 11:35 AM, David Charle wrote:
> hi kafka'ers
>
> I couldn't find anything useful where I could add partitions th
You can take a look at the "consumer rebalancing algorithm" part in
http://kafka.apache.org/documentation.html. Basically, partitions are
evenly distributed among all consumers in the same group. If there are more
consumers in a group than partitions, some consumers will never get any
data.
Thanks
nsumer API, are all threads identified as the
> same consumer? I guess they are, right?...
>
> Thanks,
> Shlomi
>
>
> On Tue, Oct 28, 2014 at 4:21 AM, Jun Rao wrote:
>
> > You can take a look at the "consumer rebalancing algorithm" part in
> > htt
What's the output of the ConsumerOffsetChecker tool?
Thanks,
Jun
On Tue, Oct 28, 2014 at 7:31 AM, Natarajan, Murugavel <
murugavel.natara...@softwareag.com> wrote:
> Hi,
>
> I have the following Kafka Setup
> Number of producer : 1
> Number of topics : 1
> Number of partitions : 2
> Number of c
The IllegalStateException typically happens if you call next() before
hasNext() on the iterator.
Thanks,
Jun
On Tue, Oct 28, 2014 at 10:50 AM, Bhavesh Mistry wrote:
> Hi Neha,
>
> Thanks for your answer. Can you please let me know how I can resolve the
> Iterator IllegalStateException ? I wo
1), 2), 3) Yes. If you use SimpleConsumer, you have to figure out the
leader of each partition and connect to the right broker. Each fetch
request can send multiple partitions (if with same leader) and you need to
examine the error code per partition.
Not all those error codes are applicable to th
The log before that will show you the cause of the error. Could you dig
that out?
Thanks,
Jun
On Wed, Oct 29, 2014 at 6:43 PM, Rajiv Kurian wrote:
> I keep seeing these errors in my code that is just trying to send some data
> using an AsyncProducer:
>
> kafka.common.FailedToSendMessageExcepti
on arg trying to
> recover", exp);
> retry--;
> }
> }
> return false;
> }
>
> Thanks,
>
> Bhavesh
>
> On Wed, Oct 29, 2014 at 6:36 PM, Jun Rao wrote:
>
> > The IllegalStateExcepti
; > > topics myTopic with correlation ids in [1729,1736]
> > > >> > >
> > > >> > > 2014-10-30T01:54:46.008Z ERROR [ProducerSendThread-
> > ]
> > > >> > > [k.producer.async.ProducerSendThread ] {}: Error in handling
> >
Thu, Oct 30, 2014 at 5:54 PM, Jun Rao wrote:
>
> > Potentially, you can pass in state related info through the properties
> and
> > use those to instantiate MyDeciderThingy.
> >
> > Thanks,
> >
> > Jun
> >
> > On Thu, Oct 30, 2014 at 11:46 AM,
*.
> This is what I see after recovery.
>
>
>
> Thanks,
>
> Bhavesh
>
> On Thu, Oct 30, 2014 at 11:42 AM, Jun Rao wrote:
>
> > Another possibility is that the consumer connector is already closed and
> > then you call hasNext() on the iterator.
> >
>
erver":type="BrokerTopicMetrics",name="topic-1-BytesInPerSec"
we will have
kafka.server:type=BrokerTopicMetrics,name=BytesInPerSec,topic=topic-1
Any objection to committing this to the 0.8.2 branch?
Thanks,
Jun
On Fri, Oct 17, 2014 at 11:54 AM, Jun Rao wrote:
> Hi
; checked in).
>
> On Fri, Oct 31, 2014 at 10:42:13AM -0700, Jun Rao wrote:
> > To circle back on this thread. The patch in kafka-1482 is almost ready.
> To
> > make the mbean names more meaningful and easier to parse, the patch will
> > use explicit key/value pairs in the
Koert,
The java consumer in 0.8.2 beta only has the api and hasn't been
implemented yet. The implementation will likely complete in 0.9.
Thanks,
Jun
On Sat, Nov 1, 2014 at 8:18 AM, Koert Kuipers wrote:
> joe,
> looking at those 0.8.2 beta javadoc I also see a Consumer api and
> KafkaConsumer
a regression in Metrics 2.2.0. IMHO Metrics
> 2.2.0 should not be used because of this. Just downgrade to Metrics 2.1.5
> and you are good.
>
> Of course, upgrading to Metrics 3 would do the trick also.
>
> Kind regards,
> Erik.
>
>
> Jun Rao schreef op 17-10-14 om
cer/blob/master/KafkaConsumerWithDelay.java
>
> Please let me know if you can help me determine the root cause. Why
> there is illegal state and blocking ?
>
> Thanks,
>
> Bhavesh
>
> On Fri, Oct 31, 2014 at 8:33 AM, Jun Rao wrote:
>
> > Do you have a simple test that ca
It seems that the consumer can't connect to the broker for some reason. Any
other error on the broker? Any issue with the network?
Thanks,
Jun
On Sat, Nov 1, 2014 at 9:46 PM, Chen Wang
wrote:
> Hello Folks,
> I am using Highlevel consumer, and it seems to drop connections
> intermittently:
>
>
Are you using the java producer?
Thanks,
Jun
On Mon, Nov 3, 2014 at 3:31 AM, Fredrik S Loekke
wrote:
> Hi
>
>
>
> We are experimenting with running kafka server on a windows machine, but
> keep getting exeptions when producing a lot of messages (in the
> neighborhood of 1 million)..
>
>
>
> k
Which version of Kafka are you using? Is the broker I/O or network
saturated? If so, that will limit the throughput that each producer can
achieve. If not, using a larger number messages per batch and/or enabling
producer side compression typically improves the producer throughput.
Thanks,
Jun
O
The log4j entries before that error should tell you the cause of the error.
Thanks,
Jun
On Tue, Nov 4, 2014 at 11:25 PM, Alex Melville wrote:
> Background:
>
> I have searched for a while online, and through the files located in the
> kafka/logs directory, trying to find where kafka writes log
For higher throughput, you want to configure the producer with a higher
batch size. You may also want to enable compression.
Thanks,
Jun
On Wed, Nov 5, 2014 at 6:46 AM, Eduardo Costa Alfaia wrote:
> Hi Dudes,
>
> I would like to know if the producer and consumer’s properties files into
> the c
The keyed messages are typically used in two cases (1) you want messages
with the same key to be in the same partition and therefore those messages
with the same key will be consumed by the same consumer instance; (2) you
want to enable the log compaction feature for retention such that the
broker
The following is how samza references the kafka test jar in gradle.
testCompile "org.apache.kafka:kafka_$scalaVersion:$kafkaVersion:test"
Thanks,
Jun
On Thu, Nov 6, 2014 at 6:38 AM, Markus Jais wrote:
> Hello,
>
> I want to use the kafka_2.10-0.8.2-beta-test.jar in my Scala project.
>
>
I am also wondering how the corruption happened. The way that we update the
OffsetCheckpoint file is to first write to a tmp file and flush the data.
We then rename the tmp file to the final file. This is done to prevent
corruption caused by a crash in the middle of the writes. In your case, was
th
1 - 100 of 1807 matches
Mail list logo