Re: Failure in Apex runner

2017-07-09 Thread Kenneth Knowles
On that same subject: in the SDK-angnostic proto for a pipeline there is no such thing as a main output [1]. The distinction between single and multiple output ParDo is SDK-specific. Kenn [1] https://github.com/apache/beam/blob/master/sdks/common/runner-api/src/main/proto/beam_runner_api.proto#L1

Re: Failure in Apex runner

2017-07-09 Thread Reuven Lax
Yes. Semantically all outputs from a ParDo are equivalent, so the watermark should traverse them all. The only reason a "default" output exists is for convenience so we don't force users to always specify an output tag. On Sun, Jul 9, 2017 at 12:03 AM, Thomas Weise wrote: > This error turns out

Re: Failure in Apex runner

2017-07-09 Thread Thomas Weise
This error turns out to be deterministic and debug friendly :) I enabled trace and found that the watermark "disappears" between the following two operators: [8/WriteCounts/WriteFiles/WriteBundles:ApexParDoOperator] [10/WriteCounts/WriteFiles/GroupUnwritten:ApexGroupByKeyOperator] GroupUnwritten

Re: Failure in Apex runner

2017-07-07 Thread Manu Zhang
Okay, that fixes my errors. On Sat, Jul 8, 2017 at 1:41 AM Pramod Immaneni wrote: > Hi Manu, > > Can you refresh your netlet dependency. There was a respin of a release > that usually doesn't happen. You could do this by deleting contents of > your ~/.m2/repository/com/datatorrent/netlet/ folder

Re: Failure in Apex runner

2017-07-07 Thread Pramod Immaneni
Hi Manu, Can you refresh your netlet dependency. There was a respin of a release that usually doesn't happen. You could do this by deleting contents of your ~/.m2/repository/com/datatorrent/netlet/ folder and rebuilding, which will fetch the netlet dependency again. Thanks On Fri, Jul 7, 2017 at

Re: Failure in Apex runner

2017-07-07 Thread Manu Zhang
Hey guys, I'd like to offer some input. The test also fails locally on my Mac with the following error. (so WriteOperation#finalize is not called) java.lang.NullPointerException at com.datatorrent.netlet.util.Slice.(Slice.java:54) at org.apache.beam.runners.apex.translation.utils.ApexStateInterna

Re: Failure in Apex runner

2017-07-06 Thread Reuven Lax
Thomas, any suggestions on what we should do? Do you have an idea what's going on, or should we exclude this test for now until you have time to look at it? Reuven On Wed, Jul 5, 2017 at 3:36 PM, Reuven Lax wrote: > I wonder if the watermark is accidentally advancing too early, causing > Apex t

Re: Failure in Apex runner

2017-07-05 Thread Reuven Lax
I wonder if the watermark is accidentally advancing too early, causing Apex to shut down the pipeline before the final finalize DoFn executes? On Wed, Jul 5, 2017 at 1:45 PM, Thomas Weise wrote: > I don't think this is a problem with the test and if anything this problem > to me shows the test i

Re: Failure in Apex runner

2017-07-05 Thread Kenneth Knowles
There is no asynchronous behavior in this test. It is basically a "batch" test, here: https://github.com/apache/beam/blob/master/runners/apex/src/test/java/org/apache/beam/runners/apex/examples/WordCountTest.java#L117 The pipeline is: p.apply("ReadLines", TextIO.read().from(options.getInputFi

Re: Failure in Apex runner

2017-07-05 Thread Thomas Weise
I don't think this is a problem with the test and if anything this problem to me shows the test is useful in catching similar issues during unit test runs. Is there any form of asynchronous/trigger based processing in this pipeline that could cause this? The Apex runner will shutdown the pipeline

Re: Failure in Apex runner

2017-07-05 Thread Kenneth Knowles
Upon further investigation, this tests always writes to ./target/wordcountresult-0-of-2 and ./target/wordcountresult-1-of-2. So after a successful test run, any further run without a `clean` will spuriously succeed. I was running via IntelliJ so did not do the ritual `mvn clean` wor

Re: Failure in Apex runner

2017-07-05 Thread Reuven Lax
I've done a bit more debugging with logging. It appears that the finalize ParDo is never being invoked in this Apex test (or at least the LOG.info in that ParDo never runs). This ParDo is run on a constant element (code snippet below), so it should always run. PCollection singletonCollection = p.a

Re: Failure in Apex runner

2017-07-05 Thread Kenneth Knowles
Data-dependent file destinations is a pretty great feature. We also have another change to make to this @Experimental feature, and it would be nice to get them both into 2.1.0 if we can unblock this quickly. I just tried this too, and failed to reproduce it. But Jenkins and Reuven both have a reli

Re: Failure in Apex runner

2017-07-05 Thread Reuven Lax
Hi Thomas, This only happens with https://github.com/apache/beam/pull/3356. Reuven On Mon, Jul 3, 2017 at 6:11 AM, Thomas Weise wrote: > Hi Reuven, > > I'm not able to reproduce the issue locally. I was hoping to see which > thread is attempting to emit the results. In Apex, only the operator

Re: Failure in Apex runner

2017-07-03 Thread Thomas Weise
Hi Reuven, I'm not able to reproduce the issue locally. I was hoping to see which thread is attempting to emit the results. In Apex, only the operator thread can emit the results, any other thread that is launched by the operator cannot. I'm not aware of ParDo managing separate threads though and

Failure in Apex runner

2017-07-01 Thread Reuven Lax
pr/3356 fails in the Apex WordCountTest. The failed test is here : Upon debugging, it looks like this is lik