You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@asterixdb.apache.org by mb...@apache.org on 2016/05/13 02:40:53 UTC

[03/12] incubator-asterixdb git commit: ASTERIXDB-1436: Big Object Support For Storage

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/asterixdb/asterix-app/src/test/resources/runtimets/testsuite.xml
----------------------------------------------------------------------
diff --git a/asterixdb/asterix-app/src/test/resources/runtimets/testsuite.xml b/asterixdb/asterix-app/src/test/resources/runtimets/testsuite.xml
index e9de6ab..c040e9d 100644
--- a/asterixdb/asterix-app/src/test/resources/runtimets/testsuite.xml
+++ b/asterixdb/asterix-app/src/test/resources/runtimets/testsuite.xml
@@ -6460,6 +6460,16 @@
         <output-dir compare="Text">big_object_load</output-dir>
       </compilation-unit>
     </test-case>
+    <test-case FilePath="big-object">
+      <compilation-unit name="big_object_bulkload">
+        <output-dir compare="Text">big_object_bulkload</output-dir>
+      </compilation-unit>
+    </test-case>
+    <test-case FilePath="big-object">
+      <compilation-unit name="big_object_insert">
+        <output-dir compare="Text">big_object_insert</output-dir>
+      </compilation-unit>
+    </test-case>
   </test-group>
   <test-group name="temporal">
     <test-case FilePath="temporal">

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/transactions/ILogRecord.java
----------------------------------------------------------------------
diff --git a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/transactions/ILogRecord.java b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/transactions/ILogRecord.java
index b86aebe..abc1211 100644
--- a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/transactions/ILogRecord.java
+++ b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/transactions/ILogRecord.java
@@ -25,10 +25,11 @@ import org.apache.hyracks.dataflow.common.data.accessors.ITupleReference;
 
 public interface ILogRecord {
 
-    public enum RECORD_STATUS {
+    enum RecordReadStatus {
         TRUNCATED,
         BAD_CHKSUM,
-        OK
+        OK,
+        LARGE_RECORD
     }
 
     public static final int JOB_TERMINATE_LOG_SIZE = 14; //JOB_COMMIT or ABORT log type
@@ -37,7 +38,7 @@ public interface ILogRecord {
     public static final int FLUSH_LOG_SIZE = 18;
     public static final int WAIT_LOG_SIZE = 14;
 
-    public LogRecord.RECORD_STATUS readLogRecord(ByteBuffer buffer);
+    public RecordReadStatus readLogRecord(ByteBuffer buffer);
 
     public void writeLogRecord(ByteBuffer buffer);
 
@@ -115,7 +116,7 @@ public interface ILogRecord {
 
     public int writeRemoteRecoveryLog(ByteBuffer buffer);
 
-    public RECORD_STATUS readRemoteLog(ByteBuffer buffer, boolean remoteRecoveryLog);
+    public RecordReadStatus readRemoteLog(ByteBuffer buffer, boolean remoteRecoveryLog);
 
     public void setReplicationThread(IReplicationThread replicationThread);
 

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/transactions/LogRecord.java
----------------------------------------------------------------------
diff --git a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/transactions/LogRecord.java b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/transactions/LogRecord.java
index 2eb8244..c35b155 100644
--- a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/transactions/LogRecord.java
+++ b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/transactions/LogRecord.java
@@ -236,12 +236,12 @@ public class LogRecord implements ILogRecord {
     }
 
     @Override
-    public RECORD_STATUS readLogRecord(ByteBuffer buffer) {
+    public RecordReadStatus readLogRecord(ByteBuffer buffer) {
         int beginOffset = buffer.position();
 
         //read common fields
-        RECORD_STATUS status = readLogCommonFields(buffer);
-        if (status != RECORD_STATUS.OK) {
+        RecordReadStatus status = readLogCommonFields(buffer);
+        if (status != RecordReadStatus.OK) {
             buffer.position(beginOffset);
             return status;
         }
@@ -249,20 +249,20 @@ public class LogRecord implements ILogRecord {
         // attempt to read checksum
         if (buffer.remaining() < CHKSUM_LEN) {
             buffer.position(beginOffset);
-            return RECORD_STATUS.TRUNCATED;
+            return RecordReadStatus.TRUNCATED;
         }
         checksum = buffer.getLong();
         if (checksum != generateChecksum(buffer, beginOffset, logSize - CHKSUM_LEN)) {
-            return RECORD_STATUS.BAD_CHKSUM;
+            return RecordReadStatus.BAD_CHKSUM;
         }
 
-        return RECORD_STATUS.OK;
+        return RecordReadStatus.OK;
     }
 
-    private RECORD_STATUS readLogCommonFields(ByteBuffer buffer) {
+    private RecordReadStatus readLogCommonFields(ByteBuffer buffer) {
         //first we need the logtype and Job ID, if the buffer isn't that big, then no dice.
         if (buffer.remaining() < ALL_RECORD_HEADER_LEN) {
-            return RECORD_STATUS.TRUNCATED;
+            return RecordReadStatus.TRUNCATED;
         }
         logSource = buffer.get();
         logType = buffer.get();
@@ -270,7 +270,7 @@ public class LogRecord implements ILogRecord {
 
         if (logType == LogType.FLUSH) {
             if (buffer.remaining() < DatasetId.BYTES) {
-                return RECORD_STATUS.TRUNCATED;
+                return RecordReadStatus.TRUNCATED;
             }
             datasetId = buffer.getInt();
             resourceId = 0l;
@@ -284,7 +284,7 @@ public class LogRecord implements ILogRecord {
             } else {
                 //attempt to read in the resourcePartition, dsid, PK hash and PK length
                 if (buffer.remaining() < ENTITYCOMMIT_UPDATE_HEADER_LEN) {
-                    return RECORD_STATUS.TRUNCATED;
+                    return RecordReadStatus.TRUNCATED;
                 }
                 resourcePartition = buffer.getInt();
                 datasetId = buffer.getInt();
@@ -292,7 +292,7 @@ public class LogRecord implements ILogRecord {
                 PKValueSize = buffer.getInt();
                 // attempt to read in the PK
                 if (buffer.remaining() < PKValueSize) {
-                    return RECORD_STATUS.TRUNCATED;
+                    return RecordReadStatus.TRUNCATED;
                 }
                 if (PKValueSize <= 0) {
                     throw new IllegalStateException("Primary Key Size is less than or equal to 0");
@@ -303,7 +303,7 @@ public class LogRecord implements ILogRecord {
             if (logType == LogType.UPDATE) {
                 // attempt to read in the previous LSN, log size, new value size, and new record type
                 if (buffer.remaining() < UPDATE_LSN_HEADER + UPDATE_BODY_HEADER) {
-                    return RECORD_STATUS.TRUNCATED;
+                    return RecordReadStatus.TRUNCATED;
                 }
                 prevLSN = buffer.getLong();
                 resourceId = buffer.getLong();
@@ -312,7 +312,10 @@ public class LogRecord implements ILogRecord {
                 newOp = buffer.get();
                 newValueSize = buffer.getInt();
                 if (buffer.remaining() < newValueSize) {
-                    return RECORD_STATUS.TRUNCATED;
+                    if (logSize > buffer.capacity()) {
+                        return RecordReadStatus.LARGE_RECORD;
+                    }
+                    return RecordReadStatus.TRUNCATED;
                 }
                 newValue = readTuple(buffer, readNewValue, fieldCnt, newValueSize);
             } else {
@@ -320,16 +323,16 @@ public class LogRecord implements ILogRecord {
             }
         }
 
-        return RECORD_STATUS.OK;
+        return RecordReadStatus.OK;
     }
 
     @Override
-    public RECORD_STATUS readRemoteLog(ByteBuffer buffer, boolean remoteRecoveryLog) {
+    public RecordReadStatus readRemoteLog(ByteBuffer buffer, boolean remoteRecoveryLog) {
         int beginOffset = buffer.position();
 
         //read common fields
-        RECORD_STATUS status = readLogCommonFields(buffer);
-        if (status != RECORD_STATUS.OK) {
+        RecordReadStatus status = readLogCommonFields(buffer);
+        if (status != RecordReadStatus.OK) {
             buffer.position(beginOffset);
             return status;
         }
@@ -346,11 +349,11 @@ public class LogRecord implements ILogRecord {
                     nodeId = new String(nodeIdBytes);
                 } else {
                     buffer.position(beginOffset);
-                    return RECORD_STATUS.TRUNCATED;
+                    return RecordReadStatus.TRUNCATED;
                 }
             } else {
                 buffer.position(beginOffset);
-                return RECORD_STATUS.TRUNCATED;
+                return RecordReadStatus.TRUNCATED;
             }
         }
 
@@ -360,11 +363,11 @@ public class LogRecord implements ILogRecord {
                 LSN = buffer.getLong();
             } else {
                 buffer.position(beginOffset);
-                return RECORD_STATUS.TRUNCATED;
+                return RecordReadStatus.TRUNCATED;
             }
         }
 
-        return RECORD_STATUS.OK;
+        return RecordReadStatus.OK;
     }
 
     private ITupleReference readPKValue(ByteBuffer buffer) {

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/asterixdb/asterix-common/src/test/java/org/apache/asterix/test/aql/TestExecutor.java
----------------------------------------------------------------------
diff --git a/asterixdb/asterix-common/src/test/java/org/apache/asterix/test/aql/TestExecutor.java b/asterixdb/asterix-common/src/test/java/org/apache/asterix/test/aql/TestExecutor.java
index e23837b..491bc70 100644
--- a/asterixdb/asterix-common/src/test/java/org/apache/asterix/test/aql/TestExecutor.java
+++ b/asterixdb/asterix-common/src/test/java/org/apache/asterix/test/aql/TestExecutor.java
@@ -168,6 +168,9 @@ public class TestExecutor {
             if (lineActual != null) {
                 throw new Exception("Result for " + scriptFile + " changed at line " + num + ":\n< \n> " + lineActual);
             }
+        } catch (Exception e) {
+            System.err.println("Actual results file: " + actualFile.toString());
+            throw e;
         } finally {
             readerExpected.close();
             readerActual.close();

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/dataflow/FeedRecordDataFlowController.java
----------------------------------------------------------------------
diff --git a/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/dataflow/FeedRecordDataFlowController.java b/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/dataflow/FeedRecordDataFlowController.java
index a092620..ac30172 100644
--- a/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/dataflow/FeedRecordDataFlowController.java
+++ b/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/dataflow/FeedRecordDataFlowController.java
@@ -78,11 +78,6 @@ public class FeedRecordDataFlowController<T> extends AbstractFeedDataFlowControl
                 tb.addFieldEndOffset();
                 addMetaPart(tb, record);
                 addPrimaryKeys(tb, record);
-                if (tb.getSize() > tupleForwarder.getMaxRecordSize()) {
-                    // log
-                    feedLogManager.logRecord(record.toString(), ExternalDataConstants.ERROR_LARGE_RECORD);
-                    continue;
-                }
                 tupleForwarder.addTuple(tb);
             }
         } catch (InterruptedException e) {

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/dataflow/FeedTupleForwarder.java
----------------------------------------------------------------------
diff --git a/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/dataflow/FeedTupleForwarder.java b/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/dataflow/FeedTupleForwarder.java
index 63787ee..60aaea2 100644
--- a/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/dataflow/FeedTupleForwarder.java
+++ b/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/dataflow/FeedTupleForwarder.java
@@ -23,10 +23,8 @@ import java.nio.ByteBuffer;
 
 import javax.annotation.Nonnull;
 
-import org.apache.asterix.common.api.IAsterixAppRuntimeContext;
 import org.apache.asterix.external.api.ITupleForwarder;
 import org.apache.asterix.external.util.DataflowUtils;
-import org.apache.asterix.external.util.ExternalDataConstants;
 import org.apache.asterix.external.util.FeedLogManager;
 import org.apache.asterix.external.util.FeedMessageUtils;
 import org.apache.hyracks.api.comm.IFrame;
@@ -41,7 +39,6 @@ import org.apache.hyracks.dataflow.common.comm.util.FrameUtils;
 public class FeedTupleForwarder implements ITupleForwarder {
 
     private final FeedLogManager feedLogManager;
-    private int maxRecordSize; // temporary until the big object in storage is solved
     private FrameTupleAppender appender;
     private IFrame frame;
     private IFrameWriter writer;
@@ -59,8 +56,6 @@ public class FeedTupleForwarder implements ITupleForwarder {
     @Override
     public void initialize(IHyracksTaskContext ctx, IFrameWriter writer) throws HyracksDataException {
         if (!initialized) {
-            this.maxRecordSize = ((IAsterixAppRuntimeContext) ctx.getJobletContext().getApplicationContext()
-                    .getApplicationObject()).getBufferCache().getPageSize() / 2;
             this.frame = new VSizeFrame(ctx);
             this.writer = writer;
             this.appender = new FrameTupleAppender(frame);
@@ -75,13 +70,6 @@ public class FeedTupleForwarder implements ITupleForwarder {
 
     @Override
     public void addTuple(ArrayTupleBuilder tb) throws HyracksDataException {
-        if (tb.getSize() > maxRecordSize) {
-            try {
-                feedLogManager.logRecord(tb.toString(), ExternalDataConstants.ERROR_LARGE_RECORD);
-            } catch (IOException e) {
-                throw new HyracksDataException(e);
-            }
-        }
         if (paused) {
             synchronized (this) {
                 while (paused) {
@@ -120,8 +108,4 @@ public class FeedTupleForwarder implements ITupleForwarder {
     public void flush() throws HyracksDataException {
         appender.flush(writer);
     }
-
-    public int getMaxRecordSize() {
-        return maxRecordSize;
-    }
 }

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/util/ExternalDataConstants.java
----------------------------------------------------------------------
diff --git a/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/util/ExternalDataConstants.java b/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/util/ExternalDataConstants.java
index b5ec27a..0fd76cc 100644
--- a/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/util/ExternalDataConstants.java
+++ b/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/util/ExternalDataConstants.java
@@ -227,6 +227,5 @@ public class ExternalDataConstants {
     public static final String FORMAT_CSV = "csv";
     public static final String TEST_RECORD_WITH_PK = "test-record-with-pk";
 
-    public static final String ERROR_LARGE_RECORD = "Record is too large";
     public static final String ERROR_PARSE_RECORD = "Parser failed to parse record";
 }

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/util/FeedUtils.java
----------------------------------------------------------------------
diff --git a/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/util/FeedUtils.java b/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/util/FeedUtils.java
index 502a432..51e7e72 100644
--- a/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/util/FeedUtils.java
+++ b/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/util/FeedUtils.java
@@ -37,7 +37,7 @@ import org.apache.hyracks.api.exceptions.HyracksDataException;
 import org.apache.hyracks.api.io.FileReference;
 import org.apache.hyracks.api.io.IIOManager;
 import org.apache.hyracks.dataflow.common.comm.io.FrameTupleAccessor;
-import org.apache.hyracks.dataflow.common.util.IntSerDeUtils;
+import org.apache.hyracks.util.IntSerDeUtils;
 import org.apache.hyracks.dataflow.std.file.FileSplit;
 
 public class FeedUtils {

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/asterixdb/asterix-lang-aql/src/main/java/org/apache/asterix/lang/aql/statement/SubscribeFeedStatement.java
----------------------------------------------------------------------
diff --git a/asterixdb/asterix-lang-aql/src/main/java/org/apache/asterix/lang/aql/statement/SubscribeFeedStatement.java b/asterixdb/asterix-lang-aql/src/main/java/org/apache/asterix/lang/aql/statement/SubscribeFeedStatement.java
index ae657d4..092bf69 100644
--- a/asterixdb/asterix-lang-aql/src/main/java/org/apache/asterix/lang/aql/statement/SubscribeFeedStatement.java
+++ b/asterixdb/asterix-lang-aql/src/main/java/org/apache/asterix/lang/aql/statement/SubscribeFeedStatement.java
@@ -85,7 +85,7 @@ public class SubscribeFeedStatement implements Statement {
         if (appliedFunction != null) {
             function = MetadataManager.INSTANCE.getFunction(mdTxnCtx, appliedFunction);
             if (function == null) {
-                throw new MetadataException(" Unknown function " + function);
+                throw new MetadataException(" Unknown function " + appliedFunction);
             } else if (function.getParams().size() > 1) {
                 throw new MetadataException(
                         " Incompatible function: " + appliedFunction + " Number if arguments must be 1");

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/asterixdb/asterix-replication/src/main/java/org/apache/asterix/replication/logging/ReplicationLogBuffer.java
----------------------------------------------------------------------
diff --git a/asterixdb/asterix-replication/src/main/java/org/apache/asterix/replication/logging/ReplicationLogBuffer.java b/asterixdb/asterix-replication/src/main/java/org/apache/asterix/replication/logging/ReplicationLogBuffer.java
index a7cfaec..588968c 100644
--- a/asterixdb/asterix-replication/src/main/java/org/apache/asterix/replication/logging/ReplicationLogBuffer.java
+++ b/asterixdb/asterix-replication/src/main/java/org/apache/asterix/replication/logging/ReplicationLogBuffer.java
@@ -156,4 +156,12 @@ public class ReplicationLogBuffer {
     public void isStop(boolean stop) {
         this.stop = stop;
     }
+
+    public int getLogBufferSize() {
+        return logBufferSize;
+    }
+
+    public ReplicationManager getReplicationManager() {
+        return replicationManager;
+    }
 }

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/asterixdb/asterix-replication/src/main/java/org/apache/asterix/replication/logging/ReplicationLogFlusher.java
----------------------------------------------------------------------
diff --git a/asterixdb/asterix-replication/src/main/java/org/apache/asterix/replication/logging/ReplicationLogFlusher.java b/asterixdb/asterix-replication/src/main/java/org/apache/asterix/replication/logging/ReplicationLogFlusher.java
index 8541eaf..3312cb1 100644
--- a/asterixdb/asterix-replication/src/main/java/org/apache/asterix/replication/logging/ReplicationLogFlusher.java
+++ b/asterixdb/asterix-replication/src/main/java/org/apache/asterix/replication/logging/ReplicationLogFlusher.java
@@ -93,7 +93,10 @@ public class ReplicationLogFlusher implements Callable<Boolean> {
                     }
                 }
                 flushPage.flush();
-                emptyQ.offer(flushPage);
+                // TODO: pool large pages
+                if (flushPage.getLogBufferSize() == flushPage.getReplicationManager().getLogPageSize()) {
+                    emptyQ.offer(flushPage);
+                }
             }
         } catch (Exception e) {
             e.printStackTrace();

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/asterixdb/asterix-replication/src/main/java/org/apache/asterix/replication/management/ReplicationManager.java
----------------------------------------------------------------------
diff --git a/asterixdb/asterix-replication/src/main/java/org/apache/asterix/replication/management/ReplicationManager.java b/asterixdb/asterix-replication/src/main/java/org/apache/asterix/replication/management/ReplicationManager.java
index 93d1085..d983a62 100644
--- a/asterixdb/asterix-replication/src/main/java/org/apache/asterix/replication/management/ReplicationManager.java
+++ b/asterixdb/asterix-replication/src/main/java/org/apache/asterix/replication/management/ReplicationManager.java
@@ -232,6 +232,14 @@ public class ReplicationManager implements IReplicationManager {
         appendToLogBuffer(logRecord);
     }
 
+    protected void getAndInitNewLargePage(int pageSize) {
+        // for now, alloc a new buffer for each large page
+        // TODO: consider pooling large pages
+        currentTxnLogBuffer = new ReplicationLogBuffer(this, pageSize);
+        currentTxnLogBuffer.setReplicationSockets(logsReplicaSockets);
+        pendingFlushLogBuffersQ.offer(currentTxnLogBuffer);
+    }
+
     protected void getAndInitNewPage() {
         currentTxnLogBuffer = null;
         while (currentTxnLogBuffer == null) {
@@ -249,7 +257,11 @@ public class ReplicationManager implements IReplicationManager {
     private synchronized void appendToLogBuffer(ILogRecord logRecord) {
         if (!currentTxnLogBuffer.hasSpace(logRecord)) {
             currentTxnLogBuffer.isFull(true);
-            getAndInitNewPage();
+            if (logRecord.getLogSize() > logManager.getLogPageSize()) {
+                getAndInitNewLargePage(logRecord.getLogSize());
+            } else {
+                getAndInitNewPage();
+            }
         }
 
         currentTxnLogBuffer.append(logRecord);
@@ -1112,6 +1124,10 @@ public class ReplicationManager implements IReplicationManager {
         }
     }
 
+    public int getLogPageSize() {
+        return logManager.getLogPageSize();
+    }
+
     //supporting classes
     /**
      * This class is responsible for processing replica events.

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/LogBuffer.java
----------------------------------------------------------------------
diff --git a/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/LogBuffer.java b/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/LogBuffer.java
index c1180a4..872adcd 100644
--- a/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/LogBuffer.java
+++ b/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/LogBuffer.java
@@ -362,4 +362,8 @@ public class LogBuffer implements ILogBuffer {
     public void isStop(boolean stop) {
         this.stop = stop;
     }
+
+    public int getLogPageSize() {
+        return logPageSize;
+    }
 }

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/LogBufferTailReader.java
----------------------------------------------------------------------
diff --git a/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/LogBufferTailReader.java b/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/LogBufferTailReader.java
index 28f21e5..5ed5724 100644
--- a/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/LogBufferTailReader.java
+++ b/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/LogBufferTailReader.java
@@ -20,7 +20,7 @@ package org.apache.asterix.transaction.management.service.logging;
 
 import java.nio.ByteBuffer;
 
-import org.apache.asterix.common.transactions.ILogRecord.RECORD_STATUS;
+import org.apache.asterix.common.transactions.ILogRecord.RecordReadStatus;
 import org.apache.asterix.common.transactions.LogRecord;
 
 public class LogBufferTailReader {
@@ -43,9 +43,9 @@ public class LogBufferTailReader {
         if (buffer.position() == endOffset) {
             return null;
         }
-        RECORD_STATUS status = logRecord.readLogRecord(buffer);
+        RecordReadStatus status = logRecord.readLogRecord(buffer);
         //underflow is not expected because we are at the very tail of the current log buffer
-        if (status != RECORD_STATUS.OK) {
+        if (status != RecordReadStatus.OK) {
             throw new IllegalStateException();
         }
         return logRecord;

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/LogManager.java
----------------------------------------------------------------------
diff --git a/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/LogManager.java b/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/LogManager.java
index 776577a..f10520b 100644
--- a/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/LogManager.java
+++ b/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/LogManager.java
@@ -114,10 +114,6 @@ public class LogManager implements ILogManager, ILifeCycleComponent {
 
     @Override
     public void log(ILogRecord logRecord) throws ACIDException {
-        if (logRecord.getLogSize() > logPageSize) {
-            throw new IllegalStateException();
-        }
-
         if (logRecord.getLogType() == LogType.FLUSH) {
             flushLogsQ.offer(logRecord);
             return;
@@ -158,7 +154,11 @@ public class LogManager implements ILogManager, ILifeCycleComponent {
             getAndInitNewPage();
         } else if (!appendPage.hasSpace(logRecord.getLogSize())) {
             appendPage.isFull(true);
-            getAndInitNewPage();
+            if (logRecord.getLogSize() > logPageSize) {
+                getAndInitNewLargePage(logRecord.getLogSize());
+            } else {
+                getAndInitNewPage();
+            }
         }
         if (logRecord.getLogType() == LogType.UPDATE) {
             logRecord.setPrevLSN(txnCtx.getLastLSN());
@@ -171,6 +171,14 @@ public class LogManager implements ILogManager, ILifeCycleComponent {
         appendLSN.addAndGet(logRecord.getLogSize());
     }
 
+    protected void getAndInitNewLargePage(int logSize) {
+        // for now, alloc a new buffer for each large page
+        // TODO: pool large pages
+        appendPage = new LogBuffer(txnSubsystem, logSize, flushLSN);
+        appendPage.setFileChannel(appendChannel);
+        flushQ.offer(appendPage);
+    }
+
     protected void getAndInitNewPage() {
         appendPage = null;
         while (appendPage == null) {
@@ -577,7 +585,9 @@ class LogFlusher implements Callable<Boolean> {
                     }
                 }
                 flushPage.flush();
-                emptyQ.offer(flushPage);
+                if (flushPage.getLogPageSize() == logMgr.getLogPageSize()) {
+                    emptyQ.offer(flushPage);
+                }
             }
         } catch (Exception e) {
             if (LOGGER.isLoggable(Level.INFO)) {

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/LogManagerWithReplication.java
----------------------------------------------------------------------
diff --git a/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/LogManagerWithReplication.java b/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/LogManagerWithReplication.java
index 4112529..951bbbd 100644
--- a/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/LogManagerWithReplication.java
+++ b/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/LogManagerWithReplication.java
@@ -108,7 +108,11 @@ public class LogManagerWithReplication extends LogManager {
             getAndInitNewPage();
         } else if (!appendPage.hasSpace(logRecord.getLogSize())) {
             appendPage.isFull(true);
-            getAndInitNewPage();
+            if (logRecord.getLogSize() > logPageSize) {
+                getAndInitNewLargePage(logRecord.getLogSize());
+            } else {
+                getAndInitNewPage();
+            }
         }
         if (logRecord.getLogSource() == LogSource.LOCAL) {
             if (logRecord.getLogType() == LogType.UPDATE) {

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/LogReader.java
----------------------------------------------------------------------
diff --git a/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/LogReader.java b/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/LogReader.java
index a018dc2..0b1d320 100644
--- a/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/LogReader.java
+++ b/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/LogReader.java
@@ -26,7 +26,7 @@ import java.util.logging.Logger;
 import org.apache.asterix.common.exceptions.ACIDException;
 import org.apache.asterix.common.transactions.ILogReader;
 import org.apache.asterix.common.transactions.ILogRecord;
-import org.apache.asterix.common.transactions.ILogRecord.RECORD_STATUS;
+import org.apache.asterix.common.transactions.ILogRecord.RecordReadStatus;
 import org.apache.asterix.common.transactions.LogRecord;
 import org.apache.asterix.common.transactions.MutableLong;
 
@@ -92,36 +92,42 @@ public class LogReader implements ILogReader {
                 return null;
             }
         }
+        ByteBuffer readBuffer = this.readBuffer;
+        boolean refilled = false;
 
-        RECORD_STATUS status = logRecord.readLogRecord(readBuffer);
-        switch(status) {
-            case TRUNCATED: {
-                //we may have just read off the end of the buffer, so try refiling it
-                if(!refillLogReadBuffer()) {
-                    return null;
-                }
-                //now see what we have in the refilled buffer
-                status = logRecord.readLogRecord(readBuffer);
-                switch(status){
-                    case TRUNCATED: {
+        while (true) {
+            RecordReadStatus status = logRecord.readLogRecord(readBuffer);
+            switch (status) {
+                case TRUNCATED: {
+                    if (!refilled) {
+                        //we may have just read off the end of the buffer, so try refiling it
+                        if (!refillLogReadBuffer()) {
+                            return null;
+                        }
+                        refilled = true;
+                        //now see what we have in the refilled buffer
+                        continue;
+                    } else {
                         LOGGER.info("Log file has truncated log records.");
                         return null;
                     }
-                    case BAD_CHKSUM:{
-                        LOGGER.severe("Transaction log contains corrupt log records (perhaps due to medium error). Stopping recovery early.");
-                        return null;
-                    }
-                    case OK: break;
                 }
-                //if we have exited the inner switch,
-                // this means status is really "OK" after buffer refill
-                break;
-            }
-            case BAD_CHKSUM:{
-                LOGGER.severe("Transaction log contains corrupt log records (perhaps due to medium error). Stopping recovery early.");
-                return null;
+                case LARGE_RECORD: {
+                    readBuffer = ByteBuffer.allocate(logRecord.getLogSize());
+                    fillLogReadBuffer(logRecord.getLogSize(), readBuffer);
+                    //now see what we have in the expanded buffer
+                    continue;
+                }
+                case BAD_CHKSUM: {
+                    LOGGER.severe("Transaction log contains corrupt log records (perhaps due to medium error). Stopping recovery early.");
+                    return null;
+                }
+                case OK:
+                    break;
+
             }
-            case OK: break;
+            // break the loop by default
+            break;
         }
         logRecord.setLSN(readLSN);
         readLSN += logRecord.getLogSize();
@@ -173,15 +179,19 @@ public class LogReader implements ILogReader {
      */
 
     private boolean fillLogReadBuffer() throws ACIDException {
+        return fillLogReadBuffer(logPageSize, readBuffer);
+    }
+
+    private boolean fillLogReadBuffer(int readSize, ByteBuffer readBuffer) throws ACIDException {
         int size=0;
         int read=0;
         readBuffer.position(0);
-        readBuffer.limit(logPageSize);
+        readBuffer.limit(readSize);
         try {
             fileChannel.position(readLSN % logFileSize);
             //We loop here because read() may return 0, but this simply means we are waiting on IO.
             //Therefore we want to break out only when either the buffer is full, or we reach EOF.
-            while( size < logPageSize && read != -1) {
+            while( size < readSize && read != -1) {
                 read = fileChannel.read(readBuffer);
                 if(read>0) {
                     size += read;
@@ -235,16 +245,27 @@ public class LogReader implements ILogReader {
                 throw new ACIDException("LSN is out of bounds");
             }
         }
-        RECORD_STATUS status = logRecord.readLogRecord(readBuffer);
-        switch(status){
-            case TRUNCATED:{
-                throw new ACIDException("LSN is out of bounds");
-            }
-            case BAD_CHKSUM:{
-                throw new ACIDException("Log record has incorrect checksum");
-            }
-            case OK: break;
+        ByteBuffer readBuffer = this.readBuffer;
+        while (true) {
+            RecordReadStatus status = logRecord.readLogRecord(readBuffer);
+            switch (status) {
+                case LARGE_RECORD: {
+                    readBuffer = ByteBuffer.allocate(logRecord.getLogSize());
+                    fillLogReadBuffer(logRecord.getLogSize(), readBuffer);
+                    //now see what we have in the refilled buffer
+                    continue;
+                }
+                case TRUNCATED: {
+                    throw new ACIDException("LSN is out of bounds");
+                }
+                case BAD_CHKSUM: {
+                    throw new ACIDException("Log record has incorrect checksum");
+                }
+                case OK:
+                    break;
 
+            }
+            break;
         }
         logRecord.setLSN(readLSN);
         readLSN += logRecord.getLogSize();

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/RemoteLogReader.java
----------------------------------------------------------------------
diff --git a/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/RemoteLogReader.java b/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/RemoteLogReader.java
index 8f88321..4dc1700 100644
--- a/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/RemoteLogReader.java
+++ b/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/service/logging/RemoteLogReader.java
@@ -25,7 +25,7 @@ import java.nio.channels.FileChannel;
 import org.apache.asterix.common.exceptions.ACIDException;
 import org.apache.asterix.common.transactions.ILogReader;
 import org.apache.asterix.common.transactions.ILogRecord;
-import org.apache.asterix.common.transactions.ILogRecord.RECORD_STATUS;
+import org.apache.asterix.common.transactions.ILogRecord.RecordReadStatus;
 import org.apache.asterix.common.transactions.LogRecord;
 
 public class RemoteLogReader implements ILogReader {
@@ -50,6 +50,10 @@ public class RemoteLogReader implements ILogReader {
     }
 
     private boolean fillLogReadBuffer() throws ACIDException {
+        return fillLogReadBuffer(logPageSize, readBuffer);
+    }
+
+    private boolean fillLogReadBuffer(int pageSize, ByteBuffer readBuffer) throws ACIDException {
         int size = 0;
         int read = 0;
         readBuffer.position(0);
@@ -58,7 +62,7 @@ public class RemoteLogReader implements ILogReader {
             fileChannel.position(readLSN);
             //We loop here because read() may return 0, but this simply means we are waiting on IO.
             //Therefore we want to break out only when either the buffer is full, or we reach EOF.
-            while (size < logPageSize && read != -1) {
+            while (size < pageSize && read != -1) {
                 read = fileChannel.read(readBuffer);
                 if (read > 0) {
                     size += read;
@@ -88,34 +92,37 @@ public class RemoteLogReader implements ILogReader {
                 return null;
             }
         }
+        ByteBuffer readBuffer = this.readBuffer;
+        boolean refilled = false;
 
-        RECORD_STATUS status = logRecord.readRemoteLog(readBuffer, true);
-        switch (status) {
-            case TRUNCATED: {
-                //we may have just read off the end of the buffer, so try refiling it
-                if (!fillLogReadBuffer()) {
+        while (true) {
+            RecordReadStatus status = logRecord.readRemoteLog(readBuffer, true);
+            switch (status) {
+                case TRUNCATED: {
+                    if (!refilled) {
+                        //we may have just read off the end of the buffer, so try refiling it
+                        if (!fillLogReadBuffer()) {
+                            return null;
+                        }
+                        refilled = true;
+                        //now see what we have in the refilled buffer
+                        continue;
+                    }
                     return null;
                 }
-                //now see what we have in the refilled buffer
-                status = logRecord.readRemoteLog(readBuffer, true);
-                switch (status) {
-                    case TRUNCATED: {
-                        return null;
-                    }
-                    case OK:
-                        break;
-                    default:
-                        break;
+                case LARGE_RECORD: {
+                    readBuffer = ByteBuffer.allocate(logRecord.getLogSize());
+                    fillLogReadBuffer(logRecord.getLogSize(), readBuffer);
+                    //now see what we have in the expanded buffer
+                    continue;
                 }
-                //if we have exited the inner switch,
-                // this means status is really "OK" after buffer refill
-                break;
-            }
-            case BAD_CHKSUM: {
-                return null;
+                case BAD_CHKSUM: {
+                    return null;
+                }
+                case OK:
+                    break;
             }
-            case OK:
-                break;
+            break;
         }
 
         readLSN += logRecord.getSerializedLogSize();

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/hyracks-fullstack/hyracks/hyracks-api/pom.xml
----------------------------------------------------------------------
diff --git a/hyracks-fullstack/hyracks/hyracks-api/pom.xml b/hyracks-fullstack/hyracks/hyracks-api/pom.xml
index 4425f3b..3961921 100644
--- a/hyracks-fullstack/hyracks/hyracks-api/pom.xml
+++ b/hyracks-fullstack/hyracks/hyracks-api/pom.xml
@@ -72,5 +72,10 @@
       <artifactId>commons-lang3</artifactId>
       <version>3.1</version>
     </dependency>
+    <dependency>
+      <groupId>org.apache.hyracks</groupId>
+      <artifactId>hyracks-util</artifactId>
+      <version>0.2.18-SNAPSHOT</version>
+    </dependency>
   </dependencies>
 </project>

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/FrameConstants.java
----------------------------------------------------------------------
diff --git a/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/FrameConstants.java b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/FrameConstants.java
index 9cf96ec..cd74659 100644
--- a/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/FrameConstants.java
+++ b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/FrameConstants.java
@@ -25,20 +25,20 @@ public interface FrameConstants {
     int SIZE_LEN = 4;
 
     /**
-     * The offset of the frame_count which is one byte indicate how many initial_frames contained in current frame.
-     * The actual frameSize = frame_count * intitialFrameSize(given by user)
+     * The offset of the frame_count which is one int indicates how many initial_frames contained in current frame.
+     * The actual frameSize = frame_count * initialFrameSize(given by user)
      */
     int META_DATA_FRAME_COUNT_OFFSET = 0;
 
     /**
-     * The start offset of the tuple data. The first byte is used to store the frame_count
+     * The start offset of the tuple data. The first int is used to store the frame_count
      */
-    int TUPLE_START_OFFSET = 1;
+    int TUPLE_START_OFFSET = 5;
 
     /**
-     * Since we use one byte to store the frame_count, the max frame_count is 255.
+     * The max frame size is Integer.MAX_VALUE.
      */
-    int MAX_NUM_MINFRAME = 255;
+    int MAX_FRAMESIZE = Integer.MAX_VALUE;
 
     /**
      * Indicate the total size of the meta data.

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/FrameHelper.java
----------------------------------------------------------------------
diff --git a/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/FrameHelper.java b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/FrameHelper.java
index 473f3ae..68533c6 100644
--- a/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/FrameHelper.java
+++ b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/FrameHelper.java
@@ -20,6 +20,8 @@ package org.apache.hyracks.api.comm;
 
 import java.nio.ByteBuffer;
 
+import org.apache.hyracks.util.IntSerDeUtils;
+
 public class FrameHelper {
     public static int getTupleCountOffset(int frameSize) {
         return frameSize - FrameConstants.SIZE_LEN;
@@ -27,24 +29,24 @@ public class FrameHelper {
 
     /**
      * The actual frameSize = frameCount * intitialFrameSize
-     * This method is used to put that frameCount into the first byte of the frame buffer.
+     * This method is used to put that frameCount into the first int of the frame buffer.
      * @param outputFrame
      * @param numberOfMinFrame
      */
-    public static void serializeFrameSize(ByteBuffer outputFrame, byte numberOfMinFrame) {
+    public static void serializeFrameSize(ByteBuffer outputFrame, int numberOfMinFrame) {
         serializeFrameSize(outputFrame, 0, numberOfMinFrame);
     }
 
-    public static void serializeFrameSize(ByteBuffer outputFrame, int start, byte numberOfMinFrame) {
-        outputFrame.array()[start + FrameConstants.META_DATA_FRAME_COUNT_OFFSET] = (byte) (numberOfMinFrame & 0xff);
+    public static void serializeFrameSize(ByteBuffer outputFrame, int start, int numberOfMinFrame) {
+        IntSerDeUtils.putInt(outputFrame.array(), start + FrameConstants.META_DATA_FRAME_COUNT_OFFSET, numberOfMinFrame);
     }
 
-    public static byte deserializeNumOfMinFrame(ByteBuffer frame) {
+    public static int deserializeNumOfMinFrame(ByteBuffer frame) {
         return deserializeNumOfMinFrame(frame, 0);
     }
 
-    public static byte deserializeNumOfMinFrame(ByteBuffer buffer, int start) {
-        return (byte) (buffer.array()[start + FrameConstants.META_DATA_FRAME_COUNT_OFFSET] & 0xff);
+    public static int deserializeNumOfMinFrame(ByteBuffer buffer, int start) {
+        return IntSerDeUtils.getInt(buffer.array(), start + FrameConstants.META_DATA_FRAME_COUNT_OFFSET);
     }
 
     /**

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/resources/memory/FrameManager.java
----------------------------------------------------------------------
diff --git a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/resources/memory/FrameManager.java b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/resources/memory/FrameManager.java
index 8df1f38..4dc6bc9 100644
--- a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/resources/memory/FrameManager.java
+++ b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/resources/memory/FrameManager.java
@@ -49,13 +49,13 @@ public class FrameManager implements IHyracksFrameMgrContext {
         if (bytes % minFrameSize != 0) {
             throw new HyracksDataException("The size should be an integral multiple of the default frame size");
         }
-        ByteBuffer buffer = ByteBuffer.allocate(bytes);
-        if (bytes / minFrameSize > FrameConstants.MAX_NUM_MINFRAME) {
+        if (bytes > FrameConstants.MAX_FRAMESIZE) {
             throw new HyracksDataException(
-                    "Unable to allocate frame larger than:" + FrameConstants.MAX_NUM_MINFRAME * minFrameSize
+                    "Unable to allocate frame larger than:" + FrameConstants.MAX_FRAMESIZE
                             + " bytes");
         }
-        FrameHelper.serializeFrameSize(buffer, (byte) (bytes / minFrameSize));
+        ByteBuffer buffer = ByteBuffer.allocate(bytes);
+        FrameHelper.serializeFrameSize(buffer, bytes / minFrameSize);
         return (ByteBuffer) buffer.clear();
     }
 
@@ -66,6 +66,10 @@ public class FrameManager implements IHyracksFrameMgrContext {
             deallocateFrames(tobeDeallocate.capacity());
             return allocateFrame(newSizeInBytes);
         } else {
+            if (newSizeInBytes > FrameConstants.MAX_FRAMESIZE) {
+                throw new HyracksDataException("Unable to allocate frame of size bigger than: "
+                        + FrameConstants.MAX_FRAMESIZE + " bytes");
+            }
             ByteBuffer buffer = allocateFrame(newSizeInBytes);
             int limit = Math.min(newSizeInBytes, tobeDeallocate.capacity());
             int pos = Math.min(limit, tobeDeallocate.position());
@@ -74,11 +78,7 @@ public class FrameManager implements IHyracksFrameMgrContext {
             buffer.put(tobeDeallocate);
             buffer.position(pos);
 
-            if (newSizeInBytes / minFrameSize > FrameConstants.MAX_NUM_MINFRAME) {
-                throw new HyracksDataException("Unable to allocate frame of size bigger than: "
-                        + FrameConstants.MAX_NUM_MINFRAME * minFrameSize + " bytes");
-            }
-            FrameHelper.serializeFrameSize(buffer, (byte) (newSizeInBytes / minFrameSize));
+            FrameHelper.serializeFrameSize(buffer, newSizeInBytes / minFrameSize);
             return buffer;
         }
     }

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/comm/io/AbstractFrameAppender.java
----------------------------------------------------------------------
diff --git a/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/comm/io/AbstractFrameAppender.java b/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/comm/io/AbstractFrameAppender.java
index 1553605..e7131d5 100644
--- a/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/comm/io/AbstractFrameAppender.java
+++ b/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/comm/io/AbstractFrameAppender.java
@@ -27,7 +27,7 @@ import org.apache.hyracks.api.comm.IFrame;
 import org.apache.hyracks.api.comm.IFrameAppender;
 import org.apache.hyracks.api.comm.IFrameWriter;
 import org.apache.hyracks.api.exceptions.HyracksDataException;
-import org.apache.hyracks.dataflow.common.util.IntSerDeUtils;
+import org.apache.hyracks.util.IntSerDeUtils;
 
 /*
  * Frame

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/comm/io/FrameFixedFieldAppender.java
----------------------------------------------------------------------
diff --git a/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/comm/io/FrameFixedFieldAppender.java b/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/comm/io/FrameFixedFieldAppender.java
index e62e9e7..8765bf1 100644
--- a/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/comm/io/FrameFixedFieldAppender.java
+++ b/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/comm/io/FrameFixedFieldAppender.java
@@ -25,7 +25,7 @@ import org.apache.hyracks.api.comm.IFrameFieldAppender;
 import org.apache.hyracks.api.comm.IFrameTupleAccessor;
 import org.apache.hyracks.api.comm.IFrameWriter;
 import org.apache.hyracks.api.exceptions.HyracksDataException;
-import org.apache.hyracks.dataflow.common.util.IntSerDeUtils;
+import org.apache.hyracks.util.IntSerDeUtils;
 
 public class FrameFixedFieldAppender extends AbstractFrameAppender implements IFrameFieldAppender {
     private final int fieldCount;

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/comm/io/FrameTupleAccessor.java
----------------------------------------------------------------------
diff --git a/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/comm/io/FrameTupleAccessor.java b/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/comm/io/FrameTupleAccessor.java
index e314cd1..d99e2f2 100644
--- a/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/comm/io/FrameTupleAccessor.java
+++ b/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/comm/io/FrameTupleAccessor.java
@@ -30,7 +30,7 @@ import org.apache.hyracks.api.dataflow.value.RecordDescriptor;
 import org.apache.hyracks.api.exceptions.HyracksDataException;
 import org.apache.hyracks.dataflow.common.comm.util.ByteBufferInputStream;
 import org.apache.hyracks.dataflow.common.data.accessors.ITupleReference;
-import org.apache.hyracks.dataflow.common.util.IntSerDeUtils;
+import org.apache.hyracks.util.IntSerDeUtils;
 
 /**
  * FrameTupleCursor is used to navigate over tuples in a Frame. A frame is

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/comm/io/FrameTupleAppender.java
----------------------------------------------------------------------
diff --git a/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/comm/io/FrameTupleAppender.java b/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/comm/io/FrameTupleAppender.java
index 136e231..1623035 100644
--- a/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/comm/io/FrameTupleAppender.java
+++ b/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/comm/io/FrameTupleAppender.java
@@ -25,7 +25,7 @@ import org.apache.hyracks.api.comm.IFrame;
 import org.apache.hyracks.api.comm.IFrameTupleAccessor;
 import org.apache.hyracks.api.comm.IFrameTupleAppender;
 import org.apache.hyracks.api.exceptions.HyracksDataException;
-import org.apache.hyracks.dataflow.common.util.IntSerDeUtils;
+import org.apache.hyracks.util.IntSerDeUtils;
 
 public class FrameTupleAppender extends AbstractFrameAppender implements IFrameTupleAppender {
 

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/comm/io/FrameTupleAppenderAccessor.java
----------------------------------------------------------------------
diff --git a/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/comm/io/FrameTupleAppenderAccessor.java b/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/comm/io/FrameTupleAppenderAccessor.java
index 0ced0fd..b464f8e 100644
--- a/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/comm/io/FrameTupleAppenderAccessor.java
+++ b/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/comm/io/FrameTupleAppenderAccessor.java
@@ -29,7 +29,7 @@ import org.apache.hyracks.api.comm.IFrameTupleAccessor;
 import org.apache.hyracks.api.dataflow.value.RecordDescriptor;
 import org.apache.hyracks.api.exceptions.HyracksDataException;
 import org.apache.hyracks.dataflow.common.comm.util.ByteBufferInputStream;
-import org.apache.hyracks.dataflow.common.util.IntSerDeUtils;
+import org.apache.hyracks.util.IntSerDeUtils;
 
 public class FrameTupleAppenderAccessor extends FrameTupleAppender implements IFrameTupleAccessor {
     private int tupleCountOffset;

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/io/MessagingFrameTupleAppender.java
----------------------------------------------------------------------
diff --git a/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/io/MessagingFrameTupleAppender.java b/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/io/MessagingFrameTupleAppender.java
index 7100c11..6c236a6 100644
--- a/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/io/MessagingFrameTupleAppender.java
+++ b/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/io/MessagingFrameTupleAppender.java
@@ -25,7 +25,7 @@ import org.apache.hyracks.api.comm.IFrameWriter;
 import org.apache.hyracks.api.context.IHyracksTaskContext;
 import org.apache.hyracks.api.exceptions.HyracksDataException;
 import org.apache.hyracks.dataflow.common.comm.io.FrameTupleAppender;
-import org.apache.hyracks.dataflow.common.util.IntSerDeUtils;
+import org.apache.hyracks.util.IntSerDeUtils;
 
 public class MessagingFrameTupleAppender extends FrameTupleAppender {
 

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/util/IntSerDeUtils.java
----------------------------------------------------------------------
diff --git a/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/util/IntSerDeUtils.java b/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/util/IntSerDeUtils.java
deleted file mode 100644
index d2100bb..0000000
--- a/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/util/IntSerDeUtils.java
+++ /dev/null
@@ -1,42 +0,0 @@
-/*
- * Licensed to the Apache Software Foundation (ASF) under one
- * or more contributor license agreements.  See the NOTICE file
- * distributed with this work for additional information
- * regarding copyright ownership.  The ASF licenses this file
- * to you under the Apache License, Version 2.0 (the
- * "License"); you may not use this file except in compliance
- * with the License.  You may obtain a copy of the License at
- *
- *   http://www.apache.org/licenses/LICENSE-2.0
- *
- * Unless required by applicable law or agreed to in writing,
- * software distributed under the License is distributed on an
- * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
- * KIND, either express or implied.  See the License for the
- * specific language governing permissions and limitations
- * under the License.
- */
-
-package org.apache.hyracks.dataflow.common.util;
-
-public class IntSerDeUtils {
-
-    public static int getInt(byte[] bytes, int offset) {
-        return ((bytes[offset] & 0xff) << 24) + ((bytes[offset + 1] & 0xff) << 16) + ((bytes[offset + 2] & 0xff) << 8)
-                + ((bytes[offset + 3] & 0xff) << 0);
-    }
-
-    /**
-     * put integer value into the array bytes at the offset offset
-     * @param bytes byte array to put data in
-     * @param offset offset from the beginning of the array to write the {@code value} in
-     * @param value value to write to {@code bytes[offset]}
-     */
-    public static void putInt(byte[] bytes, int offset, int value) {
-
-        bytes[offset++] = (byte) (value >> 24);
-        bytes[offset++] = (byte) (value >> 16);
-        bytes[offset++] = (byte) (value >> 8);
-        bytes[offset++] = (byte) (value);
-    }
-}

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/util/ReflectionUtils.java
----------------------------------------------------------------------
diff --git a/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/util/ReflectionUtils.java b/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/util/ReflectionUtils.java
deleted file mode 100644
index 13b41b0..0000000
--- a/hyracks-fullstack/hyracks/hyracks-dataflow-common/src/main/java/org/apache/hyracks/dataflow/common/util/ReflectionUtils.java
+++ /dev/null
@@ -1,33 +0,0 @@
-/*
- * Licensed to the Apache Software Foundation (ASF) under one
- * or more contributor license agreements.  See the NOTICE file
- * distributed with this work for additional information
- * regarding copyright ownership.  The ASF licenses this file
- * to you under the Apache License, Version 2.0 (the
- * "License"); you may not use this file except in compliance
- * with the License.  You may obtain a copy of the License at
- *
- *   http://www.apache.org/licenses/LICENSE-2.0
- *
- * Unless required by applicable law or agreed to in writing,
- * software distributed under the License is distributed on an
- * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
- * KIND, either express or implied.  See the License for the
- * specific language governing permissions and limitations
- * under the License.
- */
-package org.apache.hyracks.dataflow.common.util;
-
-public class ReflectionUtils {
-    public static <T> T createInstance(Class<? extends T> klass) {
-        T instance = null;
-        try {
-            instance = klass.newInstance();
-        } catch (InstantiationException e) {
-            throw new RuntimeException(e);
-        } catch (IllegalAccessException e) {
-            throw new RuntimeException(e);
-        }
-        return instance;
-    }
-}

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/buffermanager/VariableFrameMemoryManager.java
----------------------------------------------------------------------
diff --git a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/buffermanager/VariableFrameMemoryManager.java b/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/buffermanager/VariableFrameMemoryManager.java
index fe7de23..6604ba8 100644
--- a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/buffermanager/VariableFrameMemoryManager.java
+++ b/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/buffermanager/VariableFrameMemoryManager.java
@@ -25,7 +25,7 @@ import java.util.List;
 
 import org.apache.hyracks.api.comm.FrameHelper;
 import org.apache.hyracks.api.exceptions.HyracksDataException;
-import org.apache.hyracks.dataflow.common.util.IntSerDeUtils;
+import org.apache.hyracks.util.IntSerDeUtils;
 
 public class VariableFrameMemoryManager implements IFrameBufferManager {
 

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/sort/AbstractExternalSortRunMerger.java
----------------------------------------------------------------------
diff --git a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/sort/AbstractExternalSortRunMerger.java b/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/sort/AbstractExternalSortRunMerger.java
index 6e2c3f0..e032e6a 100644
--- a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/sort/AbstractExternalSortRunMerger.java
+++ b/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/sort/AbstractExternalSortRunMerger.java
@@ -75,7 +75,7 @@ public abstract class AbstractExternalSortRunMerger {
         this.recordDesc = recordDesc;
         this.framesLimit = framesLimit;
         this.writer = writer;
-        this.MAX_FRAME_SIZE = FrameConstants.MAX_NUM_MINFRAME * ctx.getInitialFrameSize();
+        this.MAX_FRAME_SIZE = FrameConstants.MAX_FRAMESIZE;
         this.topK = topK;
     }
 

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/sort/AbstractFrameSorter.java
----------------------------------------------------------------------
diff --git a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/sort/AbstractFrameSorter.java b/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/sort/AbstractFrameSorter.java
index f213b97..a6686cb 100644
--- a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/sort/AbstractFrameSorter.java
+++ b/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/sort/AbstractFrameSorter.java
@@ -36,7 +36,7 @@ import org.apache.hyracks.api.dataflow.value.RecordDescriptor;
 import org.apache.hyracks.api.exceptions.HyracksDataException;
 import org.apache.hyracks.dataflow.common.comm.io.FrameTupleAccessor;
 import org.apache.hyracks.dataflow.common.comm.io.FrameTupleAppender;
-import org.apache.hyracks.dataflow.common.util.IntSerDeUtils;
+import org.apache.hyracks.util.IntSerDeUtils;
 import org.apache.hyracks.dataflow.std.buffermanager.BufferInfo;
 import org.apache.hyracks.dataflow.std.buffermanager.IFrameBufferManager;
 import org.apache.hyracks.dataflow.common.comm.util.FrameUtils;

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/sort/util/DeletableFrameTupleAppender.java
----------------------------------------------------------------------
diff --git a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/sort/util/DeletableFrameTupleAppender.java b/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/sort/util/DeletableFrameTupleAppender.java
index 5485d5d..7d4db64 100644
--- a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/sort/util/DeletableFrameTupleAppender.java
+++ b/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/main/java/org/apache/hyracks/dataflow/std/sort/util/DeletableFrameTupleAppender.java
@@ -25,7 +25,7 @@ import org.apache.hyracks.api.comm.FrameHelper;
 import org.apache.hyracks.api.comm.IFrameTupleAccessor;
 import org.apache.hyracks.api.dataflow.value.RecordDescriptor;
 import org.apache.hyracks.api.exceptions.HyracksDataException;
-import org.apache.hyracks.dataflow.common.util.IntSerDeUtils;
+import org.apache.hyracks.util.IntSerDeUtils;
 
 /**
  * This is a special frame which is used in TupleMemoryBuffer.

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/test/java/org/apache/hyracks/dataflow/std/buffermanager/AbstractTupleMemoryManagerTest.java
----------------------------------------------------------------------
diff --git a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/test/java/org/apache/hyracks/dataflow/std/buffermanager/AbstractTupleMemoryManagerTest.java b/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/test/java/org/apache/hyracks/dataflow/std/buffermanager/AbstractTupleMemoryManagerTest.java
index cdf8834..a705222 100644
--- a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/test/java/org/apache/hyracks/dataflow/std/buffermanager/AbstractTupleMemoryManagerTest.java
+++ b/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/test/java/org/apache/hyracks/dataflow/std/buffermanager/AbstractTupleMemoryManagerTest.java
@@ -35,7 +35,7 @@ import org.apache.hyracks.dataflow.common.comm.io.FrameTupleAccessor;
 import org.apache.hyracks.dataflow.common.comm.io.FrameTupleAppender;
 import org.apache.hyracks.dataflow.common.data.marshalling.IntegerSerializerDeserializer;
 import org.apache.hyracks.dataflow.common.data.marshalling.UTF8StringSerializerDeserializer;
-import org.apache.hyracks.dataflow.common.util.IntSerDeUtils;
+import org.apache.hyracks.util.IntSerDeUtils;
 import org.apache.hyracks.dataflow.std.sort.Utility;
 import org.apache.hyracks.dataflow.std.structures.TuplePointer;
 

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/test/java/org/apache/hyracks/dataflow/std/buffermanager/VPartitionTupleBufferManagerTest.java
----------------------------------------------------------------------
diff --git a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/test/java/org/apache/hyracks/dataflow/std/buffermanager/VPartitionTupleBufferManagerTest.java b/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/test/java/org/apache/hyracks/dataflow/std/buffermanager/VPartitionTupleBufferManagerTest.java
index ce31108..d73b9ae 100644
--- a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/test/java/org/apache/hyracks/dataflow/std/buffermanager/VPartitionTupleBufferManagerTest.java
+++ b/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/test/java/org/apache/hyracks/dataflow/std/buffermanager/VPartitionTupleBufferManagerTest.java
@@ -28,7 +28,7 @@ import java.util.Map;
 import org.apache.hyracks.api.exceptions.HyracksDataException;
 import org.apache.hyracks.dataflow.common.comm.io.ArrayTupleBuilder;
 import org.apache.hyracks.dataflow.common.comm.io.FrameTupleAccessor;
-import org.apache.hyracks.dataflow.common.util.IntSerDeUtils;
+import org.apache.hyracks.util.IntSerDeUtils;
 import org.apache.hyracks.dataflow.std.structures.TuplePointer;
 import org.junit.Before;
 import org.junit.Test;

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/test/java/org/apache/hyracks/dataflow/std/buffermanager/VariableTupleMemoryManagerTest.java
----------------------------------------------------------------------
diff --git a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/test/java/org/apache/hyracks/dataflow/std/buffermanager/VariableTupleMemoryManagerTest.java b/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/test/java/org/apache/hyracks/dataflow/std/buffermanager/VariableTupleMemoryManagerTest.java
index 036cd41..618dc6d 100644
--- a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/test/java/org/apache/hyracks/dataflow/std/buffermanager/VariableTupleMemoryManagerTest.java
+++ b/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/test/java/org/apache/hyracks/dataflow/std/buffermanager/VariableTupleMemoryManagerTest.java
@@ -29,7 +29,7 @@ import java.util.Map;
 import org.apache.hyracks.api.comm.FixedSizeFrame;
 import org.apache.hyracks.api.exceptions.HyracksDataException;
 import org.apache.hyracks.dataflow.common.comm.io.FrameTupleAppender;
-import org.apache.hyracks.dataflow.common.util.IntSerDeUtils;
+import org.apache.hyracks.util.IntSerDeUtils;
 import org.apache.hyracks.dataflow.std.structures.TuplePointer;
 import org.junit.Before;
 import org.junit.Test;

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/test/java/org/apache/hyracks/dataflow/std/sort/util/DeletableFrameTupleAppenderTest.java
----------------------------------------------------------------------
diff --git a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/test/java/org/apache/hyracks/dataflow/std/sort/util/DeletableFrameTupleAppenderTest.java b/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/test/java/org/apache/hyracks/dataflow/std/sort/util/DeletableFrameTupleAppenderTest.java
index 863f778..7686540 100644
--- a/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/test/java/org/apache/hyracks/dataflow/std/sort/util/DeletableFrameTupleAppenderTest.java
+++ b/hyracks-fullstack/hyracks/hyracks-dataflow-std/src/test/java/org/apache/hyracks/dataflow/std/sort/util/DeletableFrameTupleAppenderTest.java
@@ -31,7 +31,7 @@ import org.apache.hyracks.api.exceptions.HyracksDataException;
 import org.apache.hyracks.dataflow.common.comm.io.ArrayTupleBuilder;
 import org.apache.hyracks.dataflow.common.data.marshalling.IntegerSerializerDeserializer;
 import org.apache.hyracks.dataflow.common.data.marshalling.UTF8StringSerializerDeserializer;
-import org.apache.hyracks.dataflow.common.util.IntSerDeUtils;
+import org.apache.hyracks.util.IntSerDeUtils;
 import org.apache.hyracks.dataflow.std.sort.Utility;
 import org.apache.hyracks.util.string.UTF8StringUtil;
 import org.junit.Before;

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/hyracks-fullstack/hyracks/hyracks-storage-am-btree/src/main/java/org/apache/hyracks/storage/am/btree/api/IBTreeFrame.java
----------------------------------------------------------------------
diff --git a/hyracks-fullstack/hyracks/hyracks-storage-am-btree/src/main/java/org/apache/hyracks/storage/am/btree/api/IBTreeFrame.java b/hyracks-fullstack/hyracks/hyracks-storage-am-btree/src/main/java/org/apache/hyracks/storage/am/btree/api/IBTreeFrame.java
index 6d61035..1081dc8 100644
--- a/hyracks-fullstack/hyracks/hyracks-storage-am-btree/src/main/java/org/apache/hyracks/storage/am/btree/api/IBTreeFrame.java
+++ b/hyracks-fullstack/hyracks/hyracks-storage-am-btree/src/main/java/org/apache/hyracks/storage/am/btree/api/IBTreeFrame.java
@@ -36,5 +36,7 @@ public interface IBTreeFrame extends ITreeIndexFrame {
 
     public boolean getSmFlag();
 
+    public void setLargeFlag(boolean largePage);
+
     public void validate(PageValidationInfo pvi) throws HyracksDataException;
 }

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/hyracks-fullstack/hyracks/hyracks-storage-am-btree/src/main/java/org/apache/hyracks/storage/am/btree/api/IBTreeLeafFrame.java
----------------------------------------------------------------------
diff --git a/hyracks-fullstack/hyracks/hyracks-storage-am-btree/src/main/java/org/apache/hyracks/storage/am/btree/api/IBTreeLeafFrame.java b/hyracks-fullstack/hyracks/hyracks-storage-am-btree/src/main/java/org/apache/hyracks/storage/am/btree/api/IBTreeLeafFrame.java
index f1a31e8..7648ee6 100644
--- a/hyracks-fullstack/hyracks/hyracks-storage-am-btree/src/main/java/org/apache/hyracks/storage/am/btree/api/IBTreeLeafFrame.java
+++ b/hyracks-fullstack/hyracks/hyracks-storage-am-btree/src/main/java/org/apache/hyracks/storage/am/btree/api/IBTreeLeafFrame.java
@@ -21,15 +21,18 @@ package org.apache.hyracks.storage.am.btree.api;
 
 import org.apache.hyracks.api.exceptions.HyracksDataException;
 import org.apache.hyracks.dataflow.common.data.accessors.ITupleReference;
+import org.apache.hyracks.storage.am.common.api.IMetaDataPageManager;
+import org.apache.hyracks.storage.am.common.api.ITreeIndexMetaDataFrame;
 import org.apache.hyracks.storage.am.common.api.ITreeIndexTupleReference;
 import org.apache.hyracks.storage.am.common.api.TreeIndexException;
 import org.apache.hyracks.storage.am.common.ophelpers.FindTupleMode;
 import org.apache.hyracks.storage.am.common.ophelpers.FindTupleNoExactMatchPolicy;
 import org.apache.hyracks.storage.am.common.ophelpers.MultiComparator;
+import org.apache.hyracks.storage.common.buffercache.IBufferCache;
 
 public interface IBTreeLeafFrame extends IBTreeFrame {
     public int findTupleIndex(ITupleReference searchKey, ITreeIndexTupleReference pageTuple, MultiComparator cmp,
-            FindTupleMode ftm, FindTupleNoExactMatchPolicy ftp) throws HyracksDataException;
+                              FindTupleMode ftm, FindTupleNoExactMatchPolicy ftp) throws HyracksDataException;
 
     public int findUpdateTupleIndex(ITupleReference tuple) throws TreeIndexException;
 
@@ -49,4 +52,13 @@ public interface IBTreeLeafFrame extends IBTreeFrame {
     public void setNextLeaf(int nextPage);
 
     public int getNextLeaf();
+
+    public void configureLargePage(int supplementalPages, int supplementalBlockPageId);
+
+    public boolean isLargePage();
+
+    public int getSupplementalNumPages();
+
+    void ensureCapacity(IMetaDataPageManager freePageManager, ITreeIndexMetaDataFrame metaFrame,
+                        IBufferCache bufferCache, ITupleReference tuple) throws HyracksDataException;
 }

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/hyracks-fullstack/hyracks/hyracks-storage-am-btree/src/main/java/org/apache/hyracks/storage/am/btree/frames/BTreeFieldPrefixNSMLeafFrame.java
----------------------------------------------------------------------
diff --git a/hyracks-fullstack/hyracks/hyracks-storage-am-btree/src/main/java/org/apache/hyracks/storage/am/btree/frames/BTreeFieldPrefixNSMLeafFrame.java b/hyracks-fullstack/hyracks/hyracks-storage-am-btree/src/main/java/org/apache/hyracks/storage/am/btree/frames/BTreeFieldPrefixNSMLeafFrame.java
index e908c1b..701f44f 100644
--- a/hyracks-fullstack/hyracks/hyracks-storage-am-btree/src/main/java/org/apache/hyracks/storage/am/btree/frames/BTreeFieldPrefixNSMLeafFrame.java
+++ b/hyracks-fullstack/hyracks/hyracks-storage-am-btree/src/main/java/org/apache/hyracks/storage/am/btree/frames/BTreeFieldPrefixNSMLeafFrame.java
@@ -33,9 +33,11 @@ import org.apache.hyracks.storage.am.btree.impls.BTreeOpContext.PageValidationIn
 import org.apache.hyracks.storage.am.btree.impls.FieldPrefixPrefixTupleReference;
 import org.apache.hyracks.storage.am.btree.impls.FieldPrefixSlotManager;
 import org.apache.hyracks.storage.am.btree.impls.FieldPrefixTupleReference;
+import org.apache.hyracks.storage.am.common.api.IMetaDataPageManager;
 import org.apache.hyracks.storage.am.common.api.ISplitKey;
 import org.apache.hyracks.storage.am.common.api.ITreeIndexFrame;
 import org.apache.hyracks.storage.am.common.api.ITreeIndexFrameCompressor;
+import org.apache.hyracks.storage.am.common.api.ITreeIndexMetaDataFrame;
 import org.apache.hyracks.storage.am.common.api.ITreeIndexTupleReference;
 import org.apache.hyracks.storage.am.common.api.ITreeIndexTupleWriter;
 import org.apache.hyracks.storage.am.common.api.TreeIndexException;
@@ -47,7 +49,9 @@ import org.apache.hyracks.storage.am.common.ophelpers.FindTupleNoExactMatchPolic
 import org.apache.hyracks.storage.am.common.ophelpers.MultiComparator;
 import org.apache.hyracks.storage.am.common.ophelpers.SlotOffTupleOff;
 import org.apache.hyracks.storage.am.common.tuples.TypeAwareTupleWriter;
+import org.apache.hyracks.storage.common.buffercache.IBufferCache;
 import org.apache.hyracks.storage.common.buffercache.ICachedPage;
+import org.apache.hyracks.storage.common.buffercache.ILargePageHelper;
 
 /**
  * WARNING: only works when tupleWriter is an instance of TypeAwareTupleWriter
@@ -532,6 +536,26 @@ public class BTreeFieldPrefixNSMLeafFrame implements IBTreeLeafFrame {
             buf.put(smFlagOff, (byte) 0);
     }
 
+    @Override
+    public void configureLargePage(int supplementalPages, int supplementalBlockPageId) {
+        throw new IllegalStateException();
+    }
+
+    @Override
+    public void setLargeFlag(boolean largePage) {
+        throw new IllegalStateException();
+    }
+
+    @Override
+    public boolean isLargePage() {
+        return false;
+    }
+
+    @Override
+    public int getSupplementalNumPages() {
+        throw new IllegalStateException();
+    }
+
     public int getPrefixTupleCount() {
         return buf.getInt(prefixTupleCountOff);
     }
@@ -572,7 +596,7 @@ public class BTreeFieldPrefixNSMLeafFrame implements IBTreeLeafFrame {
     }
 
     @Override
-    public void split(ITreeIndexFrame rightFrame, ITupleReference tuple, ISplitKey splitKey)
+    public void split(ITreeIndexFrame rightFrame, ITupleReference tuple, ISplitKey splitKey, IMetaDataPageManager freePageManager, ITreeIndexMetaDataFrame metaFrame, IBufferCache bufferCache)
             throws HyracksDataException {
 
         BTreeFieldPrefixNSMLeafFrame rf = (BTreeFieldPrefixNSMLeafFrame) rightFrame;
@@ -778,4 +802,15 @@ public class BTreeFieldPrefixNSMLeafFrame implements IBTreeLeafFrame {
     public void validate(PageValidationInfo pvi) {
         // Do nothing
     }
+
+    @Override
+    public ILargePageHelper getLargePageHelper() {
+        return null;
+    }
+
+    @Override
+    public void ensureCapacity(IMetaDataPageManager freePageManager, ITreeIndexMetaDataFrame metaFrame, IBufferCache bufferCache, ITupleReference tuple) throws HyracksDataException {
+        throw new IllegalStateException("nyi");
+    }
+
 }

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/hyracks-fullstack/hyracks/hyracks-storage-am-btree/src/main/java/org/apache/hyracks/storage/am/btree/frames/BTreeFieldPrefixNSMLeafFrameFactory.java
----------------------------------------------------------------------
diff --git a/hyracks-fullstack/hyracks/hyracks-storage-am-btree/src/main/java/org/apache/hyracks/storage/am/btree/frames/BTreeFieldPrefixNSMLeafFrameFactory.java b/hyracks-fullstack/hyracks/hyracks-storage-am-btree/src/main/java/org/apache/hyracks/storage/am/btree/frames/BTreeFieldPrefixNSMLeafFrameFactory.java
index beb350d..c558d30 100644
--- a/hyracks-fullstack/hyracks/hyracks-storage-am-btree/src/main/java/org/apache/hyracks/storage/am/btree/frames/BTreeFieldPrefixNSMLeafFrameFactory.java
+++ b/hyracks-fullstack/hyracks/hyracks-storage-am-btree/src/main/java/org/apache/hyracks/storage/am/btree/frames/BTreeFieldPrefixNSMLeafFrameFactory.java
@@ -22,6 +22,7 @@ package org.apache.hyracks.storage.am.btree.frames;
 import org.apache.hyracks.storage.am.btree.api.IBTreeLeafFrame;
 import org.apache.hyracks.storage.am.common.api.ITreeIndexFrameFactory;
 import org.apache.hyracks.storage.am.common.api.ITreeIndexTupleWriterFactory;
+import org.apache.hyracks.storage.common.buffercache.ILargePageHelper;
 
 public class BTreeFieldPrefixNSMLeafFrameFactory implements ITreeIndexFrameFactory {
 
@@ -42,4 +43,9 @@ public class BTreeFieldPrefixNSMLeafFrameFactory implements ITreeIndexFrameFacto
     public ITreeIndexTupleWriterFactory getTupleWriterFactory() {
         return tupleWriterFactory;
     }
+
+    @Override
+    public ILargePageHelper getLargePageHelper() {
+        return null;
+    }
 }

http://git-wip-us.apache.org/repos/asf/incubator-asterixdb/blob/1defc92a/hyracks-fullstack/hyracks/hyracks-storage-am-btree/src/main/java/org/apache/hyracks/storage/am/btree/frames/BTreeLargeFrameHelper.java
----------------------------------------------------------------------
diff --git a/hyracks-fullstack/hyracks/hyracks-storage-am-btree/src/main/java/org/apache/hyracks/storage/am/btree/frames/BTreeLargeFrameHelper.java b/hyracks-fullstack/hyracks/hyracks-storage-am-btree/src/main/java/org/apache/hyracks/storage/am/btree/frames/BTreeLargeFrameHelper.java
new file mode 100644
index 0000000..79f7f53
--- /dev/null
+++ b/hyracks-fullstack/hyracks/hyracks-storage-am-btree/src/main/java/org/apache/hyracks/storage/am/btree/frames/BTreeLargeFrameHelper.java
@@ -0,0 +1,44 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *   http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied.  See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+package org.apache.hyracks.storage.am.btree.frames;
+
+import java.nio.ByteBuffer;
+
+import org.apache.hyracks.storage.common.buffercache.CachedPage;
+import org.apache.hyracks.storage.common.buffercache.ILargePageHelper;
+
+public class BTreeLargeFrameHelper implements ILargePageHelper {
+
+    public static final BTreeLargeFrameHelper INSTANCE = new BTreeLargeFrameHelper();
+
+    private BTreeLargeFrameHelper() {
+    }
+
+    @Override
+    public int getSupplementalBlockNumPages(CachedPage cPage) {
+        ByteBuffer buffer = cPage.getBuffer();
+        return BTreeNSMLeafFrame.isLargePage(buffer) ? BTreeNSMLeafFrame.getSupplementalNumPages(buffer) : 0;
+    }
+
+    @Override
+    public int getSupplementalBlockPageId(CachedPage cPage) {
+        ByteBuffer buffer = cPage.getBuffer();
+        return BTreeNSMLeafFrame.isLargePage(buffer) ? BTreeNSMLeafFrame.getSupplementalPageId(buffer) : -1;
+    }
+}