Skip to content

Commit c24b8ec

Browse files
beyond1920TheodoreLx
authored andcommitted
[FLINK-19606][table-runtime-blink] Refactor utility class JoinConditionWithFullFilters from AbstractStreamingJoinOperator
This closes apache#15752
1 parent e419375 commit c24b8ec

File tree

2 files changed

+73
-48
lines changed

2 files changed

+73
-48
lines changed
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,66 @@
1+
/*
2+
* Licensed to the Apache Software Foundation (ASF) under one
3+
* or more contributor license agreements. See the NOTICE file
4+
* distributed with this work for additional information
5+
* regarding copyright ownership. The ASF licenses this file
6+
* to you under the Apache License, Version 2.0 (the
7+
* "License"); you may not use this file except in compliance
8+
* with the License. You may obtain a copy of the License at
9+
*
10+
* http://www.apache.org/licenses/LICENSE-2.0
11+
*
12+
* Unless required by applicable law or agreed to in writing, software
13+
* distributed under the License is distributed on an "AS IS" BASIS,
14+
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15+
* See the License for the specific language governing permissions and
16+
* limitations under the License.
17+
*/
18+
19+
package org.apache.flink.table.runtime.operators.join;
20+
21+
import org.apache.flink.api.java.operators.translation.WrappingFunction;
22+
import org.apache.flink.streaming.api.operators.KeyContext;
23+
import org.apache.flink.table.data.RowData;
24+
import org.apache.flink.table.data.binary.NullAwareGetters;
25+
import org.apache.flink.table.runtime.generated.JoinCondition;
26+
27+
/** Utility to take null filters into consideration when apply join condition. */
28+
public class JoinConditionWithNullFilters extends WrappingFunction<JoinCondition>
29+
implements JoinCondition {
30+
31+
private static final long serialVersionUID = 1L;
32+
33+
/** Should filter null keys. */
34+
private final int[] nullFilterKeys;
35+
36+
/** No keys need to filter null. */
37+
private final boolean nullSafe;
38+
39+
/** Filter null to all keys. */
40+
private final boolean filterAllNulls;
41+
42+
private final KeyContext keyContext;
43+
44+
public JoinConditionWithNullFilters(
45+
JoinCondition backingJoinCondition, boolean[] filterNullKeys, KeyContext keyContext) {
46+
super(backingJoinCondition);
47+
this.nullFilterKeys = NullAwareJoinHelper.getNullFilterKeys(filterNullKeys);
48+
this.nullSafe = nullFilterKeys.length == 0;
49+
this.filterAllNulls = nullFilterKeys.length == filterNullKeys.length;
50+
this.keyContext = keyContext;
51+
}
52+
53+
@Override
54+
public boolean apply(RowData left, RowData right) {
55+
if (!nullSafe) { // is not null safe, return false if any null exists
56+
// key is always BinaryRowData
57+
NullAwareGetters joinKey = (NullAwareGetters) keyContext.getCurrentKey();
58+
if (filterAllNulls ? joinKey.anyNull() : joinKey.anyNull(nullFilterKeys)) {
59+
// find null present, return false directly
60+
return false;
61+
}
62+
}
63+
// test condition
64+
return wrappedFunction.apply(left, right);
65+
}
66+
}

flink-table/flink-table-runtime-blink/src/main/java/org/apache/flink/table/runtime/operators/join/stream/AbstractStreamingJoinOperator.java

Lines changed: 7 additions & 48 deletions
Original file line numberDiff line numberDiff line change
@@ -18,17 +18,15 @@
1818

1919
package org.apache.flink.table.runtime.operators.join.stream;
2020

21-
import org.apache.flink.api.common.functions.AbstractRichFunction;
2221
import org.apache.flink.api.java.tuple.Tuple2;
2322
import org.apache.flink.configuration.Configuration;
2423
import org.apache.flink.streaming.api.operators.AbstractStreamOperator;
2524
import org.apache.flink.streaming.api.operators.TimestampedCollector;
2625
import org.apache.flink.streaming.api.operators.TwoInputStreamOperator;
2726
import org.apache.flink.table.data.RowData;
28-
import org.apache.flink.table.data.binary.NullAwareGetters;
2927
import org.apache.flink.table.runtime.generated.GeneratedJoinCondition;
3028
import org.apache.flink.table.runtime.generated.JoinCondition;
31-
import org.apache.flink.table.runtime.operators.join.NullAwareJoinHelper;
29+
import org.apache.flink.table.runtime.operators.join.JoinConditionWithNullFilters;
3230
import org.apache.flink.table.runtime.operators.join.stream.state.JoinInputSideSpec;
3331
import org.apache.flink.table.runtime.operators.join.stream.state.JoinRecordStateView;
3432
import org.apache.flink.table.runtime.operators.join.stream.state.OuterJoinRecordStateView;
@@ -60,14 +58,7 @@ public abstract class AbstractStreamingJoinOperator extends AbstractStreamOperat
6058
protected final JoinInputSideSpec leftInputSideSpec;
6159
protected final JoinInputSideSpec rightInputSideSpec;
6260

63-
/** Should filter null keys. */
64-
private final int[] nullFilterKeys;
65-
66-
/** No keys need to filter null. */
67-
private final boolean nullSafe;
68-
69-
/** Filter null to all keys. */
70-
private final boolean filterAllNulls;
61+
private final boolean[] filterNullKeys;
7162

7263
protected final long stateRetentionTime;
7364

@@ -88,21 +79,17 @@ public AbstractStreamingJoinOperator(
8879
this.leftInputSideSpec = leftInputSideSpec;
8980
this.rightInputSideSpec = rightInputSideSpec;
9081
this.stateRetentionTime = stateRetentionTime;
91-
this.nullFilterKeys = NullAwareJoinHelper.getNullFilterKeys(filterNullKeys);
92-
this.nullSafe = nullFilterKeys.length == 0;
93-
this.filterAllNulls = nullFilterKeys.length == filterNullKeys.length;
82+
this.filterNullKeys = filterNullKeys;
9483
}
9584

9685
@Override
9786
public void open() throws Exception {
9887
super.open();
99-
10088
JoinCondition condition =
10189
generatedJoinCondition.newInstance(getRuntimeContext().getUserCodeClassLoader());
102-
condition.setRuntimeContext(getRuntimeContext());
103-
condition.open(new Configuration());
104-
105-
this.joinCondition = new JoinConditionWithNullFilters(condition);
90+
this.joinCondition = new JoinConditionWithNullFilters(condition, filterNullKeys, this);
91+
this.joinCondition.setRuntimeContext(getRuntimeContext());
92+
this.joinCondition.open(new Configuration());
10693

10794
this.collector = new TimestampedCollector<>(output);
10895
}
@@ -111,35 +98,7 @@ public void open() throws Exception {
11198
public void close() throws Exception {
11299
super.close();
113100
if (joinCondition != null) {
114-
joinCondition.backingJoinCondition.close();
115-
}
116-
}
117-
118-
// ----------------------------------------------------------------------------------------
119-
// Utility Classes
120-
// ----------------------------------------------------------------------------------------
121-
122-
private class JoinConditionWithNullFilters extends AbstractRichFunction
123-
implements JoinCondition {
124-
125-
final JoinCondition backingJoinCondition;
126-
127-
private JoinConditionWithNullFilters(JoinCondition backingJoinCondition) {
128-
this.backingJoinCondition = backingJoinCondition;
129-
}
130-
131-
@Override
132-
public boolean apply(RowData left, RowData right) {
133-
if (!nullSafe) { // is not null safe, return false if any null exists
134-
// key is always BinaryRowData
135-
NullAwareGetters joinKey = (NullAwareGetters) getCurrentKey();
136-
if (filterAllNulls ? joinKey.anyNull() : joinKey.anyNull(nullFilterKeys)) {
137-
// find null present, return false directly
138-
return false;
139-
}
140-
}
141-
// test condition
142-
return backingJoinCondition.apply(left, right);
101+
joinCondition.close();
143102
}
144103
}
145104

0 commit comments

Comments
 (0)