This is an automated email from the ASF dual-hosted git repository.
brandonwilliams pushed a commit to branch trunk
in repository https://gitbox.apache.org/repos/asf/cassandra.git
The following commit(s) were added to refs/heads/trunk by this push:
new f3198c4 Verify sstable components on startup
f3198c4 is described below
commit f3198c45067f753444ffafbc83b6aa563cb392ac
Author: yifan-c <[email protected]>
AuthorDate: Thu Jul 16 13:12:11 2020 -0700
Verify sstable components on startup
Patch by Yifan Cai, reviewed by David Capwell and brandonwilliams for
CASSANDRA-15945
---
CHANGES.txt | 2 +
.../org/apache/cassandra/io/sstable/SSTable.java | 12 +++-
.../cassandra/io/sstable/format/SSTableReader.java | 36 +++++++++++-
.../cassandra/io/sstable/SSTableReaderTest.java | 67 ++++++++++++++++++++++
4 files changed, 115 insertions(+), 2 deletions(-)
diff --git a/CHANGES.txt b/CHANGES.txt
index 10079b3..afb3d09 100644
--- a/CHANGES.txt
+++ b/CHANGES.txt
@@ -1,3 +1,5 @@
+4.0-beta2
+ * Verify sstable components on startup (CASSANDRA-15945)
4.0-beta1
* Remove BackPressureStrategy (CASSANDRA-15375)
* Improve messaging on indexing frozen collections (CASSANDRA-15908)
diff --git a/src/java/org/apache/cassandra/io/sstable/SSTable.java
b/src/java/org/apache/cassandra/io/sstable/SSTable.java
index 348d7f5..353b624 100644
--- a/src/java/org/apache/cassandra/io/sstable/SSTable.java
+++ b/src/java/org/apache/cassandra/io/sstable/SSTable.java
@@ -306,13 +306,23 @@ public abstract class SSTable
*/
protected static Set<Component> readTOC(Descriptor descriptor) throws
IOException
{
+ return readTOC(descriptor, true);
+ }
+
+ /**
+ * Reads the list of components from the TOC component.
+ * @param skipMissing, skip adding the component to the returned set if
the corresponding file is missing.
+ * @return set of components found in the TOC
+ */
+ protected static Set<Component> readTOC(Descriptor descriptor, boolean
skipMissing) throws IOException
+ {
File tocFile = new File(descriptor.filenameFor(Component.TOC));
List<String> componentNames = Files.readLines(tocFile,
Charset.defaultCharset());
Set<Component> components =
Sets.newHashSetWithExpectedSize(componentNames.size());
for (String componentName : componentNames)
{
Component component = new
Component(Component.Type.fromRepresentation(componentName), componentName);
- if (!new File(descriptor.filenameFor(component)).exists())
+ if (skipMissing && !new
File(descriptor.filenameFor(component)).exists())
logger.error("Missing component: {}",
descriptor.filenameFor(component));
else
components.add(component);
diff --git a/src/java/org/apache/cassandra/io/sstable/format/SSTableReader.java
b/src/java/org/apache/cassandra/io/sstable/format/SSTableReader.java
index 04c4826..c7e0201 100644
--- a/src/java/org/apache/cassandra/io/sstable/format/SSTableReader.java
+++ b/src/java/org/apache/cassandra/io/sstable/format/SSTableReader.java
@@ -58,12 +58,12 @@ import org.apache.cassandra.dht.Range;
import org.apache.cassandra.dht.Token;
import org.apache.cassandra.exceptions.UnknownColumnException;
import org.apache.cassandra.io.FSError;
+import org.apache.cassandra.io.FSReadError;
import org.apache.cassandra.io.compress.CompressionMetadata;
import org.apache.cassandra.io.sstable.*;
import org.apache.cassandra.io.sstable.metadata.*;
import org.apache.cassandra.io.util.*;
import org.apache.cassandra.metrics.RestorableMeter;
-import org.apache.cassandra.metrics.StorageMetrics;
import org.apache.cassandra.schema.CachingParams;
import org.apache.cassandra.schema.Schema;
import org.apache.cassandra.schema.SchemaConstants;
@@ -402,6 +402,7 @@ public abstract class SSTableReader extends SSTable
implements SelfRefCounted<SS
// Minimum components without which we can't do anything
assert components.contains(Component.DATA) : "Data component is
missing for sstable " + descriptor;
assert components.contains(Component.PRIMARY_INDEX) : "Primary index
component is missing for sstable " + descriptor;
+ verifyCompressionInfoExistenceIfApplicable(descriptor, components);
EnumSet<MetadataType> types = EnumSet.of(MetadataType.VALIDATION,
MetadataType.STATS, MetadataType.HEADER);
Map<MetadataType, MetadataComponent> sstableMetadata;
@@ -501,6 +502,8 @@ public abstract class SSTableReader extends SSTable
implements SelfRefCounted<SS
// For the 3.0+ sstable format, the (misnomed) stats component hold
the serialization header which we need to deserialize the sstable content
assert components.contains(Component.STATS) : "Stats component is
missing for sstable " + descriptor;
+ verifyCompressionInfoExistenceIfApplicable(descriptor, components);
+
EnumSet<MetadataType> types = EnumSet.of(MetadataType.VALIDATION,
MetadataType.STATS, MetadataType.HEADER);
Map<MetadataType, MetadataComponent> sstableMetadata;
@@ -661,6 +664,37 @@ public abstract class SSTableReader extends SSTable
implements SelfRefCounted<SS
return readerFactory.open(descriptor, components, metadata,
maxDataAge, sstableMetadata, openReason, header);
}
+ /**
+ * Best-effort checking to verify the expected compression info component
exists, according to the TOC file.
+ * The verification depends on the existence of TOC file. If absent, the
verification is skipped.
+ * @param descriptor
+ * @param actualComponents, actual components listed from the file system.
+ * @throws CorruptSSTableException, if TOC expects compression info but
not found from disk.
+ * @throws FSReadError, if unable to read from TOC file.
+ */
+ public static void verifyCompressionInfoExistenceIfApplicable(Descriptor
descriptor,
+
Set<Component> actualComponents)
+ throws CorruptSSTableException, FSReadError
+ {
+ File tocFile = new File(descriptor.filenameFor(Component.TOC));
+ if (tocFile.exists())
+ {
+ try
+ {
+ Set<Component> expectedComponents = readTOC(descriptor, false);
+ if (expectedComponents.contains(Component.COMPRESSION_INFO) &&
!actualComponents.contains(Component.COMPRESSION_INFO))
+ {
+ String compressionInfoFileName =
descriptor.filenameFor(Component.COMPRESSION_INFO);
+ throw new CorruptSSTableException(new
FileNotFoundException(compressionInfoFileName), compressionInfoFileName);
+ }
+ }
+ catch (IOException e)
+ {
+ throw new FSReadError(e, tocFile);
+ }
+ }
+ }
+
protected SSTableReader(final Descriptor desc,
Set<Component> components,
TableMetadataRef metadata,
diff --git a/test/unit/org/apache/cassandra/io/sstable/SSTableReaderTest.java
b/test/unit/org/apache/cassandra/io/sstable/SSTableReaderTest.java
index 580b099..36221ed 100644
--- a/test/unit/org/apache/cassandra/io/sstable/SSTableReaderTest.java
+++ b/test/unit/org/apache/cassandra/io/sstable/SSTableReaderTest.java
@@ -18,6 +18,7 @@
package org.apache.cassandra.io.sstable;
import java.io.File;
+import java.io.FileNotFoundException;
import java.io.IOException;
import java.nio.ByteBuffer;
import java.nio.file.Files;
@@ -28,7 +29,9 @@ import java.util.concurrent.*;
import com.google.common.collect.Sets;
import org.junit.BeforeClass;
+import org.junit.Rule;
import org.junit.Test;
+import org.junit.rules.ExpectedException;
import org.junit.runner.RunWith;
import org.apache.cassandra.OrderedJUnit4ClassRunner;
@@ -48,10 +51,12 @@ import org.apache.cassandra.dht.LocalPartitioner.LocalToken;
import org.apache.cassandra.dht.Range;
import org.apache.cassandra.dht.Token;
import org.apache.cassandra.index.Index;
+import org.apache.cassandra.io.FSReadError;
import org.apache.cassandra.io.sstable.format.SSTableReader;
import org.apache.cassandra.io.util.FileDataInput;
import org.apache.cassandra.io.util.MmappedRegions;
import org.apache.cassandra.schema.CachingParams;
+import org.apache.cassandra.schema.CompressionParams;
import org.apache.cassandra.schema.KeyspaceParams;
import org.apache.cassandra.service.CacheService;
import org.apache.cassandra.utils.ByteBufferUtil;
@@ -69,6 +74,7 @@ public class SSTableReaderTest
public static final String KEYSPACE1 = "SSTableReaderTest";
public static final String CF_STANDARD = "Standard1";
public static final String CF_STANDARD2 = "Standard2";
+ public static final String CF_COMPRESSED = "Compressed";
public static final String CF_INDEXED = "Indexed1";
public static final String CF_STANDARDLOWINDEXINTERVAL =
"StandardLowIndexInterval";
@@ -87,6 +93,7 @@ public class SSTableReaderTest
KeyspaceParams.simple(1),
SchemaLoader.standardCFMD(KEYSPACE1,
CF_STANDARD),
SchemaLoader.standardCFMD(KEYSPACE1,
CF_STANDARD2),
+ SchemaLoader.standardCFMD(KEYSPACE1,
CF_COMPRESSED).compression(CompressionParams.DEFAULT),
SchemaLoader.compositeIndexCFMD(KEYSPACE1,
CF_INDEXED, true),
SchemaLoader.standardCFMD(KEYSPACE1,
CF_STANDARDLOWINDEXINTERVAL)
.minIndexInterval(8)
@@ -780,4 +787,64 @@ public class SSTableReaderTest
assertEquals(50, SSTableReader.getApproximateKeyCount(sstables));
}
}
+
+ @Rule
+ public ExpectedException expectedException = ExpectedException.none();
+
+ @Test
+ public void testVerifyCompressionInfoExistenceThrows()
+ {
+ Descriptor desc = setUpForTestVerfiyCompressionInfoExistence();
+
+ // delete the compression info, so it is corrupted.
+ File compressionInfoFile = new
File(desc.filenameFor(Component.COMPRESSION_INFO));
+ compressionInfoFile.delete();
+ assertFalse("CompressionInfo file should not exist",
compressionInfoFile.exists());
+
+ // discovert the components on disk after deletion
+ Set<Component> components = SSTable.discoverComponentsFor(desc);
+
+ expectedException.expect(CorruptSSTableException.class);
+ expectedException.expectMessage("CompressionInfo.db");
+ SSTableReader.verifyCompressionInfoExistenceIfApplicable(desc,
components);
+ }
+
+ @Test
+ public void testVerifyCompressionInfoExistenceWhenTOCUnableToOpen()
+ {
+ Descriptor desc = setUpForTestVerfiyCompressionInfoExistence();
+ Set<Component> components = SSTable.discoverComponentsFor(desc);
+ SSTableReader.verifyCompressionInfoExistenceIfApplicable(desc,
components);
+
+ // mark the toc file not readable in order to trigger the FSReadError
+ File tocFile = new File(desc.filenameFor(Component.TOC));
+ tocFile.setReadable(false);
+
+ expectedException.expect(FSReadError.class);
+ expectedException.expectMessage("TOC.txt");
+ SSTableReader.verifyCompressionInfoExistenceIfApplicable(desc,
components);
+ }
+
+ @Test
+ public void testVerifyCompressionInfoExistencePasses()
+ {
+ Descriptor desc = setUpForTestVerfiyCompressionInfoExistence();
+ Set<Component> components = SSTable.discoverComponentsFor(desc);
+ SSTableReader.verifyCompressionInfoExistenceIfApplicable(desc,
components);
+ }
+
+ private Descriptor setUpForTestVerfiyCompressionInfoExistence()
+ {
+ Keyspace keyspace = Keyspace.open(KEYSPACE1);
+ ColumnFamilyStore cfs = keyspace.getColumnFamilyStore(CF_COMPRESSED);
+ SSTableReader sstable = getNewSSTable(cfs);
+ cfs.clearUnsafe();
+ Descriptor desc = sstable.descriptor;
+
+ File compressionInfoFile = new
File(desc.filenameFor(Component.COMPRESSION_INFO));
+ File tocFile = new File(desc.filenameFor(Component.TOC));
+ assertTrue("CompressionInfo file should exist",
compressionInfoFile.exists());
+ assertTrue("TOC file should exist", tocFile.exists());
+ return desc;
+ }
}
---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]