Skip to content

Commit a93f226

Browse files
committed
Soft reject and btree durable before
1 parent c927a91 commit a93f226

47 files changed

Lines changed: 618 additions & 230 deletions

Some content is hidden

Large Commits have some content hidden by default. Use the searchbox below for content that may be hidden.

.gitmodules

Lines changed: 2 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -1,4 +1,4 @@
11
[submodule "modules/accord"]
22
path = modules/accord
3-
url = git@github.pie.apple.com:belliottsmith/accord.git
4-
branch = support-rebootstrap
3+
url = https://github.com/belliottsmith/cassandra-accord.git
4+
branch = fixes-260226

modules/accord

Submodule accord updated 115 files

src/java/org/apache/cassandra/config/AccordSpec.java

Lines changed: 7 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -40,7 +40,6 @@ public class AccordSpec
4040

4141
// TODO (expected): move to JournalSpec
4242
public volatile String journal_directory;
43-
public volatile boolean enable_journal_compaction = true;
4443

4544
/**
4645
* Enables the virtual Accord debug-only keyspace with tables
@@ -199,6 +198,11 @@ public enum CatchupMode
199198
public boolean ephemeralReadEnabled = true;
200199
public boolean state_cache_listener_jfr_enabled = false;
201200

201+
public float hard_reject_ratio = 0.5f;
202+
public int soft_reject_count = 100;
203+
public DurationSpec.LongMicrosecondsBound soft_reject_age = new DurationSpec.LongMicrosecondsBound("10s");
204+
public DurationSpec.LongMicrosecondsBound soft_reject_cumulative_age = new DurationSpec.LongMicrosecondsBound("60s");
205+
202206
public DurationSpec.IntSecondsBound catchup_on_start_success_latency = new DurationSpec.IntSecondsBound(60);
203207
public DurationSpec.IntSecondsBound catchup_on_start_fail_latency = new DurationSpec.IntSecondsBound(900);
204208
public int catchup_on_start_max_attempts = 5;
@@ -281,6 +285,7 @@ public enum StopMarkerFailurePolicy
281285
public DurationSpec.IntMillisecondsBound compactionPeriod = new DurationSpec.IntMillisecondsBound("60000ms");
282286
private volatile long flushCombinedBlockPeriod = Long.MIN_VALUE;
283287
public Version version = Version.DOWNGRADE_SAFE_VERSION;
288+
public boolean enable_compaction = true;
284289

285290
public JournalSpec setFlushPeriod(DurationSpec newFlushPeriod)
286291
{
@@ -322,7 +327,7 @@ public FlushMode flushMode()
322327
@Override
323328
public boolean enableCompaction()
324329
{
325-
return DatabaseDescriptor.getAccord().enable_journal_compaction;
330+
return enable_compaction;
326331
}
327332

328333
@Override

src/java/org/apache/cassandra/db/compaction/CompactionIterator.java

Lines changed: 7 additions & 7 deletions
Original file line numberDiff line numberDiff line change
@@ -89,10 +89,6 @@
8989
import org.apache.cassandra.schema.TableId;
9090
import org.apache.cassandra.schema.TableMetadata;
9191
import org.apache.cassandra.service.ClientState;
92-
import org.apache.cassandra.service.accord.journal.MergeSerializers;
93-
import org.apache.cassandra.service.accord.journal.Merger;
94-
import org.apache.cassandra.service.accord.journal.MergeSerializers.CommandChangeSerializer;
95-
import org.apache.cassandra.service.accord.journal.MergeSerializer;
9692
import org.apache.cassandra.service.accord.AccordKeyspace;
9793
import org.apache.cassandra.service.accord.AccordKeyspace.CommandsForKeyAccessor;
9894
import org.apache.cassandra.service.accord.AccordService;
@@ -101,10 +97,14 @@
10197
import org.apache.cassandra.service.accord.IAccordService.AccordCompactionInfos;
10298
import org.apache.cassandra.service.accord.JournalKey;
10399
import org.apache.cassandra.service.accord.api.TokenKey;
104-
import org.apache.cassandra.service.accord.journal.TopologyRecord;
105-
import org.apache.cassandra.service.accord.journal.TopologyRecord.TopologyImage;
106100
import org.apache.cassandra.service.accord.journal.CommandChangeWriter;
107101
import org.apache.cassandra.service.accord.journal.CommandChanges;
102+
import org.apache.cassandra.service.accord.journal.MergeSerializer;
103+
import org.apache.cassandra.service.accord.journal.MergeSerializers;
104+
import org.apache.cassandra.service.accord.journal.MergeSerializers.CommandChangeSerializer;
105+
import org.apache.cassandra.service.accord.journal.Merger;
106+
import org.apache.cassandra.service.accord.journal.TopologyRecord;
107+
import org.apache.cassandra.service.accord.journal.TopologyRecord.TopologyImage;
108108
import org.apache.cassandra.service.accord.serializers.Version;
109109
import org.apache.cassandra.service.paxos.PaxosRepairHistory;
110110
import org.apache.cassandra.service.paxos.uncommitted.PaxosRows;
@@ -125,9 +125,9 @@
125125
import static org.apache.cassandra.config.DatabaseDescriptor.paxosStatePurging;
126126
import static org.apache.cassandra.service.accord.AccordKeyspace.CFKAccessor;
127127
import static org.apache.cassandra.service.accord.AccordKeyspace.JournalColumns.getJournalKey;
128+
import static org.apache.cassandra.service.accord.journal.MergeSerializers.TopologySerializer.INSTANCE;
128129
import static org.apache.cassandra.service.accord.journal.TopologyRecord.Kind.Image;
129130
import static org.apache.cassandra.service.accord.journal.TopologyRecord.Kind.Repeat;
130-
import static org.apache.cassandra.service.accord.journal.MergeSerializers.TopologySerializer.INSTANCE;
131131

132132
/**
133133
* Merge multiple iterators over the content of sstable into a "compacted" iterator.

src/java/org/apache/cassandra/db/streaming/CassandraStreamReceiver.java

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -50,9 +50,9 @@
5050
import org.apache.cassandra.io.sstable.SSTableTxnSingleStreamWriter;
5151
import org.apache.cassandra.io.sstable.format.SSTableReader;
5252
import org.apache.cassandra.service.accord.AccordService;
53-
import org.apache.cassandra.service.accord.topology.AccordTopology;
5453
import org.apache.cassandra.service.accord.IAccordService;
5554
import org.apache.cassandra.service.accord.TimeOnlyRequestBookkeeping.LatencyRequestBookkeeping;
55+
import org.apache.cassandra.service.accord.topology.AccordTopology;
5656
import org.apache.cassandra.streaming.IncomingStream;
5757
import org.apache.cassandra.streaming.StreamReceiver;
5858
import org.apache.cassandra.streaming.StreamSession;

src/java/org/apache/cassandra/db/virtual/AccordDebugKeyspace.java

Lines changed: 14 additions & 14 deletions
Original file line numberDiff line numberDiff line change
@@ -653,17 +653,17 @@ private DurableBeforeTable()
653653
public void collect(PartitionsCollector collector)
654654
{
655655
DurableBefore durableBefore = AccordService.unsafeInstance().node().durableBefore();
656-
durableBefore.foldlWithBounds(
657-
(entry, ignore, start, end) -> {
658-
TableId tableId = (TableId) start.prefix();
659-
collector.row(tableId.toString(), printToken(start))
656+
durableBefore.foldl(
657+
(entry, ignore) -> {
658+
TableId tableId = (TableId) entry.prefix();
659+
collector.row(tableId.toString(), printToken(entry.start()))
660660
.lazyCollect(columns -> {
661-
columns.add("token_end", end, AccordDebugKeyspace::printToken)
662-
.add("quorum", entry.quorumBefore, TO_STRING)
663-
.add("universal", entry.universalBefore, TO_STRING);
661+
columns.add("token_end", entry.end(), AccordDebugKeyspace::printToken)
662+
.add("quorum", entry.quorum, TO_STRING)
663+
.add("universal", entry.universal, TO_STRING);
664664
});
665665
return null;
666-
}, null, ignore -> false);
666+
}, null);
667667
}
668668
}
669669

@@ -837,16 +837,16 @@ public void collect(PartitionsCollector collector)
837837
String tableIdStr = tableId.toString();
838838

839839
collector.partition(commandStoreId).collect(rows -> {
840-
maxConflicts.foldlWithBounds(
841-
(timestamp, rs, start, end) -> {
842-
rows.add(printToken(start))
840+
maxConflicts.foldl(
841+
(entry, rs) -> {
842+
rows.add(printToken(entry.start()))
843843
.lazyCollect(columns -> {
844-
columns.add("token_end", end, AccordDebugKeyspace::printToken)
844+
columns.add("token_end", entry.end(), AccordDebugKeyspace::printToken)
845845
.add("table_id", tableIdStr)
846-
.add("timestamp", timestamp, TO_STRING);
846+
.add("timestamp", entry, TO_STRING);
847847
});
848848
return rows;
849-
}, rows, ignore -> false
849+
}, rows
850850
);
851851
});
852852
}

src/java/org/apache/cassandra/index/accord/RouteIndexFormat.java

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -57,9 +57,9 @@
5757
import org.apache.cassandra.io.util.FileHandle;
5858
import org.apache.cassandra.schema.TableId;
5959
import org.apache.cassandra.serializers.UUIDSerializer;
60-
import org.apache.cassandra.service.accord.journal.AccordJournal;
6160
import org.apache.cassandra.service.accord.AccordKeyspace;
6261
import org.apache.cassandra.service.accord.JournalKey;
62+
import org.apache.cassandra.service.accord.journal.AccordJournal;
6363
import org.apache.cassandra.service.accord.journal.CommandChanges;
6464
import org.apache.cassandra.service.accord.serializers.CommandSerializers;
6565
import org.apache.cassandra.service.accord.serializers.KeySerializers;

src/java/org/apache/cassandra/index/accord/RouteJournalIndex.java

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -89,10 +89,10 @@
8989
import org.apache.cassandra.schema.TableId;
9090
import org.apache.cassandra.schema.TableMetadata;
9191
import org.apache.cassandra.service.ClientState;
92-
import org.apache.cassandra.service.accord.journal.RangeSearchManager;
9392
import org.apache.cassandra.service.accord.AccordKeyspace;
9493
import org.apache.cassandra.service.accord.JournalKey;
9594
import org.apache.cassandra.service.accord.api.TokenKey;
95+
import org.apache.cassandra.service.accord.journal.RangeSearchManager;
9696
import org.apache.cassandra.service.accord.serializers.CommandSerializers;
9797
import org.apache.cassandra.utils.AbstractIterator;
9898
import org.apache.cassandra.utils.concurrent.Future;

src/java/org/apache/cassandra/io/util/CompressedFrameDataOutputPlus.java

Lines changed: 1 addition & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -28,6 +28,7 @@
2828
import com.google.common.primitives.Shorts;
2929

3030
import accord.utils.Invariants;
31+
3132
import org.apache.cassandra.io.IVersionedSerializer;
3233
import org.apache.cassandra.io.UnversionedSerializer;
3334
import org.apache.cassandra.io.compress.ICompressor;

src/java/org/apache/cassandra/net/Verb.java

Lines changed: 5 additions & 5 deletions
Original file line numberDiff line numberDiff line change
@@ -83,17 +83,13 @@
8383
import org.apache.cassandra.schema.SchemaVersionVerbHandler;
8484
import org.apache.cassandra.service.EchoVerbHandler;
8585
import org.apache.cassandra.service.SnapshotVerbHandler;
86-
import org.apache.cassandra.service.accord.serializers.AccordSerializers;
8786
import org.apache.cassandra.service.accord.AccordService;
88-
import org.apache.cassandra.service.accord.topology.AccordSyncPropagator;
89-
import org.apache.cassandra.service.accord.topology.AccordSyncPropagator.Notification;
90-
import org.apache.cassandra.service.accord.topology.FetchTopologies;
91-
import org.apache.cassandra.service.accord.topology.WatermarkCollector;
9287
import org.apache.cassandra.service.accord.interop.AccordInteropApply;
9388
import org.apache.cassandra.service.accord.interop.AccordInteropRead;
9489
import org.apache.cassandra.service.accord.interop.AccordInteropReadRepair;
9590
import org.apache.cassandra.service.accord.interop.AccordInteropStableThenRead;
9691
import org.apache.cassandra.service.accord.serializers.AcceptSerializers;
92+
import org.apache.cassandra.service.accord.serializers.AccordSerializers;
9793
import org.apache.cassandra.service.accord.serializers.ApplySerializers;
9894
import org.apache.cassandra.service.accord.serializers.AwaitSerializers;
9995
import org.apache.cassandra.service.accord.serializers.BeginInvalidationSerializers;
@@ -111,6 +107,10 @@
111107
import org.apache.cassandra.service.accord.serializers.SetDurableSerializers;
112108
import org.apache.cassandra.service.accord.serializers.SimpleReplySerializer;
113109
import org.apache.cassandra.service.accord.serializers.Version;
110+
import org.apache.cassandra.service.accord.topology.AccordSyncPropagator;
111+
import org.apache.cassandra.service.accord.topology.AccordSyncPropagator.Notification;
112+
import org.apache.cassandra.service.accord.topology.FetchTopologies;
113+
import org.apache.cassandra.service.accord.topology.WatermarkCollector;
114114
import org.apache.cassandra.service.consensus.migration.ConsensusKeyMigrationState;
115115
import org.apache.cassandra.service.consensus.migration.ConsensusKeyMigrationState.ConsensusKeyMigrationFinished;
116116
import org.apache.cassandra.service.paxos.Commit;

0 commit comments

Comments
 (0)