You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@accumulo.apache.org by mm...@apache.org on 2022/06/08 18:38:27 UTC

[accumulo] branch main updated: Use Reference class in GC, Ample & TabletGroupWatcher (#2750)

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

mmiller pushed a commit to branch main
in repository https://gitbox.apache.org/repos/asf/accumulo.git


The following commit(s) were added to refs/heads/main by this push:
     new 37e1c6f275 Use Reference class in GC, Ample & TabletGroupWatcher (#2750)
37e1c6f275 is described below

commit 37e1c6f275cca0b118ed6ab631345c0699515552
Author: Mike Miller <mm...@apache.org>
AuthorDate: Wed Jun 8 18:38:23 2022 +0000

    Use Reference class in GC, Ample & TabletGroupWatcher (#2750)
    
    * Replace uses of String with Reference to prevent errors with
    over use of String in GC, Ample and TabletGroupWatcher
    * Make Reference comparable and implement required methods
    * Rename RelativeTabletDirectory to ReferenceDirectory
    * Rename TabletFileUtil to ValidationUtil and add new method for Reference
    * Make TabletDirectory its own independent class and clean it up
    * Create package org.apache.accumulo.core.gc and move Reference and
    ReferenceDirectory to the new package
    * Update GcVolumeUtil, MetadataTableUtil, CleanUpBulkImport to use Reference class
    * Update Upgrader and tests
    
    Co-authored-by: Dave Marion <dl...@apache.org>
---
 .../accumulo/core/{metadata => gc}/Reference.java  | 35 ++++++++++++++++++--
 .../ReferenceDirectory.java}                       |  6 ++--
 .../accumulo/core/metadata/TabletDirectory.java    | 20 ++++++++++--
 .../apache/accumulo/core/metadata/TabletFile.java  |  2 +-
 .../{TabletFileUtil.java => ValidationUtil.java}   | 10 ++++--
 .../accumulo/core/metadata/schema/Ample.java       | 13 ++++----
 .../apache/accumulo/server/gc/GcVolumeUtil.java    |  8 +++--
 .../accumulo/server/metadata/ServerAmpleImpl.java  | 15 +++++----
 .../apache/accumulo/server/util/LocalityCheck.java |  4 +--
 .../accumulo/server/util/MetadataTableUtil.java    | 12 +++----
 .../server/util/RemoveEntriesForMissingFiles.java  |  4 +--
 .../main/java/org/apache/accumulo/gc/GCRun.java    | 12 +++----
 .../accumulo/gc/GarbageCollectionAlgorithm.java    |  8 ++---
 .../accumulo/gc/GarbageCollectionEnvironment.java  |  2 +-
 .../apache/accumulo/gc/GarbageCollectionTest.java  |  6 ++--
 .../accumulo/gc/SimpleGarbageCollectorTest.java    | 16 ++++-----
 .../accumulo/manager/TabletGroupWatcher.java       | 30 +++++++++--------
 .../tableOps/bulkVer1/CleanUpBulkImport.java       |  3 +-
 .../manager/tableOps/bulkVer1/CopyFailed.java      |  4 +--
 .../tableOps/bulkVer2/CleanUpBulkImport.java       |  3 +-
 .../tableOps/tableExport/WriteExportFiles.java     |  4 +--
 .../accumulo/manager/upgrade/Upgrader9to10.java    | 28 ++++++++++++----
 .../manager/upgrade/Upgrader9to10Test.java         | 38 +++++++++++++++-------
 .../test/functional/GarbageCollectorIT.java        |  5 ++-
 .../test/upgrade/GCUpgrade9to10TestIT.java         |  3 +-
 25 files changed, 191 insertions(+), 100 deletions(-)

diff --git a/core/src/main/java/org/apache/accumulo/core/metadata/Reference.java b/core/src/main/java/org/apache/accumulo/core/gc/Reference.java
similarity index 64%
rename from core/src/main/java/org/apache/accumulo/core/metadata/Reference.java
rename to core/src/main/java/org/apache/accumulo/core/gc/Reference.java
index 0254220040..ec820b2a6a 100644
--- a/core/src/main/java/org/apache/accumulo/core/metadata/Reference.java
+++ b/core/src/main/java/org/apache/accumulo/core/gc/Reference.java
@@ -16,15 +16,15 @@
  * specific language governing permissions and limitations
  * under the License.
  */
-package org.apache.accumulo.core.metadata;
+package org.apache.accumulo.core.gc;
 
 import org.apache.accumulo.core.data.TableId;
 import org.apache.accumulo.core.metadata.schema.MetadataSchema;
 
 /**
- * A reference to a tablet file or directory.
+ * A GC reference to a tablet file or directory.
  */
-public class Reference {
+public class Reference implements Comparable<Reference> {
   // parts of an absolute URI, like "hdfs://1.2.3.4/accumulo/tables/2a/t-0003"
   public final TableId tableId; // 2a
   public final String tabletDir; // t-0003
@@ -38,4 +38,33 @@ public class Reference {
     this.metadataEntry = metadataEntry;
     this.tabletDir = metadataEntry;
   }
+
+  @Override
+  public int compareTo(Reference that) {
+    if (equals(that)) {
+      return 0;
+    } else {
+      return this.metadataEntry.compareTo(that.metadataEntry);
+    }
+  }
+
+  @Override
+  public boolean equals(Object obj) {
+    if (this == obj)
+      return true;
+    if (obj == null)
+      return false;
+    if (getClass() != obj.getClass())
+      return false;
+    Reference other = (Reference) obj;
+    if (metadataEntry == null) {
+      return other.metadataEntry == null;
+    } else
+      return metadataEntry.equals(other.metadataEntry);
+  }
+
+  @Override
+  public int hashCode() {
+    return this.metadataEntry.hashCode();
+  }
 }
diff --git a/core/src/main/java/org/apache/accumulo/core/metadata/RelativeTabletDirectory.java b/core/src/main/java/org/apache/accumulo/core/gc/ReferenceDirectory.java
similarity index 85%
rename from core/src/main/java/org/apache/accumulo/core/metadata/RelativeTabletDirectory.java
rename to core/src/main/java/org/apache/accumulo/core/gc/ReferenceDirectory.java
index ae5d53cbc5..8a9ce21899 100644
--- a/core/src/main/java/org/apache/accumulo/core/metadata/RelativeTabletDirectory.java
+++ b/core/src/main/java/org/apache/accumulo/core/gc/ReferenceDirectory.java
@@ -16,16 +16,16 @@
  * specific language governing permissions and limitations
  * under the License.
  */
-package org.apache.accumulo.core.metadata;
+package org.apache.accumulo.core.gc;
 
 import org.apache.accumulo.core.data.TableId;
 
 /**
  * Part of the Tablet File path that is definitely a directory.
  */
-public class RelativeTabletDirectory extends Reference {
+public class ReferenceDirectory extends Reference {
 
-  public RelativeTabletDirectory(TableId tableId, String dirName) {
+  public ReferenceDirectory(TableId tableId, String dirName) {
     super(tableId, dirName);
   }
 }
diff --git a/core/src/main/java/org/apache/accumulo/core/metadata/TabletDirectory.java b/core/src/main/java/org/apache/accumulo/core/metadata/TabletDirectory.java
index 18180c1b46..1fcab6e70b 100644
--- a/core/src/main/java/org/apache/accumulo/core/metadata/TabletDirectory.java
+++ b/core/src/main/java/org/apache/accumulo/core/metadata/TabletDirectory.java
@@ -18,18 +18,28 @@
  */
 package org.apache.accumulo.core.metadata;
 
+import static org.apache.accumulo.core.Constants.HDFS_TABLES_DIR;
+
 import org.apache.accumulo.core.data.TableId;
 
 /**
- * The Tablet directory that may exist in the metadata table.
+ * The Tablet directory that should exist on disk. The {@link #toString()} method only returns the
+ * tablet directory itself, the same as {@link #getTabletDir()}, which is just the name of the
+ * directory, like "t-0003". For the full directory path, use {@link #getNormalizedPath}.
  */
-public class TabletDirectory extends RelativeTabletDirectory {
+public class TabletDirectory {
   // parts of an absolute URI, like "hdfs://1.2.3.4/accumulo/tables/2a/t-0003"
   private final String volume; // hdfs://1.2.3.4/accumulo
+  private final TableId tableId; // 2a
+  private final String tabletDir; // t-0003
+  private final String normalizedPath;
 
   public TabletDirectory(String volume, TableId tableId, String tabletDir) {
-    super(tableId, tabletDir);
     this.volume = volume;
+    this.tableId = tableId;
+    this.tabletDir = tabletDir;
+    this.normalizedPath = volume + HDFS_TABLES_DIR + "/" + tableId.canonical() + "/" + tabletDir;
+
   }
 
   public String getVolume() {
@@ -44,6 +54,10 @@ public class TabletDirectory extends RelativeTabletDirectory {
     return tabletDir;
   }
 
+  public String getNormalizedPath() {
+    return normalizedPath;
+  }
+
   @Override
   public String toString() {
     return tabletDir;
diff --git a/core/src/main/java/org/apache/accumulo/core/metadata/TabletFile.java b/core/src/main/java/org/apache/accumulo/core/metadata/TabletFile.java
index 71bdfe6efc..d9c90fcec7 100644
--- a/core/src/main/java/org/apache/accumulo/core/metadata/TabletFile.java
+++ b/core/src/main/java/org/apache/accumulo/core/metadata/TabletFile.java
@@ -72,7 +72,7 @@ public class TabletFile implements Comparable<TabletFile> {
     var volume = volumePath.toString();
 
     this.tabletDir = new TabletDirectory(volume, TableId.of(id), tabletDirPath.getName());
-    this.normalizedPath = volume + HDFS_TABLES_DIR + "/" + id + "/" + tabletDir + "/" + fileName;
+    this.normalizedPath = tabletDir.getNormalizedPath() + "/" + fileName;
   }
 
   public String getVolume() {
diff --git a/core/src/main/java/org/apache/accumulo/core/metadata/TabletFileUtil.java b/core/src/main/java/org/apache/accumulo/core/metadata/ValidationUtil.java
similarity index 85%
rename from core/src/main/java/org/apache/accumulo/core/metadata/TabletFileUtil.java
rename to core/src/main/java/org/apache/accumulo/core/metadata/ValidationUtil.java
index ff92ae339e..83e57a40d2 100644
--- a/core/src/main/java/org/apache/accumulo/core/metadata/TabletFileUtil.java
+++ b/core/src/main/java/org/apache/accumulo/core/metadata/ValidationUtil.java
@@ -18,12 +18,13 @@
  */
 package org.apache.accumulo.core.metadata;
 
+import org.apache.accumulo.core.gc.Reference;
 import org.apache.hadoop.fs.Path;
 
 /**
- * Utility class for validation of metadata tablet files.
+ * Utility class for validation of tablet file paths.
  */
-public class TabletFileUtil {
+public class ValidationUtil {
 
   /**
    * Validate if string is a valid path. Return normalized string or throw exception if not valid.
@@ -36,6 +37,11 @@ public class TabletFileUtil {
     return validate(p).toString();
   }
 
+  public static Reference validate(Reference reference) {
+    validate(new Path(reference.metadataEntry));
+    return reference;
+  }
+
   public static Path validate(Path path) {
     if (path.toUri().getScheme() == null) {
       throw new IllegalArgumentException("Invalid path provided, no scheme in " + path);
diff --git a/core/src/main/java/org/apache/accumulo/core/metadata/schema/Ample.java b/core/src/main/java/org/apache/accumulo/core/metadata/schema/Ample.java
index b5a242d2e3..460f98133d 100644
--- a/core/src/main/java/org/apache/accumulo/core/metadata/schema/Ample.java
+++ b/core/src/main/java/org/apache/accumulo/core/metadata/schema/Ample.java
@@ -25,6 +25,7 @@ import java.util.stream.Stream;
 import org.apache.accumulo.core.data.Mutation;
 import org.apache.accumulo.core.data.TableId;
 import org.apache.accumulo.core.dataImpl.KeyExtent;
+import org.apache.accumulo.core.gc.Reference;
 import org.apache.accumulo.core.metadata.MetadataTable;
 import org.apache.accumulo.core.metadata.RootTable;
 import org.apache.accumulo.core.metadata.StoredTabletFile;
@@ -190,7 +191,7 @@ public interface Ample {
   /**
    * Unlike {@link #putGcCandidates(TableId, Collection)} this takes file and dir GC candidates.
    */
-  default void putGcFileAndDirCandidates(TableId tableId, Collection<String> candidates) {
+  default void putGcFileAndDirCandidates(TableId tableId, Collection<Reference> candidates) {
     throw new UnsupportedOperationException();
   }
 
@@ -217,16 +218,16 @@ public interface Ample {
   }
 
   /**
-   * Return an encoded delete marker Mutation to delete the specified TabletFile path. A String is
-   * used for the parameter because the Garbage Collector is optimized to store a directory for
-   * Tablet File. Otherwise a {@link TabletFile} object could be used. The tabletFilePathToRemove is
-   * validated and normalized before creating the mutation.
+   * Return an encoded delete marker Mutation to delete the specified TabletFile path. A Reference
+   * is used for the parameter because the Garbage Collector is optimized to store a directory for
+   * Tablet File. Otherwise, a {@link TabletFile} object could be used. The tabletFilePathToRemove
+   * is validated and normalized before creating the mutation.
    *
    * @param tabletFilePathToRemove
    *          String full path of the TabletFile
    * @return Mutation with encoded delete marker
    */
-  default Mutation createDeleteMutation(String tabletFilePathToRemove) {
+  default Mutation createDeleteMutation(Reference tabletFilePathToRemove) {
     throw new UnsupportedOperationException();
   }
 
diff --git a/server/base/src/main/java/org/apache/accumulo/server/gc/GcVolumeUtil.java b/server/base/src/main/java/org/apache/accumulo/server/gc/GcVolumeUtil.java
index aa4ac4a081..a72deb3938 100644
--- a/server/base/src/main/java/org/apache/accumulo/server/gc/GcVolumeUtil.java
+++ b/server/base/src/main/java/org/apache/accumulo/server/gc/GcVolumeUtil.java
@@ -24,6 +24,7 @@ import java.util.stream.Collectors;
 
 import org.apache.accumulo.core.Constants;
 import org.apache.accumulo.core.data.TableId;
+import org.apache.accumulo.core.gc.Reference;
 import org.apache.accumulo.core.metadata.schema.MetadataSchema.TabletsSection.ServerColumnFamily;
 import org.apache.accumulo.server.fs.VolumeManager;
 import org.apache.hadoop.fs.Path;
@@ -32,10 +33,11 @@ public class GcVolumeUtil {
   // AGCAV : Accumulo Garbage Collector All Volumes
   private static final String ALL_VOLUMES_PREFIX = "agcav:/";
 
-  public static String getDeleteTabletOnAllVolumesUri(TableId tableId, String dirName) {
+  public static Reference getDeleteTabletOnAllVolumesUri(TableId tableId, String dirName) {
     ServerColumnFamily.validateDirCol(dirName);
-    return ALL_VOLUMES_PREFIX + Constants.TABLE_DIR + Path.SEPARATOR + tableId + Path.SEPARATOR
-        + dirName;
+    String str = ALL_VOLUMES_PREFIX + Constants.TABLE_DIR + Path.SEPARATOR + tableId
+        + Path.SEPARATOR + dirName;
+    return new Reference(tableId, str);
   }
 
   public static Collection<Path> expandAllVolumesUri(VolumeManager fs, Path path) {
diff --git a/server/base/src/main/java/org/apache/accumulo/server/metadata/ServerAmpleImpl.java b/server/base/src/main/java/org/apache/accumulo/server/metadata/ServerAmpleImpl.java
index 08e20434fb..a205826c70 100644
--- a/server/base/src/main/java/org/apache/accumulo/server/metadata/ServerAmpleImpl.java
+++ b/server/base/src/main/java/org/apache/accumulo/server/metadata/ServerAmpleImpl.java
@@ -35,10 +35,11 @@ import org.apache.accumulo.core.data.Mutation;
 import org.apache.accumulo.core.data.Range;
 import org.apache.accumulo.core.data.TableId;
 import org.apache.accumulo.core.dataImpl.KeyExtent;
+import org.apache.accumulo.core.gc.Reference;
 import org.apache.accumulo.core.metadata.MetadataTable;
 import org.apache.accumulo.core.metadata.RootTable;
 import org.apache.accumulo.core.metadata.StoredTabletFile;
-import org.apache.accumulo.core.metadata.TabletFileUtil;
+import org.apache.accumulo.core.metadata.ValidationUtil;
 import org.apache.accumulo.core.metadata.schema.Ample;
 import org.apache.accumulo.core.metadata.schema.AmpleImpl;
 import org.apache.accumulo.core.metadata.schema.ExternalCompactionFinalState;
@@ -120,17 +121,18 @@ public class ServerAmpleImpl extends AmpleImpl implements Ample {
   }
 
   @Override
-  public void putGcFileAndDirCandidates(TableId tableId, Collection<String> candidates) {
+  public void putGcFileAndDirCandidates(TableId tableId, Collection<Reference> candidates) {
 
     if (RootTable.ID.equals(tableId)) {
 
       // Directories are unexpected for the root tablet, so convert to stored tablet file
-      mutateRootGcCandidates(rgcc -> rgcc.add(candidates.stream().map(StoredTabletFile::new)));
+      mutateRootGcCandidates(rgcc -> rgcc.add(
+          candidates.stream().map(reference -> new StoredTabletFile(reference.metadataEntry))));
       return;
     }
 
     try (BatchWriter writer = createWriter(tableId)) {
-      for (String fileOrDir : candidates) {
+      for (var fileOrDir : candidates) {
         writer.addMutation(createDeleteMutation(fileOrDir));
       }
     } catch (MutationsRejectedException e) {
@@ -202,9 +204,8 @@ public class ServerAmpleImpl extends AmpleImpl implements Ample {
   }
 
   @Override
-  public Mutation createDeleteMutation(String tabletFilePathToRemove) {
-    String path = TabletFileUtil.validate(tabletFilePathToRemove);
-    return createDelMutation(path);
+  public Mutation createDeleteMutation(Reference tabletFilePathToRemove) {
+    return createDelMutation(ValidationUtil.validate(tabletFilePathToRemove).metadataEntry);
   }
 
   public Mutation createDeleteMutation(StoredTabletFile pathToRemove) {
diff --git a/server/base/src/main/java/org/apache/accumulo/server/util/LocalityCheck.java b/server/base/src/main/java/org/apache/accumulo/server/util/LocalityCheck.java
index a1ff119a0e..a51d161eb6 100644
--- a/server/base/src/main/java/org/apache/accumulo/server/util/LocalityCheck.java
+++ b/server/base/src/main/java/org/apache/accumulo/server/util/LocalityCheck.java
@@ -29,7 +29,7 @@ import org.apache.accumulo.core.client.Scanner;
 import org.apache.accumulo.core.data.Key;
 import org.apache.accumulo.core.data.Value;
 import org.apache.accumulo.core.metadata.MetadataTable;
-import org.apache.accumulo.core.metadata.TabletFileUtil;
+import org.apache.accumulo.core.metadata.ValidationUtil;
 import org.apache.accumulo.core.metadata.schema.MetadataSchema.TabletsSection;
 import org.apache.accumulo.core.metadata.schema.MetadataSchema.TabletsSection.CurrentLocationColumnFamily;
 import org.apache.accumulo.core.metadata.schema.MetadataSchema.TabletsSection.DataFileColumnFamily;
@@ -76,7 +76,7 @@ public class LocalityCheck {
             addBlocks(fs, host, files, totalBlocks, localBlocks);
             files.clear();
           } else if (key.compareColumnFamily(DataFileColumnFamily.NAME) == 0) {
-            files.add(TabletFileUtil.validate(key.getColumnQualifierData().toString()));
+            files.add(ValidationUtil.validate(key.getColumnQualifierData().toString()));
           }
         }
         System.out.println(" Server         %local  total blocks");
diff --git a/server/base/src/main/java/org/apache/accumulo/server/util/MetadataTableUtil.java b/server/base/src/main/java/org/apache/accumulo/server/util/MetadataTableUtil.java
index 591cf5a021..bde6000bcc 100644
--- a/server/base/src/main/java/org/apache/accumulo/server/util/MetadataTableUtil.java
+++ b/server/base/src/main/java/org/apache/accumulo/server/util/MetadataTableUtil.java
@@ -62,11 +62,11 @@ import org.apache.accumulo.core.data.Range;
 import org.apache.accumulo.core.data.TableId;
 import org.apache.accumulo.core.data.Value;
 import org.apache.accumulo.core.dataImpl.KeyExtent;
+import org.apache.accumulo.core.gc.Reference;
 import org.apache.accumulo.core.metadata.MetadataTable;
 import org.apache.accumulo.core.metadata.RootTable;
 import org.apache.accumulo.core.metadata.StoredTabletFile;
 import org.apache.accumulo.core.metadata.TabletFile;
-import org.apache.accumulo.core.metadata.TabletFileUtil;
 import org.apache.accumulo.core.metadata.schema.Ample;
 import org.apache.accumulo.core.metadata.schema.Ample.TabletMutator;
 import org.apache.accumulo.core.metadata.schema.DataFileValue;
@@ -372,12 +372,13 @@ public class MetadataTableUtil {
           Key key = cell.getKey();
 
           if (key.getColumnFamily().equals(DataFileColumnFamily.NAME)) {
-            String ref = TabletFileUtil.validate(key.getColumnQualifierData().toString());
-            bw.addMutation(ample.createDeleteMutation(ref));
+            StoredTabletFile stf = new StoredTabletFile(key.getColumnQualifierData().toString());
+            bw.addMutation(
+                ample.createDeleteMutation(new Reference(tableId, stf.getMetaUpdateDelete())));
           }
 
           if (ServerColumnFamily.DIRECTORY_COLUMN.hasColumns(key)) {
-            String uri =
+            var uri =
                 GcVolumeUtil.getDeleteTabletOnAllVolumesUri(tableId, cell.getValue().toString());
             bw.addMutation(ample.createDeleteMutation(uri));
           }
@@ -690,8 +691,7 @@ public class MetadataTableUtil {
     m.putDelete(EMPTY_TEXT, EMPTY_TEXT);
 
     // new KeyExtent is only added to force update to write to the metadata table, not the root
-    // table
-    // because bulk loads aren't supported to the metadata table
+    // table because bulk loads aren't supported to the metadata table
     update(context, m, new KeyExtent(TableId.of("anythingNotMetadata"), null, null));
   }
 
diff --git a/server/base/src/main/java/org/apache/accumulo/server/util/RemoveEntriesForMissingFiles.java b/server/base/src/main/java/org/apache/accumulo/server/util/RemoveEntriesForMissingFiles.java
index 9a98036faa..81734791da 100644
--- a/server/base/src/main/java/org/apache/accumulo/server/util/RemoveEntriesForMissingFiles.java
+++ b/server/base/src/main/java/org/apache/accumulo/server/util/RemoveEntriesForMissingFiles.java
@@ -37,7 +37,7 @@ import org.apache.accumulo.core.data.Value;
 import org.apache.accumulo.core.dataImpl.KeyExtent;
 import org.apache.accumulo.core.metadata.MetadataTable;
 import org.apache.accumulo.core.metadata.RootTable;
-import org.apache.accumulo.core.metadata.TabletFileUtil;
+import org.apache.accumulo.core.metadata.ValidationUtil;
 import org.apache.accumulo.core.metadata.schema.MetadataSchema.TabletsSection;
 import org.apache.accumulo.core.metadata.schema.MetadataSchema.TabletsSection.DataFileColumnFamily;
 import org.apache.accumulo.core.security.Authorizations;
@@ -144,7 +144,7 @@ public class RemoveEntriesForMissingFiles {
 
       count++;
       Key key = entry.getKey();
-      Path map = new Path(TabletFileUtil.validate(key.getColumnQualifierData().toString()));
+      Path map = new Path(ValidationUtil.validate(key.getColumnQualifierData().toString()));
 
       synchronized (processing) {
         while (processing.size() >= 64 || processing.contains(map))
diff --git a/server/gc/src/main/java/org/apache/accumulo/gc/GCRun.java b/server/gc/src/main/java/org/apache/accumulo/gc/GCRun.java
index 02d376a956..ba862a4376 100644
--- a/server/gc/src/main/java/org/apache/accumulo/gc/GCRun.java
+++ b/server/gc/src/main/java/org/apache/accumulo/gc/GCRun.java
@@ -45,11 +45,11 @@ import org.apache.accumulo.core.client.TableNotFoundException;
 import org.apache.accumulo.core.conf.AccumuloConfiguration;
 import org.apache.accumulo.core.conf.Property;
 import org.apache.accumulo.core.data.TableId;
+import org.apache.accumulo.core.gc.Reference;
+import org.apache.accumulo.core.gc.ReferenceDirectory;
 import org.apache.accumulo.core.manager.state.tables.TableState;
-import org.apache.accumulo.core.metadata.Reference;
-import org.apache.accumulo.core.metadata.RelativeTabletDirectory;
 import org.apache.accumulo.core.metadata.RootTable;
-import org.apache.accumulo.core.metadata.TabletFileUtil;
+import org.apache.accumulo.core.metadata.ValidationUtil;
 import org.apache.accumulo.core.metadata.schema.Ample;
 import org.apache.accumulo.core.metadata.schema.MetadataSchema;
 import org.apache.accumulo.core.metadata.schema.TabletMetadata;
@@ -151,7 +151,7 @@ public class GCRun implements GarbageCollectionEnvironment {
           .map(f -> new Reference(tm.getTableId(), f.getMetaUpdateDelete()));
       if (tm.getDirName() != null) {
         refs = Stream.concat(refs,
-            Stream.of(new RelativeTabletDirectory(tm.getTableId(), tm.getDirName())));
+            Stream.of(new ReferenceDirectory(tm.getTableId(), tm.getDirName())));
       }
       return refs;
     });
@@ -209,9 +209,9 @@ public class GCRun implements GarbageCollectionEnvironment {
             // of deleting something that should not be deleted. Must not change value of delete
             // variable because that's what's stored in metadata table.
             log.debug("Volume replaced {} -> {}", delete, switchedDelete);
-            fullPath = TabletFileUtil.validate(switchedDelete);
+            fullPath = ValidationUtil.validate(switchedDelete);
           } else {
-            fullPath = new Path(TabletFileUtil.validate(delete));
+            fullPath = new Path(ValidationUtil.validate(delete));
           }
 
           for (Path pathToDel : GcVolumeUtil.expandAllVolumesUri(fs, fullPath)) {
diff --git a/server/gc/src/main/java/org/apache/accumulo/gc/GarbageCollectionAlgorithm.java b/server/gc/src/main/java/org/apache/accumulo/gc/GarbageCollectionAlgorithm.java
index b6e770001a..789ff37cba 100644
--- a/server/gc/src/main/java/org/apache/accumulo/gc/GarbageCollectionAlgorithm.java
+++ b/server/gc/src/main/java/org/apache/accumulo/gc/GarbageCollectionAlgorithm.java
@@ -36,8 +36,8 @@ import java.util.stream.Stream;
 import org.apache.accumulo.core.Constants;
 import org.apache.accumulo.core.client.TableNotFoundException;
 import org.apache.accumulo.core.data.TableId;
-import org.apache.accumulo.core.metadata.Reference;
-import org.apache.accumulo.core.metadata.RelativeTabletDirectory;
+import org.apache.accumulo.core.gc.Reference;
+import org.apache.accumulo.core.gc.ReferenceDirectory;
 import org.apache.accumulo.core.metadata.schema.MetadataSchema.TabletsSection.ServerColumnFamily;
 import org.apache.accumulo.core.trace.TraceUtil;
 import org.apache.accumulo.server.replication.proto.Replication.Status;
@@ -139,8 +139,8 @@ public class GarbageCollectionAlgorithm {
     while (iter.hasNext()) {
       Reference ref = iter.next();
 
-      if (ref instanceof RelativeTabletDirectory) {
-        var dirReference = (RelativeTabletDirectory) ref;
+      if (ref instanceof ReferenceDirectory) {
+        var dirReference = (ReferenceDirectory) ref;
         ServerColumnFamily.validateDirCol(dirReference.tabletDir);
 
         String dir = "/" + dirReference.tableId + "/" + dirReference.tabletDir;
diff --git a/server/gc/src/main/java/org/apache/accumulo/gc/GarbageCollectionEnvironment.java b/server/gc/src/main/java/org/apache/accumulo/gc/GarbageCollectionEnvironment.java
index 6d943c1555..fdea5887a9 100644
--- a/server/gc/src/main/java/org/apache/accumulo/gc/GarbageCollectionEnvironment.java
+++ b/server/gc/src/main/java/org/apache/accumulo/gc/GarbageCollectionEnvironment.java
@@ -31,8 +31,8 @@ import org.apache.accumulo.core.client.TableNotFoundException;
 import org.apache.accumulo.core.data.Key;
 import org.apache.accumulo.core.data.TableId;
 import org.apache.accumulo.core.data.Value;
+import org.apache.accumulo.core.gc.Reference;
 import org.apache.accumulo.core.metadata.MetadataTable;
-import org.apache.accumulo.core.metadata.Reference;
 import org.apache.accumulo.core.metadata.RootTable;
 import org.apache.accumulo.core.metadata.schema.MetadataSchema.TabletsSection.DataFileColumnFamily;
 import org.apache.accumulo.core.metadata.schema.MetadataSchema.TabletsSection.ScanFileColumnFamily;
diff --git a/server/gc/src/test/java/org/apache/accumulo/gc/GarbageCollectionTest.java b/server/gc/src/test/java/org/apache/accumulo/gc/GarbageCollectionTest.java
index 9c51fb1719..849023c385 100644
--- a/server/gc/src/test/java/org/apache/accumulo/gc/GarbageCollectionTest.java
+++ b/server/gc/src/test/java/org/apache/accumulo/gc/GarbageCollectionTest.java
@@ -37,8 +37,8 @@ import java.util.stream.Stream;
 
 import org.apache.accumulo.core.client.TableNotFoundException;
 import org.apache.accumulo.core.data.TableId;
-import org.apache.accumulo.core.metadata.Reference;
-import org.apache.accumulo.core.metadata.RelativeTabletDirectory;
+import org.apache.accumulo.core.gc.Reference;
+import org.apache.accumulo.core.gc.ReferenceDirectory;
 import org.apache.accumulo.server.replication.proto.Replication.Status;
 import org.junit.jupiter.api.Test;
 
@@ -103,7 +103,7 @@ public class GarbageCollectionTest {
     }
 
     public void addDirReference(String tableId, String endRow, String dir) {
-      references.put(tableId + ":" + endRow, new RelativeTabletDirectory(TableId.of(tableId), dir));
+      references.put(tableId + ":" + endRow, new ReferenceDirectory(TableId.of(tableId), dir));
     }
 
     public void removeDirReference(String tableId, String endRow) {
diff --git a/server/gc/src/test/java/org/apache/accumulo/gc/SimpleGarbageCollectorTest.java b/server/gc/src/test/java/org/apache/accumulo/gc/SimpleGarbageCollectorTest.java
index 4775825611..125a149edb 100644
--- a/server/gc/src/test/java/org/apache/accumulo/gc/SimpleGarbageCollectorTest.java
+++ b/server/gc/src/test/java/org/apache/accumulo/gc/SimpleGarbageCollectorTest.java
@@ -171,13 +171,13 @@ public class SimpleGarbageCollectorTest {
     confirmed.put("5a/t-0001/F0001.rf", "hdfs://nn1/accumulo/tables/5a/t-0001/F0001.rf");
     confirmed.put("5a/t-0001/F0002.rf", "hdfs://nn1/accumulo/tables/5a/t-0001/F0002.rf");
     confirmed.put("5a/t-0002/F0001.rf", "hdfs://nn1/accumulo/tables/5a/t-0002/F0001.rf");
-    confirmed.put("5b/t-0003",
-        GcVolumeUtil.getDeleteTabletOnAllVolumesUri(TableId.of("5b"), "t-0003"));
+    var uri = GcVolumeUtil.getDeleteTabletOnAllVolumesUri(TableId.of("5b"), "t-0003");
+    confirmed.put("5b/t-0003", uri.metadataEntry);
     confirmed.put("5b/t-0003/F0001.rf", "hdfs://nn1/accumulo/tables/5b/t-0003/F0001.rf");
     confirmed.put("5b/t-0003/F0002.rf", "hdfs://nn2/accumulo/tables/5b/t-0003/F0002.rf");
     confirmed.put("5b/t-0003/F0003.rf", "hdfs://nn3/accumulo/tables/5b/t-0003/F0003.rf");
-    confirmed.put("5b/t-0004",
-        GcVolumeUtil.getDeleteTabletOnAllVolumesUri(TableId.of("5b"), "t-0004"));
+    uri = GcVolumeUtil.getDeleteTabletOnAllVolumesUri(TableId.of("5b"), "t-0004");
+    confirmed.put("5b/t-0004", uri.metadataEntry);
     confirmed.put("5b/t-0004/F0001.rf", "hdfs://nn1/accumulo/tables/5b/t-0004/F0001.rf");
 
     List<String> processedDeletes = new ArrayList<>();
@@ -187,11 +187,11 @@ public class SimpleGarbageCollectorTest {
     TreeMap<String,String> expected = new TreeMap<>();
     expected.put("5a/t-0001", "hdfs://nn1/accumulo/tables/5a/t-0001");
     expected.put("5a/t-0002/F0001.rf", "hdfs://nn1/accumulo/tables/5a/t-0002/F0001.rf");
-    expected.put("5b/t-0003",
-        GcVolumeUtil.getDeleteTabletOnAllVolumesUri(TableId.of("5b"), "t-0003"));
+    uri = GcVolumeUtil.getDeleteTabletOnAllVolumesUri(TableId.of("5b"), "t-0003");
+    expected.put("5b/t-0003", uri.metadataEntry);
     expected.put("5b/t-0003/F0003.rf", "hdfs://nn3/accumulo/tables/5b/t-0003/F0003.rf");
-    expected.put("5b/t-0004",
-        GcVolumeUtil.getDeleteTabletOnAllVolumesUri(TableId.of("5b"), "t-0004"));
+    uri = GcVolumeUtil.getDeleteTabletOnAllVolumesUri(TableId.of("5b"), "t-0004");
+    expected.put("5b/t-0004", uri.metadataEntry);
 
     assertEquals(expected, confirmed);
     assertEquals(Arrays.asList("hdfs://nn1/accumulo/tables/5a/t-0001/F0001.rf",
diff --git a/server/manager/src/main/java/org/apache/accumulo/manager/TabletGroupWatcher.java b/server/manager/src/main/java/org/apache/accumulo/manager/TabletGroupWatcher.java
index fcf421aaf3..cd42491444 100644
--- a/server/manager/src/main/java/org/apache/accumulo/manager/TabletGroupWatcher.java
+++ b/server/manager/src/main/java/org/apache/accumulo/manager/TabletGroupWatcher.java
@@ -52,14 +52,16 @@ import org.apache.accumulo.core.data.Range;
 import org.apache.accumulo.core.data.TableId;
 import org.apache.accumulo.core.data.Value;
 import org.apache.accumulo.core.dataImpl.KeyExtent;
+import org.apache.accumulo.core.gc.Reference;
+import org.apache.accumulo.core.gc.ReferenceDirectory;
 import org.apache.accumulo.core.logging.TabletLogger;
 import org.apache.accumulo.core.manager.state.tables.TableState;
 import org.apache.accumulo.core.manager.thrift.ManagerState;
 import org.apache.accumulo.core.master.thrift.TabletServerStatus;
 import org.apache.accumulo.core.metadata.MetadataTable;
 import org.apache.accumulo.core.metadata.RootTable;
+import org.apache.accumulo.core.metadata.StoredTabletFile;
 import org.apache.accumulo.core.metadata.TServerInstance;
-import org.apache.accumulo.core.metadata.TabletFileUtil;
 import org.apache.accumulo.core.metadata.TabletLocationState;
 import org.apache.accumulo.core.metadata.TabletLocationState.BadLocationStateException;
 import org.apache.accumulo.core.metadata.TabletState;
@@ -635,14 +637,16 @@ abstract class TabletGroupWatcher extends AccumuloDaemonThread {
       ServerColumnFamily.TIME_COLUMN.fetch(scanner);
       scanner.fetchColumnFamily(DataFileColumnFamily.NAME);
       scanner.fetchColumnFamily(CurrentLocationColumnFamily.NAME);
-      Set<String> datafiles = new TreeSet<>();
+      Set<Reference> datafilesAndDirs = new TreeSet<>();
       for (Entry<Key,Value> entry : scanner) {
         Key key = entry.getKey();
         if (key.compareColumnFamily(DataFileColumnFamily.NAME) == 0) {
-          datafiles.add(TabletFileUtil.validate(key.getColumnQualifierData().toString()));
-          if (datafiles.size() > 1000) {
-            ample.putGcFileAndDirCandidates(extent.tableId(), datafiles);
-            datafiles.clear();
+          var stf = new StoredTabletFile(key.getColumnQualifierData().toString());
+          var tabletDirectory = new ReferenceDirectory(stf.getTableId(), stf.getTabletDir());
+          datafilesAndDirs.add(tabletDirectory);
+          if (datafilesAndDirs.size() > 1000) {
+            ample.putGcFileAndDirCandidates(extent.tableId(), datafilesAndDirs);
+            datafilesAndDirs.clear();
           }
         } else if (ServerColumnFamily.TIME_COLUMN.hasColumns(key)) {
           metadataTime = MetadataTime.parse(entry.getValue().toString());
@@ -650,16 +654,16 @@ abstract class TabletGroupWatcher extends AccumuloDaemonThread {
           throw new IllegalStateException(
               "Tablet " + key.getRow() + " is assigned during a merge!");
         } else if (ServerColumnFamily.DIRECTORY_COLUMN.hasColumns(key)) {
-          String path = GcVolumeUtil.getDeleteTabletOnAllVolumesUri(extent.tableId(),
+          Reference path = GcVolumeUtil.getDeleteTabletOnAllVolumesUri(extent.tableId(),
               entry.getValue().toString());
-          datafiles.add(path);
-          if (datafiles.size() > 1000) {
-            ample.putGcFileAndDirCandidates(extent.tableId(), datafiles);
-            datafiles.clear();
+          datafilesAndDirs.add(path);
+          if (datafilesAndDirs.size() > 1000) {
+            ample.putGcFileAndDirCandidates(extent.tableId(), datafilesAndDirs);
+            datafilesAndDirs.clear();
           }
         }
       }
-      ample.putGcFileAndDirCandidates(extent.tableId(), datafiles);
+      ample.putGcFileAndDirCandidates(extent.tableId(), datafilesAndDirs);
       BatchWriter bw = client.createBatchWriter(targetSystemTable);
       try {
         deleteTablets(info, deleteRange, bw, client);
@@ -736,7 +740,7 @@ abstract class TabletGroupWatcher extends AccumuloDaemonThread {
           maxLogicalTime =
               TabletTime.maxMetadataTime(maxLogicalTime, MetadataTime.parse(value.toString()));
         } else if (ServerColumnFamily.DIRECTORY_COLUMN.hasColumns(key)) {
-          String uri =
+          Reference uri =
               GcVolumeUtil.getDeleteTabletOnAllVolumesUri(range.tableId(), value.toString());
           bw.addMutation(manager.getContext().getAmple().createDeleteMutation(uri));
         }
diff --git a/server/manager/src/main/java/org/apache/accumulo/manager/tableOps/bulkVer1/CleanUpBulkImport.java b/server/manager/src/main/java/org/apache/accumulo/manager/tableOps/bulkVer1/CleanUpBulkImport.java
index ddc0166cf6..81acfd54ce 100644
--- a/server/manager/src/main/java/org/apache/accumulo/manager/tableOps/bulkVer1/CleanUpBulkImport.java
+++ b/server/manager/src/main/java/org/apache/accumulo/manager/tableOps/bulkVer1/CleanUpBulkImport.java
@@ -23,6 +23,7 @@ import java.util.Collections;
 import org.apache.accumulo.core.Constants;
 import org.apache.accumulo.core.client.AccumuloClient;
 import org.apache.accumulo.core.data.TableId;
+import org.apache.accumulo.core.gc.Reference;
 import org.apache.accumulo.core.master.thrift.BulkImportState;
 import org.apache.accumulo.fate.FateTxId;
 import org.apache.accumulo.fate.Repo;
@@ -61,7 +62,7 @@ public class CleanUpBulkImport extends ManagerRepo {
     MetadataTableUtil.removeBulkLoadInProgressFlag(manager.getContext(),
         "/" + bulkDir.getParent().getName() + "/" + bulkDir.getName());
     manager.getContext().getAmple().putGcFileAndDirCandidates(tableId,
-        Collections.singleton(bulkDir.toString()));
+        Collections.singleton(new Reference(tableId, bulkDir.toString())));
     log.debug("removing the metadata table markers for loaded files");
     AccumuloClient client = manager.getContext();
     MetadataTableUtil.removeBulkLoadEntries(client, tableId, tid);
diff --git a/server/manager/src/main/java/org/apache/accumulo/manager/tableOps/bulkVer1/CopyFailed.java b/server/manager/src/main/java/org/apache/accumulo/manager/tableOps/bulkVer1/CopyFailed.java
index 6f60f181c2..422507057d 100644
--- a/server/manager/src/main/java/org/apache/accumulo/manager/tableOps/bulkVer1/CopyFailed.java
+++ b/server/manager/src/main/java/org/apache/accumulo/manager/tableOps/bulkVer1/CopyFailed.java
@@ -37,7 +37,7 @@ import org.apache.accumulo.core.dataImpl.KeyExtent;
 import org.apache.accumulo.core.master.thrift.BulkImportState;
 import org.apache.accumulo.core.metadata.MetadataTable;
 import org.apache.accumulo.core.metadata.TServerInstance;
-import org.apache.accumulo.core.metadata.TabletFileUtil;
+import org.apache.accumulo.core.metadata.ValidationUtil;
 import org.apache.accumulo.core.metadata.schema.MetadataSchema.TabletsSection.BulkFileColumnFamily;
 import org.apache.accumulo.core.security.Authorizations;
 import org.apache.accumulo.fate.FateTxId;
@@ -125,7 +125,7 @@ class CopyFailed extends ManagerRepo {
       for (Entry<Key,Value> entry : mscanner) {
         if (BulkFileColumnFamily.getBulkLoadTid(entry.getValue()) == tid) {
           Path loadedFile =
-              new Path(TabletFileUtil.validate(entry.getKey().getColumnQualifierData().toString()));
+              new Path(ValidationUtil.validate(entry.getKey().getColumnQualifierData().toString()));
           if (failures.remove(loadedFile)) {
             loadedFailures.add(loadedFile);
           }
diff --git a/server/manager/src/main/java/org/apache/accumulo/manager/tableOps/bulkVer2/CleanUpBulkImport.java b/server/manager/src/main/java/org/apache/accumulo/manager/tableOps/bulkVer2/CleanUpBulkImport.java
index d7b00889fe..fe26634268 100644
--- a/server/manager/src/main/java/org/apache/accumulo/manager/tableOps/bulkVer2/CleanUpBulkImport.java
+++ b/server/manager/src/main/java/org/apache/accumulo/manager/tableOps/bulkVer2/CleanUpBulkImport.java
@@ -23,6 +23,7 @@ import java.util.Collections;
 
 import org.apache.accumulo.core.Constants;
 import org.apache.accumulo.core.client.AccumuloClient;
+import org.apache.accumulo.core.gc.Reference;
 import org.apache.accumulo.core.manager.state.tables.TableState;
 import org.apache.accumulo.core.master.thrift.BulkImportState;
 import org.apache.accumulo.fate.FateTxId;
@@ -56,7 +57,7 @@ public class CleanUpBulkImport extends ManagerRepo {
     MetadataTableUtil.removeBulkLoadInProgressFlag(manager.getContext(),
         "/" + bulkDir.getParent().getName() + "/" + bulkDir.getName());
     manager.getContext().getAmple().putGcFileAndDirCandidates(info.tableId,
-        Collections.singleton(bulkDir.toString()));
+        Collections.singleton(new Reference(info.tableId, bulkDir.toString())));
     if (info.tableState == TableState.ONLINE) {
       log.debug("removing the metadata table markers for loaded files");
       AccumuloClient client = manager.getContext();
diff --git a/server/manager/src/main/java/org/apache/accumulo/manager/tableOps/tableExport/WriteExportFiles.java b/server/manager/src/main/java/org/apache/accumulo/manager/tableOps/tableExport/WriteExportFiles.java
index 86023afe5d..97d94b52da 100644
--- a/server/manager/src/main/java/org/apache/accumulo/manager/tableOps/tableExport/WriteExportFiles.java
+++ b/server/manager/src/main/java/org/apache/accumulo/manager/tableOps/tableExport/WriteExportFiles.java
@@ -49,7 +49,7 @@ import org.apache.accumulo.core.data.Value;
 import org.apache.accumulo.core.dataImpl.KeyExtent;
 import org.apache.accumulo.core.manager.state.tables.TableState;
 import org.apache.accumulo.core.metadata.MetadataTable;
-import org.apache.accumulo.core.metadata.TabletFileUtil;
+import org.apache.accumulo.core.metadata.ValidationUtil;
 import org.apache.accumulo.core.metadata.schema.MetadataSchema.TabletsSection.CurrentLocationColumnFamily;
 import org.apache.accumulo.core.metadata.schema.MetadataSchema.TabletsSection.DataFileColumnFamily;
 import org.apache.accumulo.core.metadata.schema.MetadataSchema.TabletsSection.FutureLocationColumnFamily;
@@ -234,7 +234,7 @@ class WriteExportFiles extends ManagerRepo {
       entry.getValue().write(dataOut);
 
       if (entry.getKey().getColumnFamily().equals(DataFileColumnFamily.NAME)) {
-        String path = TabletFileUtil.validate(entry.getKey().getColumnQualifierData().toString());
+        String path = ValidationUtil.validate(entry.getKey().getColumnQualifierData().toString());
         String[] tokens = path.split("/");
         if (tokens.length < 1) {
           throw new RuntimeException("Illegal path " + path);
diff --git a/server/manager/src/main/java/org/apache/accumulo/manager/upgrade/Upgrader9to10.java b/server/manager/src/main/java/org/apache/accumulo/manager/upgrade/Upgrader9to10.java
index 62a2cecc61..dc0bd442b7 100644
--- a/server/manager/src/main/java/org/apache/accumulo/manager/upgrade/Upgrader9to10.java
+++ b/server/manager/src/main/java/org/apache/accumulo/manager/upgrade/Upgrader9to10.java
@@ -52,6 +52,7 @@ import org.apache.accumulo.core.data.Value;
 import org.apache.accumulo.core.dataImpl.KeyExtent;
 import org.apache.accumulo.core.file.FileOperations;
 import org.apache.accumulo.core.file.FileSKVIterator;
+import org.apache.accumulo.core.gc.Reference;
 import org.apache.accumulo.core.metadata.MetadataTable;
 import org.apache.accumulo.core.metadata.RootTable;
 import org.apache.accumulo.core.metadata.TServerInstance;
@@ -481,7 +482,7 @@ public class Upgrader9to10 implements Upgrader {
           log.trace("upgrading delete entry for {}", olddelete);
 
           Path absolutePath = resolveRelativeDelete(olddelete, upgradeProp);
-          String updatedDel = switchToAllVolumes(absolutePath);
+          Reference updatedDel = switchToAllVolumes(absolutePath);
 
           writer.addMutation(ample.createDeleteMutation(updatedDel));
         }
@@ -507,16 +508,29 @@ public class Upgrader9to10 implements Upgrader {
    * "tables/5a/t-0005/A0012.rf" depth = 4 will be returned as is.
    */
   @VisibleForTesting
-  static String switchToAllVolumes(Path olddelete) {
+  static Reference switchToAllVolumes(Path olddelete) {
     Path pathNoVolume = Objects.requireNonNull(VolumeManager.FileType.TABLE.removeVolume(olddelete),
         "Invalid delete marker. No volume in path: " + olddelete);
 
-    // a directory path with volume removed will have a depth of 3 so change volume to all volumes
-    if (pathNoVolume.depth() == 3 && !pathNoVolume.getName().startsWith(Constants.BULK_PREFIX)) {
-      return GcVolumeUtil.getDeleteTabletOnAllVolumesUri(
-          TableId.of(pathNoVolume.getParent().getName()), pathNoVolume.getName());
+    // a directory path with volume removed will have a depth of 3 like, "tables/5a/t-0005"
+    if (pathNoVolume.depth() == 3) {
+      String tabletDir = pathNoVolume.getName();
+      var tableId = TableId.of(pathNoVolume.getParent().getName());
+      // except bulk directories don't get an all volume prefix
+      if (pathNoVolume.getName().startsWith(Constants.BULK_PREFIX)) {
+        return new Reference(tableId, olddelete.toString());
+      } else {
+        return GcVolumeUtil.getDeleteTabletOnAllVolumesUri(tableId, tabletDir);
+      }
     } else {
-      return olddelete.toString();
+      // depth of 4 should be a file like, "tables/5a/t-0005/A0012.rf"
+      if (pathNoVolume.depth() == 4) {
+        Path tabletDirPath = pathNoVolume.getParent();
+        var tableId = TableId.of(tabletDirPath.getParent().getName());
+        return new Reference(tableId, olddelete.toString());
+      } else {
+        throw new IllegalStateException("Invalid delete marker: " + olddelete);
+      }
     }
   }
 
diff --git a/server/manager/src/test/java/org/apache/accumulo/manager/upgrade/Upgrader9to10Test.java b/server/manager/src/test/java/org/apache/accumulo/manager/upgrade/Upgrader9to10Test.java
index 0dafe696ff..60e196d11d 100644
--- a/server/manager/src/test/java/org/apache/accumulo/manager/upgrade/Upgrader9to10Test.java
+++ b/server/manager/src/test/java/org/apache/accumulo/manager/upgrade/Upgrader9to10Test.java
@@ -51,6 +51,7 @@ import org.apache.accumulo.core.data.Key;
 import org.apache.accumulo.core.data.Mutation;
 import org.apache.accumulo.core.data.TableId;
 import org.apache.accumulo.core.data.Value;
+import org.apache.accumulo.core.gc.Reference;
 import org.apache.accumulo.core.metadata.schema.Ample;
 import org.apache.accumulo.core.metadata.schema.MetadataSchema.TabletsSection.DataFileColumnFamily;
 import org.apache.accumulo.core.security.Authorizations;
@@ -72,23 +73,32 @@ public class Upgrader9to10Test {
   private static Logger log = LoggerFactory.getLogger(Upgrader9to10Test.class);
 
   private static final String VOL_PROP = "hdfs://nn1:8020/accumulo";
+  private static final TableId tableId5a = TableId.of("5a");
 
   @Test
   public void testSwitchRelativeDeletes() {
     Path resolved = Upgrader9to10.resolveRelativeDelete("/5a/t-0005", VOL_PROP);
     assertEquals(new Path(VOL_PROP + "/tables/5a/t-0005"), resolved);
-    assertEquals(GcVolumeUtil.getDeleteTabletOnAllVolumesUri(TableId.of("5a"), "t-0005"),
-        Upgrader9to10.switchToAllVolumes(resolved));
+    var ref1 = GcVolumeUtil.getDeleteTabletOnAllVolumesUri(tableId5a, "t-0005");
+    var ref2 = Upgrader9to10.switchToAllVolumes(resolved);
+    compareReferences(ref1, ref2);
 
     resolved = Upgrader9to10.resolveRelativeDelete("/5a/" + BULK_PREFIX + "0005", VOL_PROP);
     assertEquals(new Path(VOL_PROP + "/tables/5a/" + BULK_PREFIX + "0005"), resolved);
-    assertEquals(VOL_PROP + "/tables/5a/" + BULK_PREFIX + "0005",
-        Upgrader9to10.switchToAllVolumes(resolved));
+    ref1 = new Reference(tableId5a, VOL_PROP + "/tables/5a/" + BULK_PREFIX + "0005");
+    ref2 = Upgrader9to10.switchToAllVolumes(resolved);
+    compareReferences(ref1, ref2);
 
     resolved = Upgrader9to10.resolveRelativeDelete("/5a/t-0005/F0009.rf", VOL_PROP);
     assertEquals(new Path(VOL_PROP + "/tables/5a/t-0005/F0009.rf"), resolved);
-    assertEquals(VOL_PROP + "/tables/5a/t-0005/F0009.rf",
-        Upgrader9to10.switchToAllVolumes(resolved));
+    ref1 = new Reference(tableId5a, VOL_PROP + "/tables/5a/t-0005/F0009.rf");
+    ref2 = Upgrader9to10.switchToAllVolumes(resolved);
+    compareReferences(ref1, ref2);
+  }
+
+  private void compareReferences(Reference ref1, Reference ref2) {
+    assertEquals(ref1.metadataEntry, ref2.metadataEntry);
+    assertEquals(ref1.tabletDir, ref2.tabletDir);
   }
 
   @Test
@@ -107,18 +117,22 @@ public class Upgrader9to10Test {
   public void testSwitchAllVolumes() {
     Path resolved = Upgrader9to10
         .resolveRelativeDelete("hdfs://localhost:9000/accumulo/tables/5a/t-0005", VOL_PROP);
-    assertEquals(GcVolumeUtil.getDeleteTabletOnAllVolumesUri(TableId.of("5a"), "t-0005"),
-        Upgrader9to10.switchToAllVolumes(resolved));
+    var ref1 = GcVolumeUtil.getDeleteTabletOnAllVolumesUri(tableId5a, "t-0005");
+    var ref2 = Upgrader9to10.switchToAllVolumes(resolved);
+    compareReferences(ref1, ref2);
 
     resolved = Upgrader9to10.resolveRelativeDelete(
         "hdfs://localhost:9000/accumulo/tables/5a/" + BULK_PREFIX + "0005", VOL_PROP);
-    assertEquals("hdfs://localhost:9000/accumulo/tables/5a/" + BULK_PREFIX + "0005",
-        Upgrader9to10.switchToAllVolumes(resolved));
+    ref1 = new Reference(tableId5a,
+        "hdfs://localhost:9000/accumulo/tables/5a/" + BULK_PREFIX + "0005");
+    ref2 = Upgrader9to10.switchToAllVolumes(resolved);
+    compareReferences(ref1, ref2);
 
     resolved = Upgrader9to10.resolveRelativeDelete(
         "hdfs://localhost:9000/accumulo/tables/5a/t-0005/C0009.rf", VOL_PROP);
-    assertEquals("hdfs://localhost:9000/accumulo/tables/5a/t-0005/C0009.rf",
-        Upgrader9to10.switchToAllVolumes(resolved));
+    ref1 = new Reference(tableId5a, "hdfs://localhost:9000/accumulo/tables/5a/t-0005/C0009.rf");
+    ref2 = Upgrader9to10.switchToAllVolumes(resolved);
+    compareReferences(ref1, ref2);
   }
 
   @Test
diff --git a/test/src/main/java/org/apache/accumulo/test/functional/GarbageCollectorIT.java b/test/src/main/java/org/apache/accumulo/test/functional/GarbageCollectorIT.java
index cbe43ccba4..cc5773e3b5 100644
--- a/test/src/main/java/org/apache/accumulo/test/functional/GarbageCollectorIT.java
+++ b/test/src/main/java/org/apache/accumulo/test/functional/GarbageCollectorIT.java
@@ -40,7 +40,9 @@ import org.apache.accumulo.core.client.Scanner;
 import org.apache.accumulo.core.conf.Property;
 import org.apache.accumulo.core.data.Key;
 import org.apache.accumulo.core.data.Mutation;
+import org.apache.accumulo.core.data.TableId;
 import org.apache.accumulo.core.data.Value;
+import org.apache.accumulo.core.gc.Reference;
 import org.apache.accumulo.core.metadata.MetadataTable;
 import org.apache.accumulo.core.metadata.schema.Ample;
 import org.apache.accumulo.core.metadata.schema.MetadataSchema.DeletesSection;
@@ -308,7 +310,8 @@ public class GarbageCollectorIT extends ConfigurableMacBase {
       for (int i = 0; i < 100000; ++i) {
         String longpath = "aaaaaaaaaabbbbbbbbbbccccccccccddddddddddeeeeeeeeee"
             + "ffffffffffgggggggggghhhhhhhhhhiiiiiiiiiijjjjjjjjjj";
-        Mutation delFlag = ample.createDeleteMutation(String.format("file:/%020d/%s", i, longpath));
+        var path = String.format("file:/%020d/%s", i, longpath);
+        Mutation delFlag = ample.createDeleteMutation(new Reference(TableId.of("1"), path));
         bw.addMutation(delFlag);
       }
     }
diff --git a/test/src/main/java/org/apache/accumulo/test/upgrade/GCUpgrade9to10TestIT.java b/test/src/main/java/org/apache/accumulo/test/upgrade/GCUpgrade9to10TestIT.java
index bbb952ae9a..f8fd97ce75 100644
--- a/test/src/main/java/org/apache/accumulo/test/upgrade/GCUpgrade9to10TestIT.java
+++ b/test/src/main/java/org/apache/accumulo/test/upgrade/GCUpgrade9to10TestIT.java
@@ -241,7 +241,8 @@ public class GCUpgrade9to10TestIT extends ConfigurableMacBase {
         Mutation delFlag = createOldDelMutation(longpath, "", "", "");
         bw.addMutation(delFlag);
         expected.put(
-            DeletesSection.encodeRow(GcVolumeUtil.getDeleteTabletOnAllVolumesUri(tableId, dirName)),
+            DeletesSection.encodeRow(
+                GcVolumeUtil.getDeleteTabletOnAllVolumesUri(tableId, dirName).metadataEntry),
             Upgrader9to10.UPGRADED.toString());
       }