[ 
https://issues.apache.org/jira/browse/SPARK-22137?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16185527#comment-16185527
 ] 

Jia-Xuan Liu commented on SPARK-22137:
--------------------------------------

just do some testing. I'm not really sure where is the problem.

{code:java}
scala> case class UDT(`id`: Long, `features`: org.apache.spark.ml.linalg.Vector)
defined class UDT

scala> spark.sql("create table if not exists table_udt " +
     |     "(id int, features 
struct<type:tinyint,size:int,indices:array<int>,values:array<double>>)" +
     |     " row format serde 
'org.apache.hadoop.hive.ql.io.parquet.serde.ParquetHiveSerDe'"+
     |     " stored as"+
     |     " inputformat 
'org.apache.hadoop.hive.ql.io.parquet.MapredParquetInputFormat'"+
     |     " outputformat 
'org.apache.hadoop.hive.ql.io.parquet.MapredParquetOutputFormat'")

scala> spark.sql("describe table_udt").show()
+--------+--------------------+-------+
|col_name|           data_type|comment|
+--------+--------------------+-------+
|      id|                 int|   null|
|features|struct<type:tinyi...|   null|
+--------+--------------------+-------+

{code}

We can find features is a 
struct<type:tinyint,size:int,indices:array<int>,values:array<double>>.
If we create a scala dataframe stroing mllib Vector and save it as hive table, 
it will be a vector, shown as below.

{code:java}

scala> var seq = new scala.collection.mutable.ArrayBuffer[UDT]()
seq: scala.collection.mutable.ArrayBuffer[UDT] = ArrayBuffer()

scala> for (x <- 1 to 2) {
     |       seq += (new UDT(x, org.apache.spark.ml.linalg.Vectors.dense(0.2, 
0.21, 0.44)))
     | }

scala> val rowRDD = (spark.sparkContext.makeRDD[UDT](seq)).map { x => 
Row.fromSeq(Seq(x.id,x.features)) }
rowRDD: org.apache.spark.rdd.RDD[org.apache.spark.sql.Row] = 
MapPartitionsRDD[15] at map at <console>:36

scala> val schema = StructType(Array(StructField("id", 
LongType,false),StructField("features", SQLDataTypes.VectorType,false)))
schema: org.apache.spark.sql.types.StructType = 
StructType(StructField(id,LongType,false), 
StructField(features,org.apache.spark.ml.linalg.VectorUDT@3bfc3ba7,false))

scala> val df = spark.createDataFrame(rowRDD, schema)
df: org.apache.spark.sql.DataFrame = [id: bigint, features: vector]

scala> df.write.saveAsTable("test")

scala> spark.sql("select * from test").show()
+---+---------------+
| id|       features|
+---+---------------+
|  1|[0.2,0.21,0.44]|
|  2|[0.2,0.21,0.44]|
+---+---------------+

scala> spark.sql("describe test").show()
+--------+---------+-------+
|col_name|data_type|comment|
+--------+---------+-------+
|      id|   bigint|   null|
|features|   vector|   null|
+--------+---------+-------+

{code}
 
and this table can be insertInto by itself.

{code:java}

scala> val tdf = spark.table("test")
tdf: org.apache.spark.sql.DataFrame = [id: bigint, features: vector]

scala> tdf.write.insertInto("test")

scala> tdf.show()
+---+---------------+
| id|       features|
+---+---------------+
|  1|[0.2,0.21,0.44]|
|  2|[0.2,0.21,0.44]|
|  1|[0.2,0.21,0.44]|
|  2|[0.2,0.21,0.44]|
+---+---------------+

{code}

and I also try to create table with vector type but it's fail.
Maybe vector type isn't a public type.

{code:java}

scala> spark.sql("create table if not exists table_udt " +
     | "(id int, features vector)" +
     | " row format serde 
'org.apache.hadoop.hive.ql.io.parquet.serde.ParquetHiveSerDe'"+
     | " stored as"+
     | " inputformat 
'org.apache.hadoop.hive.ql.io.parquet.MapredParquetInputFormat'"+
     | " outputformat 
'org.apache.hadoop.hive.ql.io.parquet.MapredParquetOutputFormat'")
org.apache.spark.sql.catalyst.parser.ParseException:
DataType vector is not supported.(line 1, pos 54)

{code}


> Failed to insert VectorUDT to hive table with 
> DataFrameWriter.insertInto(tableName: String)
> -------------------------------------------------------------------------------------------
>
>                 Key: SPARK-22137
>                 URL: https://issues.apache.org/jira/browse/SPARK-22137
>             Project: Spark
>          Issue Type: Bug
>          Components: SQL
>    Affects Versions: 2.1.1
>            Reporter: yzheng616
>
> Failed to insert VectorUDT to hive table with 
> DataFrameWriter.insertInto(tableName: String). The issue seems similar with 
> SPARK-17765 which have been resolved in 2.1.0. 
> Error message: 
> {color:red}Exception in thread "main" org.apache.spark.sql.AnalysisException: 
> cannot resolve '`features`' due to data type mismatch: cannot cast 
> org.apache.spark.ml.linalg.VectorUDT@3bfc3ba7 to 
> StructType(StructField(type,ByteType,true), 
> StructField(size,IntegerType,true), 
> StructField(indices,ArrayType(IntegerType,true),true), 
> StructField(values,ArrayType(DoubleType,true),true));;
> 'InsertIntoTable Relation[id#21,features#22] parquet, 
> OverwriteOptions(false,Map()), false
> +- 'Project [cast(id#13L as int) AS id#27, cast(features#14 as 
> struct<type:tinyint,size:int,indices:array<int>,values:array<double>>) AS 
> features#28]
>    +- LogicalRDD [id#13L, features#14]{color}
> Reproduce code:
> {code:java}
> import scala.annotation.varargs
> import org.apache.spark.ml.linalg.SQLDataTypes
> import org.apache.spark.sql.Row
> import org.apache.spark.sql.SparkSession
> import org.apache.spark.sql.types.LongType
> import org.apache.spark.sql.types.StructField
> import org.apache.spark.sql.types.StructType
> case class UDT(`id`: Long, `features`: org.apache.spark.ml.linalg.Vector)
> object UDTTest {
>   def main(args: Array[String]): Unit = {
>     val tb = "table_udt"
>     val spark = 
> SparkSession.builder().master("local[4]").appName("UDTInsertInto").enableHiveSupport().getOrCreate()
>     spark.sql("drop table if exists " + tb)
>     
>     /*
>      * VectorUDT sql type definition:
>      * 
>      *   override def sqlType: StructType = {
>      *   StructType(Seq(
>      *        StructField("type", ByteType, nullable = false),
>      *        StructField("size", IntegerType, nullable = true),
>      *        StructField("indices", ArrayType(IntegerType, containsNull = 
> false), nullable = true),
>      *        StructField("values", ArrayType(DoubleType, containsNull = 
> false), nullable = true)))
>      *   }
>     */
>     
>     //Create Hive table base on VectorUDT sql type
>     spark.sql("create table if not exists "+tb+"(id int, features 
> struct<type:tinyint,size:int,indices:array<int>,values:array<double>>)" +
>       " row format serde 
> 'org.apache.hadoop.hive.ql.io.parquet.serde.ParquetHiveSerDe'"+
>       " stored as"+
>         " inputformat 
> 'org.apache.hadoop.hive.ql.io.parquet.MapredParquetInputFormat'"+
>         " outputformat 
> 'org.apache.hadoop.hive.ql.io.parquet.MapredParquetOutputFormat'")
>     var seq = new scala.collection.mutable.ArrayBuffer[UDT]()
>     for (x <- 1 to 2) {
>       seq += (new UDT(x, org.apache.spark.ml.linalg.Vectors.dense(0.2, 0.21, 
> 0.44)))
>     }
>     val rowRDD = (spark.sparkContext.makeRDD[UDT](seq)).map { x => 
> Row.fromSeq(Seq(x.id,x.features)) }
>     val schema = StructType(Array(StructField("id", 
> LongType,false),StructField("features", SQLDataTypes.VectorType,false)))
>     val df = spark.createDataFrame(rowRDD, schema)
>      
>     //insert into hive table
>     df.write.insertInto(tb)
>   }
> }
> {code}



--
This message was sent by Atlassian JIRA
(v6.4.14#64029)

---------------------------------------------------------------------
To unsubscribe, e-mail: issues-unsubscr...@spark.apache.org
For additional commands, e-mail: issues-h...@spark.apache.org

Reply via email to