cadonna commented on code in PR #14852:
URL: https://github.com/apache/kafka/pull/14852#discussion_r1412057240
##########
streams/src/main/java/org/apache/kafka/streams/state/internals/RocksDBStore.java:
##########
@@ -278,18 +280,74 @@ private void addValueProvidersToMetricsRecorder() {
void openRocksDB(final DBOptions dbOptions,
final ColumnFamilyOptions columnFamilyOptions) {
- final List<ColumnFamilyDescriptor> columnFamilyDescriptors
- = Collections.singletonList(new
ColumnFamilyDescriptor(RocksDB.DEFAULT_COLUMN_FAMILY, columnFamilyOptions));
- final List<ColumnFamilyHandle> columnFamilies = new
ArrayList<>(columnFamilyDescriptors.size());
+ final List<ColumnFamilyHandle> columnFamilies = openRocksDB(
+ dbOptions,
+ new ColumnFamilyDescriptor(RocksDB.DEFAULT_COLUMN_FAMILY,
columnFamilyOptions)
+ );
+
+ dbAccessor = new SingleColumnFamilyAccessor(columnFamilies.get(0));
+ }
+
+ /**
+ * Open RocksDB while automatically creating any requested column families
that don't yet exist.
+ */
+ protected List<ColumnFamilyHandle> openRocksDB(final DBOptions dbOptions,
+ final
ColumnFamilyDescriptor defaultColumnFamilyDescriptor,
+ final
ColumnFamilyDescriptor... columnFamilyDescriptors) {
+ final String absolutePath = dbDir.getAbsolutePath();
+ final List<ColumnFamilyDescriptor> extraDescriptors =
Arrays.asList(columnFamilyDescriptors);
+ final List<ColumnFamilyDescriptor> allDescriptors = new ArrayList<>(1
+ columnFamilyDescriptors.length);
+ allDescriptors.add(defaultColumnFamilyDescriptor);
+ allDescriptors.addAll(extraDescriptors);
try {
- db = RocksDB.open(dbOptions, dbDir.getAbsolutePath(),
columnFamilyDescriptors, columnFamilies);
- dbAccessor = new SingleColumnFamilyAccessor(columnFamilies.get(0));
+ final Options options = new Options(dbOptions,
defaultColumnFamilyDescriptor.getOptions());
+ final List<byte[]> allExisting =
RocksDB.listColumnFamilies(options, absolutePath);
+
+ final List<ColumnFamilyDescriptor> existingDescriptors =
allDescriptors.stream()
+ .filter(descriptor -> descriptor ==
defaultColumnFamilyDescriptor || allExisting.stream().anyMatch(existing ->
Arrays.equals(existing, descriptor.getName())))
+ .collect(Collectors.toList());
Review Comment:
nit:
Just a proposal for better readability:
```suggestion
final List<ColumnFamilyDescriptor> existingDescriptors = new
LinkedList<>();
existingDescriptors.add(defaultColumnFamilyDescriptor);
existingDescriptors.addAll(allDescriptors.stream()
.filter(descriptor -> allExisting.stream().anyMatch(existing
-> Arrays.equals(existing, descriptor.getName())))
.collect(Collectors.toList()
);
```
--
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
To unsubscribe, e-mail: [email protected]
For queries about this service, please contact Infrastructure at:
[email protected]