[ https://issues.apache.org/jira/browse/FLINK-19065?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17188427#comment-17188427 ]
Aljoscha Krettek commented on FLINK-19065: ------------------------------------------ Thanks! I found the culprit: https://github.com/apache/flink/blob/11b2fcab5841e38d712e8e919c11e6ecd4f69a5d/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/datastream/CoGroupedStreams.java#L332. The problem is that we're simulating a two-input operation by tagging elements from the two sides and then treating them differently in the real (one-input) operation. Even though I wrote this code (5 years ago) I don't like that we're using user-facing methods internally like this. A simple fix would be to set some UIDs on the operations that we create internally but we have to make sure that they are actually unique. > java.lang.IllegalStateException: Auto generated UIDs have been disabled on > join > ------------------------------------------------------------------------------- > > Key: FLINK-19065 > URL: https://issues.apache.org/jira/browse/FLINK-19065 > Project: Flink > Issue Type: Bug > Components: API / DataStream > Affects Versions: 1.11.0, 1.11.1 > Reporter: Maris > Priority: Major > > Join operation with AutoGeneratedUID disabled leads to > {code:java} > java.lang.IllegalStateException: Auto generated UIDs have been disabled but > no UID or hash has been assigned to operator Map > {code} > code to reproduce > {code:java} > class JoinSpec extends AnyFlatSpec with Matchers with Serializable { > it should "be able to join streams" in { > val env = StreamExecutionEnvironment.getExecutionEnvironment > env.getConfig.disableAutoGeneratedUIDs() > val a = env.fromCollection(List("1", "2", > "3")).name("a").uid("source-uid") > val b = env.fromCollection(List("1", "2", > "3")).name("b").uid("source-uid2") > val c = a > .join(b) > .where(identity) > .equalTo(identity) > .window(TumblingProcessingTimeWindows.of(Time.seconds(30)))((a, b) => > a+b) > .uid("joined").name("joined") > c.addSink(s => println(s)) > .name("ab") > .uid("ab") > println(env.getExecutionPlan) > env.execute > succeed > } > } > {code} -- This message was sent by Atlassian Jira (v8.3.4#803005)