Skip to content
New issue

Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.

By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.

Already on GitHub? Sign in to your account

fix: create the metastore backups directory if it does not exist #5879

Merged
merged 1 commit into from
Jul 24, 2020
Merged
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -19,11 +19,14 @@
import com.google.common.base.Ticker;
import io.confluent.ksql.rest.entity.CommandId;
import io.confluent.ksql.rest.server.computation.Command;
import io.confluent.ksql.util.KsqlException;
import io.confluent.ksql.util.KsqlConfig;
import io.confluent.ksql.util.KsqlServerException;
import io.confluent.ksql.util.Pair;
import java.io.File;
import java.io.IOException;
import java.nio.file.Files;
import java.nio.file.Paths;
import java.nio.file.attribute.PosixFilePermissions;
import java.util.Collections;
import java.util.List;
import java.util.Objects;
Expand Down Expand Up @@ -66,10 +69,7 @@ public CommandTopicBackupImpl(
final Ticker ticker
) {
final File dir = new File(Objects.requireNonNull(location, "location"));
if (!dir.exists() || !dir.isDirectory()) {
throw new KsqlException(String.format(
"Backup location '%s' does not exist or it is not a directory.", location));
}
ensureDirectoryExists(dir);

this.backupLocation = dir;
this.topicName = Objects.requireNonNull(topicName, "topicName");
Expand Down Expand Up @@ -222,4 +222,50 @@ private Optional<BackupReplayFile> latestReplayFile() {
? Optional.of(new BackupReplayFile(latestBakFile))
: Optional.empty();
}

private void ensureDirectoryExists(final File backupsDir) {
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

nit: can we de-dup this code from enforceStreamStateDirAvailability() in KsqlServerMain? Not required if it's annoying to de-dup (especially since we're trying to keep this patch small in order to cherry-pick) but it's weird to me that we have the same code in two places.

Copy link
Member Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Right, I see it. But I have the extra step to restrict the permissions to the KSQL server user only. I'd like to do that in the state directory too, but I don't know why the directory is not too restrictive. I'd rather investigate more and do it in another follow-up PR.

if (!backupsDir.exists()) {
if (!backupsDir.mkdirs()) {
throw new KsqlServerException("Couldn't create the backups directory: "
+ backupsDir.getPath()
+ "\n Make sure the directory exists and is readable/writable for KSQL server "
+ "\n or its parent directory is readable/writable by KSQL server"
+ "\n or change it to a readable/writable directory by setting '"
+ KsqlConfig.KSQL_METASTORE_BACKUP_LOCATION
+ "' config in the properties file."
);
}

try {
Files.setPosixFilePermissions(backupsDir.toPath(),
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Does it make sense to explicitly set permissions here? I see we don't do that for the Kafka Streams state directory. I'm in favor of consistency unless there's a good reason to deviate.

Copy link
Member Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

I'd like to keep the metastore directory restricted from other uses to access the data. By default, the directory created has drwxrwxr-x which is opened to any user in the system. It's up to the user to specify a directory that has restricted permissions only to the required users, but if KSQL creates it automatically, then I set those here too.

I don't know why the streams state directory is not protected. We might want to do that. I won't do it in this patch until I understand their reason.

PosixFilePermissions.fromString("rwx------"));
} catch (final IOException e) {
throw new KsqlServerException(String.format(
"Couldn't set POSIX permissions on the backups directory: %s. Error = %s",
backupsDir.getPath(), e.getMessage()));
}
}

if (!backupsDir.isDirectory()) {
throw new KsqlServerException(backupsDir.getPath()
+ " is not a directory."
+ "\n Make sure the directory exists and is readable/writable for KSQL server "
+ "\n or its parent directory is readable/writable by KSQL server"
+ "\n or change it to a readable/writable directory by setting '"
+ KsqlConfig.KSQL_METASTORE_BACKUP_LOCATION
+ "' config in the properties file."
);
}

if (!backupsDir.canWrite() || !backupsDir.canRead() || !backupsDir.canExecute()) {
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Why do we need execute permissions? I see we have a similar check for the Kafka Streams state directory so I assume there's good reason. Curious what it is.

Copy link
Member Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Directories need to have execute permissions in order to create files in it. By default, a new directory has rwx. If you remove the x, then you get a permissions denied.

$ mkdir /tmp/dir
$ chmod a-x /tmp/dir
$ ls -dl /tmp/dir
drw-rw-r-- 2 sergio sergio 4096 Jul 24 15:46 /tmp/dir
$ echo file > /tmp/dir/file
bash: /tmp/dir/file: Permission denied
$ 

throw new KsqlServerException("The backups directory is not readable/writable "
+ "for KSQL server: "
+ backupsDir.getPath()
+ "\n Make sure the directory exists and is readable/writable for KSQL server "
+ "\n or change it to a readable/writable directory by setting '"
+ KsqlConfig.KSQL_METASTORE_BACKUP_LOCATION
+ "' config in the properties file."
);
}
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -19,6 +19,7 @@
import io.confluent.ksql.rest.entity.CommandId;
import io.confluent.ksql.rest.server.computation.Command;
import io.confluent.ksql.util.KsqlException;
import io.confluent.ksql.util.KsqlServerException;
import io.confluent.ksql.util.Pair;
import org.apache.kafka.clients.consumer.ConsumerRecord;
import org.junit.Before;
Expand All @@ -31,6 +32,10 @@

import java.io.File;
import java.io.IOException;
import java.nio.file.Files;
import java.nio.file.Path;
import java.nio.file.Paths;
import java.nio.file.attribute.PosixFilePermissions;
import java.util.List;
import java.util.Optional;

Expand Down Expand Up @@ -81,31 +86,69 @@ public void shouldThrowWhenBackupLocationIsNotDirectory() throws IOException {

// When
final Exception e = assertThrows(
KsqlException.class,
KsqlServerException.class,
() -> new CommandTopicBackupImpl(file.getAbsolutePath(), COMMAND_TOPIC_NAME)
);

// Then
assertThat(e.getMessage(), containsString(String.format(
"Backup location '%s' does not exist or it is not a directory.",
"%s is not a directory.",
file.getAbsolutePath()
)));
}

@Test
public void shouldThrowWhenBackupLocationDoesNotExist() {
public void shouldThrowWhenBackupLocationIsNotWritable() throws IOException {
// Given
final File file = backupLocation.newFolder();
Files.setPosixFilePermissions(file.toPath(), PosixFilePermissions.fromString("r-x------"));

// When
final Exception e = assertThrows(
KsqlException.class,
() -> new CommandTopicBackupImpl("/not-existing-directory", COMMAND_TOPIC_NAME)
KsqlServerException.class,
() -> new CommandTopicBackupImpl(file.getAbsolutePath(), COMMAND_TOPIC_NAME)
);

// Then
assertThat(e.getMessage(), containsString(String.format(
"Backup location '/not-existing-directory' does not exist or it is not a directory."
"The backups directory is not readable/writable for KSQL server: %s",
file.getAbsolutePath()
)));
}

@Test
public void shouldThrowWhenBackupLocationIsNotReadable() throws IOException {
// Given
final File dir = backupLocation.newFolder();
Files.setPosixFilePermissions(dir.toPath(), PosixFilePermissions.fromString("-wx------"));

// When
final Exception e = assertThrows(
KsqlServerException.class,
() -> new CommandTopicBackupImpl(dir.getAbsolutePath(), COMMAND_TOPIC_NAME)
);

// Then
assertThat(e.getMessage(), containsString(String.format(
"The backups directory is not readable/writable for KSQL server: %s",
dir.getAbsolutePath()
)));
}


@Test
public void shouldCreateBackupLocationWhenDoesNotExist() throws IOException {
// Given
final Path dir = Paths.get(backupLocation.newFolder().getAbsolutePath(), "ksql-backups");
assertThat(Files.exists(dir), is(false));

// When
new CommandTopicBackupImpl(dir.toString(), COMMAND_TOPIC_NAME);

// Then
assertThat(Files.exists(dir), is(true));
}

@Test
public void shouldWriteRecordsToReplayFile() throws IOException {
// Given
Expand Down