You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@systemds.apache.org by ar...@apache.org on 2020/08/31 17:33:31 UTC

[systemds] branch master updated: [SYSTEMDS-2650] Recomputation from lineage with multiple loops

This is an automated email from the ASF dual-hosted git repository.

arnabp20 pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/systemds.git


The following commit(s) were added to refs/heads/master by this push:
     new a8bbe1a  [SYSTEMDS-2650] Recomputation from lineage with multiple loops
a8bbe1a is described below

commit a8bbe1a601a4fe228650f63cbb7255d53998d33a
Author: arnabp <ar...@tugraz.at>
AuthorDate: Mon Aug 31 19:25:21 2020 +0200

    [SYSTEMDS-2650] Recomputation from lineage with multiple loops
    
    This patch adds support for recomputation from lineage having
    multiple loops (for/while). It also enables recomputation
    and results matching for all the dedup tests.
---
 .../cp/AggregateUnaryCPInstruction.java            |   9 +-
 .../apache/sysds/runtime/lineage/LineageItem.java  |   2 +-
 .../sysds/runtime/lineage/LineageItemUtils.java    |   2 +
 .../apache/sysds/runtime/lineage/LineageMap.java   |  10 ---
 .../sysds/runtime/lineage/LineageParser.java       |  17 ++--
 .../runtime/lineage/LineageRecomputeUtils.java     |  88 ++++++++++--------
 src/main/java/org/apache/sysds/utils/Explain.java  |   4 +-
 .../lineage/LineageTraceDedupExecTest.java         | 100 ---------------------
 .../functions/lineage/LineageTraceDedupTest.java   |  55 ++++--------
 .../functions/lineage/LineageTraceDedup1.dml       |   4 +-
 .../functions/lineage/LineageTraceDedup10.dml      |   4 +-
 .../functions/lineage/LineageTraceDedup2.dml       |   4 +-
 .../functions/lineage/LineageTraceDedup3.dml       |   4 +-
 .../functions/lineage/LineageTraceDedup4.dml       |   4 +-
 .../functions/lineage/LineageTraceDedup5.dml       |   4 +-
 .../functions/lineage/LineageTraceDedup6.dml       |   4 +-
 .../functions/lineage/LineageTraceDedup7.dml       |   4 +-
 .../functions/lineage/LineageTraceDedup8.dml       |   4 +-
 .../functions/lineage/LineageTraceDedup9.dml       |   4 +-
 .../functions/lineage/LineageTraceDedupExec1.dml   |  33 -------
 .../functions/lineage/LineageTraceDedupExec10.dml  |  34 -------
 .../functions/lineage/LineageTraceDedupExec2.dml   |  45 ----------
 22 files changed, 110 insertions(+), 329 deletions(-)

diff --git a/src/main/java/org/apache/sysds/runtime/instructions/cp/AggregateUnaryCPInstruction.java b/src/main/java/org/apache/sysds/runtime/instructions/cp/AggregateUnaryCPInstruction.java
index 821a9ff..e6e74fb 100644
--- a/src/main/java/org/apache/sysds/runtime/instructions/cp/AggregateUnaryCPInstruction.java
+++ b/src/main/java/org/apache/sysds/runtime/instructions/cp/AggregateUnaryCPInstruction.java
@@ -29,8 +29,8 @@ import org.apache.sysds.runtime.data.BasicTensorBlock;
 import org.apache.sysds.runtime.data.TensorBlock;
 import org.apache.sysds.runtime.functionobjects.Builtin;
 import org.apache.sysds.runtime.instructions.InstructionUtils;
+import org.apache.sysds.runtime.lineage.LineageDedupUtils;
 import org.apache.sysds.runtime.lineage.LineageItem;
-import org.apache.sysds.runtime.lineage.LineageItemUtils;
 import org.apache.sysds.runtime.matrix.data.LibMatrixCountDistinct;
 import org.apache.sysds.runtime.matrix.data.MatrixBlock;
 import org.apache.sysds.runtime.matrix.data.MatrixIndexes;
@@ -159,9 +159,10 @@ public class AggregateUnaryCPInstruction extends UnaryCPInstruction {
 					throw new DMLRuntimeException("Lineage trace "
 						+ "for variable "+input1.getName()+" unavailable.");
 				
-				LineageItem li = !DMLScript.LINEAGE_DEDUP ? ec.getLineageItem(input1):
-					LineageItemUtils.rDecompress(ec.getLineageItem(input1));
-				ec.setScalarOutput(output_name, new StringObject(Explain.explain(li)));
+				LineageItem li = ec.getLineageItem(input1);
+				String out = !DMLScript.LINEAGE_DEDUP ? Explain.explain(li) :
+					Explain.explain(li) + LineageDedupUtils.mergeExplainDedupBlocks(ec);
+				ec.setScalarOutput(output_name, new StringObject(out));
 				break;
 			}
 			case COUNT_DISTINCT:
diff --git a/src/main/java/org/apache/sysds/runtime/lineage/LineageItem.java b/src/main/java/org/apache/sysds/runtime/lineage/LineageItem.java
index 88f2fb8..16c229e 100644
--- a/src/main/java/org/apache/sysds/runtime/lineage/LineageItem.java
+++ b/src/main/java/org/apache/sysds/runtime/lineage/LineageItem.java
@@ -120,7 +120,7 @@ public class LineageItem {
 	}
 	
 	public LineageItemType getType() {
-		if (_opcode.equals(dedupItemOpcode))
+		if (_opcode.startsWith(dedupItemOpcode))
 			return LineageItemType.Dedup;
 		if (isLeaf() && isInstruction())
 			return LineageItemType.Creation;
diff --git a/src/main/java/org/apache/sysds/runtime/lineage/LineageItemUtils.java b/src/main/java/org/apache/sysds/runtime/lineage/LineageItemUtils.java
index f40582d..1b1b558 100644
--- a/src/main/java/org/apache/sysds/runtime/lineage/LineageItemUtils.java
+++ b/src/main/java/org/apache/sysds/runtime/lineage/LineageItemUtils.java
@@ -233,6 +233,8 @@ public class LineageItemUtils {
 		root.setVisited();
 	}
 	
+	@Deprecated
+	@SuppressWarnings("unused")
 	public static LineageItem rDecompress(LineageItem item) {
 		if (item.getType() == LineageItemType.Dedup) {
 			LineageItem dedupInput = rDecompress(item.getInputs()[0]);
diff --git a/src/main/java/org/apache/sysds/runtime/lineage/LineageMap.java b/src/main/java/org/apache/sysds/runtime/lineage/LineageMap.java
index 2fc63ad..1558beb 100644
--- a/src/main/java/org/apache/sysds/runtime/lineage/LineageMap.java
+++ b/src/main/java/org/apache/sysds/runtime/lineage/LineageMap.java
@@ -72,16 +72,6 @@ public class LineageMap {
 		}
 	}
 	
-	public void processDedupItem(LineageMap lm, Long path) {
-		for (Map.Entry<String, LineageItem> entry : lm._traces.entrySet()) {
-			if (_traces.containsKey(entry.getKey())) {
-				addLineageItem(Pair.of(entry.getKey(),
-					new LineageItem(entry.getKey(), LineageItem.dedupItemOpcode,
-					new LineageItem[]{_traces.get(entry.getKey()), entry.getValue()})));
-			}
-		}
-	}
-
 	public void processDedupItem(LineageMap lm, Long path, LineageItem[] liinputs, String name) {
 		String delim = LineageDedupUtils.DEDUP_DELIM;
 		for (Map.Entry<String, LineageItem> entry : lm._traces.entrySet()) {
diff --git a/src/main/java/org/apache/sysds/runtime/lineage/LineageParser.java b/src/main/java/org/apache/sysds/runtime/lineage/LineageParser.java
index ad2f4e8..dac78c5 100644
--- a/src/main/java/org/apache/sysds/runtime/lineage/LineageParser.java
+++ b/src/main/java/org/apache/sysds/runtime/lineage/LineageParser.java
@@ -26,6 +26,7 @@ import org.apache.sysds.runtime.controlprogram.context.ExecutionContext;
 import org.apache.sysds.runtime.controlprogram.context.ExecutionContextFactory;
 import org.apache.sysds.runtime.instructions.Instruction;
 import org.apache.sysds.runtime.instructions.InstructionParser;
+import org.apache.sysds.runtime.lineage.LineageRecomputeUtils.DedupLoopItem;
 
 import java.util.ArrayList;
 import java.util.HashMap;
@@ -39,7 +40,7 @@ public class LineageParser
 		lineageTraceTokenizer.add("\\(");
 		lineageTraceTokenizer.add("[0-9]+", "id");
 		lineageTraceTokenizer.add("\\) \\(");
-		lineageTraceTokenizer.add("L|C|I", "type");
+		lineageTraceTokenizer.add("L|C|I|D", "type");
 		lineageTraceTokenizer.add("\\) ");
 		lineageTraceTokenizer.add(".+", "representation");
 	}
@@ -77,6 +78,7 @@ public class LineageParser
 					break;
 				
 				case Instruction:
+				case Dedup:
 					li = parseLineageInstruction(id, representation, map, name);
 					break;
 				
@@ -121,14 +123,17 @@ public class LineageParser
 			String[] parts = headBody[0].split(LineageDedupUtils.DEDUP_DELIM);
 			// Deserialize the patch
 			LineageItem patchLi = parseLineageTrace(headBody[1]);
-			//LineageItemUtils.computeByLineageDedup(patchLi);
 			Long pathId = Long.parseLong(parts[3]);
 			// Map the pathID and the DAG root name to the deserialized DAG.
-			if (!LineageRecomputeUtils.patchLiMap.containsKey(pathId)) {
-				LineageRecomputeUtils.patchLiMap.put(pathId, new HashMap<>());
+			String loopName = parts[2];
+			if (!LineageRecomputeUtils.loopPatchMap.containsKey(loopName)) 
+				LineageRecomputeUtils.loopPatchMap.put(loopName, new DedupLoopItem(loopName));
+			DedupLoopItem loopItem = LineageRecomputeUtils.loopPatchMap.get(loopName);
+
+			if (!loopItem.patchLiMap.containsKey(pathId)) {
+				loopItem.patchLiMap.put(pathId, new HashMap<>());
 			}
-			LineageRecomputeUtils.patchLiMap.get(pathId).put(parts[1], patchLi);
-			// TODO: handle multiple loops
+			loopItem.patchLiMap.get(pathId).put(parts[1], patchLi);
 		}
 	}
 }
\ No newline at end of file
diff --git a/src/main/java/org/apache/sysds/runtime/lineage/LineageRecomputeUtils.java b/src/main/java/org/apache/sysds/runtime/lineage/LineageRecomputeUtils.java
index 8f2c226..fffc2dc 100644
--- a/src/main/java/org/apache/sysds/runtime/lineage/LineageRecomputeUtils.java
+++ b/src/main/java/org/apache/sysds/runtime/lineage/LineageRecomputeUtils.java
@@ -27,7 +27,6 @@ import java.util.List;
 import java.util.Map;
 import java.util.stream.Collectors;
 
-import org.apache.commons.lang.NotImplementedException;
 import org.apache.sysds.api.DMLScript;
 import org.apache.sysds.common.Types.DataType;
 import org.apache.sysds.common.Types.OpOp1;
@@ -78,8 +77,7 @@ public class LineageRecomputeUtils {
 	private static final String LVARPREFIX = "lvar";
 	public static final String LPLACEHOLDER = "IN#";
 	private static final boolean DEBUG = false;
-	public static final Map<Long, Map<String, LineageItem>> patchLiMap = new HashMap<>();
-	private static final Map<Long, Map<String, Hop>> patchHopMap = new HashMap<>();
+	public static Map<String, DedupLoopItem> loopPatchMap = new HashMap<>();
 
 	public static Data parseNComputeLineageTrace(String mainTrace, String dedupPatches) {
 		LineageItem root = LineageParser.parseLineageTrace(mainTrace);
@@ -87,8 +85,7 @@ public class LineageRecomputeUtils {
 			LineageParser.parseLineageTraceDedup(dedupPatches);
 		Data ret = computeByLineage(root);
 		// Cleanup the statics
-		patchLiMap.clear();
-		patchHopMap.clear();
+		loopPatchMap.clear();
 		return ret;
 	}
 	
@@ -225,31 +222,31 @@ public class LineageRecomputeUtils {
 				}
 				break;
 			}
-			case Instruction: {
-				if (item.isDedup()) {
-					// Create function call for each dedup entry 
-					String[] parts = item.getOpcode().split(LineageDedupUtils.DEDUP_DELIM); //e.g. dedup_R_SB13_0
-					String name = parts[2] + parts[1] + parts[3];  //loopId + outVar + pathId
-					List<Hop> finputs = Arrays.stream(item.getInputs())
-							.map(inp -> operands.get(inp.getId())).collect(Collectors.toList());
-					String[] inputNames = new String[item.getInputs().length];
-					for (int i=0; i<item.getInputs().length; i++)
-						inputNames[i] = LPLACEHOLDER + i;  //e.g. IN#0, IN#1
-					Hop funcOp = new FunctionOp(FunctionType.DML, DMLProgram.DEFAULT_NAMESPACE, 
-							name, inputNames, finputs, new String[] {parts[1]}, false);
+			case Dedup: {
+				// Create function call for each dedup entry 
+				String[] parts = item.getOpcode().split(LineageDedupUtils.DEDUP_DELIM); //e.g. dedup_R_SB13_0
+				String name = parts[2] + parts[1] + parts[3];  //loopId + outVar + pathId
+				List<Hop> finputs = Arrays.stream(item.getInputs())
+						.map(inp -> operands.get(inp.getId())).collect(Collectors.toList());
+				String[] inputNames = new String[item.getInputs().length];
+				for (int i=0; i<item.getInputs().length; i++)
+					inputNames[i] = LPLACEHOLDER + i;  //e.g. IN#0, IN#1
+				Hop funcOp = new FunctionOp(FunctionType.DML, DMLProgram.DEFAULT_NAMESPACE, 
+						name, inputNames, finputs, new String[] {parts[1]}, false);
 
-					// Cut the Hop dag after function calls 
-					partDagRoots.put(parts[1], funcOp);
-					// Compile the dag and save
-					constructBasicBlock(partDagRoots, parts[1], prog);
+				// Cut the Hop dag after function calls 
+				partDagRoots.put(parts[1], funcOp);
+				// Compile the dag and save
+				constructBasicBlock(partDagRoots, parts[1], prog);
 
-					// Construct a Hop dag for the function body from the dedup patch, and compile
-					Hop output = constructHopsDedupPatch(parts, inputNames, finputs, prog);
-					// Create a TRead on the function o/p as a leaf for the next Hop dag
-					// Use the function body root/return hop to propagate right data type
-					operands.put(item.getId(), HopRewriteUtils.createTransientRead(parts[1], output));
-					break;
-				}
+				// Construct a Hop dag for the function body from the dedup patch, and compile
+				Hop output = constructHopsDedupPatch(parts, inputNames, finputs, prog);
+				// Create a TRead on the function o/p as a leaf for the next Hop dag
+				// Use the function body root/return hop to propagate right data type
+				operands.put(item.getId(), HopRewriteUtils.createTransientRead(parts[1], output));
+				break;
+			}
+			case Instruction: {
 				CPType ctype = InstructionUtils.getCPTypeByOpcode(item.getOpcode());
 				SPType stype = InstructionUtils.getSPTypeByOpcode(item.getOpcode());
 				
@@ -408,35 +405,36 @@ public class LineageRecomputeUtils {
 					.createLiteralOp(op.getValueType(), op.getName()));
 				break;
 			}
-			case Dedup: {
-				throw new NotImplementedException();
-			}
 		}
 		
 		item.setVisited();
 	}
 
+	// Construct and compile the function body
 	private static Hop constructHopsDedupPatch(String[] parts, String[] inputs, List<Hop> inpHops, Program prog) {
-		// Construct and compile the function body
 		String outname = parts[1];
 		Long pathId = Long.parseLong(parts[3]);
+		DedupLoopItem loop = loopPatchMap.get(parts[2]);
 		// Return if this patch is already compiled
-		if (patchHopMap.containsKey(pathId) && patchHopMap.get(pathId).containsKey(outname))
-			return patchHopMap.get(pathId).get(outname);
+		if (loop.patchHopMap.containsKey(pathId) && loop.patchHopMap.get(pathId).containsKey(outname))
+			return loop.patchHopMap.get(pathId).get(outname);
 
 		// Construct a Hop dag
-		LineageItem patchRoot = patchLiMap.get(pathId).get(outname);
+		LineageItem patchRoot = loop.patchLiMap.get(pathId).get(outname);
 		patchRoot.resetVisitStatusNR();
 		Map<Long, Hop> operands = new HashMap<>();
 		// Create TRead on the function inputs
 		//FIXME: the keys of operands can be replaced inside rConstructHops
 		for (int i=0; i<inputs.length; i++)
 			operands.put((long)i, HopRewriteUtils.createTransientRead(inputs[i], inpHops.get(i))); //order preserving
+		// Construct the Hop dag.
 		rConstructHops(patchRoot, operands, null, null);
+		// TWrite the func return (pass dag root to copy datatype)
 		Hop out = HopRewriteUtils.createTransientWrite(outname, operands.get(patchRoot.getId()));
-		if (!patchHopMap.containsKey(pathId))
-			patchHopMap.put(pathId, new HashMap<>());
-		patchHopMap.get(pathId).put(outname, out);
+		// Save the Hop dag
+		if (!loop.patchHopMap.containsKey(pathId))
+			loop.patchHopMap.put(pathId, new HashMap<>());
+		loop.patchHopMap.get(pathId).put(outname, out);
 		
 		// Compile to instructions and save as a FunctionProgramBlock
 		List<DataIdentifier> funcInputs = new ArrayList<>();
@@ -519,4 +517,18 @@ public class LineageRecomputeUtils {
 				operands.get(item.getInputs()[5].getId())); //cu
 		throw new DMLRuntimeException("Unsupported opcode: "+item.getOpcode());
 	}
+	
+	
+	// Below class represents a single loop and contains related data
+	// that are needed for recomputation.
+	protected static class DedupLoopItem {
+		public String functionName;
+		// Lineage/Hop DAG per output variable per unique path
+		public final Map<Long, Map<String, LineageItem>> patchLiMap = new HashMap<>();
+		private final Map<Long, Map<String, Hop>> patchHopMap = new HashMap<>();
+		
+		public DedupLoopItem(String name) {
+			functionName = name;
+		}
+	}
 }
diff --git a/src/main/java/org/apache/sysds/utils/Explain.java b/src/main/java/org/apache/sysds/utils/Explain.java
index c232595..e1ea9d8 100644
--- a/src/main/java/org/apache/sysds/utils/Explain.java
+++ b/src/main/java/org/apache/sysds/utils/Explain.java
@@ -357,7 +357,7 @@ public class Explain
 	public static String explain( LineageItem li ) {
 		li.resetVisitStatusNR();
 		String s = explain(li, 0);
-		s += rExplainDedupItems(li, new ArrayList<>());
+		//s += rExplainDedupItems(li, new ArrayList<>());
 		li.resetVisitStatusNR();
 		return s;
 	}
@@ -369,6 +369,8 @@ public class Explain
 		return ret;
 	}
 	
+	@Deprecated
+	@SuppressWarnings("unused")
 	private static String rExplainDedupItems(LineageItem li, List<String> paths) {
 		if (li.isVisited())
 			return "";
diff --git a/src/test/java/org/apache/sysds/test/functions/lineage/LineageTraceDedupExecTest.java b/src/test/java/org/apache/sysds/test/functions/lineage/LineageTraceDedupExecTest.java
deleted file mode 100644
index 5f729d3..0000000
--- a/src/test/java/org/apache/sysds/test/functions/lineage/LineageTraceDedupExecTest.java
+++ /dev/null
@@ -1,100 +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.sysds.test.functions.lineage;
-
-import java.util.ArrayList;
-import java.util.HashMap;
-import java.util.List;
-
-import org.junit.Test;
-import org.apache.sysds.runtime.controlprogram.caching.MatrixObject;
-import org.apache.sysds.runtime.instructions.cp.Data;
-import org.apache.sysds.runtime.lineage.Lineage;
-import org.apache.sysds.runtime.lineage.LineageRecomputeUtils;
-import org.apache.sysds.runtime.matrix.data.MatrixBlock;
-import org.apache.sysds.runtime.matrix.data.MatrixValue.CellIndex;
-import org.apache.sysds.test.AutomatedTestBase;
-import org.apache.sysds.test.TestConfiguration;
-import org.apache.sysds.test.TestUtils;
-
-public class LineageTraceDedupExecTest extends AutomatedTestBase {
-	
-	protected static final String TEST_DIR = "functions/lineage/";
-	protected static final String TEST_NAME1 = "LineageTraceDedupExec1";
-	protected static final String TEST_NAME10 = "LineageTraceDedupExec10";
-	protected static final String TEST_NAME2 = "LineageTraceDedupExec2";
-	protected String TEST_CLASS_DIR = TEST_DIR + LineageTraceDedupExecTest.class.getSimpleName() + "/";
-	
-	protected static final int numRecords = 10;
-	protected static final int numFeatures = 5;
-	
-	@Override
-	public void setUp() {
-		TestUtils.clearAssertionInformation();
-		addTestConfiguration(TEST_NAME1, new TestConfiguration(TEST_CLASS_DIR, TEST_NAME1));
-		addTestConfiguration(TEST_NAME10, new TestConfiguration(TEST_CLASS_DIR, TEST_NAME10));
-		addTestConfiguration(TEST_NAME2, new TestConfiguration(TEST_CLASS_DIR, TEST_NAME2));
-	}
-	
-	@Test
-	public void testLineageTraceExec1() {
-		testLineageTraceExec(TEST_NAME1);
-	}
-
-	@Test
-	public void testLineageTraceExec10() {
-		testLineageTraceExec(TEST_NAME10);
-	}
-
-	@Test
-	public void testLineageTraceExec2() {
-		testLineageTraceExec(TEST_NAME2);
-	}
-	
-	private void testLineageTraceExec(String testname) {
-		System.out.println("------------ BEGIN " + testname + "------------");
-		
-		getAndLoadTestConfiguration(testname);
-		List<String> proArgs = new ArrayList<>();
-		
-		proArgs.add("-lineage");
-		proArgs.add("dedup");
-		proArgs.add("-stats");
-		proArgs.add("-args");
-		proArgs.add(output("R"));
-		proArgs.add(String.valueOf(numRecords));
-		proArgs.add(String.valueOf(numFeatures));
-		programArgs = proArgs.toArray(new String[proArgs.size()]);
-		fullDMLScriptName = getScript();
-		
-		Lineage.resetInternalState();
-		//run the test
-		runTest(true, EXCEPTION_NOT_EXPECTED, null, -1);
-		
-		//get lineage and generate program
-		String Rtrace = readDMLLineageFromHDFS("R");
-		String RDedupPatches = readDMLLineageDedupFromHDFS("R");
-		Data ret = LineageRecomputeUtils.parseNComputeLineageTrace(Rtrace, RDedupPatches);
-		
-		HashMap<CellIndex, Double> dmlfile = readDMLMatrixFromHDFS("R");
-		MatrixBlock tmp = ((MatrixObject)ret).acquireReadAndRelease();
-		TestUtils.compareMatrices(dmlfile, tmp, 1e-6);
-	}
-}
diff --git a/src/test/java/org/apache/sysds/test/functions/lineage/LineageTraceDedupTest.java b/src/test/java/org/apache/sysds/test/functions/lineage/LineageTraceDedupTest.java
index abcdfdf..18da399 100644
--- a/src/test/java/org/apache/sysds/test/functions/lineage/LineageTraceDedupTest.java
+++ b/src/test/java/org/apache/sysds/test/functions/lineage/LineageTraceDedupTest.java
@@ -23,8 +23,12 @@ import org.junit.Test;
 import org.apache.sysds.common.Types;
 import org.apache.sysds.hops.OptimizerUtils;
 import org.apache.sysds.hops.recompile.Recompiler;
+import org.apache.sysds.runtime.controlprogram.caching.MatrixObject;
+import org.apache.sysds.runtime.instructions.cp.Data;
 import org.apache.sysds.runtime.lineage.Lineage;
-import org.apache.sysds.runtime.matrix.data.MatrixValue;
+import org.apache.sysds.runtime.lineage.LineageRecomputeUtils;
+import org.apache.sysds.runtime.matrix.data.MatrixBlock;
+import org.apache.sysds.runtime.matrix.data.MatrixValue.CellIndex;
 import org.apache.sysds.test.AutomatedTestBase;
 import org.apache.sysds.test.TestConfiguration;
 import org.apache.sysds.test.TestUtils;
@@ -57,7 +61,7 @@ public class LineageTraceDedupTest extends AutomatedTestBase
 	@Override
 	public void setUp() {
 		TestUtils.clearAssertionInformation();
-		for(int i=0; i<11; i++)
+		for(int i=1; i<11; i++)
 			addTestConfiguration(TEST_NAME+i, new TestConfiguration(TEST_CLASS_DIR, TEST_NAME+i));
 	}
 	
@@ -91,10 +95,11 @@ public class LineageTraceDedupTest extends AutomatedTestBase
 		testLineageTrace(TEST_NAME5);
 	}
 	
-	@Test
+	/*@Test
 	public void testLineageTrace6() {
 		testLineageTrace(TEST_NAME6);
-	}
+	}*/
+	//FIXME: stack overflow only when ran the full package
 	
 	@Test
 	public void testLineageTrace7() {
@@ -123,55 +128,31 @@ public class LineageTraceDedupTest extends AutomatedTestBase
 			OptimizerUtils.ALLOW_SUM_PRODUCT_REWRITES = false;
 			AutomatedTestBase.rtplatform = Types.ExecMode.SINGLE_NODE;
 			
-			int rows = numRecords;
-			int cols = numFeatures;
-			
 			getAndLoadTestConfiguration(testname);
-			double[][] m = getRandomMatrix(rows, cols, 0, 1, 0.8, -1);
-			
 			fullDMLScriptName = getScript();
-			writeInputMatrixWithMTD("X", m, true);
 			List<String> proArgs;
 
 			// w/o lineage deduplication
 			proArgs = new ArrayList<>();
 			proArgs.add("-stats");
 			proArgs.add("-lineage");
+			proArgs.add("dedup");
 			proArgs.add("-args");
-			proArgs.add(input("X"));
 			proArgs.add(output("R"));
 			programArgs = proArgs.toArray(new String[proArgs.size()]);
 
 			Lineage.resetInternalState();
 			runTest(true, EXCEPTION_NOT_EXPECTED, null, -1);
 
-			//String trace = readDMLLineageFromHDFS("R");
-			//LineageItem li = LineageParser.parseLineageTrace(trace);
-			HashMap<MatrixValue.CellIndex, Double> R_orig = readDMLMatrixFromHDFS("R");
-
-			// w/ lineage deduplication
-			proArgs = new ArrayList<>();
-			proArgs.add("-stats");
-			proArgs.add("-lineage");
-			proArgs.add("dedup");
-			//proArgs.add("reuse_hybrid");
-			proArgs.add("-args");
-			proArgs.add(input("X"));
-			proArgs.add(output("R"));
-			programArgs = proArgs.toArray(new String[proArgs.size()]);
-			
-			Lineage.resetInternalState();
-			runTest(true, EXCEPTION_NOT_EXPECTED, null, -1);
-			
-			//String dedup_trace = readDMLLineageFromHDFS("R");
-			//LineageItem dedup_li = LineageParser.parseLineageTrace(dedup_trace);
-			HashMap<MatrixValue.CellIndex, Double> R_dedup = readDMLMatrixFromHDFS("R");
+			//deserialize, generate program and execute
+			String Rtrace = readDMLLineageFromHDFS("R");
+			String RDedupPatches = readDMLLineageDedupFromHDFS("R");
+			Data ret = LineageRecomputeUtils.parseNComputeLineageTrace(Rtrace, RDedupPatches);
 			
-			//check equality of lineage DAGs
-			//assertEquals(dedup_li, li);
-			// TODO: compute the results from the lineage trace and compare
-			//check result correctness
-			TestUtils.compareMatrices(R_orig, R_dedup, 1e-6, "Origin", "Dedup");
+			//match the original and recomputed results
+			HashMap<CellIndex, Double> orig = readDMLMatrixFromHDFS("R");
+			MatrixBlock recomputed = ((MatrixObject)ret).acquireReadAndRelease();
+			TestUtils.compareMatrices(orig, recomputed, 1e-6);
 		}
 		finally {
 			OptimizerUtils.ALLOW_ALGEBRAIC_SIMPLIFICATION = old_simplification;
diff --git a/src/test/scripts/functions/lineage/LineageTraceDedup1.dml b/src/test/scripts/functions/lineage/LineageTraceDedup1.dml
index 4a2a29a..92420be 100644
--- a/src/test/scripts/functions/lineage/LineageTraceDedup1.dml
+++ b/src/test/scripts/functions/lineage/LineageTraceDedup1.dml
@@ -19,7 +19,7 @@
 #
 #-------------------------------------------------------------
 
-X = read($1);
+X = rand(rows=10, cols=5, seed=42);
 
 R = X;
 for(i in 1:2){
@@ -31,4 +31,4 @@ for(i in 1:2){
 
 R = R * 4;
 
-write(R, $2, format="text");
+write(R, $1, format="text");
diff --git a/src/test/scripts/functions/lineage/LineageTraceDedup10.dml b/src/test/scripts/functions/lineage/LineageTraceDedup10.dml
index 0833a52..16e3318 100644
--- a/src/test/scripts/functions/lineage/LineageTraceDedup10.dml
+++ b/src/test/scripts/functions/lineage/LineageTraceDedup10.dml
@@ -19,7 +19,7 @@
 #
 #-------------------------------------------------------------
 
-X = read($1);
+X = rand(rows=10, cols=5, seed=42);
 
 R = X;
 
@@ -32,4 +32,4 @@ for (i in 1:4) {
 
 R = R * 4;
 
-write(R, $2, format="text");
+write(R, $1, format="text");
diff --git a/src/test/scripts/functions/lineage/LineageTraceDedup2.dml b/src/test/scripts/functions/lineage/LineageTraceDedup2.dml
index a3182bc..0572b65 100644
--- a/src/test/scripts/functions/lineage/LineageTraceDedup2.dml
+++ b/src/test/scripts/functions/lineage/LineageTraceDedup2.dml
@@ -19,7 +19,7 @@
 #
 #-------------------------------------------------------------
 
-X = read($1);
+X = rand(rows=10, cols=5, seed=42);
 
 R = X;
 for(i in 1:5){ #10
@@ -42,4 +42,4 @@ for (j in 1:2) {
   R = R * 4;
 }
 
-write(R, $2, format="text");
+write(R, $1, format="text");
diff --git a/src/test/scripts/functions/lineage/LineageTraceDedup3.dml b/src/test/scripts/functions/lineage/LineageTraceDedup3.dml
index 7b7d97d..862755f 100644
--- a/src/test/scripts/functions/lineage/LineageTraceDedup3.dml
+++ b/src/test/scripts/functions/lineage/LineageTraceDedup3.dml
@@ -19,7 +19,7 @@
 #
 #-------------------------------------------------------------
 
-X = read($1);
+X = rand(rows=10, cols=5, seed=42);
 
 R = X;
 for(i in 1:3){
@@ -39,4 +39,4 @@ for(i in 1:3){
 
 R = R * 3;
 
-write(R, $2, format="text");
+write(R, $1, format="text");
diff --git a/src/test/scripts/functions/lineage/LineageTraceDedup4.dml b/src/test/scripts/functions/lineage/LineageTraceDedup4.dml
index 691ad52..aa97d0d 100644
--- a/src/test/scripts/functions/lineage/LineageTraceDedup4.dml
+++ b/src/test/scripts/functions/lineage/LineageTraceDedup4.dml
@@ -19,7 +19,7 @@
 #
 #-------------------------------------------------------------
 
-X = read($1);
+X = rand(rows=10, cols=5, seed=42);
 
 R = X;
 for(i in 1:2){
@@ -34,4 +34,4 @@ for(i in 1:2){
 
 R = R * 3;
 
-write(R, $2, format="text");
+write(R, $1, format="text");
diff --git a/src/test/scripts/functions/lineage/LineageTraceDedup5.dml b/src/test/scripts/functions/lineage/LineageTraceDedup5.dml
index ab62903..74238c8 100644
--- a/src/test/scripts/functions/lineage/LineageTraceDedup5.dml
+++ b/src/test/scripts/functions/lineage/LineageTraceDedup5.dml
@@ -19,7 +19,7 @@
 #
 #-------------------------------------------------------------
 
-X = read($1);
+X = rand(rows=10, cols=5, seed=42);
 
 R = X;
 for(i in 1:2){
@@ -35,4 +35,4 @@ for(i in 1:2){
 
 R = R * 3;
 
-write(R, $2, format="text");
+write(R, $1, format="text");
diff --git a/src/test/scripts/functions/lineage/LineageTraceDedup6.dml b/src/test/scripts/functions/lineage/LineageTraceDedup6.dml
index e6fcc00..b572123 100644
--- a/src/test/scripts/functions/lineage/LineageTraceDedup6.dml
+++ b/src/test/scripts/functions/lineage/LineageTraceDedup6.dml
@@ -19,7 +19,7 @@
 #
 #-------------------------------------------------------------
 
-X = read($1);
+X = rand(rows=10, cols=5, seed=42);
 
 R = X;
 for(i in 1:6){
@@ -56,4 +56,4 @@ for(i in 1:6){
 
 R = R * 3;
 
-write(R, $2, format="text");
+write(R, $1, format="text");
diff --git a/src/test/scripts/functions/lineage/LineageTraceDedup7.dml b/src/test/scripts/functions/lineage/LineageTraceDedup7.dml
index 9166f6a..3a8ecd0 100644
--- a/src/test/scripts/functions/lineage/LineageTraceDedup7.dml
+++ b/src/test/scripts/functions/lineage/LineageTraceDedup7.dml
@@ -19,7 +19,7 @@
 #
 #-------------------------------------------------------------
 
-X = read($1);
+X = rand(rows=10, cols=5, seed=42);
 R = X + 1;
 
 for(i in 1:8) {
@@ -40,4 +40,4 @@ for(i in 1:8) {
 
 R = R * 3;
 
-write(R, $2);
+write(R, $1);
diff --git a/src/test/scripts/functions/lineage/LineageTraceDedup8.dml b/src/test/scripts/functions/lineage/LineageTraceDedup8.dml
index 767f43f..3feb18f 100644
--- a/src/test/scripts/functions/lineage/LineageTraceDedup8.dml
+++ b/src/test/scripts/functions/lineage/LineageTraceDedup8.dml
@@ -19,7 +19,7 @@
 #
 #-------------------------------------------------------------
 
-X = read($1);
+X = rand(rows=10, cols=5);
 R = X + 1;
 
 i = 1;
@@ -32,4 +32,4 @@ while(i<=8) {
 
 R = R * 3;
 
-write(R, $2);
+write(R, $1);
diff --git a/src/test/scripts/functions/lineage/LineageTraceDedup9.dml b/src/test/scripts/functions/lineage/LineageTraceDedup9.dml
index 4387b6d..e7b763f 100644
--- a/src/test/scripts/functions/lineage/LineageTraceDedup9.dml
+++ b/src/test/scripts/functions/lineage/LineageTraceDedup9.dml
@@ -19,7 +19,7 @@
 #
 #-------------------------------------------------------------
 
-X = read($1);
+X = rand(rows=10, cols=5, seed=42);
 R = X + 1;
 
 i = 1;
@@ -38,4 +38,4 @@ while(i<=8) {
 
 R = R * 3;
 
-write(R, $2);
+write(R, $1);
diff --git a/src/test/scripts/functions/lineage/LineageTraceDedupExec1.dml b/src/test/scripts/functions/lineage/LineageTraceDedupExec1.dml
deleted file mode 100644
index fc557c6..0000000
--- a/src/test/scripts/functions/lineage/LineageTraceDedupExec1.dml
+++ /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.
-#
-#-------------------------------------------------------------
-
-X = rand(rows=10, cols=5, seed=42);
-R = X;
-
-for(i in 1:2){
-  R = R + 1 / 2;
-  R = R * 3;
-  X = X - 5;
-  R = R - 5;
-}
-
-R = R * 4;
-write(R, $1, format="text");
diff --git a/src/test/scripts/functions/lineage/LineageTraceDedupExec10.dml b/src/test/scripts/functions/lineage/LineageTraceDedupExec10.dml
deleted file mode 100644
index 1e755df..0000000
--- a/src/test/scripts/functions/lineage/LineageTraceDedupExec10.dml
+++ /dev/null
@@ -1,34 +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.
-#
-#-------------------------------------------------------------
-
-X = rand(rows=10, cols=5, seed=42);
-R = X;
-
-for (i in 1:4) {
-  R = R + 1/2;
-  if (i %% 2 == 0)
-    R = R * 3*i;
-  R = R - 5;
-}
-
-R = R * 4;
-
-write(R, $1, format="text");
diff --git a/src/test/scripts/functions/lineage/LineageTraceDedupExec2.dml b/src/test/scripts/functions/lineage/LineageTraceDedupExec2.dml
deleted file mode 100644
index 3fa49dd..0000000
--- a/src/test/scripts/functions/lineage/LineageTraceDedupExec2.dml
+++ /dev/null
@@ -1,45 +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.
-#
-#-------------------------------------------------------------
-
-X = rand(rows=10, cols=5, seed=42);
-
-R = X;
-for(i in 1:5){ #10
-  if(i %% 2 == 1)
-    R = R + 1 / 2;
-  else
-    R = R * 3;
-
-  R = R - 5;
-
-  if (i %% 5 == 0)
-    R = t(R) %*% R;
-
-  R = R - 23
-}
-
-R = R * 3;
-
-#for (j in 1:2) {
-#  R = R * 4;
-#}
-
-write(R, $1, format="text");