You are viewing a plain text version of this content. The canonical link for it is here.
Posted to user@beam.apache.org by amir bahmanyari <am...@yahoo.com> on 2016/09/30 04:34:34 UTC

KafkaIO() Timeout expired while fetching kafka topic metadata

Hi Colleagues,I created a kafka topic with 2048 partitions. kafka-topics.sh --describe lists everything accurately and its open for business.I start Flibk Cluster. Everything come up normal.I start my Beam app that uses KafkaIO() ...it waits for a while and then throws:org.apache.kafka.common.errors.TimeoutException: Timeout expired while fetching topic metadata
The deg of parallelism is 1024. Works with lower degree of parallelism like 512 & 1024 kafka partitions.Line 581 in my code is just p.run(); 
And this how KafkaIO.read looks like:PCollection<KV<String, String>> kafkarecords = p .apply(KafkaIO.read().withBootstrapServers("kafka01:9092").withTopics(topics) .withValueCoder(StringUtf8Coder.of()).withoutMetadata()) .apply("startBundle", ParDo.of( new DoFn<KV<byte[], String>, KV<String, String>>() {.....etc.

Thanks for your help.Amir-


...ADEBUG Completed method...ADEBUG about to run pipeline...ADEBUG Running thread  threw:  java.lang.RuntimeException: Error while translating UnboundedSource: org.apache.beam.sdk.io.kafka.KafkaIO$UnboundedKafkaSource@24b52d3e        at org.apache.beam.runners.flink.translation.FlinkStreamingTransformTranslators$UnboundedReadSourceTranslator.translateNode(FlinkStreamingTransformTranslators.java:288)        at org.apache.beam.runners.flink.translation.FlinkStreamingTransformTranslators$UnboundedReadSourceTranslator.translateNode(FlinkStreamingTransformTranslators.java:247)        at org.apache.beam.runners.flink.translation.FlinkStreamingPipelineTranslator.applyStreamingTransform(FlinkStreamingPipelineTranslator.java:106)        at org.apache.beam.runners.flink.translation.FlinkStreamingPipelineTranslator.visitPrimitiveTransform(FlinkStreamingPipelineTranslator.java:87)        at org.apache.beam.sdk.runners.TransformTreeNode.visit(TransformTreeNode.java:226)        at org.apache.beam.sdk.runners.TransformTreeNode.visit(TransformTreeNode.java:221)        at org.apache.beam.sdk.runners.TransformTreeNode.visit(TransformTreeNode.java:221)        at org.apache.beam.sdk.runners.TransformHierarchy.visit(TransformHierarchy.java:102)        at org.apache.beam.sdk.Pipeline.traverseTopologically(Pipeline.java:293)        at org.apache.beam.runners.flink.translation.FlinkPipelineTranslator.translate(FlinkPipelineTranslator.java:38)        at org.apache.beam.runners.flink.FlinkPipelineExecutionEnvironment.translate(FlinkPipelineExecutionEnvironment.java:106)        at org.apache.beam.runners.flink.FlinkRunner.run(FlinkRunner.java:105)        at org.apache.beam.runners.flink.FlinkRunner.run(FlinkRunner.java:48)        at org.apache.beam.sdk.Pipeline.run(Pipeline.java:183)        at benchmark.flinkspark.flink.BenchBeamRunners.main(BenchBeamRunners.java:581)        at sun.reflect.NativeMethodAccessorImpl.invoke0(Native Method)        at sun.reflect.NativeMethodAccessorImpl.invoke(NativeMethodAccessorImpl.java:62)        at sun.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43)        at java.lang.reflect.Method.invoke(Method.java:498)        at org.apache.flink.client.program.PackagedProgram.callMainMethod(PackagedProgram.java:505)        at org.apache.flink.client.program.PackagedProgram.invokeInteractiveModeForExecution(PackagedProgram.java:403)        at org.apache.flink.client.program.Client.runBlocking(Client.java:248)        at org.apache.flink.client.CliFrontend.executeProgramBlocking(CliFrontend.java:866)        at org.apache.flink.client.CliFrontend.run(CliFrontend.java:333)        at org.apache.flink.client.CliFrontend.parseParameters(CliFrontend.java:1189)        at org.apache.flink.client.CliFrontend.main(CliFrontend.java:1239)Caused by: org.apache.kafka.common.errors.TimeoutException: Timeout expired while fetching topic metadata

Re: KafkaIO() Timeout expired while fetching kafka topic metadata

Posted by amir bahmanyari <am...@yahoo.com>.
Am game :)

      From: Dan Halperin <dh...@google.com>
 To: user@beam.incubator.apache.org 
 Sent: Monday, October 3, 2016 11:52 AM
 Subject: Re: KafkaIO() Timeout expired while fetching kafka topic metadata
   
We have tossed around the idea starting some "Beam meetups" -- maybe now that things are really picking up we should get one off the ground!
On Sat, Oct 1, 2016 at 10:15 PM, Amir Bahmanyari <am...@yahoo.com> wrote:

Hi Sumit I diagnosed it You are right Kafka not accessible I increased ulimit to an astronomical number and passed the issue Maybe we should meet someday and share experiences I live in the Bay Area Cheers Amir

Sent from my iPhone
On Oct 1, 2016, at 10:10 PM, Chawla,Sumit <su...@gmail.com> wrote:


I have seen this error when Kafka is not accessible. Can you check connectivity to kafka? Can you read messages from Kafka using console consumer from the machine where you are running the flink job?
Regards
Sumit Chawla


On Sat, Oct 1, 2016 at 12:40 PM, amir bahmanyari <am...@yahoo.com> wrote:

Hi Stephan et al.I changed ulimit -n at all servers: Flink & Kafka to 10000 & 16000 respectively.Rebooted. Retried. Failed the same.Then restarted everything from the scratch with disk clean up etc.Started fine & no more Timeout due to not being able to fetch the topic metadata.All that started when I increased the deg of parallelism & created a Kafka topic with that number for partitions: 2048.Its running now. Am hoping to get better perf due to increasing deg paralleism.Have a great weekend & Appreciate your feedback...Amir-

      From: amir bahmanyari <am...@yahoo.com>
 To: "user@beam.incubator.apache.or g" <user@beam.incubator.apache.or g> 
 Sent: Friday, September 30, 2016 2:36 PM
 Subject: Re: KafkaIO() Timeout expired while fetching kafka topic metadata
   
Hi Stephan,yes, this program uses Beam's KafkaIO().

PCollection<KV<String, String>> kafkarecords = p .apply(KafkaIO.read().withBoot strapServers("kafka01:9092"). withTopics(topics) .withValueCoder(StringUtf8Code r.of()).withoutMetadata()) .apply("startBundle", ParDo.of( new DoFn<KV<byte[], String>, KV<String, String>>() {
Thanks+regards,Amir-

      From: Stephan Ewen <se...@apache.org>
 To: user@beam.incubator.apache.org ; amir bahmanyari <am...@yahoo.com> 
 Sent: Friday, September 30, 2016 2:13 PM
 Subject: Re: KafkaIO() Timeout expired while fetching kafka topic metadata
  
Not sure if this is caused by the Flink Kafka Consumer (or if this program uses Beam's Kafka IO).
If it is using Flink, there is a requirement that the client that submits the job can access Kafka. If it cannot, that may be a cause for such a timeout.

On Fri, Sep 30, 2016 at 7:11 PM, amir bahmanyari <am...@yahoo.com> wrote:

No more stack trace Raghu.Keeping the configs like I described, yes it fails like this since for some reason the Kafka Zookeeper gets killed.Thanks +regardsAmir-

      From: Raghu Angadi <ra...@google.com>
 To: user@beam.incubator.apache.org ; amir bahmanyari <am...@yahoo.com> 
 Sent: Thursday, September 29, 2016 11:28 PM
 Subject: Re: KafkaIO() Timeout expired while fetching kafka topic metadata
  

On Thu, Sep 29, 2016 at 9:34 PM, amir bahmanyari <am...@yahoo.com> wrote:

        at org.apache.flink.client. CliFrontend.parseParameters( CliFrontend.java:1189)        at org.apache.flink.client. CliFrontend.main(CliFrontend. java:1239)Caused by: org.apache.kafka.common. errors.TimeoutException: Timeout expired while fetching topic metadata

Is there any more stacktrace printed after this line?
Does it always fail? 


   



   

   






   

Re: KafkaIO() Timeout expired while fetching kafka topic metadata

Posted by Dan Halperin <dh...@google.com>.
We have tossed around the idea starting some "Beam meetups" -- maybe now
that things are really picking up we should get one off the ground!

On Sat, Oct 1, 2016 at 10:15 PM, Amir Bahmanyari <am...@yahoo.com>
wrote:

> Hi Sumit
> I diagnosed it
> You are right Kafka not accessible
> I increased ulimit to an astronomical number and passed the issue
> Maybe we should meet someday and share experiences
> I live in the Bay Area
> Cheers
> Amir
>
>
> Sent from my iPhone
>
> On Oct 1, 2016, at 10:10 PM, Chawla,Sumit <su...@gmail.com> wrote:
>
> I have seen this error when Kafka is not accessible. Can you check
> connectivity to kafka? Can you read messages from Kafka using console
> consumer from the machine where you are running the flink job?
>
> Regards
> Sumit Chawla
>
>
> On Sat, Oct 1, 2016 at 12:40 PM, amir bahmanyari <am...@yahoo.com>
> wrote:
>
>> Hi Stephan et al.
>> I changed ulimit -n at all servers: Flink & Kafka to 10000 & 16000
>> respectively.
>> Rebooted. Retried. Failed the same.
>> Then restarted everything from the scratch with disk clean up etc.
>> Started fine & no more Timeout due to not being able to fetch the topic
>> metadata.
>> All that started when I increased the deg of parallelism & created a
>> Kafka topic with that number for partitions: 2048.
>> Its running now. Am hoping to get better perf due to increasing deg
>> paralleism.
>> Have a great weekend & Appreciate your feedback...
>> Amir-
>>
>>
>> ------------------------------
>> *From:* amir bahmanyari <am...@yahoo.com>
>> *To:* "user@beam.incubator.apache.org" <us...@beam.incubator.apache.org>
>> *Sent:* Friday, September 30, 2016 2:36 PM
>>
>> *Subject:* Re: KafkaIO() Timeout expired while fetching kafka topic
>> metadata
>>
>> Hi Stephan,
>> yes, this program uses Beam's KafkaIO().
>>
>> PCollection<KV<String, String>> kafkarecords = p
>> .apply(*KafkaIO.read()*.withBootstrapServers("kafka01:9092").
>> withTopics(topics)
>> .withValueCoder(StringUtf8Coder.of()).withoutMetadata())
>> .apply("startBundle", ParDo.of(
>> new DoFn<KV<byte[], String>, KV<String, String>>() {
>>
>> Thanks+regards,
>> Amir-
>>
>>
>> ------------------------------
>> *From:* Stephan Ewen <se...@apache.org>
>> *To:* user@beam.incubator.apache.org; amir bahmanyari <
>> amirtousa@yahoo.com>
>> *Sent:* Friday, September 30, 2016 2:13 PM
>> *Subject:* Re: KafkaIO() Timeout expired while fetching kafka topic
>> metadata
>>
>> Not sure if this is caused by the Flink Kafka Consumer (or if this
>> program uses Beam's Kafka IO).
>>
>> If it is using Flink, there is a requirement that the client that submits
>> the job can access Kafka. If it cannot, that may be a cause for such a
>> timeout.
>>
>>
>> On Fri, Sep 30, 2016 at 7:11 PM, amir bahmanyari <am...@yahoo.com>
>> wrote:
>>
>> No more stack trace Raghu.
>> Keeping the configs like I described, yes it fails like this since for
>> some reason the Kafka Zookeeper gets killed.
>> Thanks +regards
>> Amir-
>>
>> ------------------------------
>> *From:* Raghu Angadi <ra...@google.com>
>> *To:* user@beam.incubator.apache.org ; amir bahmanyari <
>> amirtousa@yahoo.com>
>> *Sent:* Thursday, September 29, 2016 11:28 PM
>> *Subject:* Re: KafkaIO() Timeout expired while fetching kafka topic
>> metadata
>>
>>
>> On Thu, Sep 29, 2016 at 9:34 PM, amir bahmanyari <am...@yahoo.com>
>> wrote:
>>
>>         at org.apache.flink.client. CliFrontend.parseParameters(
>> CliFrontend.java:1189)
>>         at org.apache.flink.client. CliFrontend.main(CliFrontend.
>> java:1239)
>> Caused by: o*rg.apache.kafka.common. errors.TimeoutException: Timeout
>> expired while fetching topic metadata*
>>
>>
>> Is there any more stacktrace printed after this line?
>>
>> Does it always fail?
>>
>>
>>
>>
>>
>>
>>
>>
>>
>

Re: KafkaIO() Timeout expired while fetching kafka topic metadata

Posted by Amir Bahmanyari <am...@yahoo.com>.
Hi Sumit 
I diagnosed it 
You are right Kafka not accessible 
I increased ulimit to an astronomical number and passed the issue 
Maybe we should meet someday and share experiences 
I live in the Bay Area 
Cheers 
Amir


Sent from my iPhone

> On Oct 1, 2016, at 10:10 PM, Chawla,Sumit <su...@gmail.com> wrote:
> 
> I have seen this error when Kafka is not accessible. Can you check connectivity to kafka? Can you read messages from Kafka using console consumer from the machine where you are running the flink job?
> 
> Regards
> Sumit Chawla
> 
> 
>> On Sat, Oct 1, 2016 at 12:40 PM, amir bahmanyari <am...@yahoo.com> wrote:
>> Hi Stephan et al.
>> I changed ulimit -n at all servers: Flink & Kafka to 10000 & 16000 respectively.
>> Rebooted. Retried. Failed the same.
>> Then restarted everything from the scratch with disk clean up etc.
>> Started fine & no more Timeout due to not being able to fetch the topic metadata.
>> All that started when I increased the deg of parallelism & created a Kafka topic with that number for partitions: 2048.
>> Its running now. Am hoping to get better perf due to increasing deg paralleism.
>> Have a great weekend & Appreciate your feedback...
>> Amir-
>> 
>> 
>> From: amir bahmanyari <am...@yahoo.com>
>> To: "user@beam.incubator.apache.org" <us...@beam.incubator.apache.org> 
>> Sent: Friday, September 30, 2016 2:36 PM
>> 
>> Subject: Re: KafkaIO() Timeout expired while fetching kafka topic metadata
>> 
>> Hi Stephan,
>> yes, this program uses Beam's KafkaIO().
>> 
>> PCollection<KV<String, String>> kafkarecords = p
>> 					.apply(KafkaIO.read().withBootstrapServers("kafka01:9092").withTopics(topics)
>> 							.withValueCoder(StringUtf8Coder.of()).withoutMetadata())
>> 					.apply("startBundle", ParDo.of(
>> 							new DoFn<KV<byte[], String>, KV<String, String>>() {
>> 
>> Thanks+regards,
>> Amir-
>> 
>> 
>> From: Stephan Ewen <se...@apache.org>
>> To: user@beam.incubator.apache.org; amir bahmanyari <am...@yahoo.com> 
>> Sent: Friday, September 30, 2016 2:13 PM
>> Subject: Re: KafkaIO() Timeout expired while fetching kafka topic metadata
>> 
>> Not sure if this is caused by the Flink Kafka Consumer (or if this program uses Beam's Kafka IO).
>> 
>> If it is using Flink, there is a requirement that the client that submits the job can access Kafka. If it cannot, that may be a cause for such a timeout.
>> 
>> 
>> On Fri, Sep 30, 2016 at 7:11 PM, amir bahmanyari <am...@yahoo.com> wrote:
>> No more stack trace Raghu.
>> Keeping the configs like I described, yes it fails like this since for some reason the Kafka Zookeeper gets killed.
>> Thanks +regards
>> Amir-
>> 
>> From: Raghu Angadi <ra...@google.com>
>> To: user@beam.incubator.apache.org ; amir bahmanyari <am...@yahoo.com> 
>> Sent: Thursday, September 29, 2016 11:28 PM
>> Subject: Re: KafkaIO() Timeout expired while fetching kafka topic metadata
>> 
>> 
>> On Thu, Sep 29, 2016 at 9:34 PM, amir bahmanyari <am...@yahoo.com> wrote:
>>         at org.apache.flink.client. CliFrontend.parseParameters( CliFrontend.java:1189)
>>         at org.apache.flink.client. CliFrontend.main(CliFrontend. java:1239)
>> Caused by: org.apache.kafka.common. errors.TimeoutException: Timeout expired while fetching topic metadata
>> 
>> Is there any more stacktrace printed after this line?
>> 
>> Does it always fail? 
> 

Re: KafkaIO() Timeout expired while fetching kafka topic metadata

Posted by "Chawla,Sumit " <su...@gmail.com>.
I have seen this error when Kafka is not accessible. Can you check
connectivity to kafka? Can you read messages from Kafka using console
consumer from the machine where you are running the flink job?

Regards
Sumit Chawla


On Sat, Oct 1, 2016 at 12:40 PM, amir bahmanyari <am...@yahoo.com>
wrote:

> Hi Stephan et al.
> I changed ulimit -n at all servers: Flink & Kafka to 10000 & 16000
> respectively.
> Rebooted. Retried. Failed the same.
> Then restarted everything from the scratch with disk clean up etc.
> Started fine & no more Timeout due to not being able to fetch the topic
> metadata.
> All that started when I increased the deg of parallelism & created a Kafka
> topic with that number for partitions: 2048.
> Its running now. Am hoping to get better perf due to increasing deg
> paralleism.
> Have a great weekend & Appreciate your feedback...
> Amir-
>
>
> ------------------------------
> *From:* amir bahmanyari <am...@yahoo.com>
> *To:* "user@beam.incubator.apache.org" <us...@beam.incubator.apache.org>
> *Sent:* Friday, September 30, 2016 2:36 PM
>
> *Subject:* Re: KafkaIO() Timeout expired while fetching kafka topic
> metadata
>
> Hi Stephan,
> yes, this program uses Beam's KafkaIO().
>
> PCollection<KV<String, String>> kafkarecords = p
> .apply(*KafkaIO.read()*.withBootstrapServers("kafka01:
> 9092").withTopics(topics)
> .withValueCoder(StringUtf8Coder.of()).withoutMetadata())
> .apply("startBundle", ParDo.of(
> new DoFn<KV<byte[], String>, KV<String, String>>() {
>
> Thanks+regards,
> Amir-
>
>
> ------------------------------
> *From:* Stephan Ewen <se...@apache.org>
> *To:* user@beam.incubator.apache.org; amir bahmanyari <am...@yahoo.com>
>
> *Sent:* Friday, September 30, 2016 2:13 PM
> *Subject:* Re: KafkaIO() Timeout expired while fetching kafka topic
> metadata
>
> Not sure if this is caused by the Flink Kafka Consumer (or if this program
> uses Beam's Kafka IO).
>
> If it is using Flink, there is a requirement that the client that submits
> the job can access Kafka. If it cannot, that may be a cause for such a
> timeout.
>
>
> On Fri, Sep 30, 2016 at 7:11 PM, amir bahmanyari <am...@yahoo.com>
> wrote:
>
> No more stack trace Raghu.
> Keeping the configs like I described, yes it fails like this since for
> some reason the Kafka Zookeeper gets killed.
> Thanks +regards
> Amir-
>
> ------------------------------
> *From:* Raghu Angadi <ra...@google.com>
> *To:* user@beam.incubator.apache.org ; amir bahmanyari <
> amirtousa@yahoo.com>
> *Sent:* Thursday, September 29, 2016 11:28 PM
> *Subject:* Re: KafkaIO() Timeout expired while fetching kafka topic
> metadata
>
>
> On Thu, Sep 29, 2016 at 9:34 PM, amir bahmanyari <am...@yahoo.com>
> wrote:
>
>         at org.apache.flink.client. CliFrontend.parseParameters(
> CliFrontend.java:1189)
>         at org.apache.flink.client. CliFrontend.main(CliFrontend.
> java:1239)
> Caused by: o*rg.apache.kafka.common. errors.TimeoutException: Timeout
> expired while fetching topic metadata*
>
>
> Is there any more stacktrace printed after this line?
>
> Does it always fail?
>
>
>
>
>
>
>
>
>

Re: KafkaIO() Timeout expired while fetching kafka topic metadata

Posted by amir bahmanyari <am...@yahoo.com>.
Hi Stephan et al.I changed ulimit -n at all servers: Flink & Kafka to 10000 & 16000 respectively.Rebooted. Retried. Failed the same.Then restarted everything from the scratch with disk clean up etc.Started fine & no more Timeout due to not being able to fetch the topic metadata.All that started when I increased the deg of parallelism & created a Kafka topic with that number for partitions: 2048.Its running now. Am hoping to get better perf due to increasing deg paralleism.Have a great weekend & Appreciate your feedback...Amir-

      From: amir bahmanyari <am...@yahoo.com>
 To: "user@beam.incubator.apache.org" <us...@beam.incubator.apache.org> 
 Sent: Friday, September 30, 2016 2:36 PM
 Subject: Re: KafkaIO() Timeout expired while fetching kafka topic metadata
   
Hi Stephan,yes, this program uses Beam's KafkaIO().

PCollection<KV<String, String>> kafkarecords = p .apply(KafkaIO.read().withBootstrapServers("kafka01:9092").withTopics(topics) .withValueCoder(StringUtf8Coder.of()).withoutMetadata()) .apply("startBundle", ParDo.of( new DoFn<KV<byte[], String>, KV<String, String>>() {
Thanks+regards,Amir-

      From: Stephan Ewen <se...@apache.org>
 To: user@beam.incubator.apache.org; amir bahmanyari <am...@yahoo.com> 
 Sent: Friday, September 30, 2016 2:13 PM
 Subject: Re: KafkaIO() Timeout expired while fetching kafka topic metadata
  
Not sure if this is caused by the Flink Kafka Consumer (or if this program uses Beam's Kafka IO).
If it is using Flink, there is a requirement that the client that submits the job can access Kafka. If it cannot, that may be a cause for such a timeout.

On Fri, Sep 30, 2016 at 7:11 PM, amir bahmanyari <am...@yahoo.com> wrote:

No more stack trace Raghu.Keeping the configs like I described, yes it fails like this since for some reason the Kafka Zookeeper gets killed.Thanks +regardsAmir-

      From: Raghu Angadi <ra...@google.com>
 To: user@beam.incubator.apache.org ; amir bahmanyari <am...@yahoo.com> 
 Sent: Thursday, September 29, 2016 11:28 PM
 Subject: Re: KafkaIO() Timeout expired while fetching kafka topic metadata
  

On Thu, Sep 29, 2016 at 9:34 PM, amir bahmanyari <am...@yahoo.com> wrote:

        at org.apache.flink.client. CliFrontend.parseParameters( CliFrontend.java:1189)        at org.apache.flink.client. CliFrontend.main(CliFrontend. java:1239)Caused by: org.apache.kafka.common. errors.TimeoutException: Timeout expired while fetching topic metadata

Is there any more stacktrace printed after this line?
Does it always fail? 


   



   

   

Re: KafkaIO() Timeout expired while fetching kafka topic metadata

Posted by amir bahmanyari <am...@yahoo.com>.
Hi Stephan,yes, this program uses Beam's KafkaIO().

PCollection<KV<String, String>> kafkarecords = p .apply(KafkaIO.read().withBootstrapServers("kafka01:9092").withTopics(topics) .withValueCoder(StringUtf8Coder.of()).withoutMetadata()) .apply("startBundle", ParDo.of( new DoFn<KV<byte[], String>, KV<String, String>>() {
Thanks+regards,Amir-

      From: Stephan Ewen <se...@apache.org>
 To: user@beam.incubator.apache.org; amir bahmanyari <am...@yahoo.com> 
 Sent: Friday, September 30, 2016 2:13 PM
 Subject: Re: KafkaIO() Timeout expired while fetching kafka topic metadata
   
Not sure if this is caused by the Flink Kafka Consumer (or if this program uses Beam's Kafka IO).
If it is using Flink, there is a requirement that the client that submits the job can access Kafka. If it cannot, that may be a cause for such a timeout.

On Fri, Sep 30, 2016 at 7:11 PM, amir bahmanyari <am...@yahoo.com> wrote:

No more stack trace Raghu.Keeping the configs like I described, yes it fails like this since for some reason the Kafka Zookeeper gets killed.Thanks +regardsAmir-

      From: Raghu Angadi <ra...@google.com>
 To: user@beam.incubator.apache.org ; amir bahmanyari <am...@yahoo.com> 
 Sent: Thursday, September 29, 2016 11:28 PM
 Subject: Re: KafkaIO() Timeout expired while fetching kafka topic metadata
  

On Thu, Sep 29, 2016 at 9:34 PM, amir bahmanyari <am...@yahoo.com> wrote:

        at org.apache.flink.client. CliFrontend.parseParameters( CliFrontend.java:1189)        at org.apache.flink.client. CliFrontend.main(CliFrontend. java:1239)Caused by: org.apache.kafka.common. errors.TimeoutException: Timeout expired while fetching topic metadata

Is there any more stacktrace printed after this line?
Does it always fail? 


   



   

Re: KafkaIO() Timeout expired while fetching kafka topic metadata

Posted by Stephan Ewen <se...@apache.org>.
Not sure if this is caused by the Flink Kafka Consumer (or if this program
uses Beam's Kafka IO).

If it is using Flink, there is a requirement that the client that submits
the job can access Kafka. If it cannot, that may be a cause for such a
timeout.


On Fri, Sep 30, 2016 at 7:11 PM, amir bahmanyari <am...@yahoo.com>
wrote:

> No more stack trace Raghu.
> Keeping the configs like I described, yes it fails like this since for
> some reason the Kafka Zookeeper gets killed.
> Thanks +regards
> Amir-
>
> ------------------------------
> *From:* Raghu Angadi <ra...@google.com>
> *To:* user@beam.incubator.apache.org; amir bahmanyari <am...@yahoo.com>
>
> *Sent:* Thursday, September 29, 2016 11:28 PM
> *Subject:* Re: KafkaIO() Timeout expired while fetching kafka topic
> metadata
>
>
> On Thu, Sep 29, 2016 at 9:34 PM, amir bahmanyari <am...@yahoo.com>
> wrote:
>
>         at org.apache.flink.client. CliFrontend.parseParameters(
> CliFrontend.java:1189)
>         at org.apache.flink.client. CliFrontend.main(CliFrontend.
> java:1239)
> Caused by: o*rg.apache.kafka.common. errors.TimeoutException: Timeout
> expired while fetching topic metadata*
>
>
> Is there any more stacktrace printed after this line?
>
> Does it always fail?
>
>
>
>

Re: KafkaIO() Timeout expired while fetching kafka topic metadata

Posted by amir bahmanyari <am...@yahoo.com>.
No more stack trace Raghu.Keeping the configs like I described, yes it fails like this since for some reason the Kafka Zookeeper gets killed.Thanks +regardsAmir-

      From: Raghu Angadi <ra...@google.com>
 To: user@beam.incubator.apache.org; amir bahmanyari <am...@yahoo.com> 
 Sent: Thursday, September 29, 2016 11:28 PM
 Subject: Re: KafkaIO() Timeout expired while fetching kafka topic metadata
   

On Thu, Sep 29, 2016 at 9:34 PM, amir bahmanyari <am...@yahoo.com> wrote:

        at org.apache.flink.client. CliFrontend.parseParameters( CliFrontend.java:1189)        at org.apache.flink.client. CliFrontend.main(CliFrontend. java:1239)Caused by: org.apache.kafka.common. errors.TimeoutException: Timeout expired while fetching topic metadata

Is there any more stacktrace printed after this line?
Does it always fail? 


   

Re: KafkaIO() Timeout expired while fetching kafka topic metadata

Posted by Raghu Angadi <ra...@google.com>.
On Thu, Sep 29, 2016 at 9:34 PM, amir bahmanyari <am...@yahoo.com>
wrote:

>         at org.apache.flink.client.CliFrontend.parseParameters(
> CliFrontend.java:1189)
>         at org.apache.flink.client.CliFrontend.main(CliFrontend.java:1239)
> Caused by: o*rg.apache.kafka.common.errors.TimeoutException: Timeout
> expired while fetching topic metadata*
>

Is there any more stacktrace printed after this line?

Does it always fail?