Re: Unable to understand datastream error message

2016-05-16 Thread subash basnet
Hello Aljoscha, For *DataSet: * IterativeDataSet *loop* = centroids.iterate(numIterations); DataSet *newCentroids* = points.map(new SelectNearestCenter()). *withBroadcastSet*(*loop*, "*centroids*") .map(new CountAppender()).groupBy(0).reduce(new CentroidAccumulator()) .map(new CentroidAverager());

Re: Unable to understand datastream error message

2016-05-16 Thread Aljoscha Krettek
There you have your explanation. A loop actually has to be a loop for it to work in Flink. On Sat, 14 May 2016 at 16:35 subash basnet wrote: > Hello, > > I had to use, > private static IterativeStream *loop*; > loop as global variable because I cannot broadcast it like that of DataSet > API in D

Re: Unable to understand datastream error message

2016-05-14 Thread subash basnet
Hello, I had to use, private static IterativeStream *loop*; loop as global variable because I cannot broadcast it like that of DataSet API in DataStream API. I tried to use *closewith * like that of DataSet as below in DataStream but it gives me exception: DataStream finalCentroids = *loop*.close

Re: Unable to understand datastream error message

2016-05-14 Thread subash basnet
Hello Aljoscha, Below is the shorted version of StockAnalysis class which is a datastream adapation of the *KMeans.java* dataset code. public class StockAnalysis{ public static void main(String args[]){ DataStream centroids = newCentroidDataStream.map(new TupleCentroidConverter()); *

Re: Unable to understand datastream error message

2016-05-07 Thread Aljoscha Krettek
Could you please post your code. On Sat, 7 May 2016 at 19:16 subash basnet wrote: > Hello all, > > I am getting the below error on execute of StreamExecutionEnvironment. > > > *Caused by: java.lang.IllegalStateException: Iteration > FeedbackTransformation{id=15, name='Feedback', > outputType=Poj

Unable to understand datastream error message

2016-05-07 Thread subash basnet
Hello all, I am getting the below error on execute of StreamExecutionEnvironment. *Caused by: java.lang.IllegalStateException: Iteration FeedbackTransformation{id=15, name='Feedback', outputType=PojoType]>, parallelism=4} does not have any feedback edges.* The run method inside the thread class