Skip to content
Closed
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -19,6 +19,8 @@ package org.apache.spark.sql.hive.execution

import scala.collection.JavaConversions._

import java.util.{HashMap => JHashMap}

import org.apache.hadoop.hive.common.`type`.{HiveDecimal, HiveVarchar}
import org.apache.hadoop.hive.metastore.MetaStoreUtils
import org.apache.hadoop.hive.ql.Context
Expand Down Expand Up @@ -88,6 +90,12 @@ case class InsertIntoHiveTable(
val wrappedSeq = s.map(wrap(_, oi.getListElementObjectInspector))
seqAsJavaList(wrappedSeq)

case (m: Map[_, _], oi: MapObjectInspector) =>
val keyOi = oi.getMapKeyObjectInspector
val valueOi = oi.getMapValueObjectInspector
val wrappedMap = m.map { case (key, value) => wrap(key, keyOi) -> wrap(value, valueOi) }
mapAsJavaMap(wrappedMap)

case (obj, _) =>
obj
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -228,15 +228,15 @@ class HiveQuerySuite extends HiveComparisonTest {
val fixture = List(("foo", 2), ("bar", 1), ("foo", 4), ("bar", 3))
.zipWithIndex.map {case Pair(Pair(value, attr), key) => HavingRow(key, value, attr)}
TestHive.sparkContext.parallelize(fixture).registerAsTable("having_test")
val results =
val results =
hql("SELECT value, max(attr) AS attr FROM having_test GROUP BY value HAVING attr > 3")
.collect()
.map(x => Pair(x.getString(0), x.getInt(1)))

assert(results === Array(Pair("foo", 4)))
TestHive.reset()
}

test("SPARK-2180: HAVING with non-boolean clause raises no exceptions") {
hql("select key, count(*) c from src group by key having c").collect()
}
Expand Down Expand Up @@ -370,6 +370,16 @@ class HiveQuerySuite extends HiveComparisonTest {
}
}

test("SPARK-2263: Insert Map<K, V> values") {
hql("CREATE TABLE m(value MAP<INT, STRING>)")
hql("INSERT OVERWRITE TABLE m SELECT MAP(key, value) FROM src LIMIT 10")
hql("SELECT * FROM m").collect().zip(hql("SELECT * FROM src LIMIT 10").collect()).map {
case (Row(map: Map[Int, String]), Row(key: Int, value: String)) =>
assert(map.size === 1)
assert(map.head === (key, value))
}
}

test("parse HQL set commands") {
// Adapted from its SQL counterpart.
val testKey = "spark.sql.key.usedfortestonly"
Expand Down Expand Up @@ -460,7 +470,6 @@ class HiveQuerySuite extends HiveComparisonTest {

// Put tests that depend on specific Hive settings before these last two test,
// since they modify /clear stuff.

}

// for SPARK-2180 test
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -122,6 +122,3 @@ class PairUdf extends GenericUDF {

override def getDisplayString(p1: Array[String]): String = ""
}