You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@flink.apache.org by tw...@apache.org on 2017/12/01 16:36:13 UTC

flink git commit: [FLINK-8151] [table] Fix map type equality

Repository: flink
Updated Branches:
  refs/heads/master 740f711c4 -> deea4b32b


[FLINK-8151] [table] Fix map type equality

This closes #5070.


Project: http://git-wip-us.apache.org/repos/asf/flink/repo
Commit: http://git-wip-us.apache.org/repos/asf/flink/commit/deea4b32
Tree: http://git-wip-us.apache.org/repos/asf/flink/tree/deea4b32
Diff: http://git-wip-us.apache.org/repos/asf/flink/diff/deea4b32

Branch: refs/heads/master
Commit: deea4b32bab4160e2279e9f57d7272df2d91e434
Parents: 740f711
Author: Rong Rong <ro...@uber.com>
Authored: Sat Nov 25 17:17:14 2017 -0800
Committer: twalthr <tw...@apache.org>
Committed: Fri Dec 1 17:35:29 2017 +0100

----------------------------------------------------------------------
 .../table/codegen/calls/ScalarOperators.scala     |  4 ++--
 .../flink/table/expressions/collection.scala      |  4 ----
 .../flink/table/expressions/MapTypeTest.scala     | 13 +++++++++----
 .../table/expressions/utils/MapTypeTestBase.scala |  7 +++++--
 .../validation/MapTypeValidationTest.scala        | 18 +++++++++++++++++-
 5 files changed, 33 insertions(+), 13 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/flink/blob/deea4b32/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/codegen/calls/ScalarOperators.scala
----------------------------------------------------------------------
diff --git a/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/codegen/calls/ScalarOperators.scala b/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/codegen/calls/ScalarOperators.scala
index a6d77c1..7fb8eba 100644
--- a/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/codegen/calls/ScalarOperators.scala
+++ b/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/codegen/calls/ScalarOperators.scala
@@ -192,7 +192,7 @@ object ScalarOperators {
     else if (isMap(left.resultType) &&
       left.resultType.getTypeClass == right.resultType.getTypeClass) {
       generateOperatorIfNotNull(nullCheck, BOOLEAN_TYPE_INFO, left, right) {
-        (leftTerm, rightTerm) => s"java.util.Map.equals($leftTerm, $rightTerm)"
+        (leftTerm, rightTerm) => s"$leftTerm.equals($rightTerm)"
       }
     }
     // comparable types of same type
@@ -240,7 +240,7 @@ object ScalarOperators {
     else if (isMap(left.resultType) &&
       left.resultType.getTypeClass == right.resultType.getTypeClass) {
       generateOperatorIfNotNull(nullCheck, BOOLEAN_TYPE_INFO, left, right) {
-        (leftTerm, rightTerm) => s"!java.util.Map.equals($leftTerm, $rightTerm)"
+        (leftTerm, rightTerm) => s"!($leftTerm.equals($rightTerm))"
       }
     }
     // comparable types

http://git-wip-us.apache.org/repos/asf/flink/blob/deea4b32/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/expressions/collection.scala
----------------------------------------------------------------------
diff --git a/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/expressions/collection.scala b/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/expressions/collection.scala
index a3c6a54..3b65ee4 100644
--- a/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/expressions/collection.scala
+++ b/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/expressions/collection.scala
@@ -65,10 +65,6 @@ case class ArrayConstructor(elements: Seq[Expression]) extends Expression {
 case class MapConstructor(elements: Seq[Expression]) extends Expression {
   override private[flink] def children: Seq[Expression] = elements
 
-  private[flink] var mapResultType: TypeInformation[_] = new MapTypeInfo(
-    new GenericTypeInfo[AnyRef](classOf[AnyRef]),
-    new GenericTypeInfo[AnyRef](classOf[AnyRef]))
-
   override private[flink] def toRexNode(implicit relBuilder: RelBuilder): RexNode = {
     val typeFactory = relBuilder.asInstanceOf[FlinkRelBuilder].getTypeFactory
     val relDataType = typeFactory.createMapType(

http://git-wip-us.apache.org/repos/asf/flink/blob/deea4b32/flink-libraries/flink-table/src/test/scala/org/apache/flink/table/expressions/MapTypeTest.scala
----------------------------------------------------------------------
diff --git a/flink-libraries/flink-table/src/test/scala/org/apache/flink/table/expressions/MapTypeTest.scala b/flink-libraries/flink-table/src/test/scala/org/apache/flink/table/expressions/MapTypeTest.scala
index b173349..1fdfb79 100644
--- a/flink-libraries/flink-table/src/test/scala/org/apache/flink/table/expressions/MapTypeTest.scala
+++ b/flink-libraries/flink-table/src/test/scala/org/apache/flink/table/expressions/MapTypeTest.scala
@@ -141,12 +141,17 @@ class MapTypeTest extends MapTypeTestBase {
 
     // comparison
     testAllApis(
-      'f5 === 'f2.at("a"),
-      "f5 === f2.at('a')",
-      "f5 = f2['a']",
+      'f1 === 'f2,
+      "f1 === f2",
+      "f1 = f2",
+      "false")
+
+    testAllApis(
+      'f3 === 'f7,
+      "f3 === f7",
+      "f3 = f7",
       "true")
 
-    // comparison
     testAllApis(
       'f5 === 'f2.at("a"),
       "f5 === f2.at('a')",

http://git-wip-us.apache.org/repos/asf/flink/blob/deea4b32/flink-libraries/flink-table/src/test/scala/org/apache/flink/table/expressions/utils/MapTypeTestBase.scala
----------------------------------------------------------------------
diff --git a/flink-libraries/flink-table/src/test/scala/org/apache/flink/table/expressions/utils/MapTypeTestBase.scala b/flink-libraries/flink-table/src/test/scala/org/apache/flink/table/expressions/utils/MapTypeTestBase.scala
index a5df7ec..f83dd89 100644
--- a/flink-libraries/flink-table/src/test/scala/org/apache/flink/table/expressions/utils/MapTypeTestBase.scala
+++ b/flink-libraries/flink-table/src/test/scala/org/apache/flink/table/expressions/utils/MapTypeTestBase.scala
@@ -20,6 +20,7 @@ package org.apache.flink.table.expressions.utils
 
 import java.util.{HashMap => JHashMap}
 
+import com.google.common.collect.ImmutableMap
 import org.apache.flink.api.common.typeinfo.{BasicTypeInfo, TypeInformation}
 import org.apache.flink.api.java.typeutils.{MapTypeInfo, RowTypeInfo}
 import org.apache.flink.table.api.Types
@@ -34,7 +35,7 @@ class MapTypeTestBase extends ExpressionTestBase {
     val map2 = new JHashMap[Int, String]()
     map2.put(12, "a")
     map2.put(13, "b")
-    val testData = new Row(7)
+    val testData = new Row(8)
     testData.setField(0, null)
     testData.setField(1, new JHashMap[String, Int]())
     testData.setField(2, map1)
@@ -42,6 +43,7 @@ class MapTypeTestBase extends ExpressionTestBase {
     testData.setField(4, "foo")
     testData.setField(5, 12)
     testData.setField(6, Array(1.2, 1.3))
+    testData.setField(7, ImmutableMap.of(12, "a", 13, "b"))
     testData
   }
 
@@ -53,7 +55,8 @@ class MapTypeTestBase extends ExpressionTestBase {
       new MapTypeInfo(BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.STRING_TYPE_INFO),
       Types.STRING,
       Types.INT,
-      Types.OBJECT_ARRAY(Types.DOUBLE)
+      Types.OBJECT_ARRAY(Types.DOUBLE),
+      new MapTypeInfo(BasicTypeInfo.INT_TYPE_INFO, BasicTypeInfo.STRING_TYPE_INFO)
     ).asInstanceOf[TypeInformation[Any]]
   }
 }

http://git-wip-us.apache.org/repos/asf/flink/blob/deea4b32/flink-libraries/flink-table/src/test/scala/org/apache/flink/table/expressions/validation/MapTypeValidationTest.scala
----------------------------------------------------------------------
diff --git a/flink-libraries/flink-table/src/test/scala/org/apache/flink/table/expressions/validation/MapTypeValidationTest.scala b/flink-libraries/flink-table/src/test/scala/org/apache/flink/table/expressions/validation/MapTypeValidationTest.scala
index ae85b0d..8a5eaea 100644
--- a/flink-libraries/flink-table/src/test/scala/org/apache/flink/table/expressions/validation/MapTypeValidationTest.scala
+++ b/flink-libraries/flink-table/src/test/scala/org/apache/flink/table/expressions/validation/MapTypeValidationTest.scala
@@ -19,6 +19,7 @@
 package org.apache.flink.table.expressions.validation
 
 import org.apache.flink.table.api.ValidationException
+import org.apache.flink.table.api.scala._
 import org.apache.flink.table.expressions.utils.MapTypeTestBase
 import org.junit.Test
 
@@ -26,6 +27,21 @@ class MapTypeValidationTest extends MapTypeTestBase {
 
   @Test(expected = classOf[ValidationException])
   def testWrongKeyType(): Unit = {
-    testSqlApi("f2[12]", "FAIL")
+    testAllApis('f2.at(12), "f2.at(12)", "f2[12]", "FAIL")
+  }
+
+  @Test(expected = classOf[ValidationException])
+  def testIncorrectMapTypeComparison(): Unit = {
+    testAllApis('f1 === 'f3, "f1 === f3", "f1 = f3", "FAIL")
+  }
+
+  @Test(expected = classOf[ValidationException])
+  def testUnsupportedComparisonType(): Unit = {
+    testAllApis('f6 !== 'f2, "f6 !== f2", "f6 != f2", "FAIL")
+  }
+
+  @Test(expected = classOf[ValidationException])
+  def testEmptyMap(): Unit = {
+    testAllApis("FAIL", "map()", "MAP[]", "FAIL")
   }
 }