forked from apache/cassandra
-
Notifications
You must be signed in to change notification settings - Fork 21
/
Copy pathCompactionStrategyManager.java
1171 lines (1048 loc) · 40.1 KB
/
CompactionStrategyManager.java
1
2
3
4
5
6
7
8
9
10
11
12
13
14
15
16
17
18
19
20
21
22
23
24
25
26
27
28
29
30
31
32
33
34
35
36
37
38
39
40
41
42
43
44
45
46
47
48
49
50
51
52
53
54
55
56
57
58
59
60
61
62
63
64
65
66
67
68
69
70
71
72
73
74
75
76
77
78
79
80
81
82
83
84
85
86
87
88
89
90
91
92
93
94
95
96
97
98
99
100
101
102
103
104
105
106
107
108
109
110
111
112
113
114
115
116
117
118
119
120
121
122
123
124
125
126
127
128
129
130
131
132
133
134
135
136
137
138
139
140
141
142
143
144
145
146
147
148
149
150
151
152
153
154
155
156
157
158
159
160
161
162
163
164
165
166
167
168
169
170
171
172
173
174
175
176
177
178
179
180
181
182
183
184
185
186
187
188
189
190
191
192
193
194
195
196
197
198
199
200
201
202
203
204
205
206
207
208
209
210
211
212
213
214
215
216
217
218
219
220
221
222
223
224
225
226
227
228
229
230
231
232
233
234
235
236
237
238
239
240
241
242
243
244
245
246
247
248
249
250
251
252
253
254
255
256
257
258
259
260
261
262
263
264
265
266
267
268
269
270
271
272
273
274
275
276
277
278
279
280
281
282
283
284
285
286
287
288
289
290
291
292
293
294
295
296
297
298
299
300
301
302
303
304
305
306
307
308
309
310
311
312
313
314
315
316
317
318
319
320
321
322
323
324
325
326
327
328
329
330
331
332
333
334
335
336
337
338
339
340
341
342
343
344
345
346
347
348
349
350
351
352
353
354
355
356
357
358
359
360
361
362
363
364
365
366
367
368
369
370
371
372
373
374
375
376
377
378
379
380
381
382
383
384
385
386
387
388
389
390
391
392
393
394
395
396
397
398
399
400
401
402
403
404
405
406
407
408
409
410
411
412
413
414
415
416
417
418
419
420
421
422
423
424
425
426
427
428
429
430
431
432
433
434
435
436
437
438
439
440
441
442
443
444
445
446
447
448
449
450
451
452
453
454
455
456
457
458
459
460
461
462
463
464
465
466
467
468
469
470
471
472
473
474
475
476
477
478
479
480
481
482
483
484
485
486
487
488
489
490
491
492
493
494
495
496
497
498
499
500
501
502
503
504
505
506
507
508
509
510
511
512
513
514
515
516
517
518
519
520
521
522
523
524
525
526
527
528
529
530
531
532
533
534
535
536
537
538
539
540
541
542
543
544
545
546
547
548
549
550
551
552
553
554
555
556
557
558
559
560
561
562
563
564
565
566
567
568
569
570
571
572
573
574
575
576
577
578
579
580
581
582
583
584
585
586
587
588
589
590
591
592
593
594
595
596
597
598
599
600
601
602
603
604
605
606
607
608
609
610
611
612
613
614
615
616
617
618
619
620
621
622
623
624
625
626
627
628
629
630
631
632
633
634
635
636
637
638
639
640
641
642
643
644
645
646
647
648
649
650
651
652
653
654
655
656
657
658
659
660
661
662
663
664
665
666
667
668
669
670
671
672
673
674
675
676
677
678
679
680
681
682
683
684
685
686
687
688
689
690
691
692
693
694
695
696
697
698
699
700
701
702
703
704
705
706
707
708
709
710
711
712
713
714
715
716
717
718
719
720
721
722
723
724
725
726
727
728
729
730
731
732
733
734
735
736
737
738
739
740
741
742
743
744
745
746
747
748
749
750
751
752
753
754
755
756
757
758
759
760
761
762
763
764
765
766
767
768
769
770
771
772
773
774
775
776
777
778
779
780
781
782
783
784
785
786
787
788
789
790
791
792
793
794
795
796
797
798
799
800
801
802
803
804
805
806
807
808
809
810
811
812
813
814
815
816
817
818
819
820
821
822
823
824
825
826
827
828
829
830
831
832
833
834
835
836
837
838
839
840
841
842
843
844
845
846
847
848
849
850
851
852
853
854
855
856
857
858
859
860
861
862
863
864
865
866
867
868
869
870
871
872
873
874
875
876
877
878
879
880
881
882
883
884
885
886
887
888
889
890
891
892
893
894
895
896
897
898
899
900
901
902
903
904
905
906
907
908
909
910
911
912
913
914
915
916
917
918
919
920
921
922
923
924
925
926
927
928
929
930
931
932
933
934
935
936
937
938
939
940
941
942
943
944
945
946
947
948
949
950
951
952
953
954
955
956
957
958
959
960
961
962
963
964
965
966
967
968
969
970
971
972
973
974
975
976
977
978
979
980
981
982
983
984
985
986
987
988
989
990
991
992
993
994
995
996
997
998
999
1000
/*
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.cassandra.db.compaction;
import java.util.ArrayList;
import java.util.Arrays;
import java.util.Collection;
import java.util.Collections;
import java.util.ConcurrentModificationException;
import java.util.HashSet;
import java.util.LinkedHashMap;
import java.util.List;
import java.util.Map;
import java.util.Set;
import java.util.UUID;
import java.util.concurrent.locks.ReentrantReadWriteLock;
import java.util.function.Supplier;
import java.util.stream.Collectors;
import javax.annotation.Nonnull;
import javax.annotation.Nullable;
import com.google.common.annotations.VisibleForTesting;
import com.google.common.collect.ImmutableList;
import com.google.common.collect.Iterables;
import com.google.common.collect.Lists;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
import org.apache.cassandra.db.DiskBoundaries;
import org.apache.cassandra.db.SerializationHeader;
import org.apache.cassandra.db.commitlog.CommitLogPosition;
import org.apache.cassandra.db.commitlog.IntervalSet;
import org.apache.cassandra.db.compaction.AbstractStrategyHolder.TasksSupplier;
import org.apache.cassandra.db.lifecycle.LifecycleNewTracker;
import org.apache.cassandra.db.lifecycle.LifecycleTransaction;
import org.apache.cassandra.db.lifecycle.SSTableSet;
import org.apache.cassandra.dht.Range;
import org.apache.cassandra.dht.Token;
import org.apache.cassandra.index.Index;
import org.apache.cassandra.io.sstable.Descriptor;
import org.apache.cassandra.io.sstable.ISSTableScanner;
import org.apache.cassandra.io.sstable.SSTable;
import org.apache.cassandra.io.sstable.SSTableMultiWriter;
import org.apache.cassandra.io.sstable.ScannerList;
import org.apache.cassandra.io.sstable.format.SSTableReader;
import org.apache.cassandra.notifications.INotification;
import org.apache.cassandra.notifications.SSTableAddedNotification;
import org.apache.cassandra.notifications.SSTableDeletingNotification;
import org.apache.cassandra.notifications.SSTableListChangedNotification;
import org.apache.cassandra.notifications.SSTableRepairStatusChanged;
import org.apache.cassandra.schema.CompactionParams;
import org.apache.cassandra.service.ActiveRepairService;
import static org.apache.cassandra.db.compaction.AbstractStrategyHolder.GroupedSSTableContainer;
/**
* Manages the compaction strategies.
*
* SSTables are isolated from each other based on their incremental repair status (repaired, unrepaired, or pending repair)
* and directory (determined by their starting token). This class handles the routing between {@link AbstractStrategyHolder}
* instances based on repair status, and the {@link AbstractStrategyHolder} instances have separate compaction strategies
* for each directory, which it routes sstables to. Note that {@link PendingRepairHolder} also divides sstables on their
* pending repair id.
*
* Operations on this class are guarded by a {@link ReentrantReadWriteLock}. This lock performs mutual exclusion on
* reads and writes to the following variables: {@link this#repaired}, {@link this#unrepaired}, {@link this#isActive},
* {@link this#params}, {@link this#currentBoundaries}. Whenever performing reads on these variables,
* the {@link this#readLock} should be acquired. Likewise, updates to these variables should be guarded by
* {@link this#writeLock}.
*
* Whenever the {@link DiskBoundaries} change, the compaction strategies must be reloaded, so in order to ensure
* the compaction strategy placement reflect most up-to-date disk boundaries, call {@link this#maybeReloadDiskBoundaries()}
* before acquiring the read lock to access the strategies.
*
*/
public class CompactionStrategyManager implements CompactionStrategyContainer
{
private static final Logger logger = LoggerFactory.getLogger(CompactionStrategyManager.class);
public final CompactionLogger compactionLogger;
private final CompactionRealm realm;
private final boolean partitionSSTablesByTokenRange;
private final Supplier<DiskBoundaries> boundariesSupplier;
private final boolean enableAutoCompaction;
/**
* Performs mutual exclusion on the variables below
*/
private final ReentrantReadWriteLock lock = new ReentrantReadWriteLock();
private final ReentrantReadWriteLock.ReadLock readLock = lock.readLock();
private final ReentrantReadWriteLock.WriteLock writeLock = lock.writeLock();
/**
* Variables guarded by read and write lock above
*/
private final PendingRepairHolder transientRepairs;
private final PendingRepairHolder pendingRepairs;
private final CompactionStrategyHolder repaired;
private final CompactionStrategyHolder unrepaired;
private final ImmutableList<AbstractStrategyHolder> holders;
private volatile CompactionParams params;
private DiskBoundaries currentBoundaries;
private volatile boolean enabled;
private volatile boolean isActive = true;
/*
We keep a copy of the table metadata compaction parameters here to be able to decide if we
should update the compaction strategy due to a metadata change such as a schema changed
caused by an ALTER TABLE.
If a user changes the local compaction strategy via JMX and then later ALTERs a compaction parameter,
we will use the new compaction parameters but we will not override the JMX parameters if compaction
was not changed by the ALTER.
*/
@SuppressWarnings("thread-safe")
private volatile CompactionParams metadataParams;
private volatile boolean supportsEarlyOpen;
private volatile int fanout;
private volatile long maxSSTableSizeBytes;
private volatile String name;
public CompactionStrategyManager(CompactionStrategyFactory strategyFactory, boolean enableAutoCompaction)
{
this(strategyFactory,
() -> strategyFactory.getRealm().getDiskBoundaries(),
strategyFactory.getRealm().getPartitioner().splitter().isPresent(),
enableAutoCompaction);
}
@VisibleForTesting
public CompactionStrategyManager(CompactionStrategyFactory strategyFactory,
Supplier<DiskBoundaries> boundariesSupplier,
boolean partitionSSTablesByTokenRange,
boolean enableAutoCompaction)
{
AbstractStrategyHolder.DestinationRouter router = new AbstractStrategyHolder.DestinationRouter()
{
public int getIndexForSSTable(CompactionSSTable sstable)
{
return compactionStrategyIndexFor(sstable);
}
public int getIndexForSSTableDirectory(Descriptor descriptor)
{
return compactionStrategyIndexForDirectory(descriptor);
}
};
this.enableAutoCompaction = enableAutoCompaction;
realm = strategyFactory.getRealm();
transientRepairs = new PendingRepairHolder(realm, strategyFactory, router, true);
pendingRepairs = new PendingRepairHolder(realm, strategyFactory, router, false);
repaired = new CompactionStrategyHolder(realm, strategyFactory, router, true);
unrepaired = new CompactionStrategyHolder(realm, strategyFactory, router, false);
holders = ImmutableList.of(transientRepairs, pendingRepairs, repaired, unrepaired);
compactionLogger = strategyFactory.getCompactionLogger();
this.boundariesSupplier = boundariesSupplier;
this.partitionSSTablesByTokenRange = partitionSSTablesByTokenRange;
params = realm.metadata().params.compaction;
enabled = params.isEnabled();
}
public static CompactionStrategyContainer create(@Nullable CompactionStrategyContainer previous,
CompactionStrategyFactory strategyFactory,
CompactionParams compactionParams,
CompactionStrategyContainer.ReloadReason reason,
boolean enableAutoCompaction)
{
CompactionStrategyManager csm = new CompactionStrategyManager(strategyFactory, enableAutoCompaction);
csm.reload(previous != null ? previous : csm, compactionParams, reason);
return csm;
}
/**
* Return the next background task
*
* Legacy strategies will always return one task but we wrap this in a collection because new strategies
* might return multiple tasks.
*
* @return the task for the compaction strategy that needs it the most (most estimated remaining tasks) */
@Override
public Collection<AbstractCompactionTask> getNextBackgroundTasks(int gcBefore)
{
maybeReloadDiskBoundaries();
readLock.lock();
try
{
if (!isEnabled())
return ImmutableList.of();
int numPartitions = getNumTokenPartitions();
// first try to promote/demote sstables from completed repairs
Collection<AbstractCompactionTask> repairFinishedTasks;
repairFinishedTasks = pendingRepairs.getNextRepairFinishedTasks();
if (!repairFinishedTasks.isEmpty())
return repairFinishedTasks;
repairFinishedTasks = transientRepairs.getNextRepairFinishedTasks();
if (!repairFinishedTasks.isEmpty())
return repairFinishedTasks;
// sort compaction task suppliers by remaining tasks descending
List<TasksSupplier> suppliers = new ArrayList<>(numPartitions * holders.size());
for (AbstractStrategyHolder holder : holders)
suppliers.addAll(holder.getBackgroundTaskSuppliers(gcBefore));
Collections.sort(suppliers);
// return the first non-empty list, we could enhance it to return all tasks of all
// suppliers but this would change existing behavior
for (TasksSupplier supplier : suppliers)
{
Collection<AbstractCompactionTask> tasks = supplier.getTasks();
if (!tasks.isEmpty())
return tasks;
}
return ImmutableList.of();
}
finally
{
readLock.unlock();
}
}
@Override
public CompactionLogger getCompactionLogger()
{
return compactionLogger;
}
@Override
public boolean isEnabled()
{
return enableAutoCompaction && enabled && isActive;
}
@Override
public boolean isActive()
{
return isActive;
}
@Override
public void resume()
{
writeLock.lock();
try
{
isActive = true;
}
finally
{
writeLock.unlock();
}
}
/**
* pause compaction while we cancel all ongoing compactions
*
* Separate call from enable/disable to not have to save the enabled-state externally
*/
@Override
public void pause()
{
writeLock.lock();
try
{
isActive = false;
}
finally
{
writeLock.unlock();
}
}
@Override
public void startup()
{
writeLock.lock();
try
{
for (CompactionSSTable sstable : realm.getSSTables(SSTableSet.CANONICAL))
{
if (sstable.isSuitableForCompaction())
compactionStrategyFor(sstable).addSSTable(sstable);
}
holders.forEach(AbstractStrategyHolder::startup);
supportsEarlyOpen = repaired.first().supportsEarlyOpen();
maxSSTableSizeBytes = repaired.first().getMaxSSTableBytes();
name = repaired.first().getName();
}
finally
{
writeLock.unlock();
}
if (repaired.first().getOptions().isLogEnabled())
compactionLogger.enable();
}
/**
* returns differently based on the repaired status and which vnode the compaction strategy belongs to
* @param sstable
* @return the compaction strategy for the given sstable
*/
LegacyAbstractCompactionStrategy getCompactionStrategyFor(CompactionSSTable sstable)
{
maybeReloadDiskBoundaries();
return compactionStrategyFor(sstable);
}
@VisibleForTesting
LegacyAbstractCompactionStrategy compactionStrategyFor(CompactionSSTable sstable)
{
// should not call maybeReloadDiskBoundaries because it may be called from within lock
readLock.lock();
try
{
return getHolder(sstable).getStrategyFor(sstable);
}
finally
{
readLock.unlock();
}
}
/**
* Get the correct compaction strategy for the given sstable. If the first token starts within a disk boundary, we
* will add it to that compaction strategy.
*
* In the case we are upgrading, the first compaction strategy will get most files - we do not care about which disk
* the sstable is on currently (unless we don't know the local tokens yet). Once we start compacting we will write out
* sstables in the correct locations and give them to the correct compaction strategy instance.
*
* @param sstable
* @return
*/
int compactionStrategyIndexFor(CompactionSSTable sstable)
{
// should not call maybeReloadDiskBoundaries because it may be called from within lock
readLock.lock();
try
{
//We only have a single compaction strategy when sstables are not
//partitioned by token range
if (!partitionSSTablesByTokenRange)
return 0;
return currentBoundaries.getDiskIndexFromKey(sstable);
}
finally
{
readLock.unlock();
}
}
private int compactionStrategyIndexForDirectory(Descriptor descriptor)
{
readLock.lock();
try
{
return partitionSSTablesByTokenRange ? currentBoundaries.getBoundariesFromSSTableDirectory(descriptor) : 0;
}
finally
{
readLock.unlock();
}
}
@VisibleForTesting
CompactionStrategyHolder getRepairedUnsafe()
{
return repaired;
}
@VisibleForTesting
CompactionStrategyHolder getUnrepairedUnsafe()
{
return unrepaired;
}
@VisibleForTesting
PendingRepairHolder getPendingRepairsUnsafe()
{
return pendingRepairs;
}
@VisibleForTesting
PendingRepairHolder getTransientRepairsUnsafe()
{
return transientRepairs;
}
@Override
public void shutdown()
{
writeLock.lock();
try
{
isActive = false;
holders.forEach(AbstractStrategyHolder::shutdown);
compactionLogger.disable();
}
finally
{
writeLock.unlock();
}
}
/**
* Checks if the disk boundaries changed and reloads the compaction strategies
* to reflect the most up-to-date disk boundaries.
*
* This is typically called before acquiring the {@link this#readLock} to ensure the most up-to-date
* disk locations and boundaries are used.
*
* This should *never* be called inside by a thread holding the {@link this#readLock}, since it
* will potentially acquire the {@link this#writeLock} to update the compaction strategies
* what can cause a deadlock.
*/
//TODO improve this to reload after receiving a notification rather than trying to reload on every operation
@VisibleForTesting
protected void maybeReloadDiskBoundaries()
{
if (!currentBoundaries.isOutOfDate())
return;
writeLock.lock();
try
{
if (!currentBoundaries.isOutOfDate())
return;
doReload(this, params, ReloadReason.DISK_BOUNDARIES_UPDATED);
}
finally
{
writeLock.unlock();
}
}
@Override
public CompactionStrategyContainer reload(@Nonnull CompactionStrategyContainer previous, CompactionParams newCompactionParams, ReloadReason reason)
{
writeLock.lock();
try
{
doReload(previous, newCompactionParams, reason);
}
finally
{
writeLock.unlock();
}
if (previous != this)
previous.shutdown();
return this;
}
private void doReload(CompactionStrategyContainer previous, CompactionParams compactionParams, ReloadReason reason)
{
boolean updateDiskBoundaries = currentBoundaries == null || currentBoundaries.isOutOfDate();
boolean enabledOnReload = CompactionStrategyFactory.enableCompactionOnReload(previous, compactionParams, reason) && enableAutoCompaction;
logger.debug("Recreating compaction strategy for {}.{}, reason: {}, params updated: {}, disk boundaries updated: {}, enabled: {}, params: {} -> {}, metadataParams: {}",
realm.getKeyspaceName(), realm.getTableName(), reason, !compactionParams.equals(params), updateDiskBoundaries, enabledOnReload, params, compactionParams, metadataParams);
if (updateDiskBoundaries)
currentBoundaries = boundariesSupplier.get();
int numPartitions = getNumTokenPartitions();
for (AbstractStrategyHolder holder : holders)
holder.setStrategy(compactionParams, numPartitions);
params = compactionParams;
// full reload or switch from a strategy not managed by CompactionStrategyManager
if (metadataParams == null || reason == ReloadReason.FULL)
metadataParams = realm.metadata().params.compaction;
else if (reason == ReloadReason.METADATA_CHANGE)
// metadataParams are aligned with compactionParams. We do not access TableParams.COMPACTION to avoid racing with
// concurrent ALTER TABLE metadata change.
metadataParams = compactionParams;
// no-op for DISK_BOUNDARIES_UPDATED and JMX_REQUEST. DISK_BOUNDARIES_UPDATED does not change compaction params
// and JMX changes do not affect table metadata
if (params.maxCompactionThreshold() <= 0 || params.minCompactionThreshold() <= 0)
{
logger.warn("Disabling compaction strategy by setting compaction thresholds to 0 is deprecated, set the compaction option 'enabled' to 'false' instead.");
disable();
}
else if (!enabledOnReload)
disable();
else
enable();
startup();
}
private Iterable<CompactionStrategy> getAllStrategies()
{
return Iterables.concat(Iterables.transform(holders, AbstractStrategyHolder::allStrategies));
}
public int getUnleveledSSTables()
{
maybeReloadDiskBoundaries();
readLock.lock();
try
{
if (repaired.first() instanceof LeveledCompactionStrategy)
{
int count = 0;
for (CompactionStrategy strategy : getAllStrategies())
count += ((LeveledCompactionStrategy) strategy).getLevelSize(0);
return count;
}
}
finally
{
readLock.unlock();
}
return 0;
}
@Override
public int getLevelFanoutSize()
{
return repaired.first().getLevelFanoutSize();
}
@Override
public int[] getSSTableCountPerLevel()
{
maybeReloadDiskBoundaries();
readLock.lock();
try
{
if (repaired.first() instanceof LeveledCompactionStrategy)
{
int[] res = new int[LeveledGenerations.MAX_LEVEL_COUNT];
for (CompactionStrategy strategy : getAllStrategies())
{
int[] repairedCountPerLevel = ((LeveledCompactionStrategy) strategy).getAllLevelSize();
res = sumArrays(res, repairedCountPerLevel);
}
return res;
}
else
{
return new int[0];
}
}
finally
{
readLock.unlock();
}
}
static int[] sumArrays(int[] a, int[] b)
{
int[] res = new int[Math.max(a.length, b.length)];
for (int i = 0; i < res.length; i++)
{
if (i < a.length && i < b.length)
res[i] = a[i] + b[i];
else if (i < a.length)
res[i] = a[i];
else
res[i] = b[i];
}
return res;
}
/**
* Should only be called holding the readLock
*/
private void handleFlushNotification(Iterable<? extends CompactionSSTable> added)
{
for (CompactionSSTable sstable : added)
compactionStrategyFor(sstable).addSSTable(sstable);
}
private int getHolderIndex(CompactionSSTable sstable)
{
for (int i = 0; i < holders.size(); i++)
{
if (holders.get(i).managesSSTable(sstable))
return i;
}
throw new IllegalStateException("No holder claimed " + sstable);
}
private AbstractStrategyHolder getHolder(CompactionSSTable sstable)
{
for (AbstractStrategyHolder holder : holders)
{
if (holder.managesSSTable(sstable))
return holder;
}
throw new IllegalStateException("No holder claimed " + sstable);
}
private AbstractStrategyHolder getHolder(long repairedAt, UUID pendingRepair, boolean isTransient)
{
return getHolder(repairedAt != ActiveRepairService.UNREPAIRED_SSTABLE,
pendingRepair != ActiveRepairService.NO_PENDING_REPAIR,
isTransient);
}
@VisibleForTesting
AbstractStrategyHolder getHolder(boolean isRepaired, boolean isPendingRepair, boolean isTransient)
{
for (AbstractStrategyHolder holder : holders)
{
if (holder.managesRepairedGroup(isRepaired, isPendingRepair, isTransient))
return holder;
}
throw new IllegalStateException(String.format("No holder claimed isPendingRepair: %s, isPendingRepair %s",
isRepaired, isPendingRepair));
}
@VisibleForTesting
ImmutableList<AbstractStrategyHolder> getHolders()
{
return holders;
}
/**
* Split sstables into a list of grouped sstable containers, the list index an sstable
*
* lives in matches the list index of the holder that's responsible for it
*/
public <S extends CompactionSSTable>
List<GroupedSSTableContainer<S>> groupSSTables(Iterable<? extends S> sstables)
{
List<GroupedSSTableContainer<S>> classified = new ArrayList<>(holders.size());
for (AbstractStrategyHolder holder : holders)
{
classified.add(holder.createGroupedSSTableContainer());
}
for (S sstable : sstables)
{
classified.get(getHolderIndex(sstable)).add(sstable);
}
return classified;
}
/**
* Should only be called holding the readLock
*/
private void handleListChangedNotification(Iterable<? extends CompactionSSTable> added, Iterable<? extends CompactionSSTable> removed)
{
List<GroupedSSTableContainer<CompactionSSTable>> addedGroups = groupSSTables(added);
List<GroupedSSTableContainer<CompactionSSTable>> removedGroups = groupSSTables(removed);
for (int i=0; i<holders.size(); i++)
{
holders.get(i).replaceSSTables(removedGroups.get(i), addedGroups.get(i));
}
}
/**
* Should only be called holding the readLock
*/
private void handleRepairStatusChangedNotification(Iterable<? extends CompactionSSTable> sstables)
{
List<GroupedSSTableContainer<CompactionSSTable>> groups = groupSSTables(sstables);
for (int i = 0; i < holders.size(); i++)
{
GroupedSSTableContainer<CompactionSSTable> group = groups.get(i);
if (group.isEmpty())
continue;
AbstractStrategyHolder dstHolder = holders.get(i);
for (AbstractStrategyHolder holder : holders)
{
if (holder != dstHolder)
holder.removeSSTables(group);
}
// adding sstables into another strategy may change its level,
// thus it won't be removed from original LCS. We have to remove sstables first
dstHolder.addSSTables(group);
}
}
/**
* Should only be called holding the readLock
*/
private void handleDeletingNotification(CompactionSSTable deleted)
{
compactionStrategyFor(deleted).removeSSTable(deleted);
}
public void handleNotification(INotification notification, Object sender)
{
// we might race with reload adding/removing the sstables, this means that compaction strategies
// must handle double notifications.
maybeReloadDiskBoundaries();
readLock.lock();
try
{
if (notification instanceof SSTableAddedNotification)
{
SSTableAddedNotification flushedNotification = (SSTableAddedNotification) notification;
handleFlushNotification(flushedNotification.added);
}
else if (notification instanceof SSTableListChangedNotification)
{
SSTableListChangedNotification listChangedNotification = (SSTableListChangedNotification) notification;
handleListChangedNotification(listChangedNotification.added, listChangedNotification.removed);
}
else if (notification instanceof SSTableRepairStatusChanged)
{
handleRepairStatusChangedNotification(((SSTableRepairStatusChanged) notification).sstables);
}
else if (notification instanceof SSTableDeletingNotification)
{
handleDeletingNotification(((SSTableDeletingNotification) notification).deleting);
}
}
finally
{
readLock.unlock();
}
}
@Override
public void enable()
{
writeLock.lock();
try
{
// enable this last to make sure the strategies are ready to get calls.
enabled = true;
}
finally
{
writeLock.unlock();
}
}
@Override
public void disable()
{
writeLock.lock();
try
{
enabled = false;
}
finally
{
writeLock.unlock();
}
}
/**
* Create ISSTableScanners from the given sstables
*
* Delegates the call to the compaction strategies to allow LCS to create a scanner
* @param sstables
* @param ranges
* @return
*/
@SuppressWarnings("resource")
private ScannerList maybeGetScanners(Collection<SSTableReader> sstables, Collection<Range<Token>> ranges)
{
maybeReloadDiskBoundaries();
List<ISSTableScanner> scanners = new ArrayList<>(sstables.size());
readLock.lock();
try
{
List<GroupedSSTableContainer<SSTableReader>> sstableGroups = groupSSTables(sstables);
for (int i = 0; i < holders.size(); i++)
{
AbstractStrategyHolder holder = holders.get(i);
GroupedSSTableContainer<SSTableReader> group = sstableGroups.get(i);
scanners.addAll(holder.getScanners(group, ranges));
}
}
catch (PendingRepairManager.IllegalSSTableArgumentException e)
{
ISSTableScanner.closeAllAndPropagate(scanners, new ConcurrentModificationException(e));
}
finally
{
readLock.unlock();
}
return new ScannerList(scanners);
}
@Override
public ScannerList getScanners(Collection<SSTableReader> sstables, Collection<Range<Token>> ranges)
{
while (true)
{
try
{
return maybeGetScanners(sstables, ranges);
}
catch (ConcurrentModificationException e)
{
logger.debug("SSTable repairedAt/pendingRepaired values changed while getting scanners");
}
}
}
@Override
public ScannerList getScanners(Collection<SSTableReader> sstables)
{
return getScanners(sstables, null);
}
@Override
public Set<CompactionSSTable> getSSTables()
{
return getStrategies().stream()
.flatMap(strategy -> strategy.getSSTables().stream())
.collect(Collectors.toSet());
}
@Override
public Collection<Collection<CompactionSSTable>> groupSSTablesForAntiCompaction(Collection<? extends CompactionSSTable> sstablesToGroup)
{
maybeReloadDiskBoundaries();
readLock.lock();
try
{
return unrepaired.groupForAnticompaction(sstablesToGroup);
}
finally
{
readLock.unlock();
}
}
@Override
public long getMaxSSTableBytes()
{
return maxSSTableSizeBytes;
}
@Override
public AbstractCompactionTask createCompactionTask(LifecycleTransaction txn, int gcBefore, long maxSSTableBytes)
{
maybeReloadDiskBoundaries();
readLock.lock();
try
{
validateForCompaction(txn.originals());
return compactionStrategyFor(txn.originals().iterator().next()).createCompactionTask(txn, gcBefore, maxSSTableBytes);
}
finally
{
readLock.unlock();
}
}
private void validateForCompaction(Iterable<? extends CompactionSSTable> input)
{
readLock.lock();
try
{
CompactionSSTable firstSSTable = Iterables.getFirst(input, null);
assert firstSSTable != null;
boolean repaired = firstSSTable.isRepaired();
int firstIndex = compactionStrategyIndexFor(firstSSTable);
boolean isPending = firstSSTable.isPendingRepair();
UUID pendingRepair = firstSSTable.getPendingRepair();
for (CompactionSSTable sstable : input)
{
if (sstable.isRepaired() != repaired)
throw new UnsupportedOperationException("You can't mix repaired and unrepaired data in a compaction");
if (firstIndex != compactionStrategyIndexFor(sstable))
throw new UnsupportedOperationException("You can't mix sstables from different directories in a compaction");
if (isPending && !pendingRepair.equals(sstable.getPendingRepair()))
throw new UnsupportedOperationException("You can't compact sstables from different pending repair sessions");
}
}
finally
{
readLock.unlock();
}
}
@Override
public CompactionTasks getMaximalTasks(final int gcBefore, final boolean splitOutput, int permittedParallelism)
{
maybeReloadDiskBoundaries();
// runWithCompactionsDisabled cancels active compactions and disables them, then we are able
// to make the repaired/unrepaired strategies mark their own sstables as compacting. Once the
// sstables are marked the compactions are re-enabled
return realm.runWithCompactionsDisabled(() -> {
List<AbstractCompactionTask> tasks = new ArrayList<>();
readLock.lock();
try
{
for (AbstractStrategyHolder holder : holders)
{
tasks.addAll(holder.getMaximalTasks(gcBefore, splitOutput, permittedParallelism));
}
}
finally
{
readLock.unlock();
}
return CompactionTasks.create(CompositeCompactionTask.applyParallelismLimit(tasks, permittedParallelism));
}, false, false, TableOperation.StopTrigger.COMPACTION);
}
/**
* Return a list of compaction tasks corresponding to the sstables requested. Split the sstables according
* to whether they are repaired or not, and by disk location. Return a task per disk location and repair status
* group.
*
* @param sstables the sstables to compact
* @param gcBefore gc grace period, throw away tombstones older than this
* @return a list of compaction tasks corresponding to the sstables requested
*/
@Override
public CompactionTasks getUserDefinedTasks(Collection<? extends CompactionSSTable> sstables, int gcBefore)
{
maybeReloadDiskBoundaries();
List<AbstractCompactionTask> ret = new ArrayList<>();
readLock.lock();
try
{
List<GroupedSSTableContainer<CompactionSSTable>> groupedSSTables = groupSSTables(sstables);
for (int i = 0; i < holders.size(); i++)
{
ret.addAll(holders.get(i).getUserDefinedTasks(groupedSSTables.get(i), gcBefore));
}
return CompactionTasks.create(ret);
}
finally
{
readLock.unlock();
}
}
@Override
public int getEstimatedRemainingTasks()
{
return getStrategies(false).stream()
.flatMap(list -> list.stream())
.mapToInt(CompactionStrategy::getEstimatedRemainingTasks)
.sum();
}
@Override
public int getTotalCompactions()
{
return getStrategies(false).stream()
.flatMap(list -> list.stream())
.mapToInt(CompactionStrategy::getTotalCompactions)
.sum();
}
@Override
public String getName()
{
return name;
}
@Override
public List<CompactionStrategy> getStrategies()
{
return getStrategies(true).stream().flatMap(List::stream).collect(Collectors.toList());