|
Ken Krugler
2012-05-06, 15:21
Ken Krugler
2012-05-06, 16:53
Jun Rao
2012-05-07, 15:10
Ken Krugler
2012-05-09, 00:34
Jun Rao
2012-05-09, 00:40
Ken Krugler
2012-05-09, 19:01
Jun Rao
2012-05-09, 21:03
Ken Krugler
2012-05-09, 21:47
Jun Rao
2012-05-10, 00:12
Hisham Mardam-Bey
2012-05-10, 00:20
Ken Krugler
2012-05-10, 01:14
Hisham Mardam-Bey
2012-05-10, 03:00
Ken Krugler
2012-06-28, 03:19
|
-
Running Kafka in local test modeKen Krugler 2012-05-06, 15:21
Hi all,
I'm trying to run Kakfa in a minimal local test environment, but having issues gracefully shutting down. I can start up ZooKeeper/Kafka, and it's running fine. But when I try to shut it all down, I'm having trouble cleanly terminating the consumers. I think the issue is that they're blocking on ConsumerIterator.makeNext(), which doesn't seem to be paying attention to being interrupted. So then I proceed with cleaning up everything else, and shutting down the Kafka broker. Which in turn triggers a kafka.consumer.ConsumerTimeoutException from the pending hasNext() call in my consumer Runnable. What's the clean way to set up/tear down a ZooKeeper/Kafka setup that's being used indirectly by the test of another project? Thanks! -- Ken -------------------------- Ken Krugler http://www.scaleunlimited.com custom big data solutions & training Hadoop, Cascading, Mahout & Solr
-
Re: Running Kafka in local test modeKen Krugler 2012-05-06, 16:53
I may have answered my own question…
Looks like if I call ConsumerConnector#shutdown before interrupting my consumer Runnable, it works because then KafkaMessageStream#iterator#hasNext will return false, rather than blocking. I'm still interested in any examples for the right way to set up/tear down a very temporary Kafka setup for testing. For example, I clear out the ZooKeeper data & log dirs before starting it up, in an attempt to avoid occasional errors with "broker already registered". But that in turn seems to trigger Kafka logging about not finding ZK nodes for sessions: 12/05/06 09:35:51 INFO server.PrepRequestProcessor: Got user-level KeeperException when processing sessionid:0x1372301d2120001 type:create cxid:0x1 zxid:0xfffffffffffffffe txntype:unknown reqpath:n/a Error Path:/consumers/bixo-storm/ids Error:KeeperErrorCode = NoNode for /consumers/bixo-storm/ids So I assume there's also Kafka state I should be clearing out before each run, right? Thanks, -- Ken On May 6, 2012, at 8:21am, Ken Krugler wrote: > Hi all, > > I'm trying to run Kakfa in a minimal local test environment, but having issues gracefully shutting down. > > I can start up ZooKeeper/Kafka, and it's running fine. > > But when I try to shut it all down, I'm having trouble cleanly terminating the consumers. > > I think the issue is that they're blocking on ConsumerIterator.makeNext(), which doesn't seem to be paying attention to being interrupted. > > So then I proceed with cleaning up everything else, and shutting down the Kafka broker. > > Which in turn triggers a kafka.consumer.ConsumerTimeoutException from the pending hasNext() call in my consumer Runnable. > > What's the clean way to set up/tear down a ZooKeeper/Kafka setup that's being used indirectly by the test of another project? > > Thanks! > > -- Ken > > -------------------------- > Ken Krugler > http://www.scaleunlimited.com > custom big data solutions & training > Hadoop, Cascading, Mahout & Solr > > > > -------------------------- Ken Krugler +1 530-210-6378 http://www.scaleunlimited.com custom big data solutions & training Hadoop, Cascading, Mahout & Solr -------------------------- Ken Krugler http://www.scaleunlimited.com custom big data solutions & training Hadoop, Cascading, Mahout & Solr
-
Re: Running Kafka in local test modeJun Rao 2012-05-07, 15:10
Ken,
Yes, you need to call ConsumerConnector#shutdown to cleanly shutdown the consumer. Clearing ZK data and kafka log should be enough if you want to start from clean. The ZK NoNode exceptions that you saw can happen when some of the ZK paths are created for the very first time. They should only show up once though. Thanks, Jun On Sun, May 6, 2012 at 9:53 AM, Ken Krugler <[EMAIL PROTECTED]>wrote: > I may have answered my own question… > > Looks like if I call ConsumerConnector#shutdown before interrupting my > consumer Runnable, it works because then > KafkaMessageStream#iterator#hasNext will return false, rather than blocking. > > I'm still interested in any examples for the right way to set up/tear down > a very temporary Kafka setup for testing. > > For example, I clear out the ZooKeeper data & log dirs before starting it > up, in an attempt to avoid occasional errors with "broker already > registered". > > But that in turn seems to trigger Kafka logging about not finding ZK nodes > for sessions: > > 12/05/06 09:35:51 INFO server.PrepRequestProcessor: Got user-level > KeeperException when processing sessionid:0x1372301d2120001 type:create > cxid:0x1 zxid:0xfffffffffffffffe txntype:unknown reqpath:n/a Error > Path:/consumers/bixo-storm/ids Error:KeeperErrorCode = NoNode for > /consumers/bixo-storm/ids > > So I assume there's also Kafka state I should be clearing out before each > run, right? > > Thanks, > > -- Ken > > On May 6, 2012, at 8:21am, Ken Krugler wrote: > > > Hi all, > > > > I'm trying to run Kakfa in a minimal local test environment, but having > issues gracefully shutting down. > > > > I can start up ZooKeeper/Kafka, and it's running fine. > > > > But when I try to shut it all down, I'm having trouble cleanly > terminating the consumers. > > > > I think the issue is that they're blocking on > ConsumerIterator.makeNext(), which doesn't seem to be paying attention to > being interrupted. > > > > So then I proceed with cleaning up everything else, and shutting down > the Kafka broker. > > > > Which in turn triggers a kafka.consumer.ConsumerTimeoutException from > the pending hasNext() call in my consumer Runnable. > > > > What's the clean way to set up/tear down a ZooKeeper/Kafka setup that's > being used indirectly by the test of another project? > > > > Thanks! > > > > -- Ken > > > > -------------------------- > > Ken Krugler > > http://www.scaleunlimited.com > > custom big data solutions & training > > Hadoop, Cascading, Mahout & Solr > > > > > > > > > > -------------------------- > Ken Krugler > +1 530-210-6378 > http://www.scaleunlimited.com > custom big data solutions & training > Hadoop, Cascading, Mahout & Solr > > > > > > -------------------------- > Ken Krugler > http://www.scaleunlimited.com > custom big data solutions & training > Hadoop, Cascading, Mahout & Solr > > > > >
-
Re: Running Kafka in local test modeKen Krugler 2012-05-09, 00:34
On May 7, 2012, at 11:10am, Jun Rao wrote: > Ken, > > Yes, you need to call ConsumerConnector#shutdown to cleanly shutdown the > consumer. Thanks for the confirmation. > Clearing ZK data and kafka log should be enough if you want to > start from clean. The ZK NoNode exceptions that you saw can happen when > some of the ZK paths are created for the very first time. They should only > show up once though. But if I delete ZK data at the start of my unit test (to avoid getting "broker already registered" errors), then the ZK paths are gone, right? So these exception would show up every time my test runs, in that case. Is there a way to avoid the "broker already registered" error and these exceptions? Thanks, -- Ken > On Sun, May 6, 2012 at 9:53 AM, Ken Krugler <[EMAIL PROTECTED]>wrote: > >> I may have answered my own question… >> >> Looks like if I call ConsumerConnector#shutdown before interrupting my >> consumer Runnable, it works because then >> KafkaMessageStream#iterator#hasNext will return false, rather than blocking. >> >> I'm still interested in any examples for the right way to set up/tear down >> a very temporary Kafka setup for testing. >> >> For example, I clear out the ZooKeeper data & log dirs before starting it >> up, in an attempt to avoid occasional errors with "broker already >> registered". >> >> But that in turn seems to trigger Kafka logging about not finding ZK nodes >> for sessions: >> >> 12/05/06 09:35:51 INFO server.PrepRequestProcessor: Got user-level >> KeeperException when processing sessionid:0x1372301d2120001 type:create >> cxid:0x1 zxid:0xfffffffffffffffe txntype:unknown reqpath:n/a Error >> Path:/consumers/bixo-storm/ids Error:KeeperErrorCode = NoNode for >> /consumers/bixo-storm/ids >> >> So I assume there's also Kafka state I should be clearing out before each >> run, right? >> >> Thanks, >> >> -- Ken >> >> On May 6, 2012, at 8:21am, Ken Krugler wrote: >> >>> Hi all, >>> >>> I'm trying to run Kakfa in a minimal local test environment, but having >> issues gracefully shutting down. >>> >>> I can start up ZooKeeper/Kafka, and it's running fine. >>> >>> But when I try to shut it all down, I'm having trouble cleanly >> terminating the consumers. >>> >>> I think the issue is that they're blocking on >> ConsumerIterator.makeNext(), which doesn't seem to be paying attention to >> being interrupted. >>> >>> So then I proceed with cleaning up everything else, and shutting down >> the Kafka broker. >>> >>> Which in turn triggers a kafka.consumer.ConsumerTimeoutException from >> the pending hasNext() call in my consumer Runnable. >>> >>> What's the clean way to set up/tear down a ZooKeeper/Kafka setup that's >> being used indirectly by the test of another project? >>> >>> Thanks! >>> >>> -- Ken >>> >>> -------------------------- >>> Ken Krugler >>> http://www.scaleunlimited.com >>> custom big data solutions & training >>> Hadoop, Cascading, Mahout & Solr >>> >>> >>> >>> >> >> -------------------------- >> Ken Krugler >> +1 530-210-6378 >> http://www.scaleunlimited.com >> custom big data solutions & training >> Hadoop, Cascading, Mahout & Solr >> >> >> >> >> >> -------------------------- >> Ken Krugler >> http://www.scaleunlimited.com >> custom big data solutions & training >> Hadoop, Cascading, Mahout & Solr >> >> >> >> >> -------------------------- Ken Krugler http://www.scaleunlimited.com custom big data solutions & training Hadoop, Cascading, Mahout & Solr
-
Re: Running Kafka in local test modeJun Rao 2012-05-09, 00:40
The problem is probably that you didn't shut down the broker cleanly (use
kill -15 instead of kill -9). Jun On Tue, May 8, 2012 at 5:34 PM, Ken Krugler <[EMAIL PROTECTED]>wrote: > > On May 7, 2012, at 11:10am, Jun Rao wrote: > > > Ken, > > > > Yes, you need to call ConsumerConnector#shutdown to cleanly shutdown the > > consumer. > > Thanks for the confirmation. > > > Clearing ZK data and kafka log should be enough if you want to > > start from clean. The ZK NoNode exceptions that you saw can happen when > > some of the ZK paths are created for the very first time. They should > only > > show up once though. > > But if I delete ZK data at the start of my unit test (to avoid getting > "broker already registered" errors), then the ZK paths are gone, right? > > So these exception would show up every time my test runs, in that case. > > Is there a way to avoid the "broker already registered" error and these > exceptions? > > Thanks, > > -- Ken > > > > On Sun, May 6, 2012 at 9:53 AM, Ken Krugler <[EMAIL PROTECTED] > >wrote: > > > >> I may have answered my own question… > >> > >> Looks like if I call ConsumerConnector#shutdown before interrupting my > >> consumer Runnable, it works because then > >> KafkaMessageStream#iterator#hasNext will return false, rather than > blocking. > >> > >> I'm still interested in any examples for the right way to set up/tear > down > >> a very temporary Kafka setup for testing. > >> > >> For example, I clear out the ZooKeeper data & log dirs before starting > it > >> up, in an attempt to avoid occasional errors with "broker already > >> registered". > >> > >> But that in turn seems to trigger Kafka logging about not finding ZK > nodes > >> for sessions: > >> > >> 12/05/06 09:35:51 INFO server.PrepRequestProcessor: Got user-level > >> KeeperException when processing sessionid:0x1372301d2120001 type:create > >> cxid:0x1 zxid:0xfffffffffffffffe txntype:unknown reqpath:n/a Error > >> Path:/consumers/bixo-storm/ids Error:KeeperErrorCode = NoNode for > >> /consumers/bixo-storm/ids > >> > >> So I assume there's also Kafka state I should be clearing out before > each > >> run, right? > >> > >> Thanks, > >> > >> -- Ken > >> > >> On May 6, 2012, at 8:21am, Ken Krugler wrote: > >> > >>> Hi all, > >>> > >>> I'm trying to run Kakfa in a minimal local test environment, but having > >> issues gracefully shutting down. > >>> > >>> I can start up ZooKeeper/Kafka, and it's running fine. > >>> > >>> But when I try to shut it all down, I'm having trouble cleanly > >> terminating the consumers. > >>> > >>> I think the issue is that they're blocking on > >> ConsumerIterator.makeNext(), which doesn't seem to be paying attention > to > >> being interrupted. > >>> > >>> So then I proceed with cleaning up everything else, and shutting down > >> the Kafka broker. > >>> > >>> Which in turn triggers a kafka.consumer.ConsumerTimeoutException from > >> the pending hasNext() call in my consumer Runnable. > >>> > >>> What's the clean way to set up/tear down a ZooKeeper/Kafka setup that's > >> being used indirectly by the test of another project? > >>> > >>> Thanks! > >>> > >>> -- Ken > >>> > >>> -------------------------- > >>> Ken Krugler > >>> http://www.scaleunlimited.com > >>> custom big data solutions & training > >>> Hadoop, Cascading, Mahout & Solr > >>> > >>> > >>> > >>> > >> > >> -------------------------- > >> Ken Krugler > >> +1 530-210-6378 > >> http://www.scaleunlimited.com > >> custom big data solutions & training > >> Hadoop, Cascading, Mahout & Solr > >> > >> > >> > >> > >> > >> -------------------------- > >> Ken Krugler > >> http://www.scaleunlimited.com > >> custom big data solutions & training > >> Hadoop, Cascading, Mahout & Solr > >> > >> > >> > >> > >> > > -------------------------- > Ken Krugler > http://www.scaleunlimited.com > custom big data solutions & training > Hadoop, Cascading, Mahout & Solr > > > > >
-
Re: Running Kafka in local test modeKen Krugler 2012-05-09, 19:01
Hi Jun,
On May 8, 2012, at 5:40pm, Jun Rao wrote: > The problem is probably that you didn't shut down the broker cleanly (use > kill -15 instead of kill -9). Thanks - though this is for unit tests. So it needs to be something I can do via standard Java code. Is that possible, or does ZK require the Heavy Hammer to get it to terminate? Thanks, -- Ken > On Tue, May 8, 2012 at 5:34 PM, Ken Krugler <[EMAIL PROTECTED]>wrote: > >> >> On May 7, 2012, at 11:10am, Jun Rao wrote: >> >>> Ken, >>> >>> Yes, you need to call ConsumerConnector#shutdown to cleanly shutdown the >>> consumer. >> >> Thanks for the confirmation. >> >>> Clearing ZK data and kafka log should be enough if you want to >>> start from clean. The ZK NoNode exceptions that you saw can happen when >>> some of the ZK paths are created for the very first time. They should >> only >>> show up once though. >> >> But if I delete ZK data at the start of my unit test (to avoid getting >> "broker already registered" errors), then the ZK paths are gone, right? >> >> So these exception would show up every time my test runs, in that case. >> >> Is there a way to avoid the "broker already registered" error and these >> exceptions? >> >> Thanks, >> >> -- Ken >> >> >>> On Sun, May 6, 2012 at 9:53 AM, Ken Krugler <[EMAIL PROTECTED] >>> wrote: >>> >>>> I may have answered my own question… >>>> >>>> Looks like if I call ConsumerConnector#shutdown before interrupting my >>>> consumer Runnable, it works because then >>>> KafkaMessageStream#iterator#hasNext will return false, rather than >> blocking. >>>> >>>> I'm still interested in any examples for the right way to set up/tear >> down >>>> a very temporary Kafka setup for testing. >>>> >>>> For example, I clear out the ZooKeeper data & log dirs before starting >> it >>>> up, in an attempt to avoid occasional errors with "broker already >>>> registered". >>>> >>>> But that in turn seems to trigger Kafka logging about not finding ZK >> nodes >>>> for sessions: >>>> >>>> 12/05/06 09:35:51 INFO server.PrepRequestProcessor: Got user-level >>>> KeeperException when processing sessionid:0x1372301d2120001 type:create >>>> cxid:0x1 zxid:0xfffffffffffffffe txntype:unknown reqpath:n/a Error >>>> Path:/consumers/bixo-storm/ids Error:KeeperErrorCode = NoNode for >>>> /consumers/bixo-storm/ids >>>> >>>> So I assume there's also Kafka state I should be clearing out before >> each >>>> run, right? >>>> >>>> Thanks, >>>> >>>> -- Ken >>>> >>>> On May 6, 2012, at 8:21am, Ken Krugler wrote: >>>> >>>>> Hi all, >>>>> >>>>> I'm trying to run Kakfa in a minimal local test environment, but having >>>> issues gracefully shutting down. >>>>> >>>>> I can start up ZooKeeper/Kafka, and it's running fine. >>>>> >>>>> But when I try to shut it all down, I'm having trouble cleanly >>>> terminating the consumers. >>>>> >>>>> I think the issue is that they're blocking on >>>> ConsumerIterator.makeNext(), which doesn't seem to be paying attention >> to >>>> being interrupted. >>>>> >>>>> So then I proceed with cleaning up everything else, and shutting down >>>> the Kafka broker. >>>>> >>>>> Which in turn triggers a kafka.consumer.ConsumerTimeoutException from >>>> the pending hasNext() call in my consumer Runnable. >>>>> >>>>> What's the clean way to set up/tear down a ZooKeeper/Kafka setup that's >>>> being used indirectly by the test of another project? >>>>> >>>>> Thanks! >>>>> >>>>> -- Ken >>>>> >>>>> -------------------------- >>>>> Ken Krugler >>>>> http://www.scaleunlimited.com >>>>> custom big data solutions & training >>>>> Hadoop, Cascading, Mahout & Solr >>>>> >>>>> >>>>> >>>>> >>>> >>>> -------------------------- >>>> Ken Krugler >>>> +1 530-210-6378 >>>> http://www.scaleunlimited.com >>>> custom big data solutions & training >>>> Hadoop, Cascading, Mahout & Solr >>>> >>>> >>>> >>>> >>>> >>>> -------------------------- >>>> Ken Krugler >>>> http://www.scaleunlimited.com >>>> custom big data solutions & training Ken Krugler http://www.scaleunlimited.com custom big data solutions & training Hadoop, Cascading, Mahout & Solr
-
Re: Running Kafka in local test modeJun Rao 2012-05-09, 21:03
Are you starting an embedded broker in your unit test? If so, you need to
call shutdown() on the broker. Jun On Wed, May 9, 2012 at 12:01 PM, Ken Krugler <[EMAIL PROTECTED]>wrote: > Hi Jun, > > On May 8, 2012, at 5:40pm, Jun Rao wrote: > > > The problem is probably that you didn't shut down the broker cleanly (use > > kill -15 instead of kill -9). > > Thanks - though this is for unit tests. So it needs to be something I can > do via standard Java code. > > Is that possible, or does ZK require the Heavy Hammer to get it to > terminate? > > Thanks, > > -- Ken > > > On Tue, May 8, 2012 at 5:34 PM, Ken Krugler <[EMAIL PROTECTED] > >wrote: > > > >> > >> On May 7, 2012, at 11:10am, Jun Rao wrote: > >> > >>> Ken, > >>> > >>> Yes, you need to call ConsumerConnector#shutdown to cleanly shutdown > the > >>> consumer. > >> > >> Thanks for the confirmation. > >> > >>> Clearing ZK data and kafka log should be enough if you want to > >>> start from clean. The ZK NoNode exceptions that you saw can happen when > >>> some of the ZK paths are created for the very first time. They should > >> only > >>> show up once though. > >> > >> But if I delete ZK data at the start of my unit test (to avoid getting > >> "broker already registered" errors), then the ZK paths are gone, right? > >> > >> So these exception would show up every time my test runs, in that case. > >> > >> Is there a way to avoid the "broker already registered" error and these > >> exceptions? > >> > >> Thanks, > >> > >> -- Ken > >> > >> > >>> On Sun, May 6, 2012 at 9:53 AM, Ken Krugler < > [EMAIL PROTECTED] > >>> wrote: > >>> > >>>> I may have answered my own question… > >>>> > >>>> Looks like if I call ConsumerConnector#shutdown before interrupting my > >>>> consumer Runnable, it works because then > >>>> KafkaMessageStream#iterator#hasNext will return false, rather than > >> blocking. > >>>> > >>>> I'm still interested in any examples for the right way to set up/tear > >> down > >>>> a very temporary Kafka setup for testing. > >>>> > >>>> For example, I clear out the ZooKeeper data & log dirs before starting > >> it > >>>> up, in an attempt to avoid occasional errors with "broker already > >>>> registered". > >>>> > >>>> But that in turn seems to trigger Kafka logging about not finding ZK > >> nodes > >>>> for sessions: > >>>> > >>>> 12/05/06 09:35:51 INFO server.PrepRequestProcessor: Got user-level > >>>> KeeperException when processing sessionid:0x1372301d2120001 > type:create > >>>> cxid:0x1 zxid:0xfffffffffffffffe txntype:unknown reqpath:n/a Error > >>>> Path:/consumers/bixo-storm/ids Error:KeeperErrorCode = NoNode for > >>>> /consumers/bixo-storm/ids > >>>> > >>>> So I assume there's also Kafka state I should be clearing out before > >> each > >>>> run, right? > >>>> > >>>> Thanks, > >>>> > >>>> -- Ken > >>>> > >>>> On May 6, 2012, at 8:21am, Ken Krugler wrote: > >>>> > >>>>> Hi all, > >>>>> > >>>>> I'm trying to run Kakfa in a minimal local test environment, but > having > >>>> issues gracefully shutting down. > >>>>> > >>>>> I can start up ZooKeeper/Kafka, and it's running fine. > >>>>> > >>>>> But when I try to shut it all down, I'm having trouble cleanly > >>>> terminating the consumers. > >>>>> > >>>>> I think the issue is that they're blocking on > >>>> ConsumerIterator.makeNext(), which doesn't seem to be paying attention > >> to > >>>> being interrupted. > >>>>> > >>>>> So then I proceed with cleaning up everything else, and shutting down > >>>> the Kafka broker. > >>>>> > >>>>> Which in turn triggers a kafka.consumer.ConsumerTimeoutException from > >>>> the pending hasNext() call in my consumer Runnable. > >>>>> > >>>>> What's the clean way to set up/tear down a ZooKeeper/Kafka setup > that's > >>>> being used indirectly by the test of another project? > >>>>> > >>>>> Thanks! > >>>>> > >>>>> -- Ken > >>>>> > >>>>> -------------------------- > >>>>> Ken Krugler > >>>>> http://www.scaleunlimited.com > >>>>> custom big data solutions & training
-
Re: Running Kafka in local test modeKen Krugler 2012-05-09, 21:47
On May 9, 2012, at 2:03pm, Jun Rao wrote: > Are you starting an embedded broker in your unit test? That's what I'm trying to do, yes. > If so, you need to > call shutdown() on the broker. I think I do - here's the code from the run() method of my KafakRunnable that I use to run the Kafka broker in my unit test: public void run() { LOGGER.info("Starting KafkaRunnable..."); KafkaServerStartable kafkaServerStartable = new KafkaServerStartable(_serverConfig); kafkaServerStartable.startup(); _alive = true; while (!Thread.interrupted()) { try { Thread.sleep(100); } catch (InterruptedException e) { Thread.currentThread().interrupt(); } } LOGGER.info("Stopping KafkaRunnable"); kafkaServerStartable.shutdown(); _alive = false; LOGGER.info("Exiting KafkaRunnable"); } -- Ken > On Wed, May 9, 2012 at 12:01 PM, Ken Krugler <[EMAIL PROTECTED]>wrote: > >> Hi Jun, >> >> On May 8, 2012, at 5:40pm, Jun Rao wrote: >> >>> The problem is probably that you didn't shut down the broker cleanly (use >>> kill -15 instead of kill -9). >> >> Thanks - though this is for unit tests. So it needs to be something I can >> do via standard Java code. >> >> Is that possible, or does ZK require the Heavy Hammer to get it to >> terminate? >> >> Thanks, >> >> -- Ken >> >>> On Tue, May 8, 2012 at 5:34 PM, Ken Krugler <[EMAIL PROTECTED] >>> wrote: >>> >>>> >>>> On May 7, 2012, at 11:10am, Jun Rao wrote: >>>> >>>>> Ken, >>>>> >>>>> Yes, you need to call ConsumerConnector#shutdown to cleanly shutdown >> the >>>>> consumer. >>>> >>>> Thanks for the confirmation. >>>> >>>>> Clearing ZK data and kafka log should be enough if you want to >>>>> start from clean. The ZK NoNode exceptions that you saw can happen when >>>>> some of the ZK paths are created for the very first time. They should >>>> only >>>>> show up once though. >>>> >>>> But if I delete ZK data at the start of my unit test (to avoid getting >>>> "broker already registered" errors), then the ZK paths are gone, right? >>>> >>>> So these exception would show up every time my test runs, in that case. >>>> >>>> Is there a way to avoid the "broker already registered" error and these >>>> exceptions? >>>> >>>> Thanks, >>>> >>>> -- Ken >>>> >>>> >>>>> On Sun, May 6, 2012 at 9:53 AM, Ken Krugler < >> [EMAIL PROTECTED] >>>>> wrote: >>>>> >>>>>> I may have answered my own question… >>>>>> >>>>>> Looks like if I call ConsumerConnector#shutdown before interrupting my >>>>>> consumer Runnable, it works because then >>>>>> KafkaMessageStream#iterator#hasNext will return false, rather than >>>> blocking. >>>>>> >>>>>> I'm still interested in any examples for the right way to set up/tear >>>> down >>>>>> a very temporary Kafka setup for testing. >>>>>> >>>>>> For example, I clear out the ZooKeeper data & log dirs before starting >>>> it >>>>>> up, in an attempt to avoid occasional errors with "broker already >>>>>> registered". >>>>>> >>>>>> But that in turn seems to trigger Kafka logging about not finding ZK >>>> nodes >>>>>> for sessions: >>>>>> >>>>>> 12/05/06 09:35:51 INFO server.PrepRequestProcessor: Got user-level >>>>>> KeeperException when processing sessionid:0x1372301d2120001 >> type:create >>>>>> cxid:0x1 zxid:0xfffffffffffffffe txntype:unknown reqpath:n/a Error >>>>>> Path:/consumers/bixo-storm/ids Error:KeeperErrorCode = NoNode for >>>>>> /consumers/bixo-storm/ids >>>>>> >>>>>> So I assume there's also Kafka state I should be clearing out before >>>> each >>>>>> run, right? >>>>>> >>>>>> Thanks, >>>>>> >>>>>> -- Ken >>>>>> >>>>>> On May 6, 2012, at 8:21am, Ken Krugler wrote: >>>>>> >>>>>>> Hi all, >>>>>>> >>>>>>> I'm trying to run Kakfa in a minimal local test environment, but http://about.me/kkrugler +1 530-210-6378 Ken Krugler http://www.scaleunlimited.com custom big data solutions & training Hadoop, Cascading, Mahout & Solr
-
Re: Running Kafka in local test modeJun Rao 2012-05-10, 00:12
Are you embedding a ZK server too? If so, make sure you shutdown the broker
before ZK. Jun On Wed, May 9, 2012 at 2:47 PM, Ken Krugler <[EMAIL PROTECTED]>wrote: > > On May 9, 2012, at 2:03pm, Jun Rao wrote: > > > Are you starting an embedded broker in your unit test? > > That's what I'm trying to do, yes. > > > If so, you need to > > call shutdown() on the broker. > > I think I do - here's the code from the run() method of my KafakRunnable > that I use to run the Kafka broker in my unit test: > > public void run() { > LOGGER.info("Starting KafkaRunnable..."); > > KafkaServerStartable kafkaServerStartable = new > KafkaServerStartable(_serverConfig); > kafkaServerStartable.startup(); > _alive = true; > > while (!Thread.interrupted()) { > try { > Thread.sleep(100); > } catch (InterruptedException e) { > Thread.currentThread().interrupt(); > } > } > > LOGGER.info("Stopping KafkaRunnable"); > > kafkaServerStartable.shutdown(); > _alive = false; > > LOGGER.info("Exiting KafkaRunnable"); > } > > -- Ken > > > > On Wed, May 9, 2012 at 12:01 PM, Ken Krugler < > [EMAIL PROTECTED]>wrote: > > > >> Hi Jun, > >> > >> On May 8, 2012, at 5:40pm, Jun Rao wrote: > >> > >>> The problem is probably that you didn't shut down the broker cleanly > (use > >>> kill -15 instead of kill -9). > >> > >> Thanks - though this is for unit tests. So it needs to be something I > can > >> do via standard Java code. > >> > >> Is that possible, or does ZK require the Heavy Hammer to get it to > >> terminate? > >> > >> Thanks, > >> > >> -- Ken > >> > >>> On Tue, May 8, 2012 at 5:34 PM, Ken Krugler < > [EMAIL PROTECTED] > >>> wrote: > >>> > >>>> > >>>> On May 7, 2012, at 11:10am, Jun Rao wrote: > >>>> > >>>>> Ken, > >>>>> > >>>>> Yes, you need to call ConsumerConnector#shutdown to cleanly shutdown > >> the > >>>>> consumer. > >>>> > >>>> Thanks for the confirmation. > >>>> > >>>>> Clearing ZK data and kafka log should be enough if you want to > >>>>> start from clean. The ZK NoNode exceptions that you saw can happen > when > >>>>> some of the ZK paths are created for the very first time. They should > >>>> only > >>>>> show up once though. > >>>> > >>>> But if I delete ZK data at the start of my unit test (to avoid getting > >>>> "broker already registered" errors), then the ZK paths are gone, > right? > >>>> > >>>> So these exception would show up every time my test runs, in that > case. > >>>> > >>>> Is there a way to avoid the "broker already registered" error and > these > >>>> exceptions? > >>>> > >>>> Thanks, > >>>> > >>>> -- Ken > >>>> > >>>> > >>>>> On Sun, May 6, 2012 at 9:53 AM, Ken Krugler < > >> [EMAIL PROTECTED] > >>>>> wrote: > >>>>> > >>>>>> I may have answered my own question… > >>>>>> > >>>>>> Looks like if I call ConsumerConnector#shutdown before interrupting > my > >>>>>> consumer Runnable, it works because then > >>>>>> KafkaMessageStream#iterator#hasNext will return false, rather than > >>>> blocking. > >>>>>> > >>>>>> I'm still interested in any examples for the right way to set > up/tear > >>>> down > >>>>>> a very temporary Kafka setup for testing. > >>>>>> > >>>>>> For example, I clear out the ZooKeeper data & log dirs before > starting > >>>> it > >>>>>> up, in an attempt to avoid occasional errors with "broker already > >>>>>> registered". > >>>>>> > >>>>>> But that in turn seems to trigger Kafka logging about not finding ZK > >>>> nodes > >>>>>> for sessions: > >>>>>> > >>>>>> 12/05/06 09:35:51 INFO server.PrepRequestProcessor: Got user-level > >>>>>> KeeperException when processing sessionid:0x1372301d2120001 > >> type:create > >>>>>> cxid:0x1 zxid:0xfffffffffffffffe txntype:unknown reqpath:n/a Error > >>>>>> Path:/consumers/bixo-storm/ids Error:KeeperErrorCode = NoNode for > >>>>>> /consumers/bixo-storm/ids
-
Re: Running Kafka in local test modeHisham Mardam-Bey 2012-05-10, 00:20
On Wed, May 9, 2012 at 5:47 PM, Ken Krugler <[EMAIL PROTECTED]> wrote:
> kafkaServerStartable.shutdown(); Ken, When you call shutdown() on KafkaServerStartable you should also call awaitShutdown if you're expecting your app / JVM to terminate (perhaps you are not). Also take into account that if the embedded KafkaServer instance in KafkaServerStartable throws an exception on shutdown it will call Runtime.getRuntime.halt(1) (not sure if you want this behaviour). I'm embedding a Kafka broker and producer in an experimental lib doing pretty much what you're doing except I use KafkaServer directly and make no use of ZK. Hope this helps, hmb. -- Hisham Mardam-Bey A: Because it messes up the order in which people normally read text. Q: Why is top-posting such a bad thing? A: Top-posting. Q: What is the most annoying thing in e-mail? -=[ Codito Ergo Sum ]=-
-
Re: Running Kafka in local test modeKen Krugler 2012-05-10, 01:14
Hi Hisham,
Thanks for chiming in - see below. > When you call shutdown() on KafkaServerStartable you should also call > awaitShutdown if you're expecting your app / JVM to terminate (perhaps > you are not). It's been shutting down properly, but you're right that calling awaitShutdown() ensures that everything has been properly cleaned up before I continue. > Also take into account that if the embedded KafkaServer > instance in KafkaServerStartable throws an exception on shutdown it > will call Runtime.getRuntime.halt(1) (not sure if you want this > behaviour). Not really, but hopefully my unit tests aren't triggering exceptions in the KafkaServer :) > I'm embedding a Kafka broker and producer in an experimental lib doing > pretty much what you're doing except I use KafkaServer directly and > make no use of ZK. I initially tried that, but ran into the issue where Kafka consumers always create a ZooKeeper client, which expects to have a ZooKeeper server running. Do you know if it's possible to set up a complete Kafka environment (broker, producer, consumer) without ZooKeeper? -- Ken -------------------------- Ken Krugler http://www.scaleunlimited.com custom big data solutions & training Hadoop, Cascading, Mahout & Solr
-
Re: Running Kafka in local test modeHisham Mardam-Bey 2012-05-10, 03:00
Ken,
I've whipped up a quick Scala example of an embedded Kafka broker, producer, and consumer. I don't use ZK and I use the simple consumer (manual offset management). https://gist.github.com/2650743 Hopefully this helps clear things up a bit. hmb. On Wed, May 9, 2012 at 9:14 PM, Ken Krugler <[EMAIL PROTECTED]> wrote: > Hi Hisham, > > Thanks for chiming in - see below. > >> When you call shutdown() on KafkaServerStartable you should also call >> awaitShutdown if you're expecting your app / JVM to terminate (perhaps >> you are not). > > It's been shutting down properly, but you're right that calling awaitShutdown() ensures that everything has been properly cleaned up before I continue. > >> Also take into account that if the embedded KafkaServer >> instance in KafkaServerStartable throws an exception on shutdown it >> will call Runtime.getRuntime.halt(1) (not sure if you want this >> behaviour). > > Not really, but hopefully my unit tests aren't triggering exceptions in the KafkaServer :) > >> I'm embedding a Kafka broker and producer in an experimental lib doing >> pretty much what you're doing except I use KafkaServer directly and >> make no use of ZK. > > I initially tried that, but ran into the issue where Kafka consumers always create a ZooKeeper client, which expects to have a ZooKeeper server running. > > Do you know if it's possible to set up a complete Kafka environment (broker, producer, consumer) without ZooKeeper? > > -- Ken > > -------------------------- > Ken Krugler > http://www.scaleunlimited.com > custom big data solutions & training > Hadoop, Cascading, Mahout & Solr > > > > -- Hisham Mardam-Bey [ Director of Engineering ] [ Mate1 Inc. ] A: Because it messes up the order in which people normally read text. Q: Why is top-posting such a bad thing? A: Top-posting. Q: What is the most annoying thing in e-mail? -=[ Codito Ergo Sum ]=-
-
Re: Running Kafka in local test modeKen Krugler 2012-06-28, 03:19
Hi Hisham,
> I've whipped up a quick Scala example of an embedded Kafka broker, > producer, and consumer. I don't use ZK and I use the simple consumer > (manual offset management). > > https://gist.github.com/2650743 > > Hopefully this helps clear things up a bit. Yes, thanks - and sorry for the delay in responding, I had to step away from this project for a bit. I was easily able to get the equivalent Java code running. The remaining question I've got now is how to cleanly shut things down. I can terminate the Kafka server, but I can't terminate the threads that are running as consumers. Specifically when I terminate the executor that I'm using to run the consumer threads, this fails, and in the logs I see: [2012-06-27 20:07:29,696] INFO fetch reconnect due to java.nio.channels.ClosedByInterruptException (kafka.consumer.SimpleConsumer) The consumer threads are blocked on SimpleConsumer#fetch, and the fetch reconnect means that this doesn't ever return. Any input on how best to terminate properly in a test environment? Thanks! -- Ken > On Wed, May 9, 2012 at 9:14 PM, Ken Krugler <[EMAIL PROTECTED]> wrote: >> Hi Hisham, >> >> Thanks for chiming in - see below. >> >>> When you call shutdown() on KafkaServerStartable you should also call >>> awaitShutdown if you're expecting your app / JVM to terminate (perhaps >>> you are not). >> >> It's been shutting down properly, but you're right that calling awaitShutdown() ensures that everything has been properly cleaned up before I continue. >> >>> Also take into account that if the embedded KafkaServer >>> instance in KafkaServerStartable throws an exception on shutdown it >>> will call Runtime.getRuntime.halt(1) (not sure if you want this >>> behaviour). >> >> Not really, but hopefully my unit tests aren't triggering exceptions in the KafkaServer :) >> >>> I'm embedding a Kafka broker and producer in an experimental lib doing >>> pretty much what you're doing except I use KafkaServer directly and >>> make no use of ZK. >> >> I initially tried that, but ran into the issue where Kafka consumers always create a ZooKeeper client, which expects to have a ZooKeeper server running. >> >> Do you know if it's possible to set up a complete Kafka environment (broker, producer, consumer) without ZooKeeper? >> >> -- Ken >> >> -------------------------- >> Ken Krugler >> http://www.scaleunlimited.com >> custom big data solutions & training >> Hadoop, Cascading, Mahout & Solr >> >> >> >> > > > > -- > Hisham Mardam-Bey > [ Director of Engineering ] [ Mate1 Inc. ] > > A: Because it messes up the order in which people normally read text. > Q: Why is top-posting such a bad thing? > A: Top-posting. > Q: What is the most annoying thing in e-mail? > > -=[ Codito Ergo Sum ]=- -------------------------- Ken Krugler http://www.scaleunlimited.com custom big data solutions & training Hadoop, Cascading, Mahout & Solr |