Skip to content

Commit

Permalink
[KYUUBI apache#4666] Support flink varbinary type in query operation
Browse files Browse the repository at this point in the history
### _Why are the changes needed?_
closed apache#1770
Support flink `varbinary` type in query operation

### _How was this patch tested?_
- [x] Add some test cases that check the changes thoroughly including negative and positive cases if possible

- [ ] Add screenshots for manual tests if appropriate

- [x] [Run test](https://kyuubi.readthedocs.io/en/master/develop_tools/testing.html#running-tests) locally before make a pull request

Closes apache#4666 from yuruguo/support-flink-varbinary-type.

Closes apache#4666

e05675e [Ruguo Yu] Support flink varbinary type in query operation

Authored-by: Ruguo Yu <jiang7chengzitc@163.com>
Signed-off-by: Cheng Pan <chengpan@apache.org>
  • Loading branch information
yuruguo authored and pan3793 committed Apr 6, 2023
1 parent 015b800 commit 1241a38
Show file tree
Hide file tree
Showing 3 changed files with 13 additions and 2 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -177,7 +177,7 @@ class ExecuteStatement(
row.setField(i, d.toObjectArray(arrayType.getElementType))
case _ =>
}
case _: BinaryType =>
case _: BinaryType | _: VarBinaryType =>
row.setField(i, r.getBinary(i))
case _: BigIntType =>
row.setField(i, r.getLong(i))
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -307,6 +307,7 @@ object RowSet {
case _: MapType => TTypeId.MAP_TYPE
case _: RowType => TTypeId.STRUCT_TYPE
case _: BinaryType => TTypeId.BINARY_TYPE
case _: VarBinaryType => TTypeId.BINARY_TYPE
case _: TimeType => TTypeId.STRING_TYPE
case t @ (_: ZonedTimestampType | _: LocalZonedTimestampType | _: MultisetType |
_: YearMonthIntervalType | _: DayTimeIntervalType) =>
Expand Down Expand Up @@ -369,7 +370,7 @@ object RowSet {
// Only match string in nested type values
"\"" + s + "\""

case (bin: Array[Byte], _: BinaryType) =>
case (bin: Array[Byte], _ @(_: BinaryType | _: VarBinaryType)) =>
new String(bin, StandardCharsets.UTF_8)

case (other, _) =>
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -812,6 +812,16 @@ class FlinkOperationSuite extends WithFlinkSQLEngine with HiveJDBCTestHelper {
}
}

test("execute statement - select varbinary") {
withJdbcStatement() { statement =>
val resultSet = statement.executeQuery("select cast('kyuubi' as varbinary)")
assert(resultSet.next())
assert(resultSet.getString(1) == "kyuubi")
val metaData = resultSet.getMetaData
assert(metaData.getColumnType(1) === java.sql.Types.BINARY)
}
}

test("execute statement - select float") {
withJdbcStatement() { statement =>
val resultSet = statement.executeQuery("SELECT cast(0.1 as float)")
Expand Down

0 comments on commit 1241a38

Please sign in to comment.