Class HRegionFileSystem

java.lang.Object
org.apache.hadoop.hbase.regionserver.HRegionFileSystem
Direct Known Subclasses:
HRegionWALFileSystem

@Private public class HRegionFileSystem extends Object
View to an on-disk Region. Provides the set of methods necessary to interact with the on-disk region data.
  • Field Details

  • Constructor Details

    • HRegionFileSystem

      HRegionFileSystem(org.apache.hadoop.conf.Configuration conf, org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.fs.Path tableDir, RegionInfo regionInfo)
      Create a view to the on-disk region
      Parameters:
      conf - the Configuration to use
      fs - FileSystem that contains the region
      tableDir - Path to where the table is being stored
      regionInfo - RegionInfo for region
  • Method Details

    • getFileSystem

      public org.apache.hadoop.fs.FileSystem getFileSystem()
      Returns the underlying FileSystem
    • getRegionInfo

      Returns the RegionInfo that describe this on-disk region view
    • getRegionInfoForFS

    • getTableDir

      public org.apache.hadoop.fs.Path getTableDir()
      Returns Path to the region's root directory.
    • getRegionDir

      public org.apache.hadoop.fs.Path getRegionDir()
      Returns Path to the region directory.
    • getTempDir

      public org.apache.hadoop.fs.Path getTempDir()
      Returns Path to the region's temp directory, used for file creations
    • cleanupTempDir

      Clean up any temp detritus that may have been left around from previous operation attempts.
      Throws:
      IOException
    • getStoreDir

      public org.apache.hadoop.fs.Path getStoreDir(String familyName)
      Returns the directory path of the specified family
      Parameters:
      familyName - Column Family Name
      Returns:
      Path to the directory of the specified family
    • getStoreHomedir

      public static org.apache.hadoop.fs.Path getStoreHomedir(org.apache.hadoop.fs.Path tabledir, RegionInfo hri, byte[] family)
      Parameters:
      tabledir - Path to where the table is being stored
      hri - RegionInfo for the region.
      family - ColumnFamilyDescriptor describing the column family
      Returns:
      Path to family/Store home directory.
    • getStoreHomedir

      public static org.apache.hadoop.fs.Path getStoreHomedir(org.apache.hadoop.fs.Path tabledir, String encodedName, byte[] family)
      Parameters:
      tabledir - Path to where the table is being stored
      encodedName - Encoded region name.
      family - ColumnFamilyDescriptor describing the column family
      Returns:
      Path to family/Store home directory.
    • createStoreDir

      org.apache.hadoop.fs.Path createStoreDir(String familyName) throws IOException
      Create the store directory for the specified family name
      Parameters:
      familyName - Column Family Name
      Returns:
      Path to the directory of the specified family
      Throws:
      IOException - if the directory creation fails.
    • setStoragePolicy

      public void setStoragePolicy(String familyName, String policyName)
      Set the directory of CF to the specified storage policy.
      "LAZY_PERSIST", "ALL_SSD", "ONE_SSD", "HOT", "WARM", "COLD"

      See HdfsConstants for more details.
      Parameters:
      familyName - The name of column family.
      policyName - The name of the storage policy: 'HOT', 'COLD', etc. See hadoop 2.6+ org.apache.hadoop.hdfs.protocol.HdfsConstants for possible list e.g 'COLD', 'WARM', 'HOT', 'ONE_SSD', 'ALL_SSD', 'LAZY_PERSIST'.
    • setStoragePolicy

      public void setStoragePolicy(String policyName)
      Set storage policy for a whole region.
      "LAZY_PERSIST", "ALL_SSD", "ONE_SSD", "HOT", "WARM", "COLD"

      See HdfsConstants for more details.
      Parameters:
      policyName - The name of the storage policy: 'HOT', 'COLD', etc. See hadoop 2.6+ org.apache.hadoop.hdfs.protocol.HdfsConstants for possible list e.g 'COLD', 'WARM', 'HOT', 'ONE_SSD', 'ALL_SSD', 'LAZY_PERSIST'.
    • getStoragePolicyName

      @Nullable public String getStoragePolicyName(String familyName)
      Get the storage policy of the directory of CF.
      Parameters:
      familyName - The name of column family.
      Returns:
      Storage policy name, or null if not using HFileSystem or exception thrown when trying to get policy
    • getStoreFiles

      public List<StoreFileInfo> getStoreFiles(String familyName) throws IOException
      Returns the store files available for the family. This methods performs the filtering based on the valid store files.
      Parameters:
      familyName - Column Family Name
      Returns:
      a set of StoreFileInfo for the specified family.
      Throws:
      IOException
    • getStoreFiles

      public List<StoreFileInfo> getStoreFiles(String familyName, boolean validate) throws IOException
      Returns the store files available for the family. This methods performs the filtering based on the valid store files.
      Parameters:
      familyName - Column Family Name
      Returns:
      a set of StoreFileInfo for the specified family.
      Throws:
      IOException
    • getStoreFilesLocatedStatus

      public static List<org.apache.hadoop.fs.LocatedFileStatus> getStoreFilesLocatedStatus(HRegionFileSystem regionfs, String familyName, boolean validate) throws IOException
      Returns the store files' LocatedFileStatus which available for the family. This methods performs the filtering based on the valid store files.
      Parameters:
      familyName - Column Family Name
      Returns:
      a list of store files' LocatedFileStatus for the specified family.
      Throws:
      IOException
    • getStoreFilePath

      org.apache.hadoop.fs.Path getStoreFilePath(String familyName, String fileName)
      Return Qualified Path of the specified family/file
      Parameters:
      familyName - Column Family Name
      fileName - File Name
      Returns:
      The qualified Path for the specified family/file
    • getStoreFileInfo

      StoreFileInfo getStoreFileInfo(String familyName, String fileName) throws IOException
      Return the store file information of the specified family/file.
      Parameters:
      familyName - Column Family Name
      fileName - File Name
      Returns:
      The StoreFileInfo for the specified family/file
      Throws:
      IOException
    • hasReferences

      public boolean hasReferences(String familyName) throws IOException
      Returns true if the specified family has reference files
      Parameters:
      familyName - Column Family Name
      Returns:
      true if family contains reference files
      Throws:
      IOException
    • hasReferences

      public boolean hasReferences(TableDescriptor htd) throws IOException
      Check whether region has Reference file
      Parameters:
      htd - table desciptor of the region
      Returns:
      true if region has reference file
      Throws:
      IOException
    • getFamilies

      Returns the set of families present on disk n
      Throws:
      IOException
    • deleteFamily

      public void deleteFamily(String familyName) throws IOException
      Remove the region family from disk, archiving the store files.
      Parameters:
      familyName - Column Family Name
      Throws:
      IOException - if an error occours during the archiving
    • generateUniqueName

      private static String generateUniqueName(String suffix)
      Generate a unique file name, used by createTempName() and commitStoreFile()
      Parameters:
      suffix - extra information to append to the generated name
      Returns:
      Unique file name
    • createTempName

      public org.apache.hadoop.fs.Path createTempName()
      Generate a unique temporary Path. Used in conjuction with commitStoreFile() to get a safer file creation. Path file = fs.createTempName(); ...StoreFile.Writer(file)... fs.commitStoreFile("family", file);
      Returns:
      Unique Path of the temporary file
    • createTempName

      public org.apache.hadoop.fs.Path createTempName(String suffix)
      Generate a unique temporary Path. Used in conjuction with commitStoreFile() to get a safer file creation. Path file = fs.createTempName(); ...StoreFile.Writer(file)... fs.commitStoreFile("family", file);
      Parameters:
      suffix - extra information to append to the generated name
      Returns:
      Unique Path of the temporary file
    • commitStoreFile

      public org.apache.hadoop.fs.Path commitStoreFile(String familyName, org.apache.hadoop.fs.Path buildPath) throws IOException
      Move the file from a build/temp location to the main family store directory.
      Parameters:
      familyName - Family that will gain the file
      buildPath - Path to the file to commit.
      Returns:
      The new Path of the committed file
      Throws:
      IOException
    • preCommitStoreFile

      private org.apache.hadoop.fs.Path preCommitStoreFile(String familyName, org.apache.hadoop.fs.Path buildPath, long seqNum, boolean generateNewName) throws IOException
      Generate the filename in the main family store directory for moving the file from a build/temp location.
      Parameters:
      familyName - Family that will gain the file
      buildPath - Path to the file to commit.
      seqNum - Sequence Number to append to the file name (less then 0 if no sequence number)
      generateNewName - False if you want to keep the buildPath name
      Returns:
      The new Path of the to be committed file
      Throws:
      IOException
    • commitStoreFile

      org.apache.hadoop.fs.Path commitStoreFile(org.apache.hadoop.fs.Path buildPath, org.apache.hadoop.fs.Path dstPath) throws IOException
      Throws:
      IOException
    • removeStoreFile

      public void removeStoreFile(String familyName, org.apache.hadoop.fs.Path filePath) throws IOException
      Archives the specified store file from the specified family.
      Parameters:
      familyName - Family that contains the store files
      filePath - Path to the store file to remove
      Throws:
      IOException - if the archiving fails
    • removeStoreFiles

      public void removeStoreFiles(String familyName, Collection<HStoreFile> storeFiles) throws IOException
      Closes and archives the specified store files from the specified family.
      Parameters:
      familyName - Family that contains the store files
      storeFiles - set of store files to remove
      Throws:
      IOException - if the archiving fails
    • bulkLoadStoreFile

      Pair<org.apache.hadoop.fs.Path,org.apache.hadoop.fs.Path> bulkLoadStoreFile(String familyName, org.apache.hadoop.fs.Path srcPath, long seqNum) throws IOException
      Bulk load: Add a specified store file to the specified family. If the source file is on the same different file-system is moved from the source location to the destination location, otherwise is copied over.
      Parameters:
      familyName - Family that will gain the file
      srcPath - Path to the file to import
      seqNum - Bulk Load sequence number
      Returns:
      The destination Path of the bulk loaded file
      Throws:
      IOException
    • getSplitsDir

      public org.apache.hadoop.fs.Path getSplitsDir(RegionInfo hri)
    • cleanupDaughterRegion

      void cleanupDaughterRegion(RegionInfo regionInfo) throws IOException
      Remove daughter region
      Parameters:
      regionInfo - daughter RegionInfo
      Throws:
      IOException
    • commitDaughterRegion

      public org.apache.hadoop.fs.Path commitDaughterRegion(RegionInfo regionInfo, List<org.apache.hadoop.fs.Path> allRegionFiles, MasterProcedureEnv env) throws IOException
      Commit a daughter region, moving it from the split temporary directory to the proper location in the filesystem.
      Parameters:
      regionInfo - daughter RegionInfo
      Throws:
      IOException
    • insertRegionFilesIntoStoreTracker

      private void insertRegionFilesIntoStoreTracker(List<org.apache.hadoop.fs.Path> allFiles, MasterProcedureEnv env, HRegionFileSystem regionFs) throws IOException
      Throws:
      IOException
    • createSplitsDir

      public void createSplitsDir(RegionInfo daughterA, RegionInfo daughterB) throws IOException
      Creates region split daughter directories under the table dir. If the daughter regions already exist, for example, in the case of a recovery from a previous failed split procedure, this method deletes the given region dir recursively, then recreates it again.
      Throws:
      IOException
    • splitStoreFile

      public org.apache.hadoop.fs.Path splitStoreFile(RegionInfo hri, String familyName, HStoreFile f, byte[] splitRow, boolean top, RegionSplitPolicy splitPolicy) throws IOException
      Write out a split reference. Package local so it doesnt leak out of regionserver.
      Parameters:
      hri - RegionInfo of the destination
      familyName - Column Family Name
      f - File to split.
      splitRow - Split Row
      top - True if we are referring to the top half of the hfile.
      splitPolicy - A split policy instance; be careful! May not be full populated; e.g. if this method is invoked on the Master side, then the RegionSplitPolicy will NOT have a reference to a Region.
      Returns:
      Path to created reference.
      Throws:
      IOException
    • getMergesDir

      org.apache.hadoop.fs.Path getMergesDir(RegionInfo hri)
    • cleanupMergedRegion

      public void cleanupMergedRegion(RegionInfo mergedRegion) throws IOException
      Remove merged region
      Parameters:
      mergedRegion - RegionInfo
      Throws:
      IOException
    • mkdirs

      static boolean mkdirs(org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.conf.Configuration conf, org.apache.hadoop.fs.Path dir) throws IOException
      Throws:
      IOException
    • mergeStoreFile

      public org.apache.hadoop.fs.Path mergeStoreFile(RegionInfo mergingRegion, String familyName, HStoreFile f) throws IOException
      Write out a merge reference under the given merges directory.
      Parameters:
      mergingRegion - RegionInfo for one of the regions being merged.
      familyName - Column Family Name
      f - File to create reference.
      Returns:
      Path to created reference.
      Throws:
      IOException - if the merge write fails.
    • commitMergedRegion

      public void commitMergedRegion(List<org.apache.hadoop.fs.Path> allMergedFiles, MasterProcedureEnv env) throws IOException
      Commit a merged region, making it ready for use.
      Throws:
      IOException
    • getRegionInfoFileContent

      private static byte[] getRegionInfoFileContent(RegionInfo hri) throws IOException
      Returns Content of the file we write out to the filesystem under a region
      Throws:
      IOException
    • loadRegionInfoFileContent

      public static RegionInfo loadRegionInfoFileContent(org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.fs.Path regionDir) throws IOException
      Create a RegionInfo from the serialized version on-disk.
      Parameters:
      fs - FileSystem that contains the Region Info file
      regionDir - Path to the Region Directory that contains the Info file
      Returns:
      An RegionInfo instance gotten from the Region Info file.
      Throws:
      IOException - if an error occurred during file open/read operation.
    • writeRegionInfoFileContent

      private static void writeRegionInfoFileContent(org.apache.hadoop.conf.Configuration conf, org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.fs.Path regionInfoFile, byte[] content) throws IOException
      Write the .regioninfo file on-disk.

      Overwrites if exists already.

      Throws:
      IOException
    • checkRegionInfoOnFilesystem

      Write out an info file under the stored region directory. Useful recovering mangled regions. If the regionInfo already exists on-disk, then we fast exit.
      Throws:
      IOException
    • writeRegionInfoOnFilesystem

      private void writeRegionInfoOnFilesystem(boolean useTempDir) throws IOException
      Write out an info file under the region directory. Useful recovering mangled regions.
      Parameters:
      useTempDir - indicate whether or not using the region .tmp dir for a safer file creation.
      Throws:
      IOException
    • writeRegionInfoOnFilesystem

      private void writeRegionInfoOnFilesystem(byte[] regionInfoContent, boolean useTempDir) throws IOException
      Write out an info file under the region directory. Useful recovering mangled regions.
      Parameters:
      regionInfoContent - serialized version of the RegionInfo
      useTempDir - indicate whether or not using the region .tmp dir for a safer file creation.
      Throws:
      IOException
    • createRegionOnFileSystem

      public static HRegionFileSystem createRegionOnFileSystem(org.apache.hadoop.conf.Configuration conf, org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.fs.Path tableDir, RegionInfo regionInfo) throws IOException
      Create a new Region on file-system.
      Parameters:
      conf - the Configuration to use
      fs - FileSystem from which to add the region
      tableDir - Path to where the table is being stored
      regionInfo - RegionInfo for region to be added
      Throws:
      IOException - if the region creation fails due to a FileSystem exception.
    • openRegionFromFileSystem

      public static HRegionFileSystem openRegionFromFileSystem(org.apache.hadoop.conf.Configuration conf, org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.fs.Path tableDir, RegionInfo regionInfo, boolean readOnly) throws IOException
      Open Region from file-system.
      Parameters:
      conf - the Configuration to use
      fs - FileSystem from which to add the region
      tableDir - Path to where the table is being stored
      regionInfo - RegionInfo for region to be added
      readOnly - True if you don't want to edit the region data
      Throws:
      IOException - if the region creation fails due to a FileSystem exception.
    • deleteRegionFromFileSystem

      public static void deleteRegionFromFileSystem(org.apache.hadoop.conf.Configuration conf, org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.fs.Path tableDir, RegionInfo regionInfo) throws IOException
      Remove the region from the table directory, archiving the region's hfiles.
      Parameters:
      conf - the Configuration to use
      fs - FileSystem from which to remove the region
      tableDir - Path to where the table is being stored
      regionInfo - RegionInfo for region to be deleted
      Throws:
      IOException - if the request cannot be completed
    • createDir

      boolean createDir(org.apache.hadoop.fs.Path dir) throws IOException
      Creates a directory. Assumes the user has already checked for this directory existence.
      Returns:
      the result of fs.mkdirs(). In case underlying fs throws an IOException, it checks whether the directory exists or not, and returns true if it exists.
      Throws:
      IOException
    • rename

      boolean rename(org.apache.hadoop.fs.Path srcpath, org.apache.hadoop.fs.Path dstPath) throws IOException
      Renames a directory. Assumes the user has already checked for this directory existence.
      Returns:
      true if rename is successful.
      Throws:
      IOException
    • deleteDir

      boolean deleteDir(org.apache.hadoop.fs.Path dir) throws IOException
      Deletes a directory. Assumes the user has already checked for this directory existence.
      Returns:
      true if the directory is deleted.
      Throws:
      IOException
    • sleepBeforeRetry

      private void sleepBeforeRetry(String msg, int sleepMultiplier) throws InterruptedException
      sleeping logic; handles the interrupt exception.
      Throws:
      InterruptedException
    • createDirOnFileSystem

      private static boolean createDirOnFileSystem(org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.conf.Configuration conf, org.apache.hadoop.fs.Path dir) throws IOException
      Creates a directory for a filesystem and configuration object. Assumes the user has already checked for this directory existence.
      Returns:
      the result of fs.mkdirs(). In case underlying fs throws an IOException, it checks whether the directory exists or not, and returns true if it exists.
      Throws:
      IOException
    • sleepBeforeRetry

      private static void sleepBeforeRetry(String msg, int sleepMultiplier, int baseSleepBeforeRetries, int hdfsClientRetriesNumber) throws InterruptedException
      sleeping logic for static methods; handles the interrupt exception. Keeping a static version for this to avoid re-looking for the integer values.
      Throws:
      InterruptedException