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 @@ -188,6 +188,8 @@ public Object get(int ordinal, DataType dataType) {
return getInt(ordinal);
} else if (dataType instanceof TimestampType) {
return getLong(ordinal);
} else if (dataType instanceof TimestampNTZType) {
return getLong(ordinal);
} else if (dataType instanceof ArrayType) {
return getArray(ordinal);
} else if (dataType instanceof StructType) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -82,6 +82,8 @@ public InternalRow copy() {
row.setInt(i, getInt(i));
} else if (dt instanceof TimestampType) {
row.setLong(i, getLong(i));
} else if (dt instanceof TimestampNTZType) {
row.setLong(i, getLong(i));
} else if (dt instanceof StructType) {
row.update(i, getStruct(i, ((StructType) dt).fields().length).copy());
} else if (dt instanceof ArrayType) {
Expand Down Expand Up @@ -191,6 +193,8 @@ public Object get(int ordinal, DataType dataType) {
return getInt(ordinal);
} else if (dataType instanceof TimestampType) {
return getLong(ordinal);
} else if (dataType instanceof TimestampNTZType) {
return getLong(ordinal);
} else if (dataType instanceof ArrayType) {
return getArray(ordinal);
} else if (dataType instanceof StructType structType) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -274,6 +274,19 @@ class ColumnVectorSuite extends SparkFunSuite with SQLHelper {
}
}

testVectors("mutable ColumnarRow with TimestampNTZType", 10, TimestampNTZType) { testVector =>
val mutableRow = new MutableColumnarRow(Array(testVector))
(0 until 10).foreach { i =>
mutableRow.rowId = i
mutableRow.setLong(0, 10 - i)
}
(0 until 10).foreach { i =>
mutableRow.rowId = i
assert(mutableRow.get(0, TimestampNTZType) === (10 - i))
assert(mutableRow.copy().get(0, TimestampNTZType) === (10 - i))
}
}

val arrayType: ArrayType = ArrayType(IntegerType, containsNull = true)
testVectors("array", 10, arrayType) { testVector =>

Expand Down Expand Up @@ -384,18 +397,24 @@ class ColumnVectorSuite extends SparkFunSuite with SQLHelper {
}

val structType: StructType = new StructType().add("int", IntegerType).add("double", DoubleType)
.add("ts", TimestampNTZType)
testVectors("struct", 10, structType) { testVector =>
val c1 = testVector.getChild(0)
val c2 = testVector.getChild(1)
val c3 = testVector.getChild(2)
c1.putInt(0, 123)
c2.putDouble(0, 3.45)
c3.putLong(0, 1000L)
c1.putInt(1, 456)
c2.putDouble(1, 5.67)
c3.putLong(1, 2000L)

assert(testVector.getStruct(0).get(0, IntegerType) === 123)
assert(testVector.getStruct(0).get(1, DoubleType) === 3.45)
assert(testVector.getStruct(0).get(2, TimestampNTZType) === 1000L)
assert(testVector.getStruct(1).get(0, IntegerType) === 456)
assert(testVector.getStruct(1).get(1, DoubleType) === 5.67)
assert(testVector.getStruct(1).get(2, TimestampNTZType) === 2000L)
}

testVectors("SPARK-44805: getInts with dictionary", 3, IntegerType) { testVector =>
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -515,4 +515,28 @@ class ArrowColumnVectorSuite extends SparkFunSuite {
columnVector.close()
allocator.close()
}

test("struct with TimestampNTZType") {
val allocator = ArrowUtils.rootAllocator.newChildAllocator("struct", 0, Long.MaxValue)
val schema = new StructType().add("ts", TimestampNTZType)
val vector = ArrowUtils.toArrowField("struct", schema, nullable = true, null)
.createVector(allocator).asInstanceOf[StructVector]
vector.allocateNew()
val timestampVector = vector.getChildByOrdinal(0).asInstanceOf[TimeStampMicroVector]

vector.setIndexDefined(0)
timestampVector.setSafe(0, 1000L)

timestampVector.setValueCount(1)
vector.setValueCount(1)

val columnVector = new ArrowColumnVector(vector)
assert(columnVector.dataType === schema)

val row0 = columnVector.getStruct(0)
assert(row0.get(0, TimestampNTZType) === 1000L)

columnVector.close()
allocator.close()
}
}
Loading