-
Notifications
You must be signed in to change notification settings - Fork 13k
/
FlinkRelMdUniqueKeys.scala
638 lines (575 loc) · 21.7 KB
/
FlinkRelMdUniqueKeys.scala
1
2
3
4
5
6
7
8
9
10
11
12
13
14
15
16
17
18
19
20
21
22
23
24
25
26
27
28
29
30
31
32
33
34
35
36
37
38
39
40
41
42
43
44
45
46
47
48
49
50
51
52
53
54
55
56
57
58
59
60
61
62
63
64
65
66
67
68
69
70
71
72
73
74
75
76
77
78
79
80
81
82
83
84
85
86
87
88
89
90
91
92
93
94
95
96
97
98
99
100
101
102
103
104
105
106
107
108
109
110
111
112
113
114
115
116
117
118
119
120
121
122
123
124
125
126
127
128
129
130
131
132
133
134
135
136
137
138
139
140
141
142
143
144
145
146
147
148
149
150
151
152
153
154
155
156
157
158
159
160
161
162
163
164
165
166
167
168
169
170
171
172
173
174
175
176
177
178
179
180
181
182
183
184
185
186
187
188
189
190
191
192
193
194
195
196
197
198
199
200
201
202
203
204
205
206
207
208
209
210
211
212
213
214
215
216
217
218
219
220
221
222
223
224
225
226
227
228
229
230
231
232
233
234
235
236
237
238
239
240
241
242
243
244
245
246
247
248
249
250
251
252
253
254
255
256
257
258
259
260
261
262
263
264
265
266
267
268
269
270
271
272
273
274
275
276
277
278
279
280
281
282
283
284
285
286
287
288
289
290
291
292
293
294
295
296
297
298
299
300
301
302
303
304
305
306
307
308
309
310
311
312
313
314
315
316
317
318
319
320
321
322
323
324
325
326
327
328
329
330
331
332
333
334
335
336
337
338
339
340
341
342
343
344
345
346
347
348
349
350
351
352
353
354
355
356
357
358
359
360
361
362
363
364
365
366
367
368
369
370
371
372
373
374
375
376
377
378
379
380
381
382
383
384
385
386
387
388
389
390
391
392
393
394
395
396
397
398
399
400
401
402
403
404
405
406
407
408
409
410
411
412
413
414
415
416
417
418
419
420
421
422
423
424
425
426
427
428
429
430
431
432
433
434
435
436
437
438
439
440
441
442
443
444
445
446
447
448
449
450
451
452
453
454
455
456
457
458
459
460
461
462
463
464
465
466
467
468
469
470
471
472
473
474
475
476
477
478
479
480
481
482
483
484
485
486
487
488
489
490
491
492
493
494
495
496
497
498
499
500
501
502
503
504
505
506
507
508
509
510
511
512
513
514
515
516
517
518
519
520
521
522
523
524
525
526
527
528
529
530
531
532
533
534
535
536
537
538
539
540
541
542
543
544
545
546
547
548
549
550
551
552
553
554
555
556
557
558
559
560
561
562
563
564
565
566
567
568
569
570
571
572
573
574
575
576
577
578
579
580
581
582
583
584
585
586
587
588
589
590
591
592
593
594
595
596
597
598
599
600
601
602
603
604
605
606
607
608
609
610
611
612
613
614
615
616
617
618
619
620
621
622
623
624
625
626
627
628
629
630
631
632
633
634
635
636
637
638
/*
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.flink.table.planner.plan.metadata
import org.apache.flink.table.catalog.AbstractCatalogTable
import org.apache.flink.table.planner._
import org.apache.flink.table.planner.calcite.FlinkRelBuilder.PlannerNamedWindowProperty
import org.apache.flink.table.planner.calcite.FlinkTypeFactory
import org.apache.flink.table.planner.plan.nodes.calcite.{Expand, Rank, WatermarkAssigner, WindowAggregate}
import org.apache.flink.table.planner.plan.nodes.common.CommonLookupJoin
import org.apache.flink.table.planner.plan.nodes.logical._
import org.apache.flink.table.planner.plan.nodes.physical.MultipleInputRel
import org.apache.flink.table.planner.plan.nodes.physical.batch._
import org.apache.flink.table.planner.plan.nodes.physical.stream._
import org.apache.flink.table.planner.plan.schema.{FlinkPreparingTableBase, TableSourceTable}
import org.apache.flink.table.planner.plan.utils.{FlinkRelMdUtil, RankUtil}
import org.apache.flink.table.runtime.operators.rank.{ConstantRankRange, RankType}
import org.apache.flink.table.sources.TableSource
import org.apache.flink.table.types.logical.utils.LogicalTypeCasts
import com.google.common.collect.ImmutableSet
import org.apache.calcite.plan.RelOptTable
import org.apache.calcite.plan.hep.HepRelVertex
import org.apache.calcite.plan.volcano.RelSubset
import org.apache.calcite.rel.`type`.RelDataType
import org.apache.calcite.rel.core._
import org.apache.calcite.rel.metadata._
import org.apache.calcite.rel.{RelNode, SingleRel}
import org.apache.calcite.rex.{RexCall, RexInputRef, RexNode}
import org.apache.calcite.sql.fun.SqlStdOperatorTable
import org.apache.calcite.sql.{SqlKind, SqlWindowTableFunction}
import org.apache.calcite.util.{Bug, BuiltInMethod, ImmutableBitSet, Util}
import java.util
import scala.collection.JavaConversions._
class FlinkRelMdUniqueKeys private extends MetadataHandler[BuiltInMetadata.UniqueKeys] {
def getDef: MetadataDef[BuiltInMetadata.UniqueKeys] = BuiltInMetadata.UniqueKeys.DEF
def getUniqueKeys(
rel: TableScan,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = {
getTableUniqueKeys(null, rel.getTable)
}
def getUniqueKeys(
rel: FlinkLogicalLegacyTableSourceScan,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = {
getTableUniqueKeys(rel.tableSource, rel.getTable)
}
private def getTableUniqueKeys(
tableSource: TableSource[_],
relOptTable: RelOptTable): JSet[ImmutableBitSet] = {
relOptTable match {
case sourceTable: TableSourceTable =>
val catalogTable = sourceTable.catalogTable
catalogTable match {
case act: AbstractCatalogTable =>
val schema = act.getSchema
if (schema.getPrimaryKey.isPresent) {
val columns = schema.getFieldNames
val columnIndices = schema.getPrimaryKey.get().getColumns map { c =>
columns.indexOf(c)
}
val builder = ImmutableSet.builder[ImmutableBitSet]()
builder.add(ImmutableBitSet.of(columnIndices:_*))
val uniqueSet = sourceTable.uniqueKeysSet().orElse(null)
if (uniqueSet != null) {
builder.addAll(uniqueSet)
}
builder.build()
} else {
sourceTable.uniqueKeysSet.orElse(null)
}
}
case table: FlinkPreparingTableBase => table.uniqueKeysSet.orElse(null)
case _ => null
}
}
def getUniqueKeys(
rel: Project,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] =
getProjectUniqueKeys(rel.getProjects, rel.getInput, mq, ignoreNulls)
def getUniqueKeys(
rel: Filter,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = mq.getUniqueKeys(rel.getInput, ignoreNulls)
def getUniqueKeys(
calc: Calc,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = {
val input = calc.getInput
val projects = calc.getProgram.getProjectList.map(calc.getProgram.expandLocalRef)
getProjectUniqueKeys(projects, input, mq, ignoreNulls)
}
private def getProjectUniqueKeys(
projects: JList[RexNode],
input: RelNode,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = {
// LogicalProject maps a set of rows to a different set;
// Without knowledge of the mapping function(whether it
// preserves uniqueness), it is only safe to derive uniqueness
// info from the child of a project when the mapping is f(a) => a.
//
// Further more, the unique bitset coming from the child needs
val projUniqueKeySet = new JHashSet[ImmutableBitSet]()
val mapInToOutPos = new JHashMap[Int, JArrayList[Int]]()
def appendMapInToOutPos(inIndex: Int, outIndex: Int): Unit = {
if (mapInToOutPos.contains(inIndex)) {
mapInToOutPos(inIndex).add(outIndex)
} else {
val arrayBuffer = new JArrayList[Int]()
arrayBuffer.add(outIndex)
mapInToOutPos.put(inIndex, arrayBuffer)
}
}
// Build an input to output position map.
projects.zipWithIndex.foreach {
case (projExpr, i) =>
projExpr match {
case ref: RexInputRef => appendMapInToOutPos(ref.getIndex, i)
case a: RexCall if ignoreNulls && a.getOperator.equals(SqlStdOperatorTable.CAST) =>
val castOperand = a.getOperands.get(0)
castOperand match {
case castRef: RexInputRef =>
val typeFactory = input.getCluster.getTypeFactory
val castType = typeFactory.createTypeWithNullability(projExpr.getType, true)
val origType = typeFactory.createTypeWithNullability(castOperand.getType, true)
if (castType == origType) {
appendMapInToOutPos(castRef.getIndex, i)
}
case _ => // ignore
}
//rename or cast
case a: RexCall if (a.getKind.equals(SqlKind.AS) || isFidelityCast(a)) &&
a.getOperands.get(0).isInstanceOf[RexInputRef] =>
appendMapInToOutPos(a.getOperands.get(0).asInstanceOf[RexInputRef].getIndex, i)
case _ => // ignore
}
}
if (mapInToOutPos.isEmpty) {
// if there's no RexInputRef in the projected expressions
// return empty set.
return projUniqueKeySet
}
val childUniqueKeySet = mq.getUniqueKeys(input, ignoreNulls)
if (childUniqueKeySet != null) {
// Now add to the projUniqueKeySet the child keys that are fully
// projected.
childUniqueKeySet.foreach { colMask =>
val filerInToOutPos = mapInToOutPos.filter { inToOut =>
colMask.asList().contains(inToOut._1)
}
val keys = filerInToOutPos.keys
if (colMask.forall(keys.contains(_))) {
val total = filerInToOutPos.map(_._2.size).product
for (i <- 0 to total) {
val tmpMask = ImmutableBitSet.builder()
filerInToOutPos.foreach { inToOut =>
val outs = inToOut._2
tmpMask.set(outs.get(i % outs.size))
}
projUniqueKeySet.add(tmpMask.build())
}
}
}
}
projUniqueKeySet
}
/**
* Whether the [[RexCall]] is a cast that doesn't lose any information.
*/
private def isFidelityCast(call: RexCall): Boolean = {
if (call.getKind != SqlKind.CAST) {
return false
}
val originalType = FlinkTypeFactory.toLogicalType(call.getOperands.get(0).getType)
val newType = FlinkTypeFactory.toLogicalType(call.getType)
LogicalTypeCasts.supportsImplicitCast(originalType, newType)
}
def getUniqueKeys(
rel: Expand,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = {
// mapping input column index to output index for non-null value columns
val mapInputToOutput = new JHashMap[Int, Int]()
(0 until rel.getRowType.getFieldCount).filter(_ != rel.expandIdIndex).foreach { column =>
val inputRefs = FlinkRelMdUtil.getInputRefIndices(column, rel)
// expand columns corresponding to a given index should be same input ref.
if (inputRefs.size() == 1 && inputRefs.head >= 0) {
mapInputToOutput.put(inputRefs.head, column)
}
}
if (mapInputToOutput.isEmpty) {
return null
}
val inputUniqueKeys = mq.getUniqueKeys(rel.getInput, ignoreNulls)
if (inputUniqueKeys == null || inputUniqueKeys.isEmpty) {
return inputUniqueKeys
}
// values of expand_is are unique in rows expanded from a row,
// and a input unique key combined with expand_id are also unique
val outputUniqueKeys = new JHashSet[ImmutableBitSet]()
inputUniqueKeys.foreach { uniqueKey =>
val outputUniqueKeyBuilder = ImmutableBitSet.builder()
// a input unique key can be output only its values are all in `mapInputToOutput`
val canOutput = uniqueKey.toList.forall { key =>
if (mapInputToOutput.contains(key)) {
outputUniqueKeyBuilder.set(mapInputToOutput.get(key))
true
} else {
false
}
}
if (canOutput) {
// unique key from input combined with expand id are unique
outputUniqueKeyBuilder.set(rel.expandIdIndex)
outputUniqueKeys.add(outputUniqueKeyBuilder.build())
}
}
if (outputUniqueKeys.isEmpty) null else outputUniqueKeys
}
def getUniqueKeys(
rel: Exchange,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = mq.getUniqueKeys(rel.getInput, ignoreNulls)
def getUniqueKeys(
rel: Rank,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = {
val inputUniqueKeys = mq.getUniqueKeys(rel.getInput, ignoreNulls)
val rankFunColumnIndex = RankUtil.getRankNumberColumnIndex(rel).getOrElse(-1)
//TODO current deduplicate on row time is still a Rank,
// remove this after support deduplicate on row time
val canConvertToDeduplicate: Boolean = {
val rankRange = rel.rankRange
val isRowNumberType = rel.rankType == RankType.ROW_NUMBER
val isLimit1 = rankRange match {
case rankRange: ConstantRankRange =>
rankRange.getRankStart() == 1 && rankRange.getRankEnd() == 1
case _ => false
}
isRowNumberType && isLimit1
}
if (canConvertToDeduplicate) {
val retSet = new JHashSet[ImmutableBitSet]
retSet.add(rel.partitionKey)
retSet
}
else if (rankFunColumnIndex < 0) {
inputUniqueKeys
} else {
val retSet = new JHashSet[ImmutableBitSet]
rel.rankType match {
case RankType.ROW_NUMBER =>
retSet.add(rel.partitionKey.union(ImmutableBitSet.of(rankFunColumnIndex)))
case _ => // do nothing
}
if (inputUniqueKeys != null && inputUniqueKeys.nonEmpty) {
inputUniqueKeys.foreach {
uniqueKey => retSet.add(uniqueKey)
}
}
retSet
}
}
def getUniqueKeys(
rel: Sort,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = mq.getUniqueKeys(rel.getInput, ignoreNulls)
def getUniqueKeys(
rel: StreamExecDeduplicate,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = {
ImmutableSet.of(ImmutableBitSet.of(rel.getUniqueKeys.map(Integer.valueOf).toList))
}
def getUniqueKeys(
rel: StreamExecChangelogNormalize,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = {
ImmutableSet.of(ImmutableBitSet.of(rel.uniqueKeys.map(Integer.valueOf).toList))
}
def getUniqueKeys(
rel: Aggregate,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = {
getUniqueKeysOnAggregate(rel.getGroupSet.toArray, mq, ignoreNulls)
}
def getUniqueKeys(
rel: BatchExecGroupAggregateBase,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = {
if (rel.isFinal) {
getUniqueKeysOnAggregate(rel.getGrouping, mq, ignoreNulls)
} else {
null
}
}
def getUniqueKeys(
rel: StreamExecGroupAggregate,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = {
getUniqueKeysOnAggregate(rel.grouping, mq, ignoreNulls)
}
def getUniqueKeys(
rel: StreamExecLocalGroupAggregate,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = null
def getUniqueKeys(
rel: StreamExecGlobalGroupAggregate,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = {
getUniqueKeysOnAggregate(rel.grouping, mq, ignoreNulls)
}
def getUniqueKeysOnAggregate(
grouping: Array[Int],
mq: RelMetadataQuery,
ignoreNulls: Boolean): util.Set[ImmutableBitSet] = {
// group by keys form a unique key
ImmutableSet.of(ImmutableBitSet.of(grouping.indices: _*))
}
def getUniqueKeys(
rel: WindowAggregate,
mq: RelMetadataQuery,
ignoreNulls: Boolean): util.Set[ImmutableBitSet] = {
getUniqueKeysOnWindowAgg(
rel.getRowType.getFieldCount,
rel.getNamedProperties,
rel.getGroupSet.toArray,
mq,
ignoreNulls)
}
def getUniqueKeys(
rel: BatchExecWindowAggregateBase,
mq: RelMetadataQuery,
ignoreNulls: Boolean): util.Set[ImmutableBitSet] = {
if (rel.isFinal) {
getUniqueKeysOnWindowAgg(
rel.getRowType.getFieldCount,
rel.getNamedProperties,
rel.getGrouping,
mq,
ignoreNulls)
} else {
null
}
}
def getUniqueKeys(
rel: StreamExecGroupWindowAggregate,
mq: RelMetadataQuery,
ignoreNulls: Boolean): util.Set[ImmutableBitSet] = {
getUniqueKeysOnWindowAgg(
rel.getRowType.getFieldCount, rel.getWindowProperties, rel.getGrouping, mq, ignoreNulls)
}
private def getUniqueKeysOnWindowAgg(
fieldCount: Int,
namedProperties: Seq[PlannerNamedWindowProperty],
grouping: Array[Int],
mq: RelMetadataQuery,
ignoreNulls: Boolean): util.Set[ImmutableBitSet] = {
if (namedProperties.nonEmpty) {
val begin = fieldCount - namedProperties.size
val end = fieldCount - 1
//namedProperties's indexes is at the end of output record
val keys = ImmutableBitSet.of(grouping.indices: _*)
(begin to end).map {
i => keys.union(ImmutableBitSet.of(i))
}.toSet[ImmutableBitSet]
} else {
null
}
}
def getUniqueKeys(
rel: Window,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = {
getUniqueKeysOfOverAgg(rel, mq, ignoreNulls)
}
def getUniqueKeys(
rel: BatchExecOverAggregate,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = {
getUniqueKeysOfOverAgg(rel, mq, ignoreNulls)
}
def getUniqueKeys(
rel: StreamExecOverAggregate,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = {
getUniqueKeysOfOverAgg(rel, mq, ignoreNulls)
}
private def getUniqueKeysOfOverAgg(
window: SingleRel,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = {
mq.getUniqueKeys(window.getInput, ignoreNulls)
}
def getUniqueKeys(
join: Join,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = {
join.getJoinType match {
case JoinRelType.SEMI | JoinRelType.ANTI =>
// only return the unique keys from the LHS since a SEMI/ANTI join only
// returns the LHS
mq.getUniqueKeys(join.getLeft, ignoreNulls)
case _ =>
getJoinUniqueKeys(
join.analyzeCondition(), join.getJoinType, join.getLeft, join.getRight, mq, ignoreNulls)
}
}
def getUniqueKeys(
rel: StreamExecIntervalJoin,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = {
val joinInfo = JoinInfo.of(rel.getLeft, rel.getRight, rel.joinCondition)
getJoinUniqueKeys(joinInfo, rel.joinType, rel.getLeft, rel.getRight, mq, ignoreNulls)
}
def getUniqueKeys(
join: CommonLookupJoin,
mq: RelMetadataQuery,
ignoreNulls: Boolean): util.Set[ImmutableBitSet] = {
val left = join.getInput
val leftUniqueKeys = mq.getUniqueKeys(left, ignoreNulls)
val leftType = left.getRowType
getJoinUniqueKeys(
join.joinInfo, join.joinType, leftType, leftUniqueKeys, null,
mq.areColumnsUnique(left, join.joinInfo.leftSet, ignoreNulls),
// TODO get uniqueKeys from TableSchema of TableSource
null,
mq)
}
private def getJoinUniqueKeys(
joinInfo: JoinInfo,
joinRelType: JoinRelType,
left: RelNode,
right: RelNode,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = {
val leftUniqueKeys = mq.getUniqueKeys(left, ignoreNulls)
val rightUniqueKeys = mq.getUniqueKeys(right, ignoreNulls)
getJoinUniqueKeys(
joinInfo, joinRelType, left.getRowType, leftUniqueKeys, rightUniqueKeys,
mq.areColumnsUnique(left, joinInfo.leftSet, ignoreNulls),
mq.areColumnsUnique(right, joinInfo.rightSet, ignoreNulls),
mq)
}
private def getJoinUniqueKeys(
joinInfo: JoinInfo,
joinRelType: JoinRelType,
leftType: RelDataType,
leftUniqueKeys: JSet[ImmutableBitSet],
rightUniqueKeys: JSet[ImmutableBitSet],
isLeftUnique: JBoolean,
isRightUnique: JBoolean,
mq: RelMetadataQuery): JSet[ImmutableBitSet] = {
// first add the different combinations of concatenated unique keys
// from the left and the right, adjusting the right hand side keys to
// reflect the addition of the left hand side
//
// NOTE zfong 12/18/06 - If the number of tables in a join is large,
// the number of combinations of unique key sets will explode. If
// that is undesirable, use RelMetadataQuery.areColumnsUnique() as
// an alternative way of getting unique key information.
val retSet = new JHashSet[ImmutableBitSet]
val nFieldsOnLeft = leftType.getFieldCount
val rightSet = if (rightUniqueKeys != null) {
val res = new JHashSet[ImmutableBitSet]
rightUniqueKeys.foreach { colMask =>
val tmpMask = ImmutableBitSet.builder
colMask.foreach(bit => tmpMask.set(bit + nFieldsOnLeft))
res.add(tmpMask.build())
}
if (leftUniqueKeys != null) {
res.foreach { colMaskRight =>
leftUniqueKeys.foreach(colMaskLeft => retSet.add(colMaskLeft.union(colMaskRight)))
}
}
res
} else {
null
}
// determine if either or both the LHS and RHS are unique on the
// equi-join columns
val leftUnique = isLeftUnique
val rightUnique = isRightUnique
// if the right hand side is unique on its equijoin columns, then we can
// add the unique keys from left if the left hand side is not null
// generating
if (rightUnique != null
&& rightUnique
&& (leftUniqueKeys != null)
&& !joinRelType.generatesNullsOnLeft) {
retSet.addAll(leftUniqueKeys)
}
// same as above except left and right are reversed
if (leftUnique != null
&& leftUnique
&& (rightSet != null)
&& !joinRelType.generatesNullsOnRight) {
retSet.addAll(rightSet)
}
retSet
}
def getUniqueKeys(
rel: Correlate,
mq: RelMetadataQuery,
ignoreNulls: Boolean): util.Set[ImmutableBitSet] = null
def getUniqueKeys(
rel: BatchExecCorrelate,
mq: RelMetadataQuery,
ignoreNulls: Boolean): util.Set[ImmutableBitSet] = null
def getUniqueKeys(
rel: SetOp,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = {
if (!rel.all) {
ImmutableSet.of(ImmutableBitSet.range(rel.getRowType.getFieldCount))
} else {
ImmutableSet.of()
}
}
def getUniqueKeys(
multipleInput: MultipleInputRel,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] =
mq.getUniqueKeys(multipleInput.outputRel, ignoreNulls)
def getUniqueKeys(
subset: RelSubset,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = {
if (!Bug.CALCITE_1048_FIXED) {
//if the best node is null, so we can get the uniqueKeys based original node, due to
//the original node is logically equivalent as the rel.
val rel = Util.first(subset.getBest, subset.getOriginal)
mq.getUniqueKeys(rel, ignoreNulls)
} else {
throw new RuntimeException("CALCITE_1048 is fixed, so check this method again!")
}
}
def getUniqueKeys(
subset: HepRelVertex,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = {
mq.getUniqueKeys(subset.getCurrentRel, ignoreNulls)
}
def getUniqueKeys(
subset: WatermarkAssigner,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = {
mq.getUniqueKeys(subset.getInput, ignoreNulls)
}
// Catch-all rule when none of the others apply.
def getUniqueKeys(
rel: RelNode,
mq: RelMetadataQuery,
ignoreNulls: Boolean): JSet[ImmutableBitSet] = null
}
object FlinkRelMdUniqueKeys {
private val INSTANCE = new FlinkRelMdUniqueKeys
val SOURCE: RelMetadataProvider = ReflectiveRelMetadataProvider.reflectiveSource(
BuiltInMethod.UNIQUE_KEYS.method, INSTANCE)
}