Re: Exception using the new createDirectStream util method

2015-03-20 Thread Cody Koeninger
I went ahead and created https://issues.apache.org/jira/browse/SPARK-6434 to track this On Fri, Mar 20, 2015 at 2:44 AM, Alberto Rodriguez wrote: > You were absolutely right Cody!! I have just put a message in the kafka > topic before creating the DirectStream and now is working fine! > > Do y

Re: Exception using the new createDirectStream util method

2015-03-20 Thread Alberto Rodriguez
You were absolutely right Cody!! I have just put a message in the kafka topic before creating the DirectStream and now is working fine! Do you think that I should open an issue to warn that the kafka topic must contain at least one message before the DirectStream creation? Thank you very much! Yo

Re: Exception using the new createDirectStream util method

2015-03-19 Thread Cody Koeninger
Yeah, I wouldn't be shocked if Kafka's metadata apis didn't return results for topics that don't have any messages. (sorry about the triple negative, but I think you get my meaning). Try putting a message in the topic and seeing what happens. On Thu, Mar 19, 2015 at 4:38 PM, Alberto Rodriguez w

Re: Exception using the new createDirectStream util method

2015-03-19 Thread Alberto Rodriguez
Thank you for replying, Ted, I have been debuging and the getLeaderOffsets method is not appending errors because the method findLeaders that is called at the first line of getLeaderOffsets is not returning leaders. Cody, the topics do not have any messages yet. Could this be an issue?? If you g

Re: Exception using the new createDirectStream util method

2015-03-19 Thread Cody Koeninger
What is the value of your topics variable, and does it correspond to topics that already exist on the cluster and have messages in them? On Thu, Mar 19, 2015 at 3:10 PM, Ted Yu wrote: > Looking at KafkaCluster#getLeaderOffsets(): > > respMap.get(tp).foreach { por: PartitionOffsetsRespo

Re: Exception using the new createDirectStream util method

2015-03-19 Thread Ted Yu
Looking at KafkaCluster#getLeaderOffsets(): respMap.get(tp).foreach { por: PartitionOffsetsResponse => if (por.error == ErrorMapping.NoError) { ... } else { errs.append(ErrorMapping.exceptionFor(por.error)) } There should be some error ot

Exception using the new createDirectStream util method

2015-03-19 Thread Alberto Rodriguez
Hi all, I am trying to make the new kafka and spark streaming integration work (direct approach "no receivers" ). I have created an unit test where I configure and start both zookeeper and kafka. When I try to create the InputDS