blob: ac202004982c4147266e94e9db8086eb2b1ae511 [file] [log] [blame]
/*
* 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.plan.util
import com.google.common.collect.Sets
import org.apache.calcite.rel.RelNode
import scala.collection.JavaConversions._
/**
* Rewrite same rel object to different rel objects.
*
* <p>e.g.
* {{{
* Join Join
* / \ / \
* Filter1 Filter2 => Filter1 Filter2
* \ / | |
* Scan Scan1 Scan2
* }}}
* After rewrote, Scan1 and Scan2 are different object but have same digest.
*/
class SameRelObjectShuttle extends DefaultRelShuttle {
private val visitedNodes = Sets.newIdentityHashSet[RelNode]()
override def visit(node: RelNode): RelNode = {
val visited = !visitedNodes.add(node)
var change = false
val newInputs = node.getInputs.map {
input =>
val newInput = input.accept(this)
change = change || (input ne newInput)
newInput
}
if (change || visited) {
node.copy(node.getTraitSet, newInputs)
} else {
node
}
}
}