|
17 | 17 |
|
18 | 18 | package org.apache.spark.sql.catalyst.expressions; |
19 | 19 |
|
| 20 | +import java.io.Externalizable; |
| 21 | +import java.io.IOException; |
| 22 | +import java.io.ObjectInput; |
| 23 | +import java.io.ObjectOutput; |
20 | 24 | import java.math.BigDecimal; |
21 | 25 | import java.math.BigInteger; |
22 | 26 | import java.nio.ByteBuffer; |
|
30 | 34 | import org.apache.spark.unsafe.types.CalendarInterval; |
31 | 35 | import org.apache.spark.unsafe.types.UTF8String; |
32 | 36 |
|
| 37 | +import static org.apache.spark.unsafe.Platform.BYTE_ARRAY_OFFSET; |
| 38 | + |
33 | 39 | /** |
34 | 40 | * An Unsafe implementation of Array which is backed by raw memory instead of Java objects. |
35 | 41 | * |
|
52 | 58 | * Instances of `UnsafeArrayData` act as pointers to row data stored in this format. |
53 | 59 | */ |
54 | 60 |
|
55 | | -public final class UnsafeArrayData extends ArrayData { |
| 61 | +public final class UnsafeArrayData extends ArrayData implements Externalizable { |
56 | 62 | public static int calculateHeaderPortionInBytes(int numFields) { |
57 | 63 | return (int)calculateHeaderPortionInBytes((long)numFields); |
58 | 64 | } |
@@ -485,4 +491,35 @@ public static UnsafeArrayData fromPrimitiveArray(float[] arr) { |
485 | 491 | public static UnsafeArrayData fromPrimitiveArray(double[] arr) { |
486 | 492 | return fromPrimitiveArray(arr, Platform.DOUBLE_ARRAY_OFFSET, arr.length, 8); |
487 | 493 | } |
| 494 | + |
| 495 | + |
| 496 | + public byte[] getBytes() { |
| 497 | + if (baseObject instanceof byte[] |
| 498 | + && baseOffset == Platform.BYTE_ARRAY_OFFSET |
| 499 | + && (((byte[]) baseObject).length == sizeInBytes)) { |
| 500 | + return (byte[]) baseObject; |
| 501 | + } else { |
| 502 | + byte[] bytes = new byte[sizeInBytes]; |
| 503 | + Platform.copyMemory(baseObject, baseOffset, bytes, Platform.BYTE_ARRAY_OFFSET, sizeInBytes); |
| 504 | + return bytes; |
| 505 | + } |
| 506 | + } |
| 507 | + |
| 508 | + @Override |
| 509 | + public void writeExternal(ObjectOutput out) throws IOException { |
| 510 | + byte[] bytes = getBytes(); |
| 511 | + out.writeInt(bytes.length); |
| 512 | + out.writeInt(this.numElements); |
| 513 | + out.write(bytes); |
| 514 | + } |
| 515 | + |
| 516 | + @Override |
| 517 | + public void readExternal(ObjectInput in) throws IOException, ClassNotFoundException { |
| 518 | + this.baseOffset = BYTE_ARRAY_OFFSET; |
| 519 | + this.sizeInBytes = in.readInt(); |
| 520 | + this.numElements = in.readInt(); |
| 521 | + this.elementOffset = baseOffset + calculateHeaderPortionInBytes(this.numElements); |
| 522 | + this.baseObject = new byte[sizeInBytes]; |
| 523 | + in.readFully((byte[]) baseObject); |
| 524 | + } |
488 | 525 | } |
0 commit comments