You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@systemml.apache.org by mb...@apache.org on 2017/09/05 07:58:10 UTC

[2/2] systemml git commit: [SYSTEMML-1888] Remove parfor rewrite on update-in-place intermediates

[SYSTEMML-1888] Remove parfor rewrite on update-in-place intermediates

This patch removes the brittle parfor rewrite on update-in-place
intermediates that has shown to blow up for large parfor programs and
many candidates due to poor asymptotic behavior (at least squared in the
number of nodes and candidates, large constants). Furthermore, this
rewrites also relies on comparisons of hop names and line numbers to
check validity, which is dangerous in the presence of other rewrites. 

Meanwhile, we apply anyway a very robust update-in-place rewrite to all
for and parfor loops, and hence, the performance impact of this parfor
rewrite is rather limited. Therefore, we remove this rewrite for the
0.15 release and consider a reimplementation from scratch for the 1.0
release.
 

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

Branch: refs/heads/master
Commit: 1cbfdef3fad5235b692c372ec94bcade7f1b9ff9
Parents: 30caf36
Author: Matthias Boehm <mb...@gmail.com>
Authored: Tue Sep 5 00:51:52 2017 -0700
Committer: Matthias Boehm <mb...@gmail.com>
Committed: Tue Sep 5 00:51:52 2017 -0700

----------------------------------------------------------------------
 .../sysml/hops/rewrite/HopRewriteUtils.java     |  18 +-
 .../controlprogram/parfor/opt/OptNode.java      |   5 +
 .../parfor/opt/OptimizerRuleBased.java          | 903 +------------------
 .../updateinplace/UpdateInPlaceTest.java        |  46 -
 4 files changed, 52 insertions(+), 920 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/systemml/blob/1cbfdef3/src/main/java/org/apache/sysml/hops/rewrite/HopRewriteUtils.java
----------------------------------------------------------------------
diff --git a/src/main/java/org/apache/sysml/hops/rewrite/HopRewriteUtils.java b/src/main/java/org/apache/sysml/hops/rewrite/HopRewriteUtils.java
index 1bf381c..dcc740e 100644
--- a/src/main/java/org/apache/sysml/hops/rewrite/HopRewriteUtils.java
+++ b/src/main/java/org/apache/sysml/hops/rewrite/HopRewriteUtils.java
@@ -31,6 +31,7 @@ import org.apache.sysml.hops.AggBinaryOp;
 import org.apache.sysml.hops.AggUnaryOp;
 import org.apache.sysml.hops.BinaryOp;
 import org.apache.sysml.hops.DataOp;
+import org.apache.sysml.hops.FunctionOp;
 import org.apache.sysml.hops.Hop;
 import org.apache.sysml.hops.Hop.AggOp;
 import org.apache.sysml.hops.Hop.DataGenMethod;
@@ -1040,11 +1041,18 @@ public class HopRewriteUtils
 		return ret;
 	}
 	
-	public static boolean alwaysRequiresReblock(Hop hop)
-	{
-		return (    hop instanceof DataOp 
-				 && ((DataOp)hop).getDataOpType()==DataOpTypes.PERSISTENTREAD
-				 && ((DataOp)hop).getInputFormatType()!=FileFormatTypes.BINARY);
+	public static boolean alwaysRequiresReblock(Hop hop) {
+		return (hop instanceof DataOp
+			&& ((DataOp)hop).getDataOpType()==DataOpTypes.PERSISTENTREAD
+			 && ((DataOp)hop).getInputFormatType()!=FileFormatTypes.BINARY);
+	}
+	
+	public static boolean containsFunctioOp(ArrayList<Hop> candidates) {
+		if( candidates != null )
+			for( Hop cand : candidates )
+				if( cand instanceof FunctionOp )
+					return true;
+		return false;
 	}
 	
 	public static boolean rHasSimpleReadChain(Hop root, String var)

http://git-wip-us.apache.org/repos/asf/systemml/blob/1cbfdef3/src/main/java/org/apache/sysml/runtime/controlprogram/parfor/opt/OptNode.java
----------------------------------------------------------------------
diff --git a/src/main/java/org/apache/sysml/runtime/controlprogram/parfor/opt/OptNode.java b/src/main/java/org/apache/sysml/runtime/controlprogram/parfor/opt/OptNode.java
index 193ce3e..2464bf6 100644
--- a/src/main/java/org/apache/sysml/runtime/controlprogram/parfor/opt/OptNode.java
+++ b/src/main/java/org/apache/sysml/runtime/controlprogram/parfor/opt/OptNode.java
@@ -25,6 +25,7 @@ import java.util.HashMap;
 import java.util.LinkedList;
 import java.util.Set;
 
+import org.apache.commons.lang.ArrayUtils;
 import org.apache.sysml.lops.Lop;
 import org.apache.sysml.runtime.controlprogram.ParForProgramBlock;
 import org.apache.sysml.runtime.controlprogram.ParForProgramBlock.PDataPartitionFormat;
@@ -120,6 +121,10 @@ public class OptNode
 		_ntype = type;
 	}
 	
+	public boolean isNodeType(NodeType... types) {
+		return ArrayUtils.contains(types, _ntype);
+	}
+	
 	public ExecType getExecType() {
 		return _etype;
 	}

http://git-wip-us.apache.org/repos/asf/systemml/blob/1cbfdef3/src/main/java/org/apache/sysml/runtime/controlprogram/parfor/opt/OptimizerRuleBased.java
----------------------------------------------------------------------
diff --git a/src/main/java/org/apache/sysml/runtime/controlprogram/parfor/opt/OptimizerRuleBased.java b/src/main/java/org/apache/sysml/runtime/controlprogram/parfor/opt/OptimizerRuleBased.java
index 9dada01..c429bfa 100644
--- a/src/main/java/org/apache/sysml/runtime/controlprogram/parfor/opt/OptimizerRuleBased.java
+++ b/src/main/java/org/apache/sysml/runtime/controlprogram/parfor/opt/OptimizerRuleBased.java
@@ -24,9 +24,6 @@ import java.util.ArrayList;
 import java.util.Collection;
 import java.util.HashMap;
 import java.util.HashSet;
-import java.util.Iterator;
-import java.util.List;
-import java.util.Map;
 import java.util.Map.Entry;
 import java.util.Set;
 
@@ -36,12 +33,10 @@ import org.apache.sysml.api.DMLScript;
 import org.apache.sysml.conf.ConfigurationManager;
 import org.apache.sysml.conf.DMLConfig;
 import org.apache.sysml.hops.AggBinaryOp;
-import org.apache.sysml.hops.DataGenOp;
 import org.apache.sysml.hops.DataOp;
 import org.apache.sysml.hops.FunctionOp;
 import org.apache.sysml.hops.Hop;
 import org.apache.sysml.hops.AggBinaryOp.MMultMethod;
-import org.apache.sysml.hops.Hop.DataOpTypes;
 import org.apache.sysml.hops.Hop.MultiThreadedHop;
 import org.apache.sysml.hops.Hop.ParamBuiltinOp;
 import org.apache.sysml.hops.Hop.ReOrgOp;
@@ -62,18 +57,15 @@ import org.apache.sysml.hops.recompile.Recompiler;
 import org.apache.sysml.lops.LopProperties;
 import org.apache.sysml.lops.LopsException;
 import org.apache.sysml.parser.DMLProgram;
-import org.apache.sysml.parser.Expression;
 import org.apache.sysml.parser.Expression.DataType;
 import org.apache.sysml.parser.FunctionStatementBlock;
 import org.apache.sysml.parser.LanguageException;
 import org.apache.sysml.parser.ParForStatement;
 import org.apache.sysml.parser.ParForStatementBlock;
 import org.apache.sysml.parser.StatementBlock;
-import org.apache.sysml.parser.VariableSet;
 import org.apache.sysml.runtime.DMLRuntimeException;
 import org.apache.sysml.runtime.controlprogram.ForProgramBlock;
 import org.apache.sysml.runtime.controlprogram.FunctionProgramBlock;
-import org.apache.sysml.runtime.controlprogram.IfProgramBlock;
 import org.apache.sysml.runtime.controlprogram.LocalVariableMap;
 import org.apache.sysml.runtime.controlprogram.ParForProgramBlock;
 import org.apache.sysml.runtime.controlprogram.Program;
@@ -85,7 +77,6 @@ import org.apache.sysml.runtime.controlprogram.ParForProgramBlock.POptMode;
 import org.apache.sysml.runtime.controlprogram.ParForProgramBlock.PResultMerge;
 import org.apache.sysml.runtime.controlprogram.ParForProgramBlock.PTaskPartitioner;
 import org.apache.sysml.runtime.controlprogram.ParForProgramBlock.PartitionFormat;
-import org.apache.sysml.runtime.controlprogram.WhileProgramBlock;
 import org.apache.sysml.runtime.controlprogram.caching.MatrixObject;
 import org.apache.sysml.runtime.controlprogram.caching.MatrixObject.UpdateType;
 import org.apache.sysml.runtime.controlprogram.context.ExecutionContext;
@@ -107,7 +98,6 @@ import org.apache.sysml.runtime.matrix.MatrixFormatMetaData;
 import org.apache.sysml.runtime.matrix.data.MatrixBlock;
 import org.apache.sysml.runtime.matrix.data.OutputInfo;
 import org.apache.sysml.runtime.matrix.data.SparseRowVector;
-import org.apache.sysml.utils.Explain;
 import org.apache.sysml.yarn.ropt.YarnClusterAnalyzer;
 
 /**
@@ -160,8 +150,6 @@ public class OptimizerRuleBased extends Optimizer
 	public static final boolean ALLOW_REMOTE_NESTED_PARALLELISM = false;
 	public static final String FUNCTION_UNFOLD_NAMEPREFIX = "__unfold_";
 	
-	public static final boolean APPLY_REWRITE_UPDATE_INPLACE_INTERMEDIATE = true;
-	
 	public static final double PAR_K_FACTOR        = OptimizationWrapper.PAR_FACTOR_INFRASTRUCTURE; 
 	public static final double PAR_K_MR_FACTOR     = 1.0 * OptimizationWrapper.PAR_FACTOR_INFRASTRUCTURE; 
 	
@@ -179,13 +167,8 @@ public class OptimizerRuleBased extends Optimizer
 	protected double _lm = -1; //local memory constraint
 	protected double _rm = -1; //remote memory constraint (mappers)
 	protected double _rm2 = -1; //remote memory constraint (reducers)
-	
-	
+		
 	protected CostEstimator _cost = null;
-	
-	protected static ThreadLocal<ArrayList<String>> listUIPRes = new ThreadLocal<ArrayList<String>>() {
-		@Override protected ArrayList<String> initialValue() { return new ArrayList<String>(); }
-	};
 
 	@Override
 	public CostModelType getCostModelType() 
@@ -1686,13 +1669,12 @@ public class OptimizerRuleBased extends Optimizer
 			LOG.warn(getOptMode()+" OPT: Set in-place result update is only applicable for a ParFor node.");
 		
 		boolean apply = false;
-
+		
 		ParForProgramBlock pfpb = (ParForProgramBlock) OptTreeConverter
-              .getAbstractPlanMapping().getMappedProg(pn.getID())[1];
+			.getAbstractPlanMapping().getMappedProg(pn.getID())[1];
 		
 		//note currently we decide for all result vars jointly, i.e.,
 		//only if all fit pinned in remaining budget, we apply this rewrite.
-		
 		ArrayList<String> retVars = pfpb.getResultVariables();
 		
 		//compute total sum of pinned result variable memory
@@ -1700,8 +1682,7 @@ public class OptimizerRuleBased extends Optimizer
 		
 		//NOTE: currently this rule is too conservative (the result variable is assumed to be dense and
 		//most importantly counted twice if this is part of the maximum operation)
-		HashMap <String, ArrayList <UIPCandidateHop>> uipCandHopHM = new HashMap <String, ArrayList<UIPCandidateHop>>();   
-		double totalMem = Math.max((M+sum), rComputeSumMemoryIntermediates(pn, new HashSet<String>(), uipCandHopHM));
+		double totalMem = Math.max((M+sum), rComputeSumMemoryIntermediates(pn, new HashSet<String>()));
 		
 		//optimization decision
 		if( rHasOnlyInPlaceSafeLeftIndexing(pn, retVars) ) //basic correctness constraint
@@ -1722,9 +1703,6 @@ public class OptimizerRuleBased extends Optimizer
 			}
 		}
 		
-		if(APPLY_REWRITE_UPDATE_INPLACE_INTERMEDIATE && LOG.isDebugEnabled())
-			listUIPRes.remove();
-
 		//modify result variable meta data, if rewrite applied
 		if( apply ) 
 		{
@@ -1736,733 +1714,45 @@ public class OptimizerRuleBased extends Optimizer
 					((MatrixObject)dat).setUpdateType(UpdateType.INPLACE_PINNED);
 			}
 			inPlaceResultVars.addAll(retVars);
-
-			if(APPLY_REWRITE_UPDATE_INPLACE_INTERMEDIATE)
-			{
-				isUpdateInPlaceApplicable(pn, uipCandHopHM);
-	
-				boolean bAnyUIPApplicable = false;
-				for(Entry<String, ArrayList <UIPCandidateHop>> entry: uipCandHopHM.entrySet())
-				{
-					ArrayList <UIPCandidateHop> uipCandHopList = entry.getValue();
-					
-					if (uipCandHopList != null) {
-						for (UIPCandidateHop uipCandHop: uipCandHopList)
-							if(uipCandHop.isIntermediate() && uipCandHop.isLoopApplicable() && uipCandHop.isUpdateInPlace())
-							{
-								uipCandHop.getHop().setUpdateType(UpdateType.INPLACE_PINNED);
-								bAnyUIPApplicable = true;
-
-								if(LOG.isDebugEnabled())
-									listUIPRes.get().add(uipCandHop.getHop().getName());
-							}
-					}
-				}
-				if(bAnyUIPApplicable)
-					try {
-						//Recompile this block recursively if there is any update in place applicable.
-						LocalVariableMap localVaraibleMap = (LocalVariableMap) ec.getVariables().clone();
-						Recompiler.recompileProgramBlockHierarchy(pfpb.getChildBlocks(), localVaraibleMap, 0L, true);
-					}
-					catch(Exception ex){
-						throw new DMLRuntimeException(ex);
-					}
-			}
-		}
-
-		if(APPLY_REWRITE_UPDATE_INPLACE_INTERMEDIATE && LOG.isTraceEnabled())
-		{
-			LOG.trace("UpdateInPlace = " + apply + " for lines between " + pn.getBeginLine() + " and " + pn.getEndLine()
-					+ " for " + uipCandHopHM.size() + " intermediate matrix objects:" + uipCandHopHM.keySet().toString());
-				
-			for(Entry<String, ArrayList <UIPCandidateHop>> entry: uipCandHopHM.entrySet())
-			{
-				ArrayList <UIPCandidateHop> uipCandHopList = entry.getValue();
-				
-				if (uipCandHopList != null) {
-					for (UIPCandidateHop uipCandHop: uipCandHopList)
-					{
-						if(uipCandHop.getHop() != null)
-						{
-							LOG.trace("Matrix Object: Name: " + uipCandHop.getHop().getName() + "<" + uipCandHop.getHop().getBeginLine() + "," + uipCandHop.getHop().getEndLine()+ ">, InLoop:"
-								+ uipCandHop.isLoopApplicable() + ", UIPApplicable:" + uipCandHop.isUpdateInPlace() + ", HopUIPApplicable:" + uipCandHop.getHop().getUpdateType());
-							LOG.trace("Explain Candidate HOP after recompile");
-							LOG.trace(Explain.explain(uipCandHop.getHop()));
-						}
-						else
-						{
-							LOG.trace("Matrix Object: Name: " + uipCandHop.getLixHop().getName() + "<" + uipCandHop.getLixHop().getBeginLine() + "," + uipCandHop.getLixHop().getEndLine()+ ">, InLoop:"
-									+ uipCandHop.isLoopApplicable() + ", Not an Intermediate matrix object");
-						}
-					}
-				}
-			}
-		}
-							
-		LOG.debug(getOptMode()+" OPT: rewrite 'set in-place result indexing' - result="+
-		          apply+" ("+ProgramConverter.serializeStringCollection(inPlaceResultVars)+", M="+toMB(totalMem)+")" );	
-	}
-	
-	/* 
-	 * Algorithm: isUpdateInPlaceApplicable()
-	 *
-	 * Purpose of this algorithm to identify intermediate hops containing matrix objects to be marked as "UpdateInPlace" from ParforProgramBlock. 
-	 * First, list of candidates are identified. Then list is pruned based on conditions descibed below.
-	 * 
-	 * A.Identification of candidates:
-	 *  1. Candidate's identity defined with name, beginline, endline, and hop.
-	 * 	2. Operation of type LeftIndexingOp
-	 *  3. Number of consumers for Hop's first input should be one.
-	 *  4. Matrix Object on which leftindexing operation done has defined outside "Loop" A. 
-	 *  5. LeftIndexingOp operation is within a "Loop" A.
-	 *
-	 * 	Notes: 1. Loop is of type while, for, or parfor with parallelism of one.
-	 * 		   2. Some conidtions ignored at this point listed below
-	 * 			2.1 Unsure of general instructions. It will be hard to identify and iterate.
-	 * 			2.2 LeftIndexing outside "loop"
-	 * 
-	 * 
-	 * B.Pruning the list:
-	 *  Candidates are pruned based on any condition met from conditions from 1 to 3 below.
-	 *  0. Identify of candidate is defined with name, begineline, endline, and hop.
-	 * 	1. Based on the scope of candidate. If Variable (name) is defined in liveout of loop's statementblock.
-	 *  2. Based on operation type and order  
-	 *  	2.1 If hop's input variable name is same as candidate's name 
-	 *  	2.2 Location of hop is before candidate.
-	 *  	2.3 Hop's operator type is any of following 
-	 *  		2.3.1 DataOp (with operation type TransientWrite or TransientRead)
-	 *  		2.3.2 ReorgOp (with operation type Reshape or Transpose)
-	 *  		2.3.3 FunctionOp
-	 *  3. Location of consumer being affected.
-	 *  	3.1 Consumer defined before leftindexing through operation process defined in 2.3 above.
-	 *  	3.2 Consumer is being utilized after leftindexing on candidate.
-	 *  
-	 *  Notes:
-	 *  	1. No interleave operations.
-	 *  	2. Function with actual operation to be scanned for candiate exclusion list.
-	 *  	3. Operattion that does not include updated data through updateinplace. 
-	 * 	
-	 * 
-	 * @param pn:				OpNode of parfor loop
-	 * @param uipCandHopHM:		Hashmap of UIPCandidateHop with name as a key.		
-	 * @throws DMLRuntimeException
-	 */
-	private void isUpdateInPlaceApplicable(OptNode pn, HashMap <String, ArrayList <UIPCandidateHop>> uipCandHopHM)
-			throws DMLRuntimeException 
-	{
-		rIsInLoop(pn, uipCandHopHM, false);
-		
-		// Prune candidate list based on non-existance of intermediate candidate in the loop
-		Iterator<Map.Entry<String, ArrayList <UIPCandidateHop>>> uipCandHopHMIter = uipCandHopHM.entrySet().iterator();
-		while(uipCandHopHMIter.hasNext())
-		{
-			Map.Entry<String, ArrayList <UIPCandidateHop>> uipCandHopHMentry = uipCandHopHMIter.next();
-			ArrayList <UIPCandidateHop> uipCandHopList = uipCandHopHMentry.getValue();
-			
-			if (uipCandHopList != null) {
-				for (Iterator<UIPCandidateHop> uipCandHopListIter = uipCandHopList.iterator(); uipCandHopListIter.hasNext();) 
-				{
-					UIPCandidateHop uipCandHop = uipCandHopListIter.next();
-					if (!uipCandHop.isIntermediate() || !uipCandHop.isLoopApplicable())	//If Loop is not applicable then remove it from the list. 
-					{
-						uipCandHopListIter.remove();
-						if(LOG.isTraceEnabled())
-						{
-							if (!uipCandHop.isIntermediate())
-								LOG.trace("Matrix Object: Name: " + uipCandHop.getLixHop().getName() + "<" + uipCandHop.getLixHop().getBeginLine() + "," + uipCandHop.getLixHop().getEndLine()+ 
-										">, removed from the candidate list as it is not an intermediate matrix object.");
-							else
-								LOG.trace("Matrix Object: Name: " + uipCandHop.getLixHop().getName() + "<" + uipCandHop.getLixHop().getBeginLine() + "," + uipCandHop.getLixHop().getEndLine()+ 
-										">, removed from the candidate list as it does not have loop criteria applicable.");
-						}
-					}
-				}
-				if(uipCandHopList.isEmpty())
-					uipCandHopHMIter.remove();
-			}
-		}
-
-		if(!uipCandHopHM.isEmpty())
-		{
-			// Get consumer list
-			rResetVisitStatus(pn);
-			rGetUIPConsumerList(pn, uipCandHopHM);
-
-			// Prune candidate list if consumer is in function call.
-			uipCandHopHMIter = uipCandHopHM.entrySet().iterator();
-			while(uipCandHopHMIter.hasNext())
-			{
-				Map.Entry<String, ArrayList <UIPCandidateHop>> uipCandHopHMentry = uipCandHopHMIter.next();
-				ArrayList <UIPCandidateHop> uipCandHopList = uipCandHopHMentry.getValue();
-				
-				if (uipCandHopList != null) {
-					for (Iterator<UIPCandidateHop> uipCandHopListIter = uipCandHopList.iterator(); uipCandHopListIter.hasNext();) 
-					{
-						UIPCandidateHop uipCandHop = uipCandHopListIter.next();
-						// if one of the consumer is FunctionOp then remove it.
-						ArrayList<Hop> consHops = uipCandHop.getConsumerHops();
-						if(consHops != null)
-							for (Hop hop: consHops)
-							{
-								if(hop instanceof FunctionOp)
-								{
-									uipCandHopListIter.remove();
-									if(LOG.isTraceEnabled())
-										LOG.trace("Matrix Object: Name: " + uipCandHop.getLixHop().getName() + "<" + uipCandHop.getLixHop().getBeginLine() + "," + uipCandHop.getLixHop().getEndLine()+ 
-												">, removed from the candidate list as one of the consumer is FunctionOp.");
-									break;
-								}
-							}
-					}
-					if(uipCandHopList.isEmpty())
-						uipCandHopHMIter.remove();
-				}
-			}
-
-			//Validate the consumer list
-			rResetVisitStatus(pn);
-			rValidateUIPConsumerList(pn, uipCandHopHM);
-		}
-	}
-	
-	
-	
-	/* 	
-	 * This will check if candidate LeftIndexingOp are in loop (while, for or parfor).
-	 * 
-	 * @param pn:				OpNode of parfor loop
-	 * @param uipCandHopHM:		Hashmap of UIPCandidateHop with name as a key.		
-	 * @throws DMLRuntimeException
-	 */
-	private void rIsInLoop(OptNode pn, HashMap <String, ArrayList<UIPCandidateHop>> uipCandHopHM, boolean bInLoop)
-			throws DMLRuntimeException 
-	{
-		if(!pn.isLeaf())  
-		{
-			ProgramBlock pb = (ProgramBlock) OptTreeConverter.getAbstractPlanMapping().getMappedProg(pn.getID())[1];
-
-			VariableSet varUpdated = pb.getStatementBlock().variablesUpdated();
-			boolean bUIPCandHopUpdated = false;
-			for(Entry<String, ArrayList <UIPCandidateHop>> entry: uipCandHopHM.entrySet())
-			{
-				String uipCandHopID = entry.getKey();
-				
-				if (varUpdated.containsVariable(uipCandHopID))
-				{	
-					bUIPCandHopUpdated = true;
-					break;
-				}
-			}
-
-			// As none of the UIP candidates updated in this DAG, no need for further processing within this DAG
-			if(!bUIPCandHopUpdated)
-				return;
-
-			boolean bLoop = false;
-			if(	bInLoop || pb instanceof WhileProgramBlock || 
-				(pb instanceof ParForProgramBlock && ((ParForProgramBlock)pb).getDegreeOfParallelism() == 1) ||
-				(pb instanceof ForProgramBlock && !(pb instanceof ParForProgramBlock)))
-				bLoop = true;
-
-			for (OptNode optNode: pn.getChilds())
-			{
-				rIsInLoop(optNode, uipCandHopHM, bLoop);
-			}
-		}
-		else 
-		{
-			Hop hop = (Hop) OptTreeConverter.getAbstractPlanMapping().getMappedHop(pn.getID());
-
-			for(Entry<String, ArrayList <UIPCandidateHop>> entry: uipCandHopHM.entrySet())
-			{
-				ArrayList <UIPCandidateHop> uipCandHopList = entry.getValue();
-				
-				if (uipCandHopList != null) 
-				{
-					for (UIPCandidateHop uipCandHop: uipCandHopList)
-					{
-						//Identify where intermediate object has been defined.
-						if (hop instanceof DataGenOp && hop.getName().equals(uipCandHop.getLixHop().getName()))
-						{
-							uipCandHop.setHop(hop);
-							uipCandHop.setLocation(hop.getBeginLine());
-							uipCandHop.setIntermediate(true);
-						}
-							
-						//Update if candiate hop defined outside this loop, and leftindexing is within this loop.
-						if ((bInLoop) && (uipCandHop.getLocation() <= hop.getBeginLine() && uipCandHop.getLixHop().getBeginLine() <= hop.getEndLine()))
-							uipCandHop.setIsLoopApplicable(true);
-					}
-				}
-			}
 		}
 		
+		LOG.debug(getOptMode()+" OPT: rewrite 'set in-place result indexing' - result="+
+			apply+" ("+ProgramConverter.serializeStringCollection(inPlaceResultVars)+", M="+toMB(totalMem)+")" );
 	}
 	
-	
-	
-	/* 	
-	 * This will get consumer list for candidate LeftIndexingOp.
-	 * 
-	 * @param pn:				OpNode of parfor loop
-	 * @param uipCandHopHM:		Hashmap of UIPCandidateHop with name as a key.		
-	 * @throws DMLRuntimeException
-	 */
-	private void rGetUIPConsumerList(OptNode pn, HashMap <String, ArrayList<UIPCandidateHop>> uipCandHopHM)
-			throws DMLRuntimeException 
-	{
-		if(!pn.isLeaf())
-		{
-			if(pn.getNodeType() == OptNode.NodeType.FUNCCALL)
-				return;
-
-			ProgramBlock pb = (ProgramBlock) OptTreeConverter.getAbstractPlanMapping().getMappedProg(pn.getID())[1];
-			
-			VariableSet varRead = pb.getStatementBlock().variablesRead();
-			boolean bUIPCandHopRead = false;
-			for(Entry<String, ArrayList <UIPCandidateHop>> entry: uipCandHopHM.entrySet())
-			{
-				String uipCandHopID = entry.getKey();
-				
-				if (varRead.containsVariable(uipCandHopID))
-				{	
-					bUIPCandHopRead = true;
-					break;
-				}
-			}
-			
-			// As none of the UIP candidates updated in this DAG, no need for further processing within this DAG
-			if(!bUIPCandHopRead)
-				return;
-			
-			for (OptNode optNode: pn.getChilds())
-				rGetUIPConsumerList(optNode, uipCandHopHM);
-		}
-		else
-		{
-			OptTreePlanMappingAbstract map = OptTreeConverter.getAbstractPlanMapping();
-			long ppid = map.getMappedParentID(map.getMappedParentID(pn.getID()));
-			Object[] o = map.getMappedProg(ppid);
-			ProgramBlock pb = (ProgramBlock) o[1];
-			
-			Hop hop = (Hop) OptTreeConverter.getAbstractPlanMapping().getMappedHop(pn.getID());
-			rGetUIPConsumerList(hop, uipCandHopHM);
-
-			if(pb instanceof IfProgramBlock || pb instanceof WhileProgramBlock || 
-				(pb instanceof ForProgramBlock && !(pb instanceof ParForProgramBlock)))  //TODO
-				rGetUIPConsumerList(pb, uipCandHopHM);
-		} 
-	}
-	
-	
-	private void rGetUIPConsumerList(ProgramBlock pb, HashMap <String, ArrayList<UIPCandidateHop>> uipCandHopHM)
-			throws DMLRuntimeException
-	{
-		ArrayList<ProgramBlock> childBlocks = null;
-		ArrayList<ProgramBlock> elseBlocks = null;
-		if (pb instanceof WhileProgramBlock)
-			childBlocks = ((WhileProgramBlock)pb).getChildBlocks();
-		else if (pb instanceof ForProgramBlock)
-			childBlocks = ((ForProgramBlock)pb).getChildBlocks();
-		else if (pb instanceof IfProgramBlock) 
-		{
-			childBlocks = ((IfProgramBlock)pb).getChildBlocksIfBody();
-			elseBlocks = ((IfProgramBlock)pb).getChildBlocksElseBody();
-		}
-
-		if(childBlocks != null)
-			for (ProgramBlock childBlock: childBlocks)
-			{
-				rGetUIPConsumerList(childBlock, uipCandHopHM);
-				try 
-				{
-					rGetUIPConsumerList(childBlock.getStatementBlock().get_hops(), uipCandHopHM);
-				}
-				catch (Exception e) {
-					throw new DMLRuntimeException(e);
-				}
-			}
-		if(elseBlocks != null)
-			for (ProgramBlock childBlock: elseBlocks)
-			{
-				rGetUIPConsumerList(childBlock, uipCandHopHM);
-				try 
-				{
-					rGetUIPConsumerList(childBlock.getStatementBlock().get_hops(), uipCandHopHM);
-				}
-				catch (Exception e) {
-					throw new DMLRuntimeException(e);
-				}
-			}
-	}	
-		
-	private void rGetUIPConsumerList(ArrayList<Hop> hops, HashMap <String, ArrayList<UIPCandidateHop>> uipCandHopHM)
-			throws DMLRuntimeException
-	{
-		if(hops != null)
-			for (Hop hop: hops)
-				rGetUIPConsumerList(hop, uipCandHopHM);
-	}
-
-		
-	private void rGetUIPConsumerList(Hop hop, HashMap <String, ArrayList<UIPCandidateHop>> uipCandHopHM)
-		throws DMLRuntimeException
-	{
-		if(hop.isVisited())
-			return;
-		
-		if ((!(!hop.getParent().isEmpty() && hop.getParent().get(0) instanceof LeftIndexingOp)) &&
-			   ((hop instanceof DataOp && ((DataOp)hop).getDataOpType() == DataOpTypes.TRANSIENTREAD ) ||
-				(hop instanceof ReorgOp && (((ReorgOp)hop).getOp() == ReOrgOp.RESHAPE || ((ReorgOp)hop).getOp() == ReOrgOp.TRANSPOSE)) ||
-				(hop instanceof FunctionOp)))
-		{	
-			// If candidate's name is same as input hop.
-			String uipCandiateID = hop.getName();
-			ArrayList <UIPCandidateHop> uipCandHopList = uipCandHopHM.get(uipCandiateID);
-			
-			if (uipCandHopList != null) 
-			{
-				for (UIPCandidateHop uipCandHop: uipCandHopList)
-				{
-					// Add consumers for candidate hop.
-					ArrayList<Hop> consumerHops = uipCandHop.getConsumerHops();
-					if(uipCandHop.getConsumerHops() == null)
-						consumerHops = new ArrayList<Hop>();
-					consumerHops.add(getRootHop(hop));
-					uipCandHop.setConsumerHops(consumerHops);
-				}
-			}
-		}
-		
-		for(Hop hopIn: hop.getInput())
-			rGetUIPConsumerList(hopIn, uipCandHopHM);
-		
-		hop.setVisited();
-	}
-	
-
-	private Hop getRootHop(Hop hop)
-	{
-		return (!hop.getParent().isEmpty())?getRootHop(hop.getParent().get(0)):hop;
-	}
-	
-	
-	private void rResetVisitStatus(OptNode pn)
-		throws DMLRuntimeException
-	{
-		
-		if(!pn.isLeaf())
-		{
-			if(pn.getNodeType() == OptNode.NodeType.FUNCCALL)
-			{
-				Hop hopFunc = (Hop) OptTreeConverter.getAbstractPlanMapping().getMappedHop(pn.getID());
-				hopFunc.resetVisitStatus();
-				return;
-			}
-			ProgramBlock pb = (ProgramBlock) OptTreeConverter.getAbstractPlanMapping().getMappedProg(pn.getID())[1];
-			ArrayList<ProgramBlock> childBlocks = null;
-			ArrayList<ProgramBlock> elseBlocks = null;
-			if (pb instanceof WhileProgramBlock)
-				childBlocks = ((WhileProgramBlock)pb).getChildBlocks();
-			else if (pb instanceof ForProgramBlock)
-				childBlocks = ((ForProgramBlock)pb).getChildBlocks();
-			else if (pb instanceof IfProgramBlock) {
-				childBlocks = ((IfProgramBlock)pb).getChildBlocksIfBody();
-				elseBlocks = ((IfProgramBlock)pb).getChildBlocksElseBody();
-			}
-				
-			if(childBlocks != null)
-			{
-				for (ProgramBlock childBlock: childBlocks)
-				{
-					try 
-					{
-						Hop.resetVisitStatus(childBlock.getStatementBlock().get_hops());
-					}
-					catch (Exception e)
-					{
-						throw new DMLRuntimeException(e);
-					}
-				}
-			}
-			if(elseBlocks != null)
-			{
-				for (ProgramBlock childBlock: elseBlocks)
-				{
-					try 
-					{
-						Hop.resetVisitStatus(childBlock.getStatementBlock().get_hops());
-					}
-					catch (Exception e)
-					{
-						throw new DMLRuntimeException(e);
-					}
-				}
-			}
-			
-			for (OptNode optNode: pn.getChilds())
-			{
-				rResetVisitStatus(optNode);
-			}
-		}
-		else
-		{
-			Hop hop = (Hop) OptTreeConverter.getAbstractPlanMapping().getMappedHop(pn.getID());
-			if(hop != null)
-			{
-				hop.resetVisitStatus();
-			}
-		}
-	}
-	
-
-	
-	/* 	
-	 * This will validate candidate's consumer list.
-	 * 
-	 * @param pn:				OpNode of parfor loop
-	 * @param uipCandHopHM:		Hashmap of UIPCandidateHop with name as a key.		
-	 * @throws DMLRuntimeException
-	 */
-	
-	private void rValidateUIPConsumerList(OptNode pn, HashMap <String, ArrayList<UIPCandidateHop>> uipCandHopHM)
-			throws DMLRuntimeException 
-	{
-		if(!pn.isLeaf())
-		{
-			if(pn.getNodeType() == OptNode.NodeType.FUNCCALL)
-			{
-				Hop hop = (Hop) OptTreeConverter.getAbstractPlanMapping().getMappedHop(pn.getID());
-				rValidateUIPConsumerList(hop.getInput(), uipCandHopHM);
-				return;
-			}
-
-			ProgramBlock pb = (ProgramBlock) OptTreeConverter.getAbstractPlanMapping().getMappedProg(pn.getID())[1];
-			
-			VariableSet varRead = pb.getStatementBlock().variablesRead();
-			boolean bUIPCandHopRead = false;
-			for(Entry<String, ArrayList <UIPCandidateHop>> entry: uipCandHopHM.entrySet())
-			{
-				ArrayList <UIPCandidateHop> uipCandHopList = entry.getValue();
-				if (uipCandHopList != null) 
-				{
-					for (UIPCandidateHop uipCandHop: uipCandHopList)
-					{
-						ArrayList<Hop> consumerHops = uipCandHop.getConsumerHops();
-						if(consumerHops != null)
-						{
-							// If any of consumer's input (or any parent in hierachy of input) matches candiate's name, then 
-							// remove candidate from the list.
-							for (Hop consumerHop: consumerHops)
-							{
-								if (varRead.containsVariable(consumerHop.getName()))
-								{	
-									bUIPCandHopRead = true;
-									break;
-								}
-							}
-						}
-					}
-				}
-			}
-			// As none of the UIP candidates updated in this DAG, no need for further processing within this DAG
-			if(!bUIPCandHopRead)
-				return;
-
-			for (OptNode optNode: pn.getChilds())
-					rValidateUIPConsumerList(optNode, uipCandHopHM);
-		}
-		else
-		{
-			OptTreePlanMappingAbstract map = OptTreeConverter.getAbstractPlanMapping();
-			long ppid = map.getMappedParentID(map.getMappedParentID(pn.getID()));
-			Object[] o = map.getMappedProg(ppid);
-			ProgramBlock pb = (ProgramBlock) o[1];
-
-			if(pb instanceof IfProgramBlock || pb instanceof WhileProgramBlock || 
-				(pb instanceof ForProgramBlock && !(pb instanceof ParForProgramBlock)))	//TODO
-				rValidateUIPConsumerList(pb, uipCandHopHM);
-
-			long pid = map.getMappedParentID(pn.getID());
-			o = map.getMappedProg(pid);
-			pb = (ProgramBlock) o[1];
-			Hop hop = map.getMappedHop(pn.getID());
-			rValidateUIPConsumerList(hop, uipCandHopHM, pb.getStatementBlock().variablesRead());
-		}
-	}
-	
-	private void rValidateUIPConsumerList(ProgramBlock pb, HashMap <String, ArrayList<UIPCandidateHop>> uipCandHopHM)
-			throws DMLRuntimeException
-	{
-		ArrayList<ProgramBlock> childBlocks = null;
-		if (pb instanceof WhileProgramBlock)
-			childBlocks = ((WhileProgramBlock)pb).getChildBlocks();
-		else if (pb instanceof ForProgramBlock)
-			childBlocks = ((ForProgramBlock)pb).getChildBlocks();
-		else if (pb instanceof IfProgramBlock) 
-		{
-			childBlocks = ((IfProgramBlock)pb).getChildBlocksIfBody();
-			ArrayList<ProgramBlock> elseBlocks = ((IfProgramBlock)pb).getChildBlocksElseBody();
-			if(childBlocks != null && elseBlocks != null)
-				childBlocks.addAll(elseBlocks);
-			else if (childBlocks == null)
-				childBlocks = elseBlocks;
-		}
-
-		if(childBlocks != null)
-			for (ProgramBlock childBlock: childBlocks)
-			{
-				rValidateUIPConsumerList(childBlock, uipCandHopHM);
-				try 
-				{
-					rValidateUIPConsumerList(childBlock.getStatementBlock(), uipCandHopHM);
-				}
-				catch (Exception e) {
-					throw new DMLRuntimeException(e);
-				}
-			}
-	}	
-		
-
-	private void rValidateUIPConsumerList(StatementBlock sb, HashMap <String, ArrayList<UIPCandidateHop>> uipCandHopHM)
-			throws DMLRuntimeException 
-	{
-		VariableSet readVariables = sb.variablesRead();
-		
-		for(Entry<String, ArrayList <UIPCandidateHop>> entry: uipCandHopHM.entrySet())
-		{
-			ArrayList <UIPCandidateHop> uipCandHopList = entry.getValue();
-			if (uipCandHopList != null) 
-			{
-				for (UIPCandidateHop uipCandHop: uipCandHopList)
-				{
-					ArrayList<Hop> consumerHops = uipCandHop.getConsumerHops();
-					if(consumerHops != null)
-					{
-						// If consumer has read then remove candidate from the list (set flag to false).
-						for (Hop consumerHop: consumerHops)
-							if(readVariables.containsVariable(consumerHop.getName()))
-							{
-								uipCandHop.setUpdateInPlace(false);
-								break;
-							}
-					}
-				}
-			}
-		}
-	}
-	
-
-	private void rValidateUIPConsumerList(ArrayList<Hop> hops, HashMap <String, ArrayList<UIPCandidateHop>> uipCandHopHM)
-			throws DMLRuntimeException
-	{
-		if(hops != null)
-			for (Hop hop: hops)
-				rValidateUIPConsumerList(hop, uipCandHopHM);
-	}
-
-
-	private void rValidateUIPConsumerList(Hop hop, HashMap <String, ArrayList<UIPCandidateHop>> uipCandHopHM)
-			throws DMLRuntimeException 
-	{
-		if(hop.isVisited())
-			return;
-		
-		for(Entry<String, ArrayList <UIPCandidateHop>> entry: uipCandHopHM.entrySet())
-		{
-			if (entry.getValue() == null)
-				continue;
-			for (UIPCandidateHop uipCandHop: entry.getValue()) {
-				ArrayList<Hop> consumerHops = uipCandHop.getConsumerHops();
-				if(consumerHops != null) {
-					// If consumer has read then remove candidate from the list (set flag to false).
-					for (Hop consumerHop: consumerHops)
-						if(hop.getName().equals(consumerHop.getName())) {
-							uipCandHop.setUpdateInPlace(false);
-							break;
-						}
-				}
-			}
-		}
-		hop.setVisited();
-	}
-
-	private void rValidateUIPConsumerList(Hop hop, HashMap <String, ArrayList<UIPCandidateHop>> uipCandHopHM, VariableSet readVariables)
-			throws DMLRuntimeException 
-	{
-		if(hop.isVisited())
-			return;
-		
-		for(Entry<String, ArrayList <UIPCandidateHop>> entry: uipCandHopHM.entrySet())
-		{
-			if (entry.getValue() == null)
-				continue;
-			for (UIPCandidateHop uipCandHop: entry.getValue()) {
-				ArrayList<Hop> consumerHops = uipCandHop.getConsumerHops();
-				if(consumerHops != null) {
-					// If consumer has read then remove candidate from the list (set flag to false).
-					for (Hop consumerHop: consumerHops)
-						if(readVariables.containsVariable(consumerHop.getName())) {
-							uipCandHop.setUpdateInPlace(false);
-							break;
-						}
-				}
-			}
-		}
-		hop.setVisited();
-	}
-	
-	
-	public static List<String> getUIPList()
-	{
-		return listUIPRes.get();
-	}
-
 	protected boolean rHasOnlyInPlaceSafeLeftIndexing( OptNode n, ArrayList<String> retVars ) 
 		throws DMLRuntimeException
 	{
 		boolean ret = true;
-		
-		if( !n.isLeaf() )
-		{
+		if( !n.isLeaf() ) {
 			for( OptNode cn : n.getChilds() )
 				ret &= rHasOnlyInPlaceSafeLeftIndexing( cn, retVars );
 		}
-		else if(    n.getNodeType()== NodeType.HOP
-			     && n.getParam(ParamType.OPSTRING).equals(LeftIndexingOp.OPSTRING) )
-		{
+		else if( n.getNodeType()== NodeType.HOP) {
 			Hop h = OptTreeConverter.getAbstractPlanMapping().getMappedHop(n.getID());
-			if( retVars.contains( h.getInput().get(0).getName() ) )
-			{
+			if( h instanceof LeftIndexingOp && retVars.contains( h.getInput().get(0).getName() ) )
 				ret &= (h.getParent().size()==1 
-						&& h.getParent().get(0).getName().equals(h.getInput().get(0).getName()));
-			}
+					&& h.getParent().get(0).getName().equals(h.getInput().get(0).getName()));
 		}
-		
 		return ret;
 	}
 
-	private double computeTotalSizeResultVariables(ArrayList<String> retVars, LocalVariableMap vars, int k)
-	{
+	private double computeTotalSizeResultVariables(ArrayList<String> retVars, LocalVariableMap vars, int k) {
 		double sum = 1;
-		for( String var : retVars ){
+		for( String var : retVars ) {
 			Data dat = vars.get(var);
-			if( dat instanceof MatrixObject )
-			{
-				MatrixObject mo = (MatrixObject)dat;
-				double nnz = mo.getNnz();
-
-				if(nnz == 0.0) 
-					sum += OptimizerUtils.estimateSizeExactSparsity(mo.getNumRows(), mo.getNumColumns(), 1.0);
-				else {
-					double sp = mo.getSparsity();
-					sum += (k+1) * (OptimizerUtils.estimateSizeExactSparsity(mo.getNumRows(), mo.getNumColumns(),
-							Math.min((1.0/k)+sp, 1.0)));	// Every worker will consume memory for (MatrixSize/k + nnz) data.
-														// This is applicable only when there is non-zero nnz. 
-				}
+			if( !(dat instanceof MatrixObject) )
+				continue;
+			MatrixObject mo = (MatrixObject)dat;
+			if( mo.getNnz() == 0 ) 
+				sum += OptimizerUtils.estimateSizeExactSparsity(mo.getNumRows(), mo.getNumColumns(), 1.0);
+			else {
+				// Every worker will consume memory for (MatrixSize/k + nnz) data.
+				// This is applicable only when there is non-zero nnz. 
+				sum += (k+1) * (OptimizerUtils.estimateSizeExactSparsity(mo.getNumRows(), 
+					mo.getNumColumns(), Math.min((1.0/k)+mo.getSparsity(), 1.0)));
 			} 
 		}
-		
 		return sum;
 	}
 	
@@ -2477,86 +1767,48 @@ public class OptimizerRuleBased extends Optimizer
 		if( pn.getNodeType() != NodeType.PARFOR )
 			LOG.warn(getOptMode()+" OPT: Disable caching is only applicable for a ParFor node.");
 		
-		
 		ParForProgramBlock pfpb = (ParForProgramBlock) OptTreeConverter
-              .getAbstractPlanMapping().getMappedProg(pn.getID())[1];
-		
-		double M_sumInterm = rComputeSumMemoryIntermediates(pn, inplaceResultVars, null);
+			.getAbstractPlanMapping().getMappedProg(pn.getID())[1];
+		double M_sumInterm = rComputeSumMemoryIntermediates(pn, inplaceResultVars);
 		boolean apply = false;
 		
 		if( (pfpb.getExecMode() == PExecMode.REMOTE_MR_DP || pfpb.getExecMode() == PExecMode.REMOTE_MR)
 			&& M_sumInterm < _rm ) //all intermediates and operations fit into memory budget
 		{
-			pfpb.setCPCaching(false); //default is true			
+			pfpb.setCPCaching(false); //default is true
 			apply = true;
 		}
 		
-		LOG.debug(getOptMode()+" OPT: rewrite 'disable CP caching' - result="+apply+" (M="+toMB(M_sumInterm)+")" );			
+		LOG.debug(getOptMode()+" OPT: rewrite 'disable CP caching' - result="+apply+" (M="+toMB(M_sumInterm)+")" );
 	}
 
-	protected double rComputeSumMemoryIntermediates( OptNode n, HashSet<String> inplaceResultVars, 
-													HashMap <String, ArrayList <UIPCandidateHop>> uipCands )	
+	protected double rComputeSumMemoryIntermediates( OptNode n, HashSet<String> inplaceResultVars )
 		throws DMLRuntimeException
 	{
 		double sum = 0;
 		
-		if( !n.isLeaf() )
-		{
+		if( !n.isLeaf() ) {
 			for( OptNode cn : n.getChilds() )
-				sum += rComputeSumMemoryIntermediates( cn, inplaceResultVars, uipCands );
+				sum += rComputeSumMemoryIntermediates( cn, inplaceResultVars );
 		}
 		else if( n.getNodeType()== NodeType.HOP )
 		{
 			Hop h = OptTreeConverter.getAbstractPlanMapping().getMappedHop(n.getID());
-			if( uipCands != null && APPLY_REWRITE_UPDATE_INPLACE_INTERMEDIATE
-				&& h.getDataType() == Expression.DataType.MATRIX && h instanceof LeftIndexingOp 
-				&& h.getInput().get(0).getParent().size() == 1)
-			{ 
-				//get associated program block of parent node
-				long pid = OptTreeConverter.getAbstractPlanMapping().getMappedParentID(n.getID());
-				ProgramBlock pb = (ProgramBlock) OptTreeConverter.getAbstractPlanMapping().getMappedProg(pid)[1];
-				while(!(pb instanceof WhileProgramBlock || pb instanceof ForProgramBlock)) {
-					pid = OptTreeConverter.getAbstractPlanMapping().getMappedParentID(pid);
-					OptNode parent2 = OptTreeConverter.getAbstractPlanMapping().getOptNode(pid);
-					if( parent2.getNodeType() != NodeType.FUNCCALL )
-						pb = (ProgramBlock) OptTreeConverter.getAbstractPlanMapping().getMappedProg(pid)[1];
-				}
-
-				//add candidate to update-in-place candidate list
-				if( !uipCands.containsKey(h.getName()) )
-					uipCands.put(h.getName(), new ArrayList<UIPCandidateHop>());
-				uipCands.get(h.getName()).add(new UIPCandidateHop(h, pb));
-
-				//debug info update-in-place candidates
-				if(LOG.isTraceEnabled()) {
-					StatementBlock sb = (StatementBlock) OptTreeConverter.getAbstractPlanMapping().getMappedProg(OptTreeConverter.getAbstractPlanMapping().getMappedParentID(n.getID()))[0];
-					LOG.trace("Candidate Hop:" + h.getName() + "<" + h.getBeginLine() + "," + h.getEndLine() + ">,<" + 
-						h.getBeginColumn() + "," + h.getEndColumn() + "> PB:" + "<" + pb.getBeginLine() + "," + pb.getEndLine() + ">,<" + 
-						pb.getBeginColumn() + "," + pb.getEndColumn() + "> SB:" + "<" + sb.getBeginLine() + "," + sb.getEndLine() + ">,<" + 
-						sb.getBeginColumn() + "," + sb.getEndColumn() + ">");
-				}
-			}
-			
-			if(    n.getParam(ParamType.OPSTRING).equals(IndexingOp.OPSTRING)
-				&& n.getParam(ParamType.DATA_PARTITION_FORMAT) != null )
-			{
+			if( n.getParam(ParamType.OPSTRING).equals(IndexingOp.OPSTRING)
+				&& n.getParam(ParamType.DATA_PARTITION_FORMAT) != null ) {
 				//set during partitioning rewrite
 				sum += h.getMemEstimate();
 			}
-			else
-			{
+			else {
 				//base intermediate (worst-case w/ materialized intermediates)
 				sum +=   h.getOutputMemEstimate()
 					   + h.getIntermediateMemEstimate(); 
-
 				//inputs not represented in the planopttree (worst-case no CSE)
 				if( h.getInput() != null )
 					for( Hop cn : h.getInput() )
 						if( cn instanceof DataOp && ((DataOp)cn).isRead()  //read data
 							&& !inplaceResultVars.contains(cn.getName())) //except in-place result vars
-						{
-							sum += cn.getMemEstimate();	
-						}
+							sum += cn.getMemEstimate();
 			}
 		}
 		
@@ -3501,94 +2753,7 @@ public class OptimizerRuleBased extends Optimizer
 	//   Helper methods   //
 	////////////////////////
 	
-	public static String toMB( double inB )
-	{
+	public static String toMB( double inB ) {
 		return OptimizerUtils.toMB(inB) + "MB";
 	}
-
-	/*
-	 * This class stores information for the candidate hop, such as hop itself, program block.
-	 * When it gets evaluated if Matrix can be marked for "UpdateInPlace", additional properties such
-	 * as location, flag to indicate if its in loop (for, parfor, while), flag to indicate if hop can be marked as "UpdateInPlace".
-	 */
-	class UIPCandidateHop {
-		Hop hopCandidate, hopLix;
-		int iLocation = -1;
-		ProgramBlock pb;
-		Boolean bIntermediate = false, bIsLoopApplicable = false, bUpdateInPlace = true;
-		ArrayList<Hop> consumerHops = null;
-		
-		
-		UIPCandidateHop(Hop hopLix, ProgramBlock pb)
-		{
-			this.hopLix = hopLix;
-			this.pb = pb;
-		}
-		
-		Hop getLixHop()
-		{
-			return hopLix;
-		}
-		
-		Hop getHop()
-		{
-			return hopCandidate;
-		}
-		
-		void setHop(Hop hopCandidate)
-		{
-			this.hopCandidate = hopCandidate;
-		}
-
-		int getLocation()
-		{
-			return this.iLocation;
-		}
-		
-		void setLocation(int iLocation)
-		{
-			this.iLocation = iLocation;
-		}
-		
-		boolean isIntermediate()
-		{
-			return(bIntermediate);
-		}
-		
-		void setIntermediate(boolean bIntermediate)
-		{
-			this.bIntermediate = bIntermediate;
-		}
-		
-		boolean isLoopApplicable()
-		{
-			return(bIsLoopApplicable);
-		}
-
-		void setIsLoopApplicable(boolean bInWhileLoop)
-		{
-			this.bIsLoopApplicable = bInWhileLoop;
-		}
-
-		boolean isUpdateInPlace()
-		{
-			return(bUpdateInPlace);
-		}
-
-		void setUpdateInPlace(boolean bUpdateInPlace)
-		{
-			this.bUpdateInPlace = bUpdateInPlace;
-		}
-		
-		ArrayList<Hop> getConsumerHops()
-		{
-			return this.consumerHops;
-		}
-		
-		void setConsumerHops(ArrayList<Hop> consumerHops)
-		{
-			this.consumerHops = consumerHops;
-		}
-		
-	}
 }

http://git-wip-us.apache.org/repos/asf/systemml/blob/1cbfdef3/src/test/java/org/apache/sysml/test/integration/functions/updateinplace/UpdateInPlaceTest.java
----------------------------------------------------------------------
diff --git a/src/test/java/org/apache/sysml/test/integration/functions/updateinplace/UpdateInPlaceTest.java b/src/test/java/org/apache/sysml/test/integration/functions/updateinplace/UpdateInPlaceTest.java
index cc0a294..408ef46 100644
--- a/src/test/java/org/apache/sysml/test/integration/functions/updateinplace/UpdateInPlaceTest.java
+++ b/src/test/java/org/apache/sysml/test/integration/functions/updateinplace/UpdateInPlaceTest.java
@@ -22,14 +22,11 @@ package org.apache.sysml.test.integration.functions.updateinplace;
 import java.util.Arrays;
 import java.util.List;
 
-import org.junit.Assert;
 import org.junit.Test;
 import org.apache.sysml.hops.OptimizerUtils;
-import org.apache.sysml.runtime.controlprogram.parfor.opt.OptimizerRuleBased;
 import org.apache.sysml.test.integration.AutomatedTestBase;
 import org.apache.sysml.test.integration.TestConfiguration;
 import org.apache.sysml.test.utils.TestUtils;
-import org.apache.sysml.utils.Statistics;
 
 public class UpdateInPlaceTest extends AutomatedTestBase 
 {
@@ -243,49 +240,6 @@ public class UpdateInPlaceTest extends AutomatedTestBase
 			programArgs = new String[]{"-stats"}; //new String[]{"-args", input("A"), output("B") };
 			
 			runTest(true, false, null, -1);
-
-			if(OptimizerRuleBased.APPLY_REWRITE_UPDATE_INPLACE_INTERMEDIATE)
-			{
-				List<String> listUIPRes = OptimizerRuleBased.getUIPList();
-				int iUIPResCount = 0;
-				
-				// If UpdateInPlace list specified in the argument, verify the list.
-				if (listUIPExp != null)
-				{
-					if(listUIPRes != null)
-					{
-						for (String strUIPMatName: listUIPExp)
-							Assert.assertTrue("Expected UpdateInPlace matrix " + strUIPMatName 
-									+ " does not exist in the result UpdateInPlace matrix list.", 
-									listUIPRes.contains(strUIPMatName));
-						
-						iUIPResCount = listUIPRes.size();
-					}
-	
-					Assert.assertTrue("Expected # of UpdateInPlace matrix object/s " + listUIPExp.size() + 
-						" does not match with the # of matrix objects " + iUIPResCount + " from optimization result.", 
-						(iUIPResCount == listUIPExp.size()));
-				}
-				else
-				{
-					Assert.assertTrue("Expected # of UpdateInPlace matrix object/s " + "0" + 
-							" does not match with the # of matrix objects " + "0" + " from optimization result.", 
-							(listUIPRes == null || listUIPRes.size() == 0));
-				}
-				
-				Assert.assertTrue("Expected # of UpdateInPlace create variables of type matrix " + lTotalUIPVar + 
-						" does not match with the # of UpdateInPlace create variables of type matrix objects " + Statistics.getTotalUIPVar() + " from optimization result.", 
-						(Statistics.getTotalUIPVar() == lTotalUIPVar));
-				
-				Assert.assertTrue("Expected # of UpdateInPlace LeftIndexing " + lTotalLixUIP + 
-						" does not match with the # of UpdateInPlace LeftIndexing " + Statistics.getTotalLixUIP() + " from optimization result.", 
-						(Statistics.getTotalLixUIP() == lTotalLixUIP));
-				
-				Assert.assertTrue("Expected # of total LeftIndexing " + lTotalLix + 
-						" does not match with the # of total LeftIndexing " + Statistics.getTotalLix() + " from optimization result.", 
-						(Statistics.getTotalLix() == lTotalLix));
-				
-			}
 		}
 		finally {
 			OptimizerUtils.ALLOW_LOOP_UPDATE_IN_PLACE = oldinplace;