[SPARK-9815] Rename PlatformDependent.UNSAFE -> Platform.

PlatformDependent.UNSAFE is way too verbose.

Author: Reynold Xin <r...@databricks.com>

Closes #8094 from rxin/SPARK-9815 and squashes the following commits:

229b603 [Reynold Xin] [SPARK-9815] Rename PlatformDependent.UNSAFE -> Platform.


Project: http://git-wip-us.apache.org/repos/asf/spark/repo
Commit: http://git-wip-us.apache.org/repos/asf/spark/commit/d378396f
Tree: http://git-wip-us.apache.org/repos/asf/spark/tree/d378396f
Diff: http://git-wip-us.apache.org/repos/asf/spark/diff/d378396f

Branch: refs/heads/master
Commit: d378396f86f625f006738d87fe5dbc2ff8fd913d
Parents: 600031e
Author: Reynold Xin <r...@databricks.com>
Authored: Tue Aug 11 08:41:06 2015 -0700
Committer: Davies Liu <davies....@gmail.com>
Committed: Tue Aug 11 08:41:06 2015 -0700

----------------------------------------------------------------------
 .../serializer/DummySerializerInstance.java     |   6 +-
 .../unsafe/UnsafeShuffleExternalSorter.java     |  22 +--
 .../shuffle/unsafe/UnsafeShuffleWriter.java     |   4 +-
 .../spark/unsafe/map/BytesToBytesMap.java       |  20 +-
 .../unsafe/sort/PrefixComparators.java          |   5 +-
 .../unsafe/sort/UnsafeExternalSorter.java       |  22 +--
 .../unsafe/sort/UnsafeInMemorySorter.java       |   4 +-
 .../unsafe/sort/UnsafeSorterSpillReader.java    |   4 +-
 .../unsafe/sort/UnsafeSorterSpillWriter.java    |   6 +-
 .../UnsafeShuffleInMemorySorterSuite.java       |  20 +-
 .../map/AbstractBytesToBytesMapSuite.java       |  94 +++++-----
 .../unsafe/sort/UnsafeExternalSorterSuite.java  |  20 +-
 .../unsafe/sort/UnsafeInMemorySorterSuite.java  |  20 +-
 .../catalyst/expressions/UnsafeArrayData.java   |  51 ++---
 .../sql/catalyst/expressions/UnsafeReaders.java |   8 +-
 .../sql/catalyst/expressions/UnsafeRow.java     | 108 +++++------
 .../catalyst/expressions/UnsafeRowWriters.java  |  41 ++--
 .../sql/catalyst/expressions/UnsafeWriters.java |  43 ++---
 .../sql/execution/UnsafeExternalRowSorter.java  |   4 +-
 .../expressions/codegen/CodeGenerator.scala     |   4 +-
 .../codegen/GenerateUnsafeProjection.scala      |  32 ++--
 .../codegen/GenerateUnsafeRowJoiner.scala       |  16 +-
 .../catalyst/expressions/stringOperations.scala |   4 +-
 .../GenerateUnsafeRowJoinerBitsetSuite.scala    |   4 +-
 .../UnsafeFixedWidthAggregationMap.java         |   4 +-
 .../sql/execution/UnsafeKVExternalSorter.java   |   4 +-
 .../sql/execution/UnsafeRowSerializer.scala     |   6 +-
 .../sql/execution/joins/HashedRelation.scala    |  13 +-
 .../org/apache/spark/sql/UnsafeRowSuite.scala   |   4 +-
 .../java/org/apache/spark/unsafe/Platform.java  | 173 +++++++++++++++++
 .../apache/spark/unsafe/PlatformDependent.java  | 187 -------------------
 .../spark/unsafe/array/ByteArrayMethods.java    |  14 +-
 .../apache/spark/unsafe/array/LongArray.java    |   6 +-
 .../spark/unsafe/bitset/BitSetMethods.java      |  19 +-
 .../spark/unsafe/hash/Murmur3_x86_32.java       |   4 +-
 .../apache/spark/unsafe/memory/MemoryBlock.java |   4 +-
 .../unsafe/memory/UnsafeMemoryAllocator.java    |   6 +-
 .../apache/spark/unsafe/types/ByteArray.java    |  10 +-
 .../apache/spark/unsafe/types/UTF8String.java   |  30 ++-
 .../spark/unsafe/hash/Murmur3_x86_32Suite.java  |  14 +-
 40 files changed, 466 insertions(+), 594 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/spark/blob/d378396f/core/src/main/java/org/apache/spark/serializer/DummySerializerInstance.java
----------------------------------------------------------------------
diff --git 
a/core/src/main/java/org/apache/spark/serializer/DummySerializerInstance.java 
b/core/src/main/java/org/apache/spark/serializer/DummySerializerInstance.java
index 0399abc..0e58bb4 100644
--- 
a/core/src/main/java/org/apache/spark/serializer/DummySerializerInstance.java
+++ 
b/core/src/main/java/org/apache/spark/serializer/DummySerializerInstance.java
@@ -25,7 +25,7 @@ import java.nio.ByteBuffer;
 import scala.reflect.ClassTag;
 
 import org.apache.spark.annotation.Private;
-import org.apache.spark.unsafe.PlatformDependent;
+import org.apache.spark.unsafe.Platform;
 
 /**
  * Unfortunately, we need a serializer instance in order to construct a 
DiskBlockObjectWriter.
@@ -49,7 +49,7 @@ public final class DummySerializerInstance extends 
SerializerInstance {
         try {
           s.flush();
         } catch (IOException e) {
-          PlatformDependent.throwException(e);
+          Platform.throwException(e);
         }
       }
 
@@ -64,7 +64,7 @@ public final class DummySerializerInstance extends 
SerializerInstance {
         try {
           s.close();
         } catch (IOException e) {
-          PlatformDependent.throwException(e);
+          Platform.throwException(e);
         }
       }
     };

http://git-wip-us.apache.org/repos/asf/spark/blob/d378396f/core/src/main/java/org/apache/spark/shuffle/unsafe/UnsafeShuffleExternalSorter.java
----------------------------------------------------------------------
diff --git 
a/core/src/main/java/org/apache/spark/shuffle/unsafe/UnsafeShuffleExternalSorter.java
 
b/core/src/main/java/org/apache/spark/shuffle/unsafe/UnsafeShuffleExternalSorter.java
index 925b60a..3d1ef0c 100644
--- 
a/core/src/main/java/org/apache/spark/shuffle/unsafe/UnsafeShuffleExternalSorter.java
+++ 
b/core/src/main/java/org/apache/spark/shuffle/unsafe/UnsafeShuffleExternalSorter.java
@@ -37,7 +37,7 @@ import org.apache.spark.shuffle.ShuffleMemoryManager;
 import org.apache.spark.storage.BlockManager;
 import org.apache.spark.storage.DiskBlockObjectWriter;
 import org.apache.spark.storage.TempShuffleBlockId;
-import org.apache.spark.unsafe.PlatformDependent;
+import org.apache.spark.unsafe.Platform;
 import org.apache.spark.unsafe.array.ByteArrayMethods;
 import org.apache.spark.unsafe.memory.MemoryBlock;
 import org.apache.spark.unsafe.memory.TaskMemoryManager;
@@ -211,16 +211,12 @@ final class UnsafeShuffleExternalSorter {
       final long recordPointer = 
sortedRecords.packedRecordPointer.getRecordPointer();
       final Object recordPage = taskMemoryManager.getPage(recordPointer);
       final long recordOffsetInPage = 
taskMemoryManager.getOffsetInPage(recordPointer);
-      int dataRemaining = PlatformDependent.UNSAFE.getInt(recordPage, 
recordOffsetInPage);
+      int dataRemaining = Platform.getInt(recordPage, recordOffsetInPage);
       long recordReadPosition = recordOffsetInPage + 4; // skip over record 
length
       while (dataRemaining > 0) {
         final int toTransfer = Math.min(DISK_WRITE_BUFFER_SIZE, dataRemaining);
-        PlatformDependent.copyMemory(
-          recordPage,
-          recordReadPosition,
-          writeBuffer,
-          PlatformDependent.BYTE_ARRAY_OFFSET,
-          toTransfer);
+        Platform.copyMemory(
+          recordPage, recordReadPosition, writeBuffer, 
Platform.BYTE_ARRAY_OFFSET, toTransfer);
         writer.write(writeBuffer, 0, toTransfer);
         recordReadPosition += toTransfer;
         dataRemaining -= toTransfer;
@@ -447,14 +443,10 @@ final class UnsafeShuffleExternalSorter {
 
     final long recordAddress =
       taskMemoryManager.encodePageNumberAndOffset(dataPage, dataPagePosition);
-    PlatformDependent.UNSAFE.putInt(dataPageBaseObject, dataPagePosition, 
lengthInBytes);
+    Platform.putInt(dataPageBaseObject, dataPagePosition, lengthInBytes);
     dataPagePosition += 4;
-    PlatformDependent.copyMemory(
-      recordBaseObject,
-      recordBaseOffset,
-      dataPageBaseObject,
-      dataPagePosition,
-      lengthInBytes);
+    Platform.copyMemory(
+      recordBaseObject, recordBaseOffset, dataPageBaseObject, 
dataPagePosition, lengthInBytes);
     assert(inMemSorter != null);
     inMemSorter.insertRecord(recordAddress, partitionId);
   }

http://git-wip-us.apache.org/repos/asf/spark/blob/d378396f/core/src/main/java/org/apache/spark/shuffle/unsafe/UnsafeShuffleWriter.java
----------------------------------------------------------------------
diff --git 
a/core/src/main/java/org/apache/spark/shuffle/unsafe/UnsafeShuffleWriter.java 
b/core/src/main/java/org/apache/spark/shuffle/unsafe/UnsafeShuffleWriter.java
index 02084f9..2389c28 100644
--- 
a/core/src/main/java/org/apache/spark/shuffle/unsafe/UnsafeShuffleWriter.java
+++ 
b/core/src/main/java/org/apache/spark/shuffle/unsafe/UnsafeShuffleWriter.java
@@ -53,7 +53,7 @@ import org.apache.spark.shuffle.ShuffleMemoryManager;
 import org.apache.spark.shuffle.ShuffleWriter;
 import org.apache.spark.storage.BlockManager;
 import org.apache.spark.storage.TimeTrackingOutputStream;
-import org.apache.spark.unsafe.PlatformDependent;
+import org.apache.spark.unsafe.Platform;
 import org.apache.spark.unsafe.memory.TaskMemoryManager;
 
 @Private
@@ -244,7 +244,7 @@ public class UnsafeShuffleWriter<K, V> extends 
ShuffleWriter<K, V> {
     assert (serializedRecordSize > 0);
 
     sorter.insertRecord(
-      serBuffer.getBuf(), PlatformDependent.BYTE_ARRAY_OFFSET, 
serializedRecordSize, partitionId);
+      serBuffer.getBuf(), Platform.BYTE_ARRAY_OFFSET, serializedRecordSize, 
partitionId);
   }
 
   @VisibleForTesting

http://git-wip-us.apache.org/repos/asf/spark/blob/d378396f/core/src/main/java/org/apache/spark/unsafe/map/BytesToBytesMap.java
----------------------------------------------------------------------
diff --git 
a/core/src/main/java/org/apache/spark/unsafe/map/BytesToBytesMap.java 
b/core/src/main/java/org/apache/spark/unsafe/map/BytesToBytesMap.java
index 7f79cd1..85b46ec 100644
--- a/core/src/main/java/org/apache/spark/unsafe/map/BytesToBytesMap.java
+++ b/core/src/main/java/org/apache/spark/unsafe/map/BytesToBytesMap.java
@@ -270,10 +270,10 @@ public final class BytesToBytesMap {
 
     @Override
     public Location next() {
-      int totalLength = PlatformDependent.UNSAFE.getInt(pageBaseObject, 
offsetInPage);
+      int totalLength = Platform.getInt(pageBaseObject, offsetInPage);
       if (totalLength == END_OF_PAGE_MARKER) {
         advanceToNextPage();
-        totalLength = PlatformDependent.UNSAFE.getInt(pageBaseObject, 
offsetInPage);
+        totalLength = Platform.getInt(pageBaseObject, offsetInPage);
       }
       loc.with(currentPage, offsetInPage);
       offsetInPage += 4 + totalLength;
@@ -402,9 +402,9 @@ public final class BytesToBytesMap {
 
     private void updateAddressesAndSizes(final Object page, final long 
offsetInPage) {
       long position = offsetInPage;
-      final int totalLength = PlatformDependent.UNSAFE.getInt(page, position);
+      final int totalLength = Platform.getInt(page, position);
       position += 4;
-      keyLength = PlatformDependent.UNSAFE.getInt(page, position);
+      keyLength = Platform.getInt(page, position);
       position += 4;
       valueLength = totalLength - keyLength - 4;
 
@@ -572,7 +572,7 @@ public final class BytesToBytesMap {
           // There wasn't enough space in the current page, so write an 
end-of-page marker:
           final Object pageBaseObject = currentDataPage.getBaseObject();
           final long lengthOffsetInPage = currentDataPage.getBaseOffset() + 
pageCursor;
-          PlatformDependent.UNSAFE.putInt(pageBaseObject, lengthOffsetInPage, 
END_OF_PAGE_MARKER);
+          Platform.putInt(pageBaseObject, lengthOffsetInPage, 
END_OF_PAGE_MARKER);
         }
         final long memoryGranted = 
shuffleMemoryManager.tryToAcquire(pageSizeBytes);
         if (memoryGranted != pageSizeBytes) {
@@ -608,21 +608,21 @@ public final class BytesToBytesMap {
       final long valueDataOffsetInPage = insertCursor;
       insertCursor += valueLengthBytes; // word used to store the value size
 
-      PlatformDependent.UNSAFE.putInt(dataPageBaseObject, recordOffset,
+      Platform.putInt(dataPageBaseObject, recordOffset,
         keyLengthBytes + valueLengthBytes + 4);
-      PlatformDependent.UNSAFE.putInt(dataPageBaseObject, keyLengthOffset, 
keyLengthBytes);
+      Platform.putInt(dataPageBaseObject, keyLengthOffset, keyLengthBytes);
       // Copy the key
-      PlatformDependent.copyMemory(
+      Platform.copyMemory(
         keyBaseObject, keyBaseOffset, dataPageBaseObject, keyDataOffsetInPage, 
keyLengthBytes);
       // Copy the value
-      PlatformDependent.copyMemory(valueBaseObject, valueBaseOffset, 
dataPageBaseObject,
+      Platform.copyMemory(valueBaseObject, valueBaseOffset, dataPageBaseObject,
         valueDataOffsetInPage, valueLengthBytes);
 
       // --- Update bookeeping data structures 
-----------------------------------------------------
 
       if (useOverflowPage) {
         // Store the end-of-page marker at the end of the data page
-        PlatformDependent.UNSAFE.putInt(dataPageBaseObject, insertCursor, 
END_OF_PAGE_MARKER);
+        Platform.putInt(dataPageBaseObject, insertCursor, END_OF_PAGE_MARKER);
       } else {
         pageCursor += requiredSize;
       }

http://git-wip-us.apache.org/repos/asf/spark/blob/d378396f/core/src/main/java/org/apache/spark/util/collection/unsafe/sort/PrefixComparators.java
----------------------------------------------------------------------
diff --git 
a/core/src/main/java/org/apache/spark/util/collection/unsafe/sort/PrefixComparators.java
 
b/core/src/main/java/org/apache/spark/util/collection/unsafe/sort/PrefixComparators.java
index 5e002ae..71b76d5 100644
--- 
a/core/src/main/java/org/apache/spark/util/collection/unsafe/sort/PrefixComparators.java
+++ 
b/core/src/main/java/org/apache/spark/util/collection/unsafe/sort/PrefixComparators.java
@@ -20,10 +20,9 @@ package org.apache.spark.util.collection.unsafe.sort;
 import com.google.common.primitives.UnsignedLongs;
 
 import org.apache.spark.annotation.Private;
-import org.apache.spark.unsafe.PlatformDependent;
+import org.apache.spark.unsafe.Platform;
 import org.apache.spark.unsafe.types.UTF8String;
 import org.apache.spark.util.Utils;
-import static org.apache.spark.unsafe.PlatformDependent.BYTE_ARRAY_OFFSET;
 
 @Private
 public class PrefixComparators {
@@ -73,7 +72,7 @@ public class PrefixComparators {
         final int minLen = Math.min(bytes.length, 8);
         long p = 0;
         for (int i = 0; i < minLen; ++i) {
-          p |= (128L + PlatformDependent.UNSAFE.getByte(bytes, 
BYTE_ARRAY_OFFSET + i))
+          p |= (128L + Platform.getByte(bytes, Platform.BYTE_ARRAY_OFFSET + i))
               << (56 - 8 * i);
         }
         return p;

http://git-wip-us.apache.org/repos/asf/spark/blob/d378396f/core/src/main/java/org/apache/spark/util/collection/unsafe/sort/UnsafeExternalSorter.java
----------------------------------------------------------------------
diff --git 
a/core/src/main/java/org/apache/spark/util/collection/unsafe/sort/UnsafeExternalSorter.java
 
b/core/src/main/java/org/apache/spark/util/collection/unsafe/sort/UnsafeExternalSorter.java
index 5ebbf9b..9601aaf 100644
--- 
a/core/src/main/java/org/apache/spark/util/collection/unsafe/sort/UnsafeExternalSorter.java
+++ 
b/core/src/main/java/org/apache/spark/util/collection/unsafe/sort/UnsafeExternalSorter.java
@@ -35,7 +35,7 @@ import org.apache.spark.executor.ShuffleWriteMetrics;
 import org.apache.spark.shuffle.ShuffleMemoryManager;
 import org.apache.spark.storage.BlockManager;
 import org.apache.spark.unsafe.array.ByteArrayMethods;
-import org.apache.spark.unsafe.PlatformDependent;
+import org.apache.spark.unsafe.Platform;
 import org.apache.spark.unsafe.memory.MemoryBlock;
 import org.apache.spark.unsafe.memory.TaskMemoryManager;
 import org.apache.spark.util.Utils;
@@ -427,14 +427,10 @@ public final class UnsafeExternalSorter {
 
     final long recordAddress =
       taskMemoryManager.encodePageNumberAndOffset(dataPage, dataPagePosition);
-    PlatformDependent.UNSAFE.putInt(dataPageBaseObject, dataPagePosition, 
lengthInBytes);
+    Platform.putInt(dataPageBaseObject, dataPagePosition, lengthInBytes);
     dataPagePosition += 4;
-    PlatformDependent.copyMemory(
-      recordBaseObject,
-      recordBaseOffset,
-      dataPageBaseObject,
-      dataPagePosition,
-      lengthInBytes);
+    Platform.copyMemory(
+      recordBaseObject, recordBaseOffset, dataPageBaseObject, 
dataPagePosition, lengthInBytes);
     assert(inMemSorter != null);
     inMemSorter.insertRecord(recordAddress, prefix);
   }
@@ -493,18 +489,16 @@ public final class UnsafeExternalSorter {
 
     final long recordAddress =
       taskMemoryManager.encodePageNumberAndOffset(dataPage, dataPagePosition);
-    PlatformDependent.UNSAFE.putInt(dataPageBaseObject, dataPagePosition, 
keyLen + valueLen + 4);
+    Platform.putInt(dataPageBaseObject, dataPagePosition, keyLen + valueLen + 
4);
     dataPagePosition += 4;
 
-    PlatformDependent.UNSAFE.putInt(dataPageBaseObject, dataPagePosition, 
keyLen);
+    Platform.putInt(dataPageBaseObject, dataPagePosition, keyLen);
     dataPagePosition += 4;
 
-    PlatformDependent.copyMemory(
-      keyBaseObj, keyOffset, dataPageBaseObject, dataPagePosition, keyLen);
+    Platform.copyMemory(keyBaseObj, keyOffset, dataPageBaseObject, 
dataPagePosition, keyLen);
     dataPagePosition += keyLen;
 
-    PlatformDependent.copyMemory(
-      valueBaseObj, valueOffset, dataPageBaseObject, dataPagePosition, 
valueLen);
+    Platform.copyMemory(valueBaseObj, valueOffset, dataPageBaseObject, 
dataPagePosition, valueLen);
 
     assert(inMemSorter != null);
     inMemSorter.insertRecord(recordAddress, prefix);

http://git-wip-us.apache.org/repos/asf/spark/blob/d378396f/core/src/main/java/org/apache/spark/util/collection/unsafe/sort/UnsafeInMemorySorter.java
----------------------------------------------------------------------
diff --git 
a/core/src/main/java/org/apache/spark/util/collection/unsafe/sort/UnsafeInMemorySorter.java
 
b/core/src/main/java/org/apache/spark/util/collection/unsafe/sort/UnsafeInMemorySorter.java
index 1e4b8a1..f7787e1 100644
--- 
a/core/src/main/java/org/apache/spark/util/collection/unsafe/sort/UnsafeInMemorySorter.java
+++ 
b/core/src/main/java/org/apache/spark/util/collection/unsafe/sort/UnsafeInMemorySorter.java
@@ -19,7 +19,7 @@ package org.apache.spark.util.collection.unsafe.sort;
 
 import java.util.Comparator;
 
-import org.apache.spark.unsafe.PlatformDependent;
+import org.apache.spark.unsafe.Platform;
 import org.apache.spark.util.collection.Sorter;
 import org.apache.spark.unsafe.memory.TaskMemoryManager;
 
@@ -164,7 +164,7 @@ public final class UnsafeInMemorySorter {
       final long recordPointer = sortBuffer[position];
       baseObject = memoryManager.getPage(recordPointer);
       baseOffset = memoryManager.getOffsetInPage(recordPointer) + 4;  // Skip 
over record length
-      recordLength = PlatformDependent.UNSAFE.getInt(baseObject, baseOffset - 
4);
+      recordLength = Platform.getInt(baseObject, baseOffset - 4);
       keyPrefix = sortBuffer[position + 1];
       position += 2;
     }

http://git-wip-us.apache.org/repos/asf/spark/blob/d378396f/core/src/main/java/org/apache/spark/util/collection/unsafe/sort/UnsafeSorterSpillReader.java
----------------------------------------------------------------------
diff --git 
a/core/src/main/java/org/apache/spark/util/collection/unsafe/sort/UnsafeSorterSpillReader.java
 
b/core/src/main/java/org/apache/spark/util/collection/unsafe/sort/UnsafeSorterSpillReader.java
index ca1cced..4989b05 100644
--- 
a/core/src/main/java/org/apache/spark/util/collection/unsafe/sort/UnsafeSorterSpillReader.java
+++ 
b/core/src/main/java/org/apache/spark/util/collection/unsafe/sort/UnsafeSorterSpillReader.java
@@ -23,7 +23,7 @@ import com.google.common.io.ByteStreams;
 
 import org.apache.spark.storage.BlockId;
 import org.apache.spark.storage.BlockManager;
-import org.apache.spark.unsafe.PlatformDependent;
+import org.apache.spark.unsafe.Platform;
 
 /**
  * Reads spill files written by {@link UnsafeSorterSpillWriter} (see that 
class for a description
@@ -42,7 +42,7 @@ final class UnsafeSorterSpillReader extends 
UnsafeSorterIterator {
 
   private byte[] arr = new byte[1024 * 1024];
   private Object baseObject = arr;
-  private final long baseOffset = PlatformDependent.BYTE_ARRAY_OFFSET;
+  private final long baseOffset = Platform.BYTE_ARRAY_OFFSET;
 
   public UnsafeSorterSpillReader(
       BlockManager blockManager,

http://git-wip-us.apache.org/repos/asf/spark/blob/d378396f/core/src/main/java/org/apache/spark/util/collection/unsafe/sort/UnsafeSorterSpillWriter.java
----------------------------------------------------------------------
diff --git 
a/core/src/main/java/org/apache/spark/util/collection/unsafe/sort/UnsafeSorterSpillWriter.java
 
b/core/src/main/java/org/apache/spark/util/collection/unsafe/sort/UnsafeSorterSpillWriter.java
index 44cf6c7..e59a84f 100644
--- 
a/core/src/main/java/org/apache/spark/util/collection/unsafe/sort/UnsafeSorterSpillWriter.java
+++ 
b/core/src/main/java/org/apache/spark/util/collection/unsafe/sort/UnsafeSorterSpillWriter.java
@@ -28,7 +28,7 @@ import org.apache.spark.storage.BlockId;
 import org.apache.spark.storage.BlockManager;
 import org.apache.spark.storage.DiskBlockObjectWriter;
 import org.apache.spark.storage.TempLocalBlockId;
-import org.apache.spark.unsafe.PlatformDependent;
+import org.apache.spark.unsafe.Platform;
 
 /**
  * Spills a list of sorted records to disk. Spill files have the following 
format:
@@ -117,11 +117,11 @@ final class UnsafeSorterSpillWriter {
     long recordReadPosition = baseOffset;
     while (dataRemaining > 0) {
       final int toTransfer = Math.min(freeSpaceInWriteBuffer, dataRemaining);
-      PlatformDependent.copyMemory(
+      Platform.copyMemory(
         baseObject,
         recordReadPosition,
         writeBuffer,
-        PlatformDependent.BYTE_ARRAY_OFFSET + (DISK_WRITE_BUFFER_SIZE - 
freeSpaceInWriteBuffer),
+        Platform.BYTE_ARRAY_OFFSET + (DISK_WRITE_BUFFER_SIZE - 
freeSpaceInWriteBuffer),
         toTransfer);
       writer.write(writeBuffer, 0, (DISK_WRITE_BUFFER_SIZE - 
freeSpaceInWriteBuffer) + toTransfer);
       recordReadPosition += toTransfer;

http://git-wip-us.apache.org/repos/asf/spark/blob/d378396f/core/src/test/java/org/apache/spark/shuffle/unsafe/UnsafeShuffleInMemorySorterSuite.java
----------------------------------------------------------------------
diff --git 
a/core/src/test/java/org/apache/spark/shuffle/unsafe/UnsafeShuffleInMemorySorterSuite.java
 
b/core/src/test/java/org/apache/spark/shuffle/unsafe/UnsafeShuffleInMemorySorterSuite.java
index 8fa7259..40fefe2 100644
--- 
a/core/src/test/java/org/apache/spark/shuffle/unsafe/UnsafeShuffleInMemorySorterSuite.java
+++ 
b/core/src/test/java/org/apache/spark/shuffle/unsafe/UnsafeShuffleInMemorySorterSuite.java
@@ -24,7 +24,7 @@ import org.junit.Assert;
 import org.junit.Test;
 
 import org.apache.spark.HashPartitioner;
-import org.apache.spark.unsafe.PlatformDependent;
+import org.apache.spark.unsafe.Platform;
 import org.apache.spark.unsafe.memory.ExecutorMemoryManager;
 import org.apache.spark.unsafe.memory.MemoryAllocator;
 import org.apache.spark.unsafe.memory.MemoryBlock;
@@ -34,11 +34,7 @@ public class UnsafeShuffleInMemorySorterSuite {
 
   private static String getStringFromDataPage(Object baseObject, long 
baseOffset, int strLength) {
     final byte[] strBytes = new byte[strLength];
-    PlatformDependent.copyMemory(
-      baseObject,
-      baseOffset,
-      strBytes,
-      PlatformDependent.BYTE_ARRAY_OFFSET, strLength);
+    Platform.copyMemory(baseObject, baseOffset, strBytes, 
Platform.BYTE_ARRAY_OFFSET, strLength);
     return new String(strBytes);
   }
 
@@ -74,14 +70,10 @@ public class UnsafeShuffleInMemorySorterSuite {
     for (String str : dataToSort) {
       final long recordAddress = 
memoryManager.encodePageNumberAndOffset(dataPage, position);
       final byte[] strBytes = str.getBytes("utf-8");
-      PlatformDependent.UNSAFE.putInt(baseObject, position, strBytes.length);
+      Platform.putInt(baseObject, position, strBytes.length);
       position += 4;
-      PlatformDependent.copyMemory(
-        strBytes,
-        PlatformDependent.BYTE_ARRAY_OFFSET,
-        baseObject,
-        position,
-        strBytes.length);
+      Platform.copyMemory(
+        strBytes, Platform.BYTE_ARRAY_OFFSET, baseObject, position, 
strBytes.length);
       position += strBytes.length;
       sorter.insertRecord(recordAddress, hashPartitioner.getPartition(str));
     }
@@ -98,7 +90,7 @@ public class UnsafeShuffleInMemorySorterSuite {
       Assert.assertTrue("Partition id " + partitionId + " should be >= prev id 
" + prevPartitionId,
         partitionId >= prevPartitionId);
       final long recordAddress = iter.packedRecordPointer.getRecordPointer();
-      final int recordLength = PlatformDependent.UNSAFE.getInt(
+      final int recordLength = Platform.getInt(
         memoryManager.getPage(recordAddress), 
memoryManager.getOffsetInPage(recordAddress));
       final String str = getStringFromDataPage(
         memoryManager.getPage(recordAddress),

http://git-wip-us.apache.org/repos/asf/spark/blob/d378396f/core/src/test/java/org/apache/spark/unsafe/map/AbstractBytesToBytesMapSuite.java
----------------------------------------------------------------------
diff --git 
a/core/src/test/java/org/apache/spark/unsafe/map/AbstractBytesToBytesMapSuite.java
 
b/core/src/test/java/org/apache/spark/unsafe/map/AbstractBytesToBytesMapSuite.java
index e56a3f0..1a79c20 100644
--- 
a/core/src/test/java/org/apache/spark/unsafe/map/AbstractBytesToBytesMapSuite.java
+++ 
b/core/src/test/java/org/apache/spark/unsafe/map/AbstractBytesToBytesMapSuite.java
@@ -32,9 +32,7 @@ import static org.mockito.Mockito.*;
 import org.apache.spark.shuffle.ShuffleMemoryManager;
 import org.apache.spark.unsafe.array.ByteArrayMethods;
 import org.apache.spark.unsafe.memory.*;
-import org.apache.spark.unsafe.PlatformDependent;
-import static org.apache.spark.unsafe.PlatformDependent.BYTE_ARRAY_OFFSET;
-import static org.apache.spark.unsafe.PlatformDependent.LONG_ARRAY_OFFSET;
+import org.apache.spark.unsafe.Platform;
 
 
 public abstract class AbstractBytesToBytesMapSuite {
@@ -80,13 +78,8 @@ public abstract class AbstractBytesToBytesMapSuite {
 
   private static byte[] getByteArray(MemoryLocation loc, int size) {
     final byte[] arr = new byte[size];
-    PlatformDependent.copyMemory(
-      loc.getBaseObject(),
-      loc.getBaseOffset(),
-      arr,
-      BYTE_ARRAY_OFFSET,
-      size
-    );
+    Platform.copyMemory(
+      loc.getBaseObject(), loc.getBaseOffset(), arr, 
Platform.BYTE_ARRAY_OFFSET, size);
     return arr;
   }
 
@@ -108,7 +101,7 @@ public abstract class AbstractBytesToBytesMapSuite {
       long actualLengthBytes) {
     return (actualLengthBytes == expected.length) && 
ByteArrayMethods.arrayEquals(
       expected,
-      BYTE_ARRAY_OFFSET,
+      Platform.BYTE_ARRAY_OFFSET,
       actualAddr.getBaseObject(),
       actualAddr.getBaseOffset(),
       expected.length
@@ -124,7 +117,7 @@ public abstract class AbstractBytesToBytesMapSuite {
       final int keyLengthInWords = 10;
       final int keyLengthInBytes = keyLengthInWords * 8;
       final byte[] key = getRandomByteArray(keyLengthInWords);
-      Assert.assertFalse(map.lookup(key, BYTE_ARRAY_OFFSET, 
keyLengthInBytes).isDefined());
+      Assert.assertFalse(map.lookup(key, Platform.BYTE_ARRAY_OFFSET, 
keyLengthInBytes).isDefined());
       Assert.assertFalse(map.iterator().hasNext());
     } finally {
       map.free();
@@ -141,14 +134,14 @@ public abstract class AbstractBytesToBytesMapSuite {
     final byte[] valueData = getRandomByteArray(recordLengthWords);
     try {
       final BytesToBytesMap.Location loc =
-        map.lookup(keyData, BYTE_ARRAY_OFFSET, recordLengthBytes);
+        map.lookup(keyData, Platform.BYTE_ARRAY_OFFSET, recordLengthBytes);
       Assert.assertFalse(loc.isDefined());
       Assert.assertTrue(loc.putNewKey(
         keyData,
-        BYTE_ARRAY_OFFSET,
+        Platform.BYTE_ARRAY_OFFSET,
         recordLengthBytes,
         valueData,
-        BYTE_ARRAY_OFFSET,
+        Platform.BYTE_ARRAY_OFFSET,
         recordLengthBytes
       ));
       // After storing the key and value, the other location methods should 
return results that
@@ -159,7 +152,8 @@ public abstract class AbstractBytesToBytesMapSuite {
       Assert.assertArrayEquals(valueData, getByteArray(loc.getValueAddress(), 
recordLengthBytes));
 
       // After calling lookup() the location should still point to the correct 
data.
-      Assert.assertTrue(map.lookup(keyData, BYTE_ARRAY_OFFSET, 
recordLengthBytes).isDefined());
+      Assert.assertTrue(
+        map.lookup(keyData, Platform.BYTE_ARRAY_OFFSET, 
recordLengthBytes).isDefined());
       Assert.assertEquals(recordLengthBytes, loc.getKeyLength());
       Assert.assertEquals(recordLengthBytes, loc.getValueLength());
       Assert.assertArrayEquals(keyData, getByteArray(loc.getKeyAddress(), 
recordLengthBytes));
@@ -168,10 +162,10 @@ public abstract class AbstractBytesToBytesMapSuite {
       try {
         Assert.assertTrue(loc.putNewKey(
           keyData,
-          BYTE_ARRAY_OFFSET,
+          Platform.BYTE_ARRAY_OFFSET,
           recordLengthBytes,
           valueData,
-          BYTE_ARRAY_OFFSET,
+          Platform.BYTE_ARRAY_OFFSET,
           recordLengthBytes
         ));
         Assert.fail("Should not be able to set a new value for a key");
@@ -191,25 +185,25 @@ public abstract class AbstractBytesToBytesMapSuite {
       for (long i = 0; i < size; i++) {
         final long[] value = new long[] { i };
         final BytesToBytesMap.Location loc =
-          map.lookup(value, PlatformDependent.LONG_ARRAY_OFFSET, 8);
+          map.lookup(value, Platform.LONG_ARRAY_OFFSET, 8);
         Assert.assertFalse(loc.isDefined());
         // Ensure that we store some zero-length keys
         if (i % 5 == 0) {
           Assert.assertTrue(loc.putNewKey(
             null,
-            PlatformDependent.LONG_ARRAY_OFFSET,
+            Platform.LONG_ARRAY_OFFSET,
             0,
             value,
-            PlatformDependent.LONG_ARRAY_OFFSET,
+            Platform.LONG_ARRAY_OFFSET,
             8
           ));
         } else {
           Assert.assertTrue(loc.putNewKey(
             value,
-            PlatformDependent.LONG_ARRAY_OFFSET,
+            Platform.LONG_ARRAY_OFFSET,
             8,
             value,
-            PlatformDependent.LONG_ARRAY_OFFSET,
+            Platform.LONG_ARRAY_OFFSET,
             8
           ));
         }
@@ -228,14 +222,13 @@ public abstract class AbstractBytesToBytesMapSuite {
         Assert.assertTrue(loc.isDefined());
         final MemoryLocation keyAddress = loc.getKeyAddress();
         final MemoryLocation valueAddress = loc.getValueAddress();
-        final long value = PlatformDependent.UNSAFE.getLong(
+        final long value = Platform.getLong(
           valueAddress.getBaseObject(), valueAddress.getBaseOffset());
         final long keyLength = loc.getKeyLength();
         if (keyLength == 0) {
           Assert.assertTrue("value " + value + " was not divisible by 5", 
value % 5 == 0);
         } else {
-          final long key = PlatformDependent.UNSAFE.getLong(
-            keyAddress.getBaseObject(), keyAddress.getBaseOffset());
+          final long key = Platform.getLong(keyAddress.getBaseObject(), 
keyAddress.getBaseOffset());
           Assert.assertEquals(value, key);
         }
         valuesSeen.set((int) value);
@@ -284,16 +277,16 @@ public abstract class AbstractBytesToBytesMapSuite {
         final long[] value = new long[] { i, i, i, i, i }; // 5 * 8 = 40 bytes
         final BytesToBytesMap.Location loc = map.lookup(
           key,
-          LONG_ARRAY_OFFSET,
+          Platform.LONG_ARRAY_OFFSET,
           KEY_LENGTH
         );
         Assert.assertFalse(loc.isDefined());
         Assert.assertTrue(loc.putNewKey(
           key,
-          LONG_ARRAY_OFFSET,
+          Platform.LONG_ARRAY_OFFSET,
           KEY_LENGTH,
           value,
-          LONG_ARRAY_OFFSET,
+          Platform.LONG_ARRAY_OFFSET,
           VALUE_LENGTH
         ));
       }
@@ -308,18 +301,18 @@ public abstract class AbstractBytesToBytesMapSuite {
         Assert.assertTrue(loc.isDefined());
         Assert.assertEquals(KEY_LENGTH, loc.getKeyLength());
         Assert.assertEquals(VALUE_LENGTH, loc.getValueLength());
-        PlatformDependent.copyMemory(
+        Platform.copyMemory(
           loc.getKeyAddress().getBaseObject(),
           loc.getKeyAddress().getBaseOffset(),
           key,
-          LONG_ARRAY_OFFSET,
+          Platform.LONG_ARRAY_OFFSET,
           KEY_LENGTH
         );
-        PlatformDependent.copyMemory(
+        Platform.copyMemory(
           loc.getValueAddress().getBaseObject(),
           loc.getValueAddress().getBaseOffset(),
           value,
-          LONG_ARRAY_OFFSET,
+          Platform.LONG_ARRAY_OFFSET,
           VALUE_LENGTH
         );
         for (long j : key) {
@@ -354,16 +347,16 @@ public abstract class AbstractBytesToBytesMapSuite {
           expected.put(ByteBuffer.wrap(key), value);
           final BytesToBytesMap.Location loc = map.lookup(
             key,
-            BYTE_ARRAY_OFFSET,
+            Platform.BYTE_ARRAY_OFFSET,
             key.length
           );
           Assert.assertFalse(loc.isDefined());
           Assert.assertTrue(loc.putNewKey(
             key,
-            BYTE_ARRAY_OFFSET,
+            Platform.BYTE_ARRAY_OFFSET,
             key.length,
             value,
-            BYTE_ARRAY_OFFSET,
+            Platform.BYTE_ARRAY_OFFSET,
             value.length
           ));
           // After calling putNewKey, the following should be true, even 
before calling
@@ -379,7 +372,8 @@ public abstract class AbstractBytesToBytesMapSuite {
       for (Map.Entry<ByteBuffer, byte[]> entry : expected.entrySet()) {
         final byte[] key = entry.getKey().array();
         final byte[] value = entry.getValue();
-        final BytesToBytesMap.Location loc = map.lookup(key, 
BYTE_ARRAY_OFFSET, key.length);
+        final BytesToBytesMap.Location loc =
+          map.lookup(key, Platform.BYTE_ARRAY_OFFSET, key.length);
         Assert.assertTrue(loc.isDefined());
         Assert.assertTrue(arrayEquals(key, loc.getKeyAddress(), 
loc.getKeyLength()));
         Assert.assertTrue(arrayEquals(value, loc.getValueAddress(), 
loc.getValueLength()));
@@ -405,16 +399,16 @@ public abstract class AbstractBytesToBytesMapSuite {
           expected.put(ByteBuffer.wrap(key), value);
           final BytesToBytesMap.Location loc = map.lookup(
             key,
-            BYTE_ARRAY_OFFSET,
+            Platform.BYTE_ARRAY_OFFSET,
             key.length
           );
           Assert.assertFalse(loc.isDefined());
           Assert.assertTrue(loc.putNewKey(
             key,
-            BYTE_ARRAY_OFFSET,
+            Platform.BYTE_ARRAY_OFFSET,
             key.length,
             value,
-            BYTE_ARRAY_OFFSET,
+            Platform.BYTE_ARRAY_OFFSET,
             value.length
           ));
           // After calling putNewKey, the following should be true, even 
before calling
@@ -429,7 +423,8 @@ public abstract class AbstractBytesToBytesMapSuite {
       for (Map.Entry<ByteBuffer, byte[]> entry : expected.entrySet()) {
         final byte[] key = entry.getKey().array();
         final byte[] value = entry.getValue();
-        final BytesToBytesMap.Location loc = map.lookup(key, 
BYTE_ARRAY_OFFSET, key.length);
+        final BytesToBytesMap.Location loc =
+          map.lookup(key, Platform.BYTE_ARRAY_OFFSET, key.length);
         Assert.assertTrue(loc.isDefined());
         Assert.assertTrue(arrayEquals(key, loc.getKeyAddress(), 
loc.getKeyLength()));
         Assert.assertTrue(arrayEquals(value, loc.getValueAddress(), 
loc.getValueLength()));
@@ -447,12 +442,10 @@ public abstract class AbstractBytesToBytesMapSuite {
     try {
       final long[] emptyArray = new long[0];
       final BytesToBytesMap.Location loc =
-        map.lookup(emptyArray, PlatformDependent.LONG_ARRAY_OFFSET, 0);
+        map.lookup(emptyArray, Platform.LONG_ARRAY_OFFSET, 0);
       Assert.assertFalse(loc.isDefined());
       Assert.assertFalse(loc.putNewKey(
-        emptyArray, LONG_ARRAY_OFFSET, 0,
-        emptyArray, LONG_ARRAY_OFFSET, 0
-      ));
+        emptyArray, Platform.LONG_ARRAY_OFFSET, 0, emptyArray, 
Platform.LONG_ARRAY_OFFSET, 0));
     } finally {
       map.free();
     }
@@ -468,8 +461,9 @@ public abstract class AbstractBytesToBytesMapSuite {
       int i;
       for (i = 0; i < 1024; i++) {
         final long[] arr = new long[]{i};
-        final BytesToBytesMap.Location loc = map.lookup(arr, 
PlatformDependent.LONG_ARRAY_OFFSET, 8);
-        success = loc.putNewKey(arr, LONG_ARRAY_OFFSET, 8, arr, 
LONG_ARRAY_OFFSET, 8);
+        final BytesToBytesMap.Location loc = map.lookup(arr, 
Platform.LONG_ARRAY_OFFSET, 8);
+        success =
+          loc.putNewKey(arr, Platform.LONG_ARRAY_OFFSET, 8, arr, 
Platform.LONG_ARRAY_OFFSET, 8);
         if (!success) {
           break;
         }
@@ -541,12 +535,12 @@ public abstract class AbstractBytesToBytesMapSuite {
     try {
       for (long i = 0; i < numRecordsPerPage * 10; i++) {
         final long[] value = new long[]{i};
-        map.lookup(value, PlatformDependent.LONG_ARRAY_OFFSET, 8).putNewKey(
+        map.lookup(value, Platform.LONG_ARRAY_OFFSET, 8).putNewKey(
           value,
-          PlatformDependent.LONG_ARRAY_OFFSET,
+          Platform.LONG_ARRAY_OFFSET,
           8,
           value,
-          PlatformDependent.LONG_ARRAY_OFFSET,
+          Platform.LONG_ARRAY_OFFSET,
           8);
         newPeakMemory = map.getPeakMemoryUsedBytes();
         if (i % numRecordsPerPage == 0) {

http://git-wip-us.apache.org/repos/asf/spark/blob/d378396f/core/src/test/java/org/apache/spark/util/collection/unsafe/sort/UnsafeExternalSorterSuite.java
----------------------------------------------------------------------
diff --git 
a/core/src/test/java/org/apache/spark/util/collection/unsafe/sort/UnsafeExternalSorterSuite.java
 
b/core/src/test/java/org/apache/spark/util/collection/unsafe/sort/UnsafeExternalSorterSuite.java
index 83049b8..445a37b 100644
--- 
a/core/src/test/java/org/apache/spark/util/collection/unsafe/sort/UnsafeExternalSorterSuite.java
+++ 
b/core/src/test/java/org/apache/spark/util/collection/unsafe/sort/UnsafeExternalSorterSuite.java
@@ -49,7 +49,7 @@ import org.apache.spark.executor.TaskMetrics;
 import org.apache.spark.serializer.SerializerInstance;
 import org.apache.spark.shuffle.ShuffleMemoryManager;
 import org.apache.spark.storage.*;
-import org.apache.spark.unsafe.PlatformDependent;
+import org.apache.spark.unsafe.Platform;
 import org.apache.spark.unsafe.memory.ExecutorMemoryManager;
 import org.apache.spark.unsafe.memory.MemoryAllocator;
 import org.apache.spark.unsafe.memory.TaskMemoryManager;
@@ -166,14 +166,14 @@ public class UnsafeExternalSorterSuite {
 
   private static void insertNumber(UnsafeExternalSorter sorter, int value) 
throws Exception {
     final int[] arr = new int[]{ value };
-    sorter.insertRecord(arr, PlatformDependent.INT_ARRAY_OFFSET, 4, value);
+    sorter.insertRecord(arr, Platform.INT_ARRAY_OFFSET, 4, value);
   }
 
   private static void insertRecord(
       UnsafeExternalSorter sorter,
       int[] record,
       long prefix) throws IOException {
-    sorter.insertRecord(record, PlatformDependent.INT_ARRAY_OFFSET, 
record.length * 4, prefix);
+    sorter.insertRecord(record, Platform.INT_ARRAY_OFFSET, record.length * 4, 
prefix);
   }
 
   private UnsafeExternalSorter newSorter() throws IOException {
@@ -205,7 +205,7 @@ public class UnsafeExternalSorterSuite {
       iter.loadNext();
       assertEquals(i, iter.getKeyPrefix());
       assertEquals(4, iter.getRecordLength());
-      assertEquals(i, PlatformDependent.UNSAFE.getInt(iter.getBaseObject(), 
iter.getBaseOffset()));
+      assertEquals(i, Platform.getInt(iter.getBaseObject(), 
iter.getBaseOffset()));
     }
 
     sorter.cleanupResources();
@@ -253,7 +253,7 @@ public class UnsafeExternalSorterSuite {
       iter.loadNext();
       assertEquals(i, iter.getKeyPrefix());
       assertEquals(4, iter.getRecordLength());
-      assertEquals(i, PlatformDependent.UNSAFE.getInt(iter.getBaseObject(), 
iter.getBaseOffset()));
+      assertEquals(i, Platform.getInt(iter.getBaseObject(), 
iter.getBaseOffset()));
       i++;
     }
     sorter.cleanupResources();
@@ -265,7 +265,7 @@ public class UnsafeExternalSorterSuite {
     final UnsafeExternalSorter sorter = newSorter();
     byte[] record = new byte[16];
     while (sorter.getNumberOfAllocatedPages() < 2) {
-      sorter.insertRecord(record, PlatformDependent.BYTE_ARRAY_OFFSET, 
record.length, 0);
+      sorter.insertRecord(record, Platform.BYTE_ARRAY_OFFSET, record.length, 
0);
     }
     sorter.cleanupResources();
     assertSpillFilesWereCleanedUp();
@@ -292,25 +292,25 @@ public class UnsafeExternalSorterSuite {
     iter.loadNext();
     assertEquals(123, iter.getKeyPrefix());
     assertEquals(smallRecord.length * 4, iter.getRecordLength());
-    assertEquals(123, PlatformDependent.UNSAFE.getInt(iter.getBaseObject(), 
iter.getBaseOffset()));
+    assertEquals(123, Platform.getInt(iter.getBaseObject(), 
iter.getBaseOffset()));
     // Small record
     assertTrue(iter.hasNext());
     iter.loadNext();
     assertEquals(123, iter.getKeyPrefix());
     assertEquals(smallRecord.length * 4, iter.getRecordLength());
-    assertEquals(123, PlatformDependent.UNSAFE.getInt(iter.getBaseObject(), 
iter.getBaseOffset()));
+    assertEquals(123, Platform.getInt(iter.getBaseObject(), 
iter.getBaseOffset()));
     // Large record
     assertTrue(iter.hasNext());
     iter.loadNext();
     assertEquals(456, iter.getKeyPrefix());
     assertEquals(largeRecord.length * 4, iter.getRecordLength());
-    assertEquals(456, PlatformDependent.UNSAFE.getInt(iter.getBaseObject(), 
iter.getBaseOffset()));
+    assertEquals(456, Platform.getInt(iter.getBaseObject(), 
iter.getBaseOffset()));
     // Large record
     assertTrue(iter.hasNext());
     iter.loadNext();
     assertEquals(456, iter.getKeyPrefix());
     assertEquals(largeRecord.length * 4, iter.getRecordLength());
-    assertEquals(456, PlatformDependent.UNSAFE.getInt(iter.getBaseObject(), 
iter.getBaseOffset()));
+    assertEquals(456, Platform.getInt(iter.getBaseObject(), 
iter.getBaseOffset()));
 
     assertFalse(iter.hasNext());
     sorter.cleanupResources();

http://git-wip-us.apache.org/repos/asf/spark/blob/d378396f/core/src/test/java/org/apache/spark/util/collection/unsafe/sort/UnsafeInMemorySorterSuite.java
----------------------------------------------------------------------
diff --git 
a/core/src/test/java/org/apache/spark/util/collection/unsafe/sort/UnsafeInMemorySorterSuite.java
 
b/core/src/test/java/org/apache/spark/util/collection/unsafe/sort/UnsafeInMemorySorterSuite.java
index 9095009..778e813 100644
--- 
a/core/src/test/java/org/apache/spark/util/collection/unsafe/sort/UnsafeInMemorySorterSuite.java
+++ 
b/core/src/test/java/org/apache/spark/util/collection/unsafe/sort/UnsafeInMemorySorterSuite.java
@@ -26,7 +26,7 @@ import static org.junit.Assert.*;
 import static org.mockito.Mockito.mock;
 
 import org.apache.spark.HashPartitioner;
-import org.apache.spark.unsafe.PlatformDependent;
+import org.apache.spark.unsafe.Platform;
 import org.apache.spark.unsafe.memory.ExecutorMemoryManager;
 import org.apache.spark.unsafe.memory.MemoryAllocator;
 import org.apache.spark.unsafe.memory.MemoryBlock;
@@ -36,11 +36,7 @@ public class UnsafeInMemorySorterSuite {
 
   private static String getStringFromDataPage(Object baseObject, long 
baseOffset, int length) {
     final byte[] strBytes = new byte[length];
-    PlatformDependent.copyMemory(
-      baseObject,
-      baseOffset,
-      strBytes,
-      PlatformDependent.BYTE_ARRAY_OFFSET, length);
+    Platform.copyMemory(baseObject, baseOffset, strBytes, 
Platform.BYTE_ARRAY_OFFSET, length);
     return new String(strBytes);
   }
 
@@ -76,14 +72,10 @@ public class UnsafeInMemorySorterSuite {
     long position = dataPage.getBaseOffset();
     for (String str : dataToSort) {
       final byte[] strBytes = str.getBytes("utf-8");
-      PlatformDependent.UNSAFE.putInt(baseObject, position, strBytes.length);
+      Platform.putInt(baseObject, position, strBytes.length);
       position += 4;
-      PlatformDependent.copyMemory(
-        strBytes,
-        PlatformDependent.BYTE_ARRAY_OFFSET,
-        baseObject,
-        position,
-        strBytes.length);
+      Platform.copyMemory(
+        strBytes, Platform.BYTE_ARRAY_OFFSET, baseObject, position, 
strBytes.length);
       position += strBytes.length;
     }
     // Since the key fits within the 8-byte prefix, we don't need to do any 
record comparison, so
@@ -113,7 +105,7 @@ public class UnsafeInMemorySorterSuite {
     position = dataPage.getBaseOffset();
     for (int i = 0; i < dataToSort.length; i++) {
       // position now points to the start of a record (which holds its length).
-      final int recordLength = PlatformDependent.UNSAFE.getInt(baseObject, 
position);
+      final int recordLength = Platform.getInt(baseObject, position);
       final long address = memoryManager.encodePageNumberAndOffset(dataPage, 
position);
       final String str = getStringFromDataPage(baseObject, position + 4, 
recordLength);
       final int partitionId = hashPartitioner.getPartition(str);

http://git-wip-us.apache.org/repos/asf/spark/blob/d378396f/sql/catalyst/src/main/java/org/apache/spark/sql/catalyst/expressions/UnsafeArrayData.java
----------------------------------------------------------------------
diff --git 
a/sql/catalyst/src/main/java/org/apache/spark/sql/catalyst/expressions/UnsafeArrayData.java
 
b/sql/catalyst/src/main/java/org/apache/spark/sql/catalyst/expressions/UnsafeArrayData.java
index 0374846..501dff0 100644
--- 
a/sql/catalyst/src/main/java/org/apache/spark/sql/catalyst/expressions/UnsafeArrayData.java
+++ 
b/sql/catalyst/src/main/java/org/apache/spark/sql/catalyst/expressions/UnsafeArrayData.java
@@ -22,7 +22,7 @@ import java.math.BigInteger;
 
 import org.apache.spark.sql.catalyst.InternalRow;
 import org.apache.spark.sql.types.*;
-import org.apache.spark.unsafe.PlatformDependent;
+import org.apache.spark.unsafe.Platform;
 import org.apache.spark.unsafe.array.ByteArrayMethods;
 import org.apache.spark.unsafe.hash.Murmur3_x86_32;
 import org.apache.spark.unsafe.types.CalendarInterval;
@@ -59,7 +59,7 @@ public class UnsafeArrayData extends ArrayData {
   private int sizeInBytes;
 
   private int getElementOffset(int ordinal) {
-    return PlatformDependent.UNSAFE.getInt(baseObject, baseOffset + ordinal * 
4L);
+    return Platform.getInt(baseObject, baseOffset + ordinal * 4L);
   }
 
   private int getElementSize(int offset, int ordinal) {
@@ -157,7 +157,7 @@ public class UnsafeArrayData extends ArrayData {
     assertIndexIsValid(ordinal);
     final int offset = getElementOffset(ordinal);
     if (offset < 0) return false;
-    return PlatformDependent.UNSAFE.getBoolean(baseObject, baseOffset + 
offset);
+    return Platform.getBoolean(baseObject, baseOffset + offset);
   }
 
   @Override
@@ -165,7 +165,7 @@ public class UnsafeArrayData extends ArrayData {
     assertIndexIsValid(ordinal);
     final int offset = getElementOffset(ordinal);
     if (offset < 0) return 0;
-    return PlatformDependent.UNSAFE.getByte(baseObject, baseOffset + offset);
+    return Platform.getByte(baseObject, baseOffset + offset);
   }
 
   @Override
@@ -173,7 +173,7 @@ public class UnsafeArrayData extends ArrayData {
     assertIndexIsValid(ordinal);
     final int offset = getElementOffset(ordinal);
     if (offset < 0) return 0;
-    return PlatformDependent.UNSAFE.getShort(baseObject, baseOffset + offset);
+    return Platform.getShort(baseObject, baseOffset + offset);
   }
 
   @Override
@@ -181,7 +181,7 @@ public class UnsafeArrayData extends ArrayData {
     assertIndexIsValid(ordinal);
     final int offset = getElementOffset(ordinal);
     if (offset < 0) return 0;
-    return PlatformDependent.UNSAFE.getInt(baseObject, baseOffset + offset);
+    return Platform.getInt(baseObject, baseOffset + offset);
   }
 
   @Override
@@ -189,7 +189,7 @@ public class UnsafeArrayData extends ArrayData {
     assertIndexIsValid(ordinal);
     final int offset = getElementOffset(ordinal);
     if (offset < 0) return 0;
-    return PlatformDependent.UNSAFE.getLong(baseObject, baseOffset + offset);
+    return Platform.getLong(baseObject, baseOffset + offset);
   }
 
   @Override
@@ -197,7 +197,7 @@ public class UnsafeArrayData extends ArrayData {
     assertIndexIsValid(ordinal);
     final int offset = getElementOffset(ordinal);
     if (offset < 0) return 0;
-    return PlatformDependent.UNSAFE.getFloat(baseObject, baseOffset + offset);
+    return Platform.getFloat(baseObject, baseOffset + offset);
   }
 
   @Override
@@ -205,7 +205,7 @@ public class UnsafeArrayData extends ArrayData {
     assertIndexIsValid(ordinal);
     final int offset = getElementOffset(ordinal);
     if (offset < 0) return 0;
-    return PlatformDependent.UNSAFE.getDouble(baseObject, baseOffset + offset);
+    return Platform.getDouble(baseObject, baseOffset + offset);
   }
 
   @Override
@@ -215,7 +215,7 @@ public class UnsafeArrayData extends ArrayData {
     if (offset < 0) return null;
 
     if (precision <= Decimal.MAX_LONG_DIGITS()) {
-      final long value = PlatformDependent.UNSAFE.getLong(baseObject, 
baseOffset + offset);
+      final long value = Platform.getLong(baseObject, baseOffset + offset);
       return Decimal.apply(value, precision, scale);
     } else {
       final byte[] bytes = getBinary(ordinal);
@@ -241,12 +241,7 @@ public class UnsafeArrayData extends ArrayData {
     if (offset < 0) return null;
     final int size = getElementSize(offset, ordinal);
     final byte[] bytes = new byte[size];
-    PlatformDependent.copyMemory(
-      baseObject,
-      baseOffset + offset,
-      bytes,
-      PlatformDependent.BYTE_ARRAY_OFFSET,
-      size);
+    Platform.copyMemory(baseObject, baseOffset + offset, bytes, 
Platform.BYTE_ARRAY_OFFSET, size);
     return bytes;
   }
 
@@ -255,9 +250,8 @@ public class UnsafeArrayData extends ArrayData {
     assertIndexIsValid(ordinal);
     final int offset = getElementOffset(ordinal);
     if (offset < 0) return null;
-    final int months = (int) PlatformDependent.UNSAFE.getLong(baseObject, 
baseOffset + offset);
-    final long microseconds =
-      PlatformDependent.UNSAFE.getLong(baseObject, baseOffset + offset + 8);
+    final int months = (int) Platform.getLong(baseObject, baseOffset + offset);
+    final long microseconds = Platform.getLong(baseObject, baseOffset + offset 
+ 8);
     return new CalendarInterval(months, microseconds);
   }
 
@@ -307,27 +301,16 @@ public class UnsafeArrayData extends ArrayData {
   }
 
   public void writeToMemory(Object target, long targetOffset) {
-    PlatformDependent.copyMemory(
-      baseObject,
-      baseOffset,
-      target,
-      targetOffset,
-      sizeInBytes
-    );
+    Platform.copyMemory(baseObject, baseOffset, target, targetOffset, 
sizeInBytes);
   }
 
   @Override
   public UnsafeArrayData copy() {
     UnsafeArrayData arrayCopy = new UnsafeArrayData();
     final byte[] arrayDataCopy = new byte[sizeInBytes];
-    PlatformDependent.copyMemory(
-      baseObject,
-      baseOffset,
-      arrayDataCopy,
-      PlatformDependent.BYTE_ARRAY_OFFSET,
-      sizeInBytes
-    );
-    arrayCopy.pointTo(arrayDataCopy, PlatformDependent.BYTE_ARRAY_OFFSET, 
numElements, sizeInBytes);
+    Platform.copyMemory(
+      baseObject, baseOffset, arrayDataCopy, Platform.BYTE_ARRAY_OFFSET, 
sizeInBytes);
+    arrayCopy.pointTo(arrayDataCopy, Platform.BYTE_ARRAY_OFFSET, numElements, 
sizeInBytes);
     return arrayCopy;
   }
 }

http://git-wip-us.apache.org/repos/asf/spark/blob/d378396f/sql/catalyst/src/main/java/org/apache/spark/sql/catalyst/expressions/UnsafeReaders.java
----------------------------------------------------------------------
diff --git 
a/sql/catalyst/src/main/java/org/apache/spark/sql/catalyst/expressions/UnsafeReaders.java
 
b/sql/catalyst/src/main/java/org/apache/spark/sql/catalyst/expressions/UnsafeReaders.java
index b521b70..7b03185 100644
--- 
a/sql/catalyst/src/main/java/org/apache/spark/sql/catalyst/expressions/UnsafeReaders.java
+++ 
b/sql/catalyst/src/main/java/org/apache/spark/sql/catalyst/expressions/UnsafeReaders.java
@@ -17,13 +17,13 @@
 
 package org.apache.spark.sql.catalyst.expressions;
 
-import org.apache.spark.unsafe.PlatformDependent;
+import org.apache.spark.unsafe.Platform;
 
 public class UnsafeReaders {
 
   public static UnsafeArrayData readArray(Object baseObject, long baseOffset, 
int numBytes) {
     // Read the number of elements from first 4 bytes.
-    final int numElements = PlatformDependent.UNSAFE.getInt(baseObject, 
baseOffset);
+    final int numElements = Platform.getInt(baseObject, baseOffset);
     final UnsafeArrayData array = new UnsafeArrayData();
     // Skip the first 4 bytes.
     array.pointTo(baseObject, baseOffset + 4, numElements, numBytes - 4);
@@ -32,9 +32,9 @@ public class UnsafeReaders {
 
   public static UnsafeMapData readMap(Object baseObject, long baseOffset, int 
numBytes) {
     // Read the number of elements from first 4 bytes.
-    final int numElements = PlatformDependent.UNSAFE.getInt(baseObject, 
baseOffset);
+    final int numElements = Platform.getInt(baseObject, baseOffset);
     // Read the numBytes of key array in second 4 bytes.
-    final int keyArraySize = PlatformDependent.UNSAFE.getInt(baseObject, 
baseOffset + 4);
+    final int keyArraySize = Platform.getInt(baseObject, baseOffset + 4);
     final int valueArraySize = numBytes - 8 - keyArraySize;
 
     final UnsafeArrayData keyArray = new UnsafeArrayData();

http://git-wip-us.apache.org/repos/asf/spark/blob/d378396f/sql/catalyst/src/main/java/org/apache/spark/sql/catalyst/expressions/UnsafeRow.java
----------------------------------------------------------------------
diff --git 
a/sql/catalyst/src/main/java/org/apache/spark/sql/catalyst/expressions/UnsafeRow.java
 
b/sql/catalyst/src/main/java/org/apache/spark/sql/catalyst/expressions/UnsafeRow.java
index e829acb..7fd9477 100644
--- 
a/sql/catalyst/src/main/java/org/apache/spark/sql/catalyst/expressions/UnsafeRow.java
+++ 
b/sql/catalyst/src/main/java/org/apache/spark/sql/catalyst/expressions/UnsafeRow.java
@@ -27,7 +27,7 @@ import java.util.HashSet;
 import java.util.Set;
 
 import org.apache.spark.sql.types.*;
-import org.apache.spark.unsafe.PlatformDependent;
+import org.apache.spark.unsafe.Platform;
 import org.apache.spark.unsafe.array.ByteArrayMethods;
 import org.apache.spark.unsafe.bitset.BitSetMethods;
 import org.apache.spark.unsafe.hash.Murmur3_x86_32;
@@ -169,7 +169,7 @@ public final class UnsafeRow extends MutableRow {
    * @param sizeInBytes the number of bytes valid in the byte array
    */
   public void pointTo(byte[] buf, int numFields, int sizeInBytes) {
-    pointTo(buf, PlatformDependent.BYTE_ARRAY_OFFSET, numFields, sizeInBytes);
+    pointTo(buf, Platform.BYTE_ARRAY_OFFSET, numFields, sizeInBytes);
   }
 
   @Override
@@ -179,7 +179,7 @@ public final class UnsafeRow extends MutableRow {
     // To preserve row equality, zero out the value when setting the column to 
null.
     // Since this row does does not currently support updates to 
variable-length values, we don't
     // have to worry about zeroing out that data.
-    PlatformDependent.UNSAFE.putLong(baseObject, getFieldOffset(i), 0);
+    Platform.putLong(baseObject, getFieldOffset(i), 0);
   }
 
   @Override
@@ -191,14 +191,14 @@ public final class UnsafeRow extends MutableRow {
   public void setInt(int ordinal, int value) {
     assertIndexIsValid(ordinal);
     setNotNullAt(ordinal);
-    PlatformDependent.UNSAFE.putInt(baseObject, getFieldOffset(ordinal), 
value);
+    Platform.putInt(baseObject, getFieldOffset(ordinal), value);
   }
 
   @Override
   public void setLong(int ordinal, long value) {
     assertIndexIsValid(ordinal);
     setNotNullAt(ordinal);
-    PlatformDependent.UNSAFE.putLong(baseObject, getFieldOffset(ordinal), 
value);
+    Platform.putLong(baseObject, getFieldOffset(ordinal), value);
   }
 
   @Override
@@ -208,28 +208,28 @@ public final class UnsafeRow extends MutableRow {
     if (Double.isNaN(value)) {
       value = Double.NaN;
     }
-    PlatformDependent.UNSAFE.putDouble(baseObject, getFieldOffset(ordinal), 
value);
+    Platform.putDouble(baseObject, getFieldOffset(ordinal), value);
   }
 
   @Override
   public void setBoolean(int ordinal, boolean value) {
     assertIndexIsValid(ordinal);
     setNotNullAt(ordinal);
-    PlatformDependent.UNSAFE.putBoolean(baseObject, getFieldOffset(ordinal), 
value);
+    Platform.putBoolean(baseObject, getFieldOffset(ordinal), value);
   }
 
   @Override
   public void setShort(int ordinal, short value) {
     assertIndexIsValid(ordinal);
     setNotNullAt(ordinal);
-    PlatformDependent.UNSAFE.putShort(baseObject, getFieldOffset(ordinal), 
value);
+    Platform.putShort(baseObject, getFieldOffset(ordinal), value);
   }
 
   @Override
   public void setByte(int ordinal, byte value) {
     assertIndexIsValid(ordinal);
     setNotNullAt(ordinal);
-    PlatformDependent.UNSAFE.putByte(baseObject, getFieldOffset(ordinal), 
value);
+    Platform.putByte(baseObject, getFieldOffset(ordinal), value);
   }
 
   @Override
@@ -239,7 +239,7 @@ public final class UnsafeRow extends MutableRow {
     if (Float.isNaN(value)) {
       value = Float.NaN;
     }
-    PlatformDependent.UNSAFE.putFloat(baseObject, getFieldOffset(ordinal), 
value);
+    Platform.putFloat(baseObject, getFieldOffset(ordinal), value);
   }
 
   /**
@@ -263,23 +263,23 @@ public final class UnsafeRow extends MutableRow {
       long cursor = getLong(ordinal) >>> 32;
       assert cursor > 0 : "invalid cursor " + cursor;
       // zero-out the bytes
-      PlatformDependent.UNSAFE.putLong(baseObject, baseOffset + cursor, 0L);
-      PlatformDependent.UNSAFE.putLong(baseObject, baseOffset + cursor + 8, 
0L);
+      Platform.putLong(baseObject, baseOffset + cursor, 0L);
+      Platform.putLong(baseObject, baseOffset + cursor + 8, 0L);
 
       if (value == null) {
         setNullAt(ordinal);
         // keep the offset for future update
-        PlatformDependent.UNSAFE.putLong(baseObject, getFieldOffset(ordinal), 
cursor << 32);
+        Platform.putLong(baseObject, getFieldOffset(ordinal), cursor << 32);
       } else {
 
         final BigInteger integer = value.toJavaBigDecimal().unscaledValue();
-        final int[] mag = (int[]) 
PlatformDependent.UNSAFE.getObjectVolatile(integer,
-          PlatformDependent.BIG_INTEGER_MAG_OFFSET);
+        final int[] mag = (int[]) Platform.getObjectVolatile(integer,
+          Platform.BIG_INTEGER_MAG_OFFSET);
         assert(mag.length <= 4);
 
         // Write the bytes to the variable length portion.
-        PlatformDependent.copyMemory(mag, PlatformDependent.INT_ARRAY_OFFSET,
-          baseObject, baseOffset + cursor, mag.length * 4);
+        Platform.copyMemory(
+          mag, Platform.INT_ARRAY_OFFSET, baseObject, baseOffset + cursor, 
mag.length * 4);
         setLong(ordinal, (cursor << 32) | ((long) (((integer.signum() + 1) << 
8) + mag.length)));
       }
     }
@@ -336,43 +336,43 @@ public final class UnsafeRow extends MutableRow {
   @Override
   public boolean getBoolean(int ordinal) {
     assertIndexIsValid(ordinal);
-    return PlatformDependent.UNSAFE.getBoolean(baseObject, 
getFieldOffset(ordinal));
+    return Platform.getBoolean(baseObject, getFieldOffset(ordinal));
   }
 
   @Override
   public byte getByte(int ordinal) {
     assertIndexIsValid(ordinal);
-    return PlatformDependent.UNSAFE.getByte(baseObject, 
getFieldOffset(ordinal));
+    return Platform.getByte(baseObject, getFieldOffset(ordinal));
   }
 
   @Override
   public short getShort(int ordinal) {
     assertIndexIsValid(ordinal);
-    return PlatformDependent.UNSAFE.getShort(baseObject, 
getFieldOffset(ordinal));
+    return Platform.getShort(baseObject, getFieldOffset(ordinal));
   }
 
   @Override
   public int getInt(int ordinal) {
     assertIndexIsValid(ordinal);
-    return PlatformDependent.UNSAFE.getInt(baseObject, 
getFieldOffset(ordinal));
+    return Platform.getInt(baseObject, getFieldOffset(ordinal));
   }
 
   @Override
   public long getLong(int ordinal) {
     assertIndexIsValid(ordinal);
-    return PlatformDependent.UNSAFE.getLong(baseObject, 
getFieldOffset(ordinal));
+    return Platform.getLong(baseObject, getFieldOffset(ordinal));
   }
 
   @Override
   public float getFloat(int ordinal) {
     assertIndexIsValid(ordinal);
-    return PlatformDependent.UNSAFE.getFloat(baseObject, 
getFieldOffset(ordinal));
+    return Platform.getFloat(baseObject, getFieldOffset(ordinal));
   }
 
   @Override
   public double getDouble(int ordinal) {
     assertIndexIsValid(ordinal);
-    return PlatformDependent.UNSAFE.getDouble(baseObject, 
getFieldOffset(ordinal));
+    return Platform.getDouble(baseObject, getFieldOffset(ordinal));
   }
 
   private static byte[] EMPTY = new byte[0];
@@ -391,13 +391,13 @@ public final class UnsafeRow extends MutableRow {
       assert signum >=0 && signum <= 2 : "invalid signum " + signum;
       int size = (int) (offsetAndSize & 0xff);
       int[] mag = new int[size];
-      PlatformDependent.copyMemory(baseObject, baseOffset + offset,
-        mag, PlatformDependent.INT_ARRAY_OFFSET, size * 4);
+      Platform.copyMemory(
+        baseObject, baseOffset + offset, mag, Platform.INT_ARRAY_OFFSET, size 
* 4);
 
       // create a BigInteger using signum and mag
       BigInteger v = new BigInteger(0, EMPTY);  // create the initial object
-      PlatformDependent.UNSAFE.putInt(v, 
PlatformDependent.BIG_INTEGER_SIGNUM_OFFSET, signum - 1);
-      PlatformDependent.UNSAFE.putObjectVolatile(v, 
PlatformDependent.BIG_INTEGER_MAG_OFFSET, mag);
+      Platform.putInt(v, Platform.BIG_INTEGER_SIGNUM_OFFSET, signum - 1);
+      Platform.putObjectVolatile(v, Platform.BIG_INTEGER_MAG_OFFSET, mag);
       return Decimal.apply(new BigDecimal(v, scale), precision, scale);
     }
   }
@@ -420,11 +420,11 @@ public final class UnsafeRow extends MutableRow {
       final int offset = (int) (offsetAndSize >> 32);
       final int size = (int) (offsetAndSize & ((1L << 32) - 1));
       final byte[] bytes = new byte[size];
-      PlatformDependent.copyMemory(
+      Platform.copyMemory(
         baseObject,
         baseOffset + offset,
         bytes,
-        PlatformDependent.BYTE_ARRAY_OFFSET,
+        Platform.BYTE_ARRAY_OFFSET,
         size
       );
       return bytes;
@@ -438,9 +438,8 @@ public final class UnsafeRow extends MutableRow {
     } else {
       final long offsetAndSize = getLong(ordinal);
       final int offset = (int) (offsetAndSize >> 32);
-      final int months = (int) PlatformDependent.UNSAFE.getLong(baseObject, 
baseOffset + offset);
-      final long microseconds =
-        PlatformDependent.UNSAFE.getLong(baseObject, baseOffset + offset + 8);
+      final int months = (int) Platform.getLong(baseObject, baseOffset + 
offset);
+      final long microseconds = Platform.getLong(baseObject, baseOffset + 
offset + 8);
       return new CalendarInterval(months, microseconds);
     }
   }
@@ -491,14 +490,14 @@ public final class UnsafeRow extends MutableRow {
   public UnsafeRow copy() {
     UnsafeRow rowCopy = new UnsafeRow();
     final byte[] rowDataCopy = new byte[sizeInBytes];
-    PlatformDependent.copyMemory(
+    Platform.copyMemory(
       baseObject,
       baseOffset,
       rowDataCopy,
-      PlatformDependent.BYTE_ARRAY_OFFSET,
+      Platform.BYTE_ARRAY_OFFSET,
       sizeInBytes
     );
-    rowCopy.pointTo(rowDataCopy, PlatformDependent.BYTE_ARRAY_OFFSET, 
numFields, sizeInBytes);
+    rowCopy.pointTo(rowDataCopy, Platform.BYTE_ARRAY_OFFSET, numFields, 
sizeInBytes);
     return rowCopy;
   }
 
@@ -518,18 +517,13 @@ public final class UnsafeRow extends MutableRow {
    */
   public void copyFrom(UnsafeRow row) {
     // copyFrom is only available for UnsafeRow created from byte array.
-    assert (baseObject instanceof byte[]) && baseOffset == 
PlatformDependent.BYTE_ARRAY_OFFSET;
+    assert (baseObject instanceof byte[]) && baseOffset == 
Platform.BYTE_ARRAY_OFFSET;
     if (row.sizeInBytes > this.sizeInBytes) {
       // resize the underlying byte[] if it's not large enough.
       this.baseObject = new byte[row.sizeInBytes];
     }
-    PlatformDependent.copyMemory(
-      row.baseObject,
-      row.baseOffset,
-      this.baseObject,
-      this.baseOffset,
-      row.sizeInBytes
-    );
+    Platform.copyMemory(
+      row.baseObject, row.baseOffset, this.baseObject, this.baseOffset, 
row.sizeInBytes);
     // update the sizeInBytes.
     this.sizeInBytes = row.sizeInBytes;
   }
@@ -544,19 +538,15 @@ public final class UnsafeRow extends MutableRow {
    */
   public void writeToStream(OutputStream out, byte[] writeBuffer) throws 
IOException {
     if (baseObject instanceof byte[]) {
-      int offsetInByteArray = (int) (PlatformDependent.BYTE_ARRAY_OFFSET - 
baseOffset);
+      int offsetInByteArray = (int) (Platform.BYTE_ARRAY_OFFSET - baseOffset);
       out.write((byte[]) baseObject, offsetInByteArray, sizeInBytes);
     } else {
       int dataRemaining = sizeInBytes;
       long rowReadPosition = baseOffset;
       while (dataRemaining > 0) {
         int toTransfer = Math.min(writeBuffer.length, dataRemaining);
-        PlatformDependent.copyMemory(
-          baseObject,
-          rowReadPosition,
-          writeBuffer,
-          PlatformDependent.BYTE_ARRAY_OFFSET,
-          toTransfer);
+        Platform.copyMemory(
+          baseObject, rowReadPosition, writeBuffer, 
Platform.BYTE_ARRAY_OFFSET, toTransfer);
         out.write(writeBuffer, 0, toTransfer);
         rowReadPosition += toTransfer;
         dataRemaining -= toTransfer;
@@ -584,13 +574,12 @@ public final class UnsafeRow extends MutableRow {
    * Returns the underlying bytes for this UnsafeRow.
    */
   public byte[] getBytes() {
-    if (baseObject instanceof byte[] && baseOffset == 
PlatformDependent.BYTE_ARRAY_OFFSET
+    if (baseObject instanceof byte[] && baseOffset == 
Platform.BYTE_ARRAY_OFFSET
       && (((byte[]) baseObject).length == sizeInBytes)) {
       return (byte[]) baseObject;
     } else {
       byte[] bytes = new byte[sizeInBytes];
-      PlatformDependent.copyMemory(baseObject, baseOffset, bytes,
-        PlatformDependent.BYTE_ARRAY_OFFSET, sizeInBytes);
+      Platform.copyMemory(baseObject, baseOffset, bytes, 
Platform.BYTE_ARRAY_OFFSET, sizeInBytes);
       return bytes;
     }
   }
@@ -600,8 +589,7 @@ public final class UnsafeRow extends MutableRow {
   public String toString() {
     StringBuilder build = new StringBuilder("[");
     for (int i = 0; i < sizeInBytes; i += 8) {
-      build.append(java.lang.Long.toHexString(
-        PlatformDependent.UNSAFE.getLong(baseObject, baseOffset + i)));
+      build.append(java.lang.Long.toHexString(Platform.getLong(baseObject, 
baseOffset + i)));
       build.append(',');
     }
     build.append(']');
@@ -619,12 +607,6 @@ public final class UnsafeRow extends MutableRow {
    * bytes in this string.
    */
   public void writeToMemory(Object target, long targetOffset) {
-    PlatformDependent.copyMemory(
-      baseObject,
-      baseOffset,
-      target,
-      targetOffset,
-      sizeInBytes
-    );
+    Platform.copyMemory(baseObject, baseOffset, target, targetOffset, 
sizeInBytes);
   }
 }

http://git-wip-us.apache.org/repos/asf/spark/blob/d378396f/sql/catalyst/src/main/java/org/apache/spark/sql/catalyst/expressions/UnsafeRowWriters.java
----------------------------------------------------------------------
diff --git 
a/sql/catalyst/src/main/java/org/apache/spark/sql/catalyst/expressions/UnsafeRowWriters.java
 
b/sql/catalyst/src/main/java/org/apache/spark/sql/catalyst/expressions/UnsafeRowWriters.java
index 28e7ec0..005351f 100644
--- 
a/sql/catalyst/src/main/java/org/apache/spark/sql/catalyst/expressions/UnsafeRowWriters.java
+++ 
b/sql/catalyst/src/main/java/org/apache/spark/sql/catalyst/expressions/UnsafeRowWriters.java
@@ -21,7 +21,7 @@ import java.math.BigInteger;
 
 import org.apache.spark.sql.catalyst.InternalRow;
 import org.apache.spark.sql.types.Decimal;
-import org.apache.spark.unsafe.PlatformDependent;
+import org.apache.spark.unsafe.Platform;
 import org.apache.spark.unsafe.array.ByteArrayMethods;
 import org.apache.spark.unsafe.types.ByteArray;
 import org.apache.spark.unsafe.types.CalendarInterval;
@@ -58,27 +58,27 @@ public class UnsafeRowWriters {
       final Object base = target.getBaseObject();
       final long offset = target.getBaseOffset() + cursor;
       // zero-out the bytes
-      PlatformDependent.UNSAFE.putLong(base, offset, 0L);
-      PlatformDependent.UNSAFE.putLong(base, offset + 8, 0L);
+      Platform.putLong(base, offset, 0L);
+      Platform.putLong(base, offset + 8, 0L);
 
       if (input == null) {
         target.setNullAt(ordinal);
         // keep the offset and length for update
         int fieldOffset = 
UnsafeRow.calculateBitSetWidthInBytes(target.numFields()) + ordinal * 8;
-        PlatformDependent.UNSAFE.putLong(base, target.getBaseOffset() + 
fieldOffset,
+        Platform.putLong(base, target.getBaseOffset() + fieldOffset,
           ((long) cursor) << 32);
         return SIZE;
       }
 
       final BigInteger integer = input.toJavaBigDecimal().unscaledValue();
       int signum = integer.signum() + 1;
-      final int[] mag = (int[]) 
PlatformDependent.UNSAFE.getObjectVolatile(integer,
-        PlatformDependent.BIG_INTEGER_MAG_OFFSET);
+      final int[] mag = (int[]) Platform.getObjectVolatile(
+        integer, Platform.BIG_INTEGER_MAG_OFFSET);
       assert(mag.length <= 4);
 
       // Write the bytes to the variable length portion.
-      PlatformDependent.copyMemory(mag, PlatformDependent.INT_ARRAY_OFFSET,
-        base, target.getBaseOffset() + cursor, mag.length * 4);
+      Platform.copyMemory(
+        mag, Platform.INT_ARRAY_OFFSET, base, target.getBaseOffset() + cursor, 
mag.length * 4);
       // Set the fixed length portion.
       target.setLong(ordinal, (((long) cursor) << 32) | ((long) ((signum << 8) 
+ mag.length)));
 
@@ -99,8 +99,7 @@ public class UnsafeRowWriters {
 
       // zero-out the padding bytes
       if ((numBytes & 0x07) > 0) {
-        PlatformDependent.UNSAFE.putLong(
-          target.getBaseObject(), offset + ((numBytes >> 3) << 3), 0L);
+        Platform.putLong(target.getBaseObject(), offset + ((numBytes >> 3) << 
3), 0L);
       }
 
       // Write the bytes to the variable length portion.
@@ -125,8 +124,7 @@ public class UnsafeRowWriters {
 
       // zero-out the padding bytes
       if ((numBytes & 0x07) > 0) {
-        PlatformDependent.UNSAFE.putLong(
-          target.getBaseObject(), offset + ((numBytes >> 3) << 3), 0L);
+        Platform.putLong(target.getBaseObject(), offset + ((numBytes >> 3) << 
3), 0L);
       }
 
       // Write the bytes to the variable length portion.
@@ -167,8 +165,7 @@ public class UnsafeRowWriters {
 
         // zero-out the padding bytes
         if ((numBytes & 0x07) > 0) {
-          PlatformDependent.UNSAFE.putLong(
-            target.getBaseObject(), offset + ((numBytes >> 3) << 3), 0L);
+          Platform.putLong(target.getBaseObject(), offset + ((numBytes >> 3) 
<< 3), 0L);
         }
 
         // Write the bytes to the variable length portion.
@@ -191,8 +188,8 @@ public class UnsafeRowWriters {
       final long offset = target.getBaseOffset() + cursor;
 
       // Write the months and microseconds fields of Interval to the variable 
length portion.
-      PlatformDependent.UNSAFE.putLong(target.getBaseObject(), offset, 
input.months);
-      PlatformDependent.UNSAFE.putLong(target.getBaseObject(), offset + 8, 
input.microseconds);
+      Platform.putLong(target.getBaseObject(), offset, input.months);
+      Platform.putLong(target.getBaseObject(), offset + 8, input.microseconds);
 
       // Set the fixed length portion.
       target.setLong(ordinal, ((long) cursor) << 32);
@@ -212,12 +209,11 @@ public class UnsafeRowWriters {
       final long offset = target.getBaseOffset() + cursor;
 
       // write the number of elements into first 4 bytes.
-      PlatformDependent.UNSAFE.putInt(target.getBaseObject(), offset, 
input.numElements());
+      Platform.putInt(target.getBaseObject(), offset, input.numElements());
 
       // zero-out the padding bytes
       if ((numBytes & 0x07) > 0) {
-        PlatformDependent.UNSAFE.putLong(
-          target.getBaseObject(), offset + ((numBytes >> 3) << 3), 0L);
+        Platform.putLong(target.getBaseObject(), offset + ((numBytes >> 3) << 
3), 0L);
       }
 
       // Write the bytes to the variable length portion.
@@ -247,14 +243,13 @@ public class UnsafeRowWriters {
       final int numBytes = 4 + 4 + keysNumBytes + valuesNumBytes;
 
       // write the number of elements into first 4 bytes.
-      PlatformDependent.UNSAFE.putInt(target.getBaseObject(), offset, 
input.numElements());
+      Platform.putInt(target.getBaseObject(), offset, input.numElements());
       // write the numBytes of key array into second 4 bytes.
-      PlatformDependent.UNSAFE.putInt(target.getBaseObject(), offset + 4, 
keysNumBytes);
+      Platform.putInt(target.getBaseObject(), offset + 4, keysNumBytes);
 
       // zero-out the padding bytes
       if ((numBytes & 0x07) > 0) {
-        PlatformDependent.UNSAFE.putLong(
-          target.getBaseObject(), offset + ((numBytes >> 3) << 3), 0L);
+        Platform.putLong(target.getBaseObject(), offset + ((numBytes >> 3) << 
3), 0L);
       }
 
       // Write the bytes of key array to the variable length portion.

http://git-wip-us.apache.org/repos/asf/spark/blob/d378396f/sql/catalyst/src/main/java/org/apache/spark/sql/catalyst/expressions/UnsafeWriters.java
----------------------------------------------------------------------
diff --git 
a/sql/catalyst/src/main/java/org/apache/spark/sql/catalyst/expressions/UnsafeWriters.java
 
b/sql/catalyst/src/main/java/org/apache/spark/sql/catalyst/expressions/UnsafeWriters.java
index 0e8e405..cd83695 100644
--- 
a/sql/catalyst/src/main/java/org/apache/spark/sql/catalyst/expressions/UnsafeWriters.java
+++ 
b/sql/catalyst/src/main/java/org/apache/spark/sql/catalyst/expressions/UnsafeWriters.java
@@ -18,8 +18,7 @@
 package org.apache.spark.sql.catalyst.expressions;
 
 import org.apache.spark.sql.types.Decimal;
-import org.apache.spark.unsafe.PlatformDependent;
-import org.apache.spark.unsafe.array.ByteArrayMethods;
+import org.apache.spark.unsafe.Platform;
 import org.apache.spark.unsafe.types.CalendarInterval;
 import org.apache.spark.unsafe.types.UTF8String;
 
@@ -36,17 +35,11 @@ public class UnsafeWriters {
 
     // zero-out the padding bytes
 //    if ((numBytes & 0x07) > 0) {
-//      PlatformDependent.UNSAFE.putLong(targetObject, targetOffset + 
((numBytes >> 3) << 3), 0L);
+//      Platform.putLong(targetObject, targetOffset + ((numBytes >> 3) << 3), 
0L);
 //    }
 
     // Write the UnsafeData to the target memory.
-    PlatformDependent.copyMemory(
-      inputObject,
-      inputOffset,
-      targetObject,
-      targetOffset,
-      numBytes
-    );
+    Platform.copyMemory(inputObject, inputOffset, targetObject, targetOffset, 
numBytes);
   }
 
   public static int getRoundedSize(int size) {
@@ -68,16 +61,11 @@ public class UnsafeWriters {
       assert(numBytes <= 16);
 
       // zero-out the bytes
-      PlatformDependent.UNSAFE.putLong(targetObject, targetOffset, 0L);
-      PlatformDependent.UNSAFE.putLong(targetObject, targetOffset + 8, 0L);
+      Platform.putLong(targetObject, targetOffset, 0L);
+      Platform.putLong(targetObject, targetOffset + 8, 0L);
 
       // Write the bytes to the variable length portion.
-      PlatformDependent.copyMemory(bytes,
-        PlatformDependent.BYTE_ARRAY_OFFSET,
-        targetObject,
-        targetOffset,
-        numBytes);
-
+      Platform.copyMemory(bytes, Platform.BYTE_ARRAY_OFFSET, targetObject, 
targetOffset, numBytes);
       return 16;
     }
   }
@@ -111,8 +99,7 @@ public class UnsafeWriters {
       final int numBytes = input.length;
 
       // Write the bytes to the variable length portion.
-      writeToMemory(input, PlatformDependent.BYTE_ARRAY_OFFSET,
-        targetObject, targetOffset, numBytes);
+      writeToMemory(input, Platform.BYTE_ARRAY_OFFSET, targetObject, 
targetOffset, numBytes);
 
       return getRoundedSize(numBytes);
     }
@@ -144,11 +131,9 @@ public class UnsafeWriters {
     }
 
     public static int write(Object targetObject, long targetOffset, 
CalendarInterval input) {
-
       // Write the months and microseconds fields of Interval to the variable 
length portion.
-      PlatformDependent.UNSAFE.putLong(targetObject, targetOffset, 
input.months);
-      PlatformDependent.UNSAFE.putLong(targetObject, targetOffset + 8, 
input.microseconds);
-
+      Platform.putLong(targetObject, targetOffset, input.months);
+      Platform.putLong(targetObject, targetOffset + 8, input.microseconds);
       return 16;
     }
   }
@@ -165,11 +150,11 @@ public class UnsafeWriters {
       final int numBytes = input.getSizeInBytes();
 
       // write the number of elements into first 4 bytes.
-      PlatformDependent.UNSAFE.putInt(targetObject, targetOffset, 
input.numElements());
+      Platform.putInt(targetObject, targetOffset, input.numElements());
 
       // Write the bytes to the variable length portion.
-      writeToMemory(input.getBaseObject(), input.getBaseOffset(),
-        targetObject, targetOffset + 4, numBytes);
+      writeToMemory(
+        input.getBaseObject(), input.getBaseOffset(), targetObject, 
targetOffset + 4, numBytes);
 
       return getRoundedSize(numBytes + 4);
     }
@@ -190,9 +175,9 @@ public class UnsafeWriters {
       final int numBytes = 4 + 4 + keysNumBytes + valuesNumBytes;
 
       // write the number of elements into first 4 bytes.
-      PlatformDependent.UNSAFE.putInt(targetObject, targetOffset, 
input.numElements());
+      Platform.putInt(targetObject, targetOffset, input.numElements());
       // write the numBytes of key array into second 4 bytes.
-      PlatformDependent.UNSAFE.putInt(targetObject, targetOffset + 4, 
keysNumBytes);
+      Platform.putInt(targetObject, targetOffset + 4, keysNumBytes);
 
       // Write the bytes of key array to the variable length portion.
       writeToMemory(keyArray.getBaseObject(), keyArray.getBaseOffset(),

http://git-wip-us.apache.org/repos/asf/spark/blob/d378396f/sql/catalyst/src/main/java/org/apache/spark/sql/execution/UnsafeExternalRowSorter.java
----------------------------------------------------------------------
diff --git 
a/sql/catalyst/src/main/java/org/apache/spark/sql/execution/UnsafeExternalRowSorter.java
 
b/sql/catalyst/src/main/java/org/apache/spark/sql/execution/UnsafeExternalRowSorter.java
index a5ae2b9..1d27182 100644
--- 
a/sql/catalyst/src/main/java/org/apache/spark/sql/execution/UnsafeExternalRowSorter.java
+++ 
b/sql/catalyst/src/main/java/org/apache/spark/sql/execution/UnsafeExternalRowSorter.java
@@ -31,7 +31,7 @@ import org.apache.spark.sql.catalyst.InternalRow;
 import org.apache.spark.sql.catalyst.expressions.UnsafeProjection;
 import org.apache.spark.sql.catalyst.expressions.UnsafeRow;
 import org.apache.spark.sql.types.StructType;
-import org.apache.spark.unsafe.PlatformDependent;
+import org.apache.spark.unsafe.Platform;
 import org.apache.spark.util.collection.unsafe.sort.PrefixComparator;
 import org.apache.spark.util.collection.unsafe.sort.RecordComparator;
 import org.apache.spark.util.collection.unsafe.sort.UnsafeExternalSorter;
@@ -157,7 +157,7 @@ final class UnsafeExternalRowSorter {
             cleanupResources();
             // Scala iterators don't declare any checked exceptions, so we 
need to use this hack
             // to re-throw the exception:
-            PlatformDependent.throwException(e);
+            Platform.throwException(e);
           }
           throw new RuntimeException("Exception should have been re-thrown in 
next()");
         };

http://git-wip-us.apache.org/repos/asf/spark/blob/d378396f/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/codegen/CodeGenerator.scala
----------------------------------------------------------------------
diff --git 
a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/codegen/CodeGenerator.scala
 
b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/codegen/CodeGenerator.scala
index c21f4d6..bf96248 100644
--- 
a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/codegen/CodeGenerator.scala
+++ 
b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/codegen/CodeGenerator.scala
@@ -28,7 +28,7 @@ import org.apache.spark.Logging
 import org.apache.spark.sql.catalyst.InternalRow
 import org.apache.spark.sql.catalyst.expressions._
 import org.apache.spark.sql.types._
-import org.apache.spark.unsafe.PlatformDependent
+import org.apache.spark.unsafe.Platform
 import org.apache.spark.unsafe.types._
 
 
@@ -371,7 +371,7 @@ abstract class CodeGenerator[InType <: AnyRef, OutType <: 
AnyRef] extends Loggin
     // Cannot be under package codegen, or fail with 
java.lang.InstantiationException
     
evaluator.setClassName("org.apache.spark.sql.catalyst.expressions.GeneratedClass")
     evaluator.setDefaultImports(Array(
-      classOf[PlatformDependent].getName,
+      classOf[Platform].getName,
       classOf[InternalRow].getName,
       classOf[UnsafeRow].getName,
       classOf[UTF8String].getName,

http://git-wip-us.apache.org/repos/asf/spark/blob/d378396f/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/codegen/GenerateUnsafeProjection.scala
----------------------------------------------------------------------
diff --git 
a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/codegen/GenerateUnsafeProjection.scala
 
b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/codegen/GenerateUnsafeProjection.scala
index 29f6a7b..b2fb913 100644
--- 
a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/codegen/GenerateUnsafeProjection.scala
+++ 
b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/codegen/GenerateUnsafeProjection.scala
@@ -145,12 +145,11 @@ object GenerateUnsafeProjection extends 
CodeGenerator[Seq[Expression], UnsafePro
           if ($buffer.length < $numBytes) {
             // This will not happen frequently, because the buffer is re-used.
             byte[] $tmp = new byte[$numBytes * 2];
-            PlatformDependent.copyMemory($buffer, 
PlatformDependent.BYTE_ARRAY_OFFSET,
-              $tmp, PlatformDependent.BYTE_ARRAY_OFFSET, $buffer.length);
+            Platform.copyMemory($buffer, Platform.BYTE_ARRAY_OFFSET,
+              $tmp, Platform.BYTE_ARRAY_OFFSET, $buffer.length);
             $buffer = $tmp;
           }
-          $output.pointTo($buffer, PlatformDependent.BYTE_ARRAY_OFFSET,
-            ${inputTypes.length}, $numBytes);
+          $output.pointTo($buffer, Platform.BYTE_ARRAY_OFFSET, 
${inputTypes.length}, $numBytes);
          """
       } else {
         ""
@@ -183,7 +182,7 @@ object GenerateUnsafeProjection extends 
CodeGenerator[Seq[Expression], UnsafePro
 
     val code = s"""
       $cursor = $fixedSize;
-      $output.pointTo($buffer, PlatformDependent.BYTE_ARRAY_OFFSET, 
${inputTypes.length}, $cursor);
+      $output.pointTo($buffer, Platform.BYTE_ARRAY_OFFSET, 
${inputTypes.length}, $cursor);
       ${ctx.splitExpressions(row, convertedFields)}
       """
     GeneratedExpressionCode(code, "false", output)
@@ -267,17 +266,17 @@ object GenerateUnsafeProjection extends 
CodeGenerator[Seq[Expression], UnsafePro
         // Should we do word align?
         val elementSize = elementType.defaultSize
         s"""
-          PlatformDependent.UNSAFE.put${ctx.primitiveTypeName(elementType)}(
+          Platform.put${ctx.primitiveTypeName(elementType)}(
             $buffer,
-            PlatformDependent.BYTE_ARRAY_OFFSET + $cursor,
+            Platform.BYTE_ARRAY_OFFSET + $cursor,
             ${convertedElement.primitive});
           $cursor += $elementSize;
         """
       case t: DecimalType if t.precision <= Decimal.MAX_LONG_DIGITS =>
         s"""
-          PlatformDependent.UNSAFE.putLong(
+          Platform.putLong(
             $buffer,
-            PlatformDependent.BYTE_ARRAY_OFFSET + $cursor,
+            Platform.BYTE_ARRAY_OFFSET + $cursor,
             ${convertedElement.primitive}.toUnscaledLong());
           $cursor += 8;
         """
@@ -286,7 +285,7 @@ object GenerateUnsafeProjection extends 
CodeGenerator[Seq[Expression], UnsafePro
         s"""
           $cursor += $writer.write(
             $buffer,
-            PlatformDependent.BYTE_ARRAY_OFFSET + $cursor,
+            Platform.BYTE_ARRAY_OFFSET + $cursor,
             $elements[$index]);
         """
     }
@@ -320,23 +319,16 @@ object GenerateUnsafeProjection extends 
CodeGenerator[Seq[Expression], UnsafePro
           for (int $index = 0; $index < $numElements; $index++) {
             if ($checkNull) {
               // If element is null, write the negative value address into 
offset region.
-              PlatformDependent.UNSAFE.putInt(
-                $buffer,
-                PlatformDependent.BYTE_ARRAY_OFFSET + 4 * $index,
-                -$cursor);
+              Platform.putInt($buffer, Platform.BYTE_ARRAY_OFFSET + 4 * 
$index, -$cursor);
             } else {
-              PlatformDependent.UNSAFE.putInt(
-                $buffer,
-                PlatformDependent.BYTE_ARRAY_OFFSET + 4 * $index,
-                $cursor);
-
+              Platform.putInt($buffer, Platform.BYTE_ARRAY_OFFSET + 4 * 
$index, $cursor);
               $writeElement
             }
           }
 
           $output.pointTo(
             $buffer,
-            PlatformDependent.BYTE_ARRAY_OFFSET,
+            Platform.BYTE_ARRAY_OFFSET,
             $numElements,
             $numBytes);
         }

http://git-wip-us.apache.org/repos/asf/spark/blob/d378396f/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/codegen/GenerateUnsafeRowJoiner.scala
----------------------------------------------------------------------
diff --git 
a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/codegen/GenerateUnsafeRowJoiner.scala
 
b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/codegen/GenerateUnsafeRowJoiner.scala
index 8aaa5b4..da91ff2 100644
--- 
a/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/codegen/GenerateUnsafeRowJoiner.scala
+++ 
b/sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/codegen/GenerateUnsafeRowJoiner.scala
@@ -19,7 +19,7 @@ package org.apache.spark.sql.catalyst.expressions.codegen
 
 import org.apache.spark.sql.catalyst.expressions.{UnsafeRow, Attribute}
 import org.apache.spark.sql.types.StructType
-import org.apache.spark.unsafe.PlatformDependent
+import org.apache.spark.unsafe.Platform
 
 
 abstract class UnsafeRowJoiner {
@@ -52,9 +52,9 @@ object GenerateUnsafeRowJoiner extends 
CodeGenerator[(StructType, StructType), U
   }
 
   def create(schema1: StructType, schema2: StructType): UnsafeRowJoiner = {
-    val offset = PlatformDependent.BYTE_ARRAY_OFFSET
-    val getLong = "PlatformDependent.UNSAFE.getLong"
-    val putLong = "PlatformDependent.UNSAFE.putLong"
+    val offset = Platform.BYTE_ARRAY_OFFSET
+    val getLong = "Platform.getLong"
+    val putLong = "Platform.putLong"
 
     val bitset1Words = (schema1.size + 63) / 64
     val bitset2Words = (schema2.size + 63) / 64
@@ -96,7 +96,7 @@ object GenerateUnsafeRowJoiner extends 
CodeGenerator[(StructType, StructType), U
     var cursor = offset + outputBitsetWords * 8
     val copyFixedLengthRow1 = s"""
        |// Copy fixed length data for row1
-       |PlatformDependent.copyMemory(
+       |Platform.copyMemory(
        |  obj1, offset1 + ${bitset1Words * 8},
        |  buf, $cursor,
        |  ${schema1.size * 8});
@@ -106,7 +106,7 @@ object GenerateUnsafeRowJoiner extends 
CodeGenerator[(StructType, StructType), U
     // --------------------- copy fixed length portion from row 2 
----------------------- //
     val copyFixedLengthRow2 = s"""
        |// Copy fixed length data for row2
-       |PlatformDependent.copyMemory(
+       |Platform.copyMemory(
        |  obj2, offset2 + ${bitset2Words * 8},
        |  buf, $cursor,
        |  ${schema2.size * 8});
@@ -118,7 +118,7 @@ object GenerateUnsafeRowJoiner extends 
CodeGenerator[(StructType, StructType), U
     val copyVariableLengthRow1 = s"""
        |// Copy variable length data for row1
        |long numBytesVariableRow1 = row1.getSizeInBytes() - 
$numBytesBitsetAndFixedRow1;
-       |PlatformDependent.copyMemory(
+       |Platform.copyMemory(
        |  obj1, offset1 + ${(bitset1Words + schema1.size) * 8},
        |  buf, $cursor,
        |  numBytesVariableRow1);
@@ -129,7 +129,7 @@ object GenerateUnsafeRowJoiner extends 
CodeGenerator[(StructType, StructType), U
     val copyVariableLengthRow2 = s"""
        |// Copy variable length data for row2
        |long numBytesVariableRow2 = row2.getSizeInBytes() - 
$numBytesBitsetAndFixedRow2;
-       |PlatformDependent.copyMemory(
+       |Platform.copyMemory(
        |  obj2, offset2 + ${(bitset2Words + schema2.size) * 8},
        |  buf, $cursor + numBytesVariableRow1,
        |  numBytesVariableRow2);


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

Reply via email to