treewide: port everything to blockdev

This gets rid of most ad-hoc block device code, using blockdev for
everything. It also gets rid of diskfs for everything but tests. This
enables Metropolis to be installed on non-512-byte block sizes.

Change-Id: I644b5b68bb7bed8106585df3179674789031687a
Reviewed-on: https://review.monogon.dev/c/monogon/+/1873
Tested-by: Jenkins CI
Reviewed-by: Serge Bazanski <serge@monogon.tech>
diff --git a/metropolis/node/build/mkimage/BUILD.bazel b/metropolis/node/build/mkimage/BUILD.bazel
index 2bf8229..2c7d699 100644
--- a/metropolis/node/build/mkimage/BUILD.bazel
+++ b/metropolis/node/build/mkimage/BUILD.bazel
@@ -5,7 +5,11 @@
     srcs = ["main.go"],
     importpath = "source.monogon.dev/metropolis/node/build/mkimage",
     visibility = ["//visibility:private"],
-    deps = ["//metropolis/node/build/mkimage/osimage"],
+    deps = [
+        "//metropolis/node/build/mkimage/osimage",
+        "//metropolis/pkg/blkio",
+        "//metropolis/pkg/blockdev",
+    ],
 )
 
 go_binary(
diff --git a/metropolis/node/build/mkimage/main.go b/metropolis/node/build/mkimage/main.go
index b885618..077348e 100644
--- a/metropolis/node/build/mkimage/main.go
+++ b/metropolis/node/build/mkimage/main.go
@@ -32,6 +32,8 @@
 	"os"
 
 	"source.monogon.dev/metropolis/node/build/mkimage/osimage"
+	"source.monogon.dev/metropolis/pkg/blkio"
+	"source.monogon.dev/metropolis/pkg/blockdev"
 )
 
 func main() {
@@ -40,26 +42,27 @@
 		efiPayload  string
 		systemImage string
 		nodeParams  string
+		outputPath  string
+		diskUUID    string
 		cfg         osimage.Params
 	)
 	flag.StringVar(&efiPayload, "efi", "", "Path to the UEFI payload used")
 	flag.StringVar(&systemImage, "system", "", "Path to the system partition image used")
 	flag.StringVar(&nodeParams, "node_parameters", "", "Path to Node Parameters to be written to the ESP (default: don't write Node Parameters)")
-	flag.StringVar(&cfg.OutputPath, "out", "", "Path to the resulting disk image or block device")
-	flag.Uint64Var(&cfg.PartitionSize.Data, "data_partition_size", 2048, "Override the data partition size (default 2048 MiB). Used only when generating image files.")
-	flag.Uint64Var(&cfg.PartitionSize.ESP, "esp_partition_size", 128, "Override the ESP partition size (default: 128MiB)")
-	flag.Uint64Var(&cfg.PartitionSize.System, "system_partition_size", 1024, "Override the System partition size (default: 1024MiB)")
-	flag.StringVar(&cfg.DiskGUID, "GUID", "", "Disk GUID marked in the resulting image's partition table (default: randomly generated)")
+	flag.StringVar(&outputPath, "out", "", "Path to the resulting disk image or block device")
+	flag.Int64Var(&cfg.PartitionSize.Data, "data_partition_size", 2048, "Override the data partition size (default 2048 MiB). Used only when generating image files.")
+	flag.Int64Var(&cfg.PartitionSize.ESP, "esp_partition_size", 128, "Override the ESP partition size (default: 128MiB)")
+	flag.Int64Var(&cfg.PartitionSize.System, "system_partition_size", 1024, "Override the System partition size (default: 1024MiB)")
+	flag.StringVar(&diskUUID, "GUID", "", "Disk GUID marked in the resulting image's partition table (default: randomly generated)")
 	flag.Parse()
 
 	// Open the input files for osimage.Create, fill in reader objects and
 	// metadata in osimage.Params.
 	// Start with the EFI Payload the OS will boot from.
-	p, err := os.Open(efiPayload)
+	p, err := blkio.NewFileReader(efiPayload)
 	if err != nil {
 		log.Fatalf("while opening the EFI payload at %q: %v", efiPayload, err)
 	}
-	defer p.Close()
 	cfg.EFIPayload = p
 
 	// Attempt to open the system image if its path is set. In case the path
@@ -76,14 +79,19 @@
 
 	// Attempt to open the node parameters file if its path is set.
 	if nodeParams != "" {
-		np, err := os.Open(nodeParams)
+		np, err := blkio.NewFileReader(nodeParams)
 		if err != nil {
 			log.Fatalf("while opening node parameters at %q: %v", nodeParams, err)
 		}
-		defer np.Close()
 		cfg.NodeParameters = np
 	}
 
+	// TODO(#254): Build and use dynamically-grown block devices
+	cfg.Output, err = blockdev.CreateFile(outputPath, 512, 10*1024*1024)
+	if err != nil {
+		panic(err)
+	}
+
 	// Write the parametrized OS image.
 	if _, err := osimage.Create(&cfg); err != nil {
 		log.Fatalf("while creating a Metropolis OS image: %v", err)
diff --git a/metropolis/node/build/mkimage/osimage/BUILD.bazel b/metropolis/node/build/mkimage/osimage/BUILD.bazel
index 35bfb9c..cfbf736 100644
--- a/metropolis/node/build/mkimage/osimage/BUILD.bazel
+++ b/metropolis/node/build/mkimage/osimage/BUILD.bazel
@@ -6,11 +6,10 @@
     importpath = "source.monogon.dev/metropolis/node/build/mkimage/osimage",
     visibility = ["//visibility:public"],
     deps = [
+        "//metropolis/pkg/blockdev",
         "//metropolis/pkg/efivarfs",
-        "@com_github_diskfs_go_diskfs//:go-diskfs",
-        "@com_github_diskfs_go_diskfs//disk",
-        "@com_github_diskfs_go_diskfs//filesystem",
-        "@com_github_diskfs_go_diskfs//partition/gpt",
+        "//metropolis/pkg/fat32",
+        "//metropolis/pkg/gpt",
         "@com_github_google_uuid//:uuid",
     ],
 )
diff --git a/metropolis/node/build/mkimage/osimage/osimage.go b/metropolis/node/build/mkimage/osimage/osimage.go
index 97fa7c0..7098439 100644
--- a/metropolis/node/build/mkimage/osimage/osimage.go
+++ b/metropolis/node/build/mkimage/osimage/osimage.go
@@ -21,379 +21,157 @@
 import (
 	"fmt"
 	"io"
-	"os"
+	"strings"
 
-	diskfs "github.com/diskfs/go-diskfs"
-	"github.com/diskfs/go-diskfs/disk"
-	"github.com/diskfs/go-diskfs/filesystem"
-	"github.com/diskfs/go-diskfs/partition/gpt"
 	"github.com/google/uuid"
 
+	"source.monogon.dev/metropolis/pkg/blockdev"
 	"source.monogon.dev/metropolis/pkg/efivarfs"
+	"source.monogon.dev/metropolis/pkg/fat32"
+	"source.monogon.dev/metropolis/pkg/gpt"
+)
+
+var (
+	SystemAType = uuid.MustParse("ee96054b-f6d0-4267-aaaa-724b2afea74c")
+	SystemBType = uuid.MustParse("ee96054b-f6d0-4267-bbbb-724b2afea74c")
+
+	DataType = uuid.MustParse("9eeec464-6885-414a-b278-4305c51f7966")
 )
 
 const (
-	systemPartitionType = gpt.Type("ee96055b-f6d0-4267-8bbb-724b2afea74c")
-	SystemVolumeLabel   = "METROPOLIS-SYSTEM"
-
-	dataPartitionType = gpt.Type("9eeec464-6885-414a-b278-4305c51f7966")
-	DataVolumeLabel   = "METROPOLIS-NODE-DATA"
-
-	ESPVolumeLabel = "ESP"
+	SystemLabel = "METROPOLIS-SYSTEM"
+	DataLabel   = "METROPOLIS-NODE-DATA"
+	ESPLabel    = "ESP"
 
 	EFIPayloadPath = "/EFI/BOOT/BOOTx64.EFI"
-	nodeParamsPath = "/metropolis/parameters.pb"
-
-	mib = 1024 * 1024
+	nodeParamsPath = "metropolis/parameters.pb"
 )
 
-// put creates a file on the target filesystem fs and fills it with
-// contents read from an io.Reader object src.
-func put(fs filesystem.FileSystem, dst string, src io.Reader) error {
-	target, err := fs.OpenFile(dst, os.O_CREATE|os.O_RDWR)
-	if err != nil {
-		return fmt.Errorf("while opening %q: %w", dst, err)
-	}
-
-	// If this is streamed (e.g. using io.Copy) it exposes a bug in diskfs, so
-	// do it in one go.
-	// TODO(mateusz@monogon.tech): Investigate the bug.
-	data, err := io.ReadAll(src)
-	if err != nil {
-		return fmt.Errorf("while reading %q: %w", src, err)
-	}
-	if _, err := target.Write(data); err != nil {
-		return fmt.Errorf("while writing to %q: %w", dst, err)
-	}
-	return nil
-}
-
-// initializeESP creates an ESP filesystem in a partition specified by
-// index. It then creates the EFI executable and copies into it contents
-// of the reader object exec, which must not be nil. The node parameters
-// file is optionally created if params is not nil. initializeESP may return
-// an error.
-func initializeESP(image *disk.Disk, index int, exec, params io.Reader) error {
-	// Create a FAT ESP filesystem inside a partition pointed to by
-	// index.
-	spec := disk.FilesystemSpec{
-		Partition:   index,
-		FSType:      filesystem.TypeFat32,
-		VolumeLabel: ESPVolumeLabel,
-	}
-	fs, err := image.CreateFilesystem(spec)
-	if err != nil {
-		return fmt.Errorf("while creating an ESP filesystem: %w", err)
-	}
-
-	// Create the EFI partition structure.
-	for _, dir := range []string{"/EFI", "/EFI/BOOT", "/metropolis"} {
-		if err := fs.Mkdir(dir); err != nil {
-			return fmt.Errorf("while creating %q: %w", dir, err)
-		}
-	}
-
-	// Copy the EFI payload to the newly created filesystem.
-	if exec == nil {
-		return fmt.Errorf("exec must not be nil")
-	}
-	if err := put(fs, EFIPayloadPath, exec); err != nil {
-		return fmt.Errorf("while writing an EFI payload: %w", err)
-	}
-
-	if params != nil {
-		// Copy Node Parameters into the ESP.
-		if err := put(fs, nodeParamsPath, params); err != nil {
-			return fmt.Errorf("while writing node parameters: %w", err)
-		}
-	}
-	return nil
-}
-
-// zeroSrc is a source of null bytes implementing io.Reader. It acts as a
-// helper data type.
-type zeroSrc struct{}
-
-// Read implements io.Reader for zeroSrc. It fills b with zero bytes. The
-// returned error is always nil.
-func (_ zeroSrc) Read(b []byte) (n int, err error) {
-	for i := range b {
-		b[i] = 0
-	}
-	return len(b), nil
-}
-
-// initializeSystemPartition copies system partition contents into a partition
-// at index. The remaining partition space is zero-padded. This function may
-// return an error.
-func initializeSystemPartition(image *disk.Disk, index int, contents io.Reader) error {
-	// Check the parameters.
-	if contents == nil {
-		return fmt.Errorf("system partition contents parameter must not be nil")
-	}
-	if index <= 0 {
-		return fmt.Errorf("partition index must be greater than zero")
-	}
-
-	// Get the system partition's size.
-	table, err := image.GetPartitionTable()
-	if err != nil {
-		return fmt.Errorf("while accessing a go-diskfs partition table: %w", err)
-	}
-	partitions := table.GetPartitions()
-	if index > len(partitions) {
-		return fmt.Errorf("partition index out of bounds")
-	}
-	size := partitions[index-1].GetSize()
-
-	// Copy the contents of the Metropolis system image into the system partition
-	// at partitionIndex. Zero-pad the remaining space.
-	var zero zeroSrc
-	sys := io.LimitReader(io.MultiReader(contents, zero), size)
-	if _, err := image.WritePartitionContents(index, sys); err != nil {
-		return fmt.Errorf("while copying the system partition: %w", err)
-	}
-	return nil
-}
-
-// mibToSectors converts a size expressed in mebibytes to a number of
-// sectors needed to store data of that size. sectorSize parameter
-// specifies the size of a logical sector.
-func mibToSectors(size, sectorSize uint64) uint64 {
-	return (size * mib) / sectorSize
-}
-
 // PartitionSizeInfo contains parameters used during partition table
 // initialization and, in case of image files, space allocation.
 type PartitionSizeInfo struct {
 	// Size of the EFI System Partition (ESP), in mebibytes. The size must
 	// not be zero.
-	ESP uint64
+	ESP int64
 	// Size of the Metropolis system partition, in mebibytes. The partition
 	// won't be created if the size is zero.
-	System uint64
+	System int64
 	// Size of the Metropolis data partition, in mebibytes. The partition
 	// won't be created if the size is zero. If the image is output to a
 	// block device, the partition will be extended to fill the remaining
 	// space.
-	Data uint64
-}
-
-// partitionList stores partition definitions in an ascending order.
-type partitionList []*gpt.Partition
-
-// appendPartition puts a new partition at the end of a partitionList,
-// automatically calculating its start and end markers based on data in
-// the list and the argument psize. A partition type and a name are
-// assigned to the partition. The containing image is used to calculate
-// sector addresses based on its logical block size.
-func (pl *partitionList) appendPartition(image *disk.Disk, ptype gpt.Type, pname string, psize uint64) {
-	// Calculate the start and end marker.
-	var pstart, pend uint64
-	if len(*pl) != 0 {
-		pstart = (*pl)[len(*pl)-1].End + 1
-	} else {
-		pstart = mibToSectors(1, uint64(image.LogicalBlocksize))
-	}
-	pend = pstart + mibToSectors(psize, uint64(image.LogicalBlocksize)) - 1
-
-	// Put the new partition at the end of the list.
-	*pl = append(*pl, &gpt.Partition{
-		Type:  ptype,
-		Name:  pname,
-		Start: pstart,
-		End:   pend,
-	})
-}
-
-// extendLastPartition moves the end marker of the last partition in a
-// partitionList to the end of image, assigning all of the remaining free
-// space to it. It may return an error.
-func (pl *partitionList) extendLastPartition(image *disk.Disk) error {
-	if len(*pl) == 0 {
-		return fmt.Errorf("no partitions defined")
-	}
-	if image.Size == 0 {
-		return fmt.Errorf("the image size mustn't be zero")
-	}
-	if image.LogicalBlocksize == 0 {
-		return fmt.Errorf("the image's logical block size mustn't be zero")
-	}
-
-	// The last 33 blocks are occupied by the Secondary GPT.
-	(*pl)[len(*pl)-1].End = uint64(image.Size/image.LogicalBlocksize) - 33
-	return nil
-}
-
-// initializePartitionTable applies a Metropolis-compatible GPT partition
-// table to an image. Logical and physical sector sizes are based on
-// block sizes exposed by Disk. Partition extents are defined by the size
-// argument. A diskGUID is associated with the partition table. In an event
-// the table couldn't have been applied, the function will return an error.
-func initializePartitionTable(image *disk.Disk, size *PartitionSizeInfo, diskGUID string) error {
-	// Start with preparing a partition list.
-	var pl partitionList
-	// Create the ESP.
-	if size.ESP == 0 {
-		return fmt.Errorf("ESP size mustn't be zero")
-	}
-	pl.appendPartition(image, gpt.EFISystemPartition, ESPVolumeLabel, size.ESP)
-	// Create the system partition only if its size is specified.
-	if size.System != 0 {
-		pl.appendPartition(image, systemPartitionType, SystemVolumeLabel, size.System)
-	}
-	// Create the data partition only if its size is specified.
-	if size.Data != 0 {
-		// Don't specify the last partition's length, as it will be extended
-		// to fit the image size anyway. size.Data will still be used in the
-		// space allocation step.
-		pl.appendPartition(image, dataPartitionType, DataVolumeLabel, 0)
-		if err := pl.extendLastPartition(image); err != nil {
-			return fmt.Errorf("while extending the last partition: %w", err)
-		}
-	}
-
-	// Build and apply the partition table.
-	table := &gpt.Table{
-		LogicalSectorSize:  int(image.LogicalBlocksize),
-		PhysicalSectorSize: int(image.PhysicalBlocksize),
-		ProtectiveMBR:      true,
-		GUID:               diskGUID,
-		Partitions:         pl,
-	}
-	if err := image.Partition(table); err != nil {
-		// Return the error unwrapped.
-		return err
-	}
-	return nil
+	Data int64
 }
 
 // Params contains parameters used by Create to build a Metropolis OS
 // image.
 type Params struct {
-	// OutputPath is the path an OS image will be written to. If the path
-	// points to an existing block device, the data partition will be
-	// extended to fill it entirely. Otherwise, a regular image file will
-	// be created at OutputPath. The path must not point to an existing
-	// regular file.
-	OutputPath string
+	// Output is the block device to which the OS image is written.
+	Output blockdev.BlockDev
 	// EFIPayload provides contents of the EFI payload file. It must not be
 	// nil.
-	EFIPayload io.Reader
+	EFIPayload fat32.SizedReader
 	// SystemImage provides contents of the Metropolis system partition.
 	// If nil, no contents will be copied into the partition.
 	SystemImage io.Reader
 	// NodeParameters provides contents of the node parameters file. If nil,
 	// the node parameters file won't be created in the target ESP
 	// filesystem.
-	NodeParameters io.Reader
-	// DiskGUID is a unique identifier of the image and a part of GPT
+	NodeParameters fat32.SizedReader
+	// DiskGUID is a unique identifier of the image and a part of Table
 	// header. It's optional and can be left blank if the identifier is
 	// to be randomly generated. Setting it to a predetermined value can
 	// help in implementing reproducible builds.
-	DiskGUID string
+	DiskGUID uuid.UUID
 	// PartitionSize specifies a size for the ESP, Metropolis System and
 	// Metropolis data partition.
 	PartitionSize PartitionSizeInfo
 }
 
-// Create writes a Metropolis OS image to either a newly created regular
-// file or a block device. The image is parametrized by the params
-// argument. In case a regular file already exists at params.OutputPath,
-// the function will fail. It returns nil on success or an error, if one
-// did occur.
+const Mi = 1024 * 1024
+
+// Create writes a Metropolis OS image to a block device.
 func Create(params *Params) (*efivarfs.LoadOption, error) {
-	// Validate each parameter before use.
-	if params.OutputPath == "" {
-		return nil, fmt.Errorf("image output path must be set")
+	// Discard the entire device, we're going to write new data over it.
+	// Ignore errors, this is only advisory.
+	params.Output.Discard(0, params.Output.BlockCount())
+
+	tbl, err := gpt.New(params.Output)
+	if err != nil {
+		return nil, fmt.Errorf("invalid block device: %w", err)
+	}
+	tbl.ID = params.DiskGUID
+	esp := gpt.Partition{
+		Type: gpt.PartitionTypeEFISystem,
+		Name: ESPLabel,
+	}
+	if err := tbl.AddPartition(&esp, params.PartitionSize.ESP*Mi); err != nil {
+		return nil, fmt.Errorf("failed to allocate ESP: %w", err)
 	}
 
-	// Learn whether we're creating a new image or writing to an existing
-	// block device by stat-ing the output path parameter.
-	outInfo, err := os.Stat(params.OutputPath)
-	if err != nil && !os.IsNotExist(err) {
+	rootInode := fat32.Inode{
+		Attrs: fat32.AttrDirectory,
+	}
+	if err := rootInode.PlaceFile(strings.TrimPrefix(EFIPayloadPath, "/"), params.EFIPayload); err != nil {
 		return nil, err
 	}
-
-	// Calculate the image size (bytes) by summing up partition sizes
-	// (mebibytes).
-	minSize := (int64(params.PartitionSize.ESP) +
-		int64(params.PartitionSize.System) +
-		int64(params.PartitionSize.Data) + 1) * mib
-	var diskImg *disk.Disk
-	if !os.IsNotExist(err) && outInfo.Mode()&os.ModeDevice != 0 {
-		// Open the block device. The data partition size parameter won't
-		// matter in this case, as said partition will be extended till the
-		// end of device.
-		diskImg, err = diskfs.Open(params.OutputPath)
-		if err != nil {
-			return nil, fmt.Errorf("couldn't open the block device at %q: %w", params.OutputPath, err)
-		}
-		// Make sure there's at least minSize space available on the block
-		// device.
-		if minSize > diskImg.Size {
-			return nil, fmt.Errorf("not enough space available on the block device at %q", params.OutputPath)
-		}
-	} else {
-		// Attempt to create an image file at params.OutputPath. diskfs.Create
-		// will abort in case a file already exists at the given path.
-		// Calculate the image size expressed in bytes by summing up declared
-		// partition lengths.
-		diskImg, err = diskfs.Create(params.OutputPath, minSize, diskfs.Raw)
-		if err != nil {
-			return nil, fmt.Errorf("couldn't create a disk image at %q: %w", params.OutputPath, err)
+	if params.NodeParameters != nil {
+		if err := rootInode.PlaceFile(nodeParamsPath, params.NodeParameters); err != nil {
+			return nil, err
 		}
 	}
-
-	// Go through the initialization steps, starting with applying a
-	// partition table according to params.
-	if err := initializePartitionTable(diskImg, &params.PartitionSize, params.DiskGUID); err != nil {
-		return nil, fmt.Errorf("failed to initialize the partition table: %w", err)
+	if err := fat32.WriteFS(blockdev.NewRWS(esp), rootInode, fat32.Options{
+		BlockSize:  uint16(esp.BlockSize()),
+		BlockCount: uint32(esp.BlockCount()),
+		Label:      "MNGN_BOOT",
+	}); err != nil {
+		return nil, fmt.Errorf("failed to write FAT32: %w", err)
 	}
-	// The system partition will be created only if its specified size isn't
-	// equal to zero, making the initialization step optional as well. In
-	// addition, params.SystemImage must be set.
+
+	// Create the system partition only if its size is specified.
 	if params.PartitionSize.System != 0 && params.SystemImage != nil {
-		if err := initializeSystemPartition(diskImg, 2, params.SystemImage); err != nil {
-			return nil, fmt.Errorf("failed to initialize the system partition: %w", err)
+		systemPartitionA := gpt.Partition{
+			Type: SystemAType,
+			Name: SystemLabel,
+		}
+		if err := tbl.AddPartition(&systemPartitionA, params.PartitionSize.System*Mi); err != nil {
+			return nil, fmt.Errorf("failed to allocate system partition A: %w", err)
+		}
+		if _, err := io.Copy(blockdev.NewRWS(systemPartitionA), params.SystemImage); err != nil {
+			return nil, fmt.Errorf("failed to write system partition A: %w", err)
 		}
 	} else if params.PartitionSize.System == 0 && params.SystemImage != nil {
 		// Safeguard against contradicting parameters.
 		return nil, fmt.Errorf("the system image parameter was passed while the associated partition size is zero")
 	}
-	// Attempt to initialize the ESP unconditionally, as it's the only
-	// partition guaranteed to be created regardless of params.PartitionSize.
-	if err := initializeESP(diskImg, 1, params.EFIPayload, params.NodeParameters); err != nil {
-		return nil, fmt.Errorf("failed to initialize the ESP: %w", err)
+	// Create the data partition only if its size is specified.
+	if params.PartitionSize.Data != 0 {
+		dataPartition := gpt.Partition{
+			Type: DataType,
+			Name: DataLabel,
+		}
+		if err := tbl.AddPartition(&dataPartition, -1); err != nil {
+			return nil, fmt.Errorf("failed to allocate data partition: %w", err)
+		}
 	}
-	// The data partition, even if created, is always left uninitialized.
 
-	// Build an EFI boot entry pointing to the image's ESP. go-diskfs won't let
-	// you do that after you close the image.
-	t, err := diskImg.GetPartitionTable()
-	p := t.GetPartitions()
-	esp := (p[0]).(*gpt.Partition)
-	guid, err := uuid.Parse(esp.GUID)
-	if err != nil {
-		return nil, fmt.Errorf("couldn't parse the GPT GUID: %w", err)
+	if err := tbl.Write(); err != nil {
+		return nil, fmt.Errorf("failed to write Table: %w", err)
 	}
-	be := efivarfs.LoadOption{
-		Description: "Metropolis",
+
+	// Build an EFI boot entry pointing to the image's ESP.
+	return &efivarfs.LoadOption{
+		Description: "Metropolis Slot A",
 		FilePath: efivarfs.DevicePath{
 			&efivarfs.HardDrivePath{
 				PartitionNumber:     1,
-				PartitionStartBlock: esp.Start,
-				PartitionSizeBlocks: esp.End - esp.Start + 1,
+				PartitionStartBlock: esp.FirstBlock,
+				PartitionSizeBlocks: esp.SizeBlocks(),
 				PartitionMatch: efivarfs.PartitionGPT{
-					PartitionUUID: guid,
+					PartitionUUID: esp.ID,
 				},
 			},
 			efivarfs.FilePath(EFIPayloadPath),
 		},
-	}
-	// Close the image and return the EFI boot entry.
-	if err := diskImg.File.Close(); err != nil {
-		return nil, fmt.Errorf("failed to finalize image: %w", err)
-	}
-	return &be, nil
+	}, nil
 }
diff --git a/metropolis/node/core/localstorage/crypt/BUILD.bazel b/metropolis/node/core/localstorage/crypt/BUILD.bazel
index b330bf1..44188d1 100644
--- a/metropolis/node/core/localstorage/crypt/BUILD.bazel
+++ b/metropolis/node/core/localstorage/crypt/BUILD.bazel
@@ -13,6 +13,7 @@
     importpath = "source.monogon.dev/metropolis/node/core/localstorage/crypt",
     visibility = ["//metropolis/node/core/localstorage:__subpackages__"],
     deps = [
+        "//metropolis/pkg/blockdev",
         "//metropolis/pkg/devicemapper",
         "//metropolis/pkg/efivarfs",
         "//metropolis/pkg/gpt",
diff --git a/metropolis/node/core/localstorage/crypt/blockdev.go b/metropolis/node/core/localstorage/crypt/blockdev.go
index 353164a..0dadb6d 100644
--- a/metropolis/node/core/localstorage/crypt/blockdev.go
+++ b/metropolis/node/core/localstorage/crypt/blockdev.go
@@ -23,11 +23,11 @@
 	"path/filepath"
 	"strconv"
 	"strings"
-	"unsafe"
 
 	"github.com/google/uuid"
 	"golang.org/x/sys/unix"
 
+	"source.monogon.dev/metropolis/pkg/blockdev"
 	"source.monogon.dev/metropolis/pkg/efivarfs"
 	"source.monogon.dev/metropolis/pkg/gpt"
 	"source.monogon.dev/metropolis/pkg/supervisor"
@@ -91,14 +91,15 @@
 		return nil
 	}
 
-	table, err := data.readPartitionTable()
+	blkdev, err := blockdev.Open(fmt.Sprintf("/dev/%v", data["DEVNAME"]))
 	if err != nil {
-		return fmt.Errorf("when reading disk info: %w", err)
+		return fmt.Errorf("failed to open block device: %w", err)
 	}
+	defer blkdev.Close()
 
-	// Not a normal block device or not a gpt table.
-	if table == nil {
-		return nil
+	table, err := gpt.Read(blkdev)
+	if err != nil {
+		return nil // Probably just not a GPT-partitioned disk
 	}
 
 	skipDisk := false
@@ -149,7 +150,6 @@
 		return err
 	}
 
-	// TODO(tim): Is this safe? Are we actually using the partition number for the slice index?
 	part := table.Partitions[pi.partNumber-1]
 
 	nodePath := nodePathForPartitionType(part.Type)
@@ -251,35 +251,3 @@
 
 	return
 }
-
-// readPartitionTable tries to read a GPT partition table based on the blockUEvent
-// data. It returns nil when either the block device is not a regular device
-// or it fails to parse the GPT table.
-func (b blockUEvent) readPartitionTable() (*gpt.Table, error) {
-	// TODO(lorenz): This extraction code is all a bit hairy, will get
-	// replaced by blockdev shortly.
-	blkdev, err := os.Open(fmt.Sprintf("/dev/%v", b["DEVNAME"]))
-	if err != nil {
-		return nil, fmt.Errorf("failed to open block device: %w", err)
-	}
-	defer blkdev.Close()
-
-	blockSize, err := unix.IoctlGetUint32(int(blkdev.Fd()), unix.BLKSSZGET)
-	if err != nil {
-		return nil, nil // This is not a regular block device
-	}
-
-	var sizeBytes uint64
-	_, _, err = unix.Syscall(unix.SYS_IOCTL, blkdev.Fd(), unix.BLKGETSIZE64, uintptr(unsafe.Pointer(&sizeBytes)))
-	if err != unix.Errno(0) {
-		return nil, fmt.Errorf("failed to get device size: %w", err)
-	}
-
-	blkdev.Seek(int64(blockSize), 0)
-	table, err := gpt.Read(blkdev, int64(blockSize), int64(sizeBytes)/int64(blockSize))
-	if err != nil {
-		return nil, nil // Probably just not a GPT-partitioned disk
-	}
-
-	return table, nil
-}
diff --git a/metropolis/node/core/localstorage/crypt/crypt.go b/metropolis/node/core/localstorage/crypt/crypt.go
index af7451b..0336832 100644
--- a/metropolis/node/core/localstorage/crypt/crypt.go
+++ b/metropolis/node/core/localstorage/crypt/crypt.go
@@ -36,13 +36,9 @@
 package crypt
 
 import (
-	"errors"
 	"fmt"
-	"os"
-	"syscall"
-	"unsafe"
 
-	"golang.org/x/sys/unix"
+	"source.monogon.dev/metropolis/pkg/blockdev"
 )
 
 // Mode of block device encryption and/or authentication, if any. See the
@@ -93,37 +89,6 @@
 	panic("invalid mode " + m)
 }
 
-// getSizeBytes returns the size of a block device in bytes.
-func getSizeBytes(path string) (uint64, error) {
-	blkdev, err := os.Open(path)
-	if err != nil {
-		return 0, fmt.Errorf("failed to open block device: %w", err)
-	}
-	defer blkdev.Close()
-
-	var sizeBytes uint64
-	_, _, err = unix.Syscall(unix.SYS_IOCTL, blkdev.Fd(), unix.BLKGETSIZE64, uintptr(unsafe.Pointer(&sizeBytes)))
-	if err != unix.Errno(0) {
-		return 0, fmt.Errorf("failed to get device size: %w", err)
-	}
-	return sizeBytes, nil
-}
-
-// getBlockSize returns the size of a block device's sector in bytes.
-func getBlockSize(path string) (uint32, error) {
-	blkdev, err := os.Open(path)
-	if err != nil {
-		return 0, fmt.Errorf("failed to open block device: %w", err)
-	}
-	defer blkdev.Close()
-
-	blockSize, err := unix.IoctlGetUint32(int(blkdev.Fd()), unix.BLKSSZGET)
-	if err != nil {
-		return 0, fmt.Errorf("BLKSSZGET: %w", err)
-	}
-	return blockSize, nil
-}
-
 // Map sets up an underlying block device (at path 'underlying') for access.
 // Depending on the given mode, authentication/integrity device-mapper targets
 // will be set up, and the top-level new block device path will be returned.
@@ -236,30 +201,14 @@
 
 	// Zero out device if authentication is enabled.
 	if mode.authenticated() {
-		blockSize, err := getBlockSize(device)
+		blkdev, err := blockdev.Open(device)
 		if err != nil {
 			return "", err
 		}
-
-		blkdev, err := os.OpenFile(device, unix.O_DIRECT|os.O_WRONLY, 0000)
+		err = blkdev.Zero(0, blkdev.BlockCount()*blkdev.BlockSize())
+		blkdev.Close()
 		if err != nil {
-			return "", fmt.Errorf("failed to open new device for zeroing: %w", err)
-		}
-
-		// Use a multiple of the block size to make the initial zeroing faster.
-		zeroedBuf := make([]byte, blockSize*256)
-		for {
-			_, err := blkdev.Write(zeroedBuf)
-			if errors.Is(err, syscall.ENOSPC) {
-				break
-			}
-			if err != nil {
-				blkdev.Close()
-				return "", fmt.Errorf("failed to zero-initalize new device: %w", err)
-			}
-		}
-		if err := blkdev.Close(); err != nil {
-			return "", fmt.Errorf("failed to close initialized device: %w", err)
+			return "", fmt.Errorf("failed to zero-initalize new device: %w", err)
 		}
 	}
 
diff --git a/metropolis/node/core/localstorage/crypt/crypt_encryption.go b/metropolis/node/core/localstorage/crypt/crypt_encryption.go
index 4fd6061..c5f246c 100644
--- a/metropolis/node/core/localstorage/crypt/crypt_encryption.go
+++ b/metropolis/node/core/localstorage/crypt/crypt_encryption.go
@@ -7,6 +7,7 @@
 
 	"golang.org/x/sys/unix"
 
+	"source.monogon.dev/metropolis/pkg/blockdev"
 	"source.monogon.dev/metropolis/pkg/devicemapper"
 )
 
@@ -19,14 +20,11 @@
 }
 
 func mapEncryption(name, underlying string, encryptionKey []byte, authenticated bool) (string, error) {
-	sizeBytes, err := getSizeBytes(underlying)
+	blkdev, err := blockdev.Open(underlying)
 	if err != nil {
-		return "", fmt.Errorf("getting size of block device failed: %w", err)
+		return "", fmt.Errorf("opening underlying block device failed: %w", err)
 	}
-	blockSize, err := getBlockSize(underlying)
-	if err != nil {
-		return "", fmt.Errorf("getting block size failed: %w", err)
-	}
+	defer blkdev.Close()
 
 	optParams := []string{
 		"no_read_workqueue", "no_write_workqueue",
@@ -49,7 +47,7 @@
 
 	cryptDev, err := devicemapper.CreateActiveDevice(encryptionDMName(name), false, []devicemapper.Target{
 		{
-			Length:     sizeBytes / uint64(blockSize),
+			Length:     uint64(blkdev.BlockCount()),
 			Type:       "crypt",
 			Parameters: params,
 		},
diff --git a/metropolis/node/core/localstorage/crypt/crypt_integrity.go b/metropolis/node/core/localstorage/crypt/crypt_integrity.go
index 4130aef..7276a3e 100644
--- a/metropolis/node/core/localstorage/crypt/crypt_integrity.go
+++ b/metropolis/node/core/localstorage/crypt/crypt_integrity.go
@@ -7,6 +7,7 @@
 
 	"golang.org/x/sys/unix"
 
+	"source.monogon.dev/metropolis/pkg/blockdev"
 	"source.monogon.dev/metropolis/pkg/devicemapper"
 )
 
@@ -25,20 +26,19 @@
 // This is described in further detail in
 // https://docs.kernel.org/admin-guide/device-mapper/dm-integrity.html.
 func readIntegrityDataSectors(path string) (uint64, error) {
-	integrityPartition, err := os.Open(path)
+	integrityPartition, err := blockdev.Open(path)
 	if err != nil {
 		return 0, err
 	}
 	defer integrityPartition.Close()
+
+	firstBlock := make([]byte, integrityPartition.BlockSize())
+	if _, err = integrityPartition.ReadAt(firstBlock, 0); err != nil {
+		return 0, err
+	}
 	// Based on structure defined in
 	//   https://git.kernel.org/pub/scm/linux/kernel/git/torvalds/linux.git/tree/drivers/md/dm-integrity.c#n59
-	if _, err := integrityPartition.Seek(16, 0); err != nil {
-		return 0, err
-	}
-	var providedDataSectors uint64
-	if err := binary.Read(integrityPartition, binary.LittleEndian, &providedDataSectors); err != nil {
-		return 0, err
-	}
+	providedDataSectors := binary.LittleEndian.Uint64(firstBlock[16:24])
 
 	// Let's perform some simple checks on the read value to make sure the returned
 	// data isn't corrupted or has been tampered with.
@@ -47,17 +47,8 @@
 		return 0, fmt.Errorf("invalid data sector count of zero")
 	}
 
-	underlyingSizeBytes, err := getSizeBytes(path)
-	if err != nil {
-		return 0, fmt.Errorf("getting underlying block device size failed: %w", err)
-	}
-	underlyingBlockSize, err := getBlockSize(path)
-	if err != nil {
-		return 0, fmt.Errorf("getting underlying block device block size failed: %w", err)
-	}
-	underlyingSectors := underlyingSizeBytes / uint64(underlyingBlockSize)
-	if providedDataSectors > underlyingSectors {
-		return 0, fmt.Errorf("device claims %d data sectors but underlying device only has %d", providedDataSectors, underlyingSectors)
+	if providedDataSectors > uint64(integrityPartition.BlockCount()) {
+		return 0, fmt.Errorf("device claims %d data sectors but underlying device only has %d", providedDataSectors, integrityPartition.BlockCount())
 	}
 	return providedDataSectors, nil
 }
diff --git a/metropolis/node/core/localstorage/directory_root.go b/metropolis/node/core/localstorage/directory_root.go
index 44af268..c385f70 100644
--- a/metropolis/node/core/localstorage/directory_root.go
+++ b/metropolis/node/core/localstorage/directory_root.go
@@ -62,6 +62,7 @@
 		r.Ephemeral.Consensus,
 		r.Ephemeral.Containerd, r.Ephemeral.Containerd.Tmp, r.Ephemeral.Containerd.RunSC, r.Ephemeral.Containerd.IPAM,
 		r.Ephemeral.FlexvolumePlugins,
+		r.ESP.Metropolis,
 	} {
 		err := d.MkdirAll(0700)
 		if err != nil {