[jira] [Created] (FLINK-4807) ResourceManager clean up JobManager's registration
Kurt Young created FLINK-4807: - Summary: ResourceManager clean up JobManager's registration Key: FLINK-4807 URL: https://issues.apache.org/jira/browse/FLINK-4807 Project: Flink Issue Type: Sub-task Reporter: Kurt Young When RM received a JM's registration, it will record it either with some leaderid or leadership listener. We should make sure the finished / failed JM can properly unregister itself with RM. We can make it happen by doing these two things: 1. If JM finds out job reaches a terminate state(either success or fail), it should send an unregistration request to RM. 2. If (1) does not happen for various reasons, RM can rely on the heartbeat manager to find out timeout JM and clear it up. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Created] (FLINK-4806) ResourceManager stop listening JobManager's leader address
Kurt Young created FLINK-4806: - Summary: ResourceManager stop listening JobManager's leader address Key: FLINK-4806 URL: https://issues.apache.org/jira/browse/FLINK-4806 Project: Flink Issue Type: Sub-task Reporter: Kurt Young Currently in flip-6 branch, when RM receives a registration from JM, it will verify the leader session id of JM and attach a JobManagerLeaderListener with it for monitoring the future changes. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Created] (FLINK-4805) Stringify() crashes with Python3 (run with pyflink3)
Yakov Goldberg created FLINK-4805: - Summary: Stringify() crashes with Python3 (run with pyflink3) Key: FLINK-4805 URL: https://issues.apache.org/jira/browse/FLINK-4805 Project: Flink Issue Type: Bug Reporter: Yakov Goldberg {code} Caused by: java.lang.RuntimeException: External process for task MapPartition (PythonMap) terminated prematurely due to an error. Traceback (most recent call last): File "/tmp/flink-dist-cache-299804c6-813a-44de-9f62-c5f4cf415990/1527bd1cc45d6f67695c180762c614ef/flink/plan.py", line 548, in env.execute(local=True) File "/tmp/flink-dist-cache-299804c6-813a-44de-9f62-c5f4cf415990/1527bd1cc45d6f67695c180762c614ef/flink/flink/plan/Environment.py", line 181, in execute operator._go() File "/tmp/flink-dist-cache-299804c6-813a-44de-9f62-c5f4cf415990/1527bd1cc45d6f67695c180762c614ef/flink/flink/functions/Function.py", line 64, in _go self._run() File "/tmp/flink-dist-cache-299804c6-813a-44de-9f62-c5f4cf415990/1527bd1cc45d6f67695c180762c614ef/flink/flink/functions/MapFunction.py", line 29, in _run collector.collect(function(value)) File "/tmp/flink-dist-cache-299804c6-813a-44de-9f62-c5f4cf415990/1527bd1cc45d6f67695c180762c614ef/flink/flink/plan/DataSet.py", line 38, in map return "(" + b", ".join([self.map(x) for x in value]) + ")" File "/tmp/flink-dist-cache-299804c6-813a-44de-9f62-c5f4cf415990/1527bd1cc45d6f67695c180762c614ef/flink/flink/plan/DataSet.py", line 38, in return "(" + b", ".join([self.map(x) for x in value]) + ")" File "/tmp/flink-dist-cache-299804c6-813a-44de-9f62-c5f4cf415990/1527bd1cc45d6f67695c180762c614ef/flink/flink/plan/DataSet.py", line 38, in map return "(" + b", ".join([self.map(x) for x in value]) + ")" TypeError: sequence item 0: expected bytes, bytearray, or an object with the buffer interface, str found at org.apache.flink.python.api.streaming.data.PythonStreamer.streamBufferWithoutGroups(PythonStreamer.java:268) at org.apache.flink.python.api.functions.PythonMapPartition.mapPartition(PythonMapPartition.java:54) at org.apache.flink.runtime.operators.MapPartitionDriver.run(MapPartitionDriver.java:98) at org.apache.flink.runtime.operators.BatchTask.run(BatchTask.java:480) at org.apache.flink.runtime.operators.BatchTask.invoke(BatchTask.java:345) at org.apache.flink.runtime.taskmanager.Task.run(Task.java:559) at java.lang.Thread.run(Thread.java:745) {code} -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Created] (FLINK-4804) Grouping.first() function usage fails
Yakov Goldberg created FLINK-4804: - Summary: Grouping.first() function usage fails Key: FLINK-4804 URL: https://issues.apache.org/jira/browse/FLINK-4804 Project: Flink Issue Type: Bug Reporter: Yakov Goldberg Trying to use Grouping.first() in following example: {code} dd2 = env.from_elements((1, "data"), (1, "hello"), (1, "z")) #dd2 = env.from_elements("data", "hello", "z", "tree","hello","world","hello", "car") dd2 \.group_by(0) \.sort_group(1, Order.ASCENDING) \ .first(2) \.reduce_group(PlainReduce(), combinable=True) {code} -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Created] (FLINK-4803) Job Cancel can hang forever waiting for OutputFormat.close()
Shannon Carey created FLINK-4803: Summary: Job Cancel can hang forever waiting for OutputFormat.close() Key: FLINK-4803 URL: https://issues.apache.org/jira/browse/FLINK-4803 Project: Flink Issue Type: Bug Affects Versions: 1.1.1 Reporter: Shannon Carey If the Flink job uses a badly-behaved OutputFormat (in this example, a HadoopOutputFormat containing a CqlBulkOutputFormat), where the close() method blocks forever, it is impossible to cancel the Flink job even though the blocked thread would respond to an interrupt. The stack traces below show the state of the important threads when a job is canceled and the OutputFormat is blocking forever inside of close(). I suggest that `DataSinkTask.cancel()` method be updated to add a timeout on `this.format.close()`. When the timeout is reached, the Task thread should be interrupted. {code} java.lang.Thread.State: BLOCKED (on object monitor) at org.apache.flink.api.java.hadoop.mapred.HadoopOutputFormatBase.close(HadoopOutputFormatBase.java:158) - waiting to lock <0x0006bae5f788> (a java.lang.Object) at org.apache.flink.runtime.operators.DataSinkTask.cancel(DataSinkTask.java:268) at org.apache.flink.runtime.taskmanager.Task$TaskCanceler.run(Task.java:1149) at java.lang.Thread.run(Thread.java:745) "DataSink (org.apache.flink.api.scala.hadoop.mapred.HadoopOutputFormat@13bf17a2) (2/5)" #6410 daemon prio=5 os_prio=0 tid=0x7fb7e79a4800 nid=0x2ad8 waiting on condition [0x7fb7bdf78000] java.lang.Thread.State: TIMED_WAITING (parking) at sun.misc.Unsafe.park(Native Method) - parking to wait for <0x0006c5ab5e20> (a java.util.concurrent.SynchronousQueue$TransferStack) at java.util.concurrent.locks.LockSupport.parkNanos(LockSupport.java:215) at java.util.concurrent.SynchronousQueue$TransferStack.awaitFulfill(SynchronousQueue.java:460) at java.util.concurrent.SynchronousQueue$TransferStack.transfer(SynchronousQueue.java:362) at java.util.concurrent.SynchronousQueue.offer(SynchronousQueue.java:895) at org.apache.cassandra.io.sstable.SSTableSimpleUnsortedWriter.put(SSTableSimpleUnsortedWriter.java:194) at org.apache.cassandra.io.sstable.SSTableSimpleUnsortedWriter.sync(SSTableSimpleUnsortedWriter.java:180) at org.apache.cassandra.io.sstable.SSTableSimpleUnsortedWriter.close(SSTableSimpleUnsortedWriter.java:156) at org.apache.cassandra.io.sstable.CQLSSTableWriter.close(CQLSSTableWriter.java:275) at org.apache.cassandra.hadoop.AbstractBulkRecordWriter.close(AbstractBulkRecordWriter.java:133) at org.apache.cassandra.hadoop.AbstractBulkRecordWriter.close(AbstractBulkRecordWriter.java:126) at org.apache.flink.api.java.hadoop.mapred.HadoopOutputFormatBase.close(HadoopOutputFormatBase.java:158) - locked <0x0006bae5f788> (a java.lang.Object) at org.apache.flink.runtime.operators.DataSinkTask.invoke(DataSinkTask.java:234) at org.apache.flink.runtime.taskmanager.Task.run(Task.java:584) at java.lang.Thread.run(Thread.java:745) {code} -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Created] (FLINK-4802) distinct() implicitly uses 0th field, when called without a parameter
Yakov Goldberg created FLINK-4802: - Summary: distinct() implicitly uses 0th field, when called without a parameter Key: FLINK-4802 URL: https://issues.apache.org/jira/browse/FLINK-4802 Project: Flink Issue Type: Bug Components: Python API Reporter: Yakov Goldberg Check this code in DataSet.py def distinct(self, *fields): f = None if len(fields) == 0: f = lambda x: (x,) fields = (0,) -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: Type problem in RichFlatMapFunction when using GenericArray type
I identified the problem and opened a issue for it: https://issues.apache.org/jira/browse/FLINK-4801 Am 11/10/16 um 15:31 schrieb Timo Walther: I will also have a look at this issue. Am 11/10/16 um 09:10 schrieb Chesnay Schepler: Yes, i think a JIRA issue would be good for this. On 11.10.2016 08:42, Martin Junghanns wrote: Shall I open an issue for that? The Exception gets thrown when using RichFlatJoinFunction or RichFlatMapFunction (updated the Gist) and the first field of the tuple is an array type. I can look into it once the issue is there. Cheers, Martin On 10.10.2016 13:39, Chesnay Schepler wrote: Hello Martin, Could you include the error you are getting? Regards, Chesnay On 10.10.2016 13:31, Martin Junghanns wrote: Hi, I ran into a problem when using generic arrays in a tuple. I wrote a minimal program to reproduce the error [1]. The problem seems to be related to the order of tuple fields. When I switch Tuple2to Tuple2 and perform the join on field 0, everything works as expected. Using Flink 1.1.2. Cheers, Martin [1] https://gist.github.com/s1ck/37aefb19198cd01a8b998fab354c2cfd -- Freundliche Grüße / Kind Regards Timo Walther Follow me: @twalthr https://www.linkedin.com/in/twalthr
[jira] [Created] (FLINK-4801) Input type inference is faulty with custom Tuples and RichFunctions
Timo Walther created FLINK-4801: --- Summary: Input type inference is faulty with custom Tuples and RichFunctions Key: FLINK-4801 URL: https://issues.apache.org/jira/browse/FLINK-4801 Project: Flink Issue Type: Bug Components: Table API & SQL Reporter: Timo Walther Assignee: Timo Walther This issue has been discussed on the ML: http://apache-flink-mailing-list-archive.1008284.n3.nabble.com/Type-problem-in-RichFlatMapFunction-when-using-GenericArray-type-td13929.html This returns the wrong type: {code} public static class Foo extends Tuple2{ public Foo() { } public Foo(K[] value0, K value1) { super(value0, value1); } } DataSource fooDataSource = env.fromElements(foo); DataSet ds = fooDataSource.join(fooDataSource) .where(field).equalTo(field) .with(new RichFlatJoinFunction () { @Override public void join(Foo first, Foo second, Collector out) throws Exception { out.collect(first); } }); {code} -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: S3/S3A support
Hi! The "truncate()" functionality is only needed for the rolling/bucketing sink. The core checkpoint functionality does not need any truncate() behavior... Best, Stephan On Tue, Oct 11, 2016 at 5:22 PM, Vijay Srinivasaraghavan < vijikar...@yahoo.com.invalid> wrote: > Thanks Stephan. My understanding is checkpoint uses truncate API but S3A > does not support it. Will this have any impact? > Some of the known S3A client limitations are captured in Hortonworks site > https://hortonworks.github.io/hdp-aws/s3-s3aclient/index.html and > wondering if that has any impact on Flink deployment using S3? > RegardsVijay > > > > On Tuesday, October 11, 2016 1:46 AM, Stephan Ewen> wrote: > > > Hi! > In 1.2-SNAPSHOT, we recently fixed issues due to the "eventual > consistency" nature of S3. The fix is not in v1.1 - that is the only known > issue I can think of. > It results in occasional (seldom) periods of heavy restart retries, until > all files are visible to all participants. > If you run into that issue, may be worthwhile to look at Flink > 1.2-SNAPSHOT. > Best, > Stephan > > On Tue, Oct 11, 2016 at 12:13 AM, Vijay Srinivasaraghavan > wrote: > > Hello, > Per documentation (https://ci.apache.org/ projects/flink/flink-docs- > master/setup/aws.html), it looks like S3/S3A FS implementation is supported > using standard Hadoop S3 FS client APIs. > In the absence of using standard HCFS and going with S3/S3A, > 1) Are there any known limitations/issues? > 2) Does checkpoint/savepoint works properly? > Regards > Vijay > > > > >
Re: S3/S3A support
Thanks Stephan. My understanding is checkpoint uses truncate API but S3A does not support it. Will this have any impact? Some of the known S3A client limitations are captured in Hortonworks site https://hortonworks.github.io/hdp-aws/s3-s3aclient/index.html and wondering if that has any impact on Flink deployment using S3? RegardsVijay On Tuesday, October 11, 2016 1:46 AM, Stephan Ewenwrote: Hi! In 1.2-SNAPSHOT, we recently fixed issues due to the "eventual consistency" nature of S3. The fix is not in v1.1 - that is the only known issue I can think of. It results in occasional (seldom) periods of heavy restart retries, until all files are visible to all participants. If you run into that issue, may be worthwhile to look at Flink 1.2-SNAPSHOT. Best, Stephan On Tue, Oct 11, 2016 at 12:13 AM, Vijay Srinivasaraghavan wrote: Hello, Per documentation (https://ci.apache.org/ projects/flink/flink-docs- master/setup/aws.html), it looks like S3/S3A FS implementation is supported using standard Hadoop S3 FS client APIs. In the absence of using standard HCFS and going with S3/S3A, 1) Are there any known limitations/issues? 2) Does checkpoint/savepoint works properly? Regards Vijay
Re: [VOTE] Release Apache Flink 1.1.3 (RC2)
+1 for releasing this as Flink 1.1.3 - Checked the staging repository for hadoop2 / hadoop1 mixup; quickstart version; build a test project against repository - Checked the artifacts: - src doesn't contain any binaries - started Flink locally & executed example & checked web interface On Mon, Oct 10, 2016 at 6:52 PM, Ufuk Celebiwrote: > Dear Flink community, > > Please vote on releasing the following candidate as Apache Flink version > 1.1.3. > > The commit to be voted on: > 8e8d454 (http://git-wip-us.apache.org/repos/asf/flink/commit/8e8d454) > > Branch: > release-1.1.3-rc2 > (https://git1-us-west.apache.org/repos/asf/flink/repo?p=flin > k.git;a=shortlog;h=refs/heads/release-1.1.3-rc2) > > The release artifacts to be voted on can be found at: > http://people.apache.org/~uce/flink-1.1.3-rc2/ > > The release artifacts are signed with the key with fingerprint 9D403309: > http://www.apache.org/dist/flink/KEYS > > The staging repository for this release can be found at: > https://repository.apache.org/content/repositories/orgapacheflink-1106 > > - > > RC2 adds two new commits since RC1. If there are no objections, I > would like to reduce the voting time to (at least) 2 days. The vote > passes if a majority of at least three +1 PMC votes are cast. > > The vote ends on Wed, October 12th, 2016. > > [ ] +1 Release this package as Apache Flink 1.1.3 > [ ] -1 Do not release this package, because ... >
[jira] [Created] (FLINK-4800) Refactor the ContinuousFileMonitoringFunction code and the related tests.
Kostas Kloudas created FLINK-4800: - Summary: Refactor the ContinuousFileMonitoringFunction code and the related tests. Key: FLINK-4800 URL: https://issues.apache.org/jira/browse/FLINK-4800 Project: Flink Issue Type: Bug Components: Streaming Connectors Reporter: Kostas Kloudas Assignee: Kostas Kloudas Priority: Minor Currently the code in the FileMonitoringFunction can be simplified. The same holds for the test code. This is the goal of this issue. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: Type problem in RichFlatMapFunction when using GenericArray type
I will also have a look at this issue. Am 11/10/16 um 09:10 schrieb Chesnay Schepler: Yes, i think a JIRA issue would be good for this. On 11.10.2016 08:42, Martin Junghanns wrote: Shall I open an issue for that? The Exception gets thrown when using RichFlatJoinFunction or RichFlatMapFunction (updated the Gist) and the first field of the tuple is an array type. I can look into it once the issue is there. Cheers, Martin On 10.10.2016 13:39, Chesnay Schepler wrote: Hello Martin, Could you include the error you are getting? Regards, Chesnay On 10.10.2016 13:31, Martin Junghanns wrote: Hi, I ran into a problem when using generic arrays in a tuple. I wrote a minimal program to reproduce the error [1]. The problem seems to be related to the order of tuple fields. When I switch Tuple2to Tuple2 and perform the join on field 0, everything works as expected. Using Flink 1.1.2. Cheers, Martin [1] https://gist.github.com/s1ck/37aefb19198cd01a8b998fab354c2cfd -- Freundliche Grüße / Kind Regards Timo Walther Follow me: @twalthr https://www.linkedin.com/in/twalthr
[jira] [Created] (FLINK-4799) Re-add build-target symlink to project root
Maximilian Michels created FLINK-4799: - Summary: Re-add build-target symlink to project root Key: FLINK-4799 URL: https://issues.apache.org/jira/browse/FLINK-4799 Project: Flink Issue Type: Wish Components: Build System Affects Versions: 1.2.0, 1.1.3 Reporter: Maximilian Michels Assignee: Maximilian Michels Priority: Minor Fix For: 1.2.0 We have previously removed the plugin which created the 'build-target' link to the build target directory. See FLINK-4732. At least one user has requested to re-add the link. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: [DISCUSS] Removing delete*Timer from the WindowOperator.Context
+Konstantin Knauflooping you in directly because you used the "delete timer" feature in the past and even did some changes to the timer system. Are you still relying on the fact that deleted timers are actually deleted. The main reason for wanting to get rid of delete timer is IMHO that deleting a timer is difficult, depending on the data structure that you use for timers. Especially if you want a data structure that can grow out of core. By the way, the current data structure for timers is a Java Queue (a heap) so deletes from this are O(n), i.e. possibly slow. On Wed, 28 Sep 2016 at 15:21 Maximilian Michels wrote: > What are the use cases where you actually need to delete a timer? How > > about we only let users delete timers which they created themselves? > > > > I guessing most of these use cases will be obsolete with the new > > Trigger DSL because the trigger logic can be expressed more easily. So > > +1 for removing the delete methods from the context. > > > > On Tue, Sep 27, 2016 at 3:43 PM, Kostas Kloudas > > wrote: > > > Hi all, > > > > > > As the title of this email suggests, I am proposing to remove the > methods > > > deleteProcessingTimeTimer(long time) and deleteEventTimeTimer(long time) > > > from the WindowOperator.Context. With this change, registered timers that > > > have nothing to do (e.g. because their state has already been cleaned up) > > > will be simply ignored by the windowOperator, when their time comes. > > > > > > The reason for the change is that by allowing custom user code, e.g. a > custom Trigger, > > > to delete timers we may have unpredictable behavior. > > > > > > As an example, one can imagine the case where we have allowed_lateness = > 0 and the cleanup > > > timer for a window collides with the end_of_window one. In this case, by > deleting the end_of_window > > > timer from the trigger (possibly a custom one), we end up also deleting > the cleanup one, > > > which in turn can lead to the window state never being garbage collected. > > > > > > To see what can be the consequences apart from memory leaks, this can > easily lead > > > to wrong session windows, as a session that should have been garbage > collected, will > > > still be around and ready to accept new data. > > > > > > With this change, timers that should correctly be deleted will now > remain in the queue of > > > pending timers, but they will do nothing, while cleanup timers will > cleanup the state of their > > > corresponding window. > > > > > > Other possible solutions like keeping a separate list for cleanup timers > would complicate > > > the codebase and also introduce memory overheads which can be avoided > using the > > > solution above (i.e. just ignoring timers the have nothing to do > anymore). > > > > > > What do you think? > > > > > > Kostas > > > > >
[jira] [Created] (FLINK-4798) CEPITCase.testSimpleKeyedPatternCEP test failure
Robert Metzger created FLINK-4798: - Summary: CEPITCase.testSimpleKeyedPatternCEP test failure Key: FLINK-4798 URL: https://issues.apache.org/jira/browse/FLINK-4798 Project: Flink Issue Type: Bug Components: CEP Affects Versions: 1.2.0 Reporter: Robert Metzger {code} --- T E S T S --- Running org.apache.flink.cep.CEPITCase Tests run: 8, Failures: 1, Errors: 0, Skipped: 0, Time elapsed: 6.627 sec <<< FAILURE! - in org.apache.flink.cep.CEPITCase testSimpleKeyedPatternCEP(org.apache.flink.cep.CEPITCase) Time elapsed: 0.312 sec <<< FAILURE! java.lang.AssertionError: Different number of lines in expected and obtained result. expected:<3> but was:<1> at org.junit.Assert.fail(Assert.java:88) at org.junit.Assert.failNotEquals(Assert.java:834) at org.junit.Assert.assertEquals(Assert.java:645) at org.apache.flink.test.util.TestBaseUtils.compareResultsByLinesInMemory(TestBaseUtils.java:316) at org.apache.flink.test.util.TestBaseUtils.compareResultsByLinesInMemory(TestBaseUtils.java:302) at org.apache.flink.cep.CEPITCase.after(CEPITCase.java:61) {code} in https://api.travis-ci.org/jobs/166676733/log.txt?deansi=true -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: S3/S3A support
Hi! In 1.2-SNAPSHOT, we recently fixed issues due to the "eventual consistency" nature of S3. The fix is not in v1.1 - that is the only known issue I can think of. It results in occasional (seldom) periods of heavy restart retries, until all files are visible to all participants. If you run into that issue, may be worthwhile to look at Flink 1.2-SNAPSHOT. Best, Stephan On Tue, Oct 11, 2016 at 12:13 AM, Vijay Srinivasaraghavan < vijikar...@yahoo.com.invalid> wrote: > Hello, > Per documentation (https://ci.apache.org/projects/flink/flink-docs- > master/setup/aws.html), it looks like S3/S3A FS implementation is > supported using standard Hadoop S3 FS client APIs. > In the absence of using standard HCFS and going with S3/S3A, > 1) Are there any known limitations/issues? > 2) Does checkpoint/savepoint works properly? > Regards > Vijay
[jira] [Created] (FLINK-4797) Add Flink 1.1 savepoint backwards compatability
Ufuk Celebi created FLINK-4797: -- Summary: Add Flink 1.1 savepoint backwards compatability Key: FLINK-4797 URL: https://issues.apache.org/jira/browse/FLINK-4797 Project: Flink Issue Type: Improvement Components: State Backends, Checkpointing Reporter: Ufuk Celebi Assignee: Ufuk Celebi Make sure that we can resume from Flink 1.1 savepoints on the job manager side. This means that we need to be able to read the savepoint header file on the job manager and create a completed checkpoint from it. This will not yet mean that we can resume from 1.1 savepoints as the operator/task manager side compatability is missing. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
[jira] [Created] (FLINK-4796) Extend SinkFunction to include also the event timestamp
Robert Metzger created FLINK-4796: - Summary: Extend SinkFunction to include also the event timestamp Key: FLINK-4796 URL: https://issues.apache.org/jira/browse/FLINK-4796 Project: Flink Issue Type: Improvement Components: DataStream API Affects Versions: 1.2.0 Reporter: Robert Metzger The Kafka 0.10 connector supports writing event timestamps to Kafka. Currently, the regular DataStream APIs don't allow user code to access the event timestamp easily. That's why the Kafka connector is using a custom operator ({{transform()}}) to access the event time. With this JIRA, I would like to provide the event timestamp in the regular DataStream APIs. Once I'll look into the issue, I'll post some proposals how to add the timestamp. -- This message was sent by Atlassian JIRA (v6.3.4#6332)
Re: Type problem in RichFlatMapFunction when using GenericArray type
Yes, i think a JIRA issue would be good for this. On 11.10.2016 08:42, Martin Junghanns wrote: Shall I open an issue for that? The Exception gets thrown when using RichFlatJoinFunction or RichFlatMapFunction (updated the Gist) and the first field of the tuple is an array type. I can look into it once the issue is there. Cheers, Martin On 10.10.2016 13:39, Chesnay Schepler wrote: Hello Martin, Could you include the error you are getting? Regards, Chesnay On 10.10.2016 13:31, Martin Junghanns wrote: Hi, I ran into a problem when using generic arrays in a tuple. I wrote a minimal program to reproduce the error [1]. The problem seems to be related to the order of tuple fields. When I switch Tuple2to Tuple2 and perform the join on field 0, everything works as expected. Using Flink 1.1.2. Cheers, Martin [1] https://gist.github.com/s1ck/37aefb19198cd01a8b998fab354c2cfd
Re: Type problem in RichFlatMapFunction when using GenericArray type
Shall I open an issue for that? The Exception gets thrown when using RichFlatJoinFunction or RichFlatMapFunction (updated the Gist) and the first field of the tuple is an array type. I can look into it once the issue is there. Cheers, Martin On 10.10.2016 13:39, Chesnay Schepler wrote: Hello Martin, Could you include the error you are getting? Regards, Chesnay On 10.10.2016 13:31, Martin Junghanns wrote: Hi, I ran into a problem when using generic arrays in a tuple. I wrote a minimal program to reproduce the error [1]. The problem seems to be related to the order of tuple fields. When I switch Tuple2to Tuple2 and perform the join on field 0, everything works as expected. Using Flink 1.1.2. Cheers, Martin [1] https://gist.github.com/s1ck/37aefb19198cd01a8b998fab354c2cfd