Merge pull request #9239 from jiangliu/cri-multi-snapshotters
CRI: use (snapshotter_id, snapshot_key) to uniquely identify snapshots
This commit is contained in:
commit
dc7dba9c20
@ -81,11 +81,21 @@ func (c *criService) getMetricsHandler(ctx context.Context, sandboxID string) (m
|
|||||||
return nil, err
|
return nil, err
|
||||||
}
|
}
|
||||||
|
|
||||||
|
ociRuntime, err := c.getSandboxRuntime(sandbox.Config, sandbox.RuntimeHandler)
|
||||||
|
if err != nil {
|
||||||
|
return nil, fmt.Errorf("failed to get runtimeHandler %q: %w", sandbox.RuntimeHandler, err)
|
||||||
|
}
|
||||||
|
snapshotter := c.RuntimeSnapshotter(ctx, ociRuntime)
|
||||||
|
|
||||||
switch p.OS {
|
switch p.OS {
|
||||||
case "windows":
|
case "windows":
|
||||||
return c.windowsContainerMetrics, nil
|
return func(meta containerstore.Metadata, stats *types.Metric) (*runtime.ContainerStats, error) {
|
||||||
|
return c.windowsContainerMetrics(meta, stats, snapshotter)
|
||||||
|
}, nil
|
||||||
case "linux":
|
case "linux":
|
||||||
return c.linuxContainerMetrics, nil
|
return func(meta containerstore.Metadata, stats *types.Metric) (*runtime.ContainerStats, error) {
|
||||||
|
return c.linuxContainerMetrics(meta, stats, snapshotter)
|
||||||
|
}, nil
|
||||||
default:
|
default:
|
||||||
return nil, fmt.Errorf("container metrics for platform %+v: %w", p, errdefs.ErrNotImplemented)
|
return nil, fmt.Errorf("container metrics for platform %+v: %w", p, errdefs.ErrNotImplemented)
|
||||||
}
|
}
|
||||||
@ -267,10 +277,11 @@ func matchLabelSelector(selector, labels map[string]string) bool {
|
|||||||
func (c *criService) windowsContainerMetrics(
|
func (c *criService) windowsContainerMetrics(
|
||||||
meta containerstore.Metadata,
|
meta containerstore.Metadata,
|
||||||
stats *types.Metric,
|
stats *types.Metric,
|
||||||
|
snapshotter string,
|
||||||
) (*runtime.ContainerStats, error) {
|
) (*runtime.ContainerStats, error) {
|
||||||
var cs runtime.ContainerStats
|
var cs runtime.ContainerStats
|
||||||
var usedBytes, inodesUsed uint64
|
var usedBytes, inodesUsed uint64
|
||||||
sn, err := c.GetSnapshot(meta.ID)
|
sn, err := c.GetSnapshot(meta.ID, snapshotter)
|
||||||
// If snapshotstore doesn't have cached snapshot information
|
// If snapshotstore doesn't have cached snapshot information
|
||||||
// set WritableLayer usage to zero
|
// set WritableLayer usage to zero
|
||||||
if err == nil {
|
if err == nil {
|
||||||
@ -280,7 +291,7 @@ func (c *criService) windowsContainerMetrics(
|
|||||||
cs.WritableLayer = &runtime.FilesystemUsage{
|
cs.WritableLayer = &runtime.FilesystemUsage{
|
||||||
Timestamp: sn.Timestamp,
|
Timestamp: sn.Timestamp,
|
||||||
FsId: &runtime.FilesystemIdentifier{
|
FsId: &runtime.FilesystemIdentifier{
|
||||||
Mountpoint: c.imageFSPath,
|
Mountpoint: c.imageFSPaths[snapshotter],
|
||||||
},
|
},
|
||||||
UsedBytes: &runtime.UInt64Value{Value: usedBytes},
|
UsedBytes: &runtime.UInt64Value{Value: usedBytes},
|
||||||
InodesUsed: &runtime.UInt64Value{Value: inodesUsed},
|
InodesUsed: &runtime.UInt64Value{Value: inodesUsed},
|
||||||
@ -322,10 +333,11 @@ func (c *criService) windowsContainerMetrics(
|
|||||||
func (c *criService) linuxContainerMetrics(
|
func (c *criService) linuxContainerMetrics(
|
||||||
meta containerstore.Metadata,
|
meta containerstore.Metadata,
|
||||||
stats *types.Metric,
|
stats *types.Metric,
|
||||||
|
snapshotter string,
|
||||||
) (*runtime.ContainerStats, error) {
|
) (*runtime.ContainerStats, error) {
|
||||||
var cs runtime.ContainerStats
|
var cs runtime.ContainerStats
|
||||||
var usedBytes, inodesUsed uint64
|
var usedBytes, inodesUsed uint64
|
||||||
sn, err := c.GetSnapshot(meta.ID)
|
sn, err := c.GetSnapshot(meta.ID, snapshotter)
|
||||||
// If snapshotstore doesn't have cached snapshot information
|
// If snapshotstore doesn't have cached snapshot information
|
||||||
// set WritableLayer usage to zero
|
// set WritableLayer usage to zero
|
||||||
if err == nil {
|
if err == nil {
|
||||||
@ -335,7 +347,7 @@ func (c *criService) linuxContainerMetrics(
|
|||||||
cs.WritableLayer = &runtime.FilesystemUsage{
|
cs.WritableLayer = &runtime.FilesystemUsage{
|
||||||
Timestamp: sn.Timestamp,
|
Timestamp: sn.Timestamp,
|
||||||
FsId: &runtime.FilesystemIdentifier{
|
FsId: &runtime.FilesystemIdentifier{
|
||||||
Mountpoint: c.imageFSPath,
|
Mountpoint: c.imageFSPaths[snapshotter],
|
||||||
},
|
},
|
||||||
UsedBytes: &runtime.UInt64Value{Value: usedBytes},
|
UsedBytes: &runtime.UInt64Value{Value: usedBytes},
|
||||||
InodesUsed: &runtime.UInt64Value{Value: inodesUsed},
|
InodesUsed: &runtime.UInt64Value{Value: inodesUsed},
|
||||||
|
@ -278,7 +278,7 @@ func (s *fakeImageService) UpdateImage(ctx context.Context, r string) error { re
|
|||||||
|
|
||||||
func (s *fakeImageService) GetImage(id string) (imagestore.Image, error) { return s.imageStore.Get(id) }
|
func (s *fakeImageService) GetImage(id string) (imagestore.Image, error) { return s.imageStore.Get(id) }
|
||||||
|
|
||||||
func (s *fakeImageService) GetSnapshot(key string) (snapshotstore.Snapshot, error) {
|
func (s *fakeImageService) GetSnapshot(key, snapshotter string) (snapshotstore.Snapshot, error) {
|
||||||
return snapshotstore.Snapshot{}, errors.New("not implemented")
|
return snapshotstore.Snapshot{}, errors.New("not implemented")
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -20,32 +20,64 @@ import (
|
|||||||
"context"
|
"context"
|
||||||
"time"
|
"time"
|
||||||
|
|
||||||
|
"github.com/containerd/containerd/pkg/cri/store/snapshot"
|
||||||
runtime "k8s.io/cri-api/pkg/apis/runtime/v1"
|
runtime "k8s.io/cri-api/pkg/apis/runtime/v1"
|
||||||
)
|
)
|
||||||
|
|
||||||
// ImageFsInfo returns information of the filesystem that is used to store images.
|
// ImageFsInfo returns information of the filesystem that is used to store images.
|
||||||
// TODO(windows): Usage for windows is always 0 right now. Support this for windows.
|
// TODO(windows): Usage for windows is always 0 right now. Support this for windows.
|
||||||
|
// TODO(random-liu): Handle storage consumed by content store
|
||||||
func (c *CRIImageService) ImageFsInfo(ctx context.Context, r *runtime.ImageFsInfoRequest) (*runtime.ImageFsInfoResponse, error) {
|
func (c *CRIImageService) ImageFsInfo(ctx context.Context, r *runtime.ImageFsInfoRequest) (*runtime.ImageFsInfoResponse, error) {
|
||||||
snapshots := c.snapshotStore.List()
|
snapshots := c.snapshotStore.List()
|
||||||
timestamp := time.Now().UnixNano()
|
snapshotterFSInfos := map[string]snapshot.Snapshot{}
|
||||||
var usedBytes, inodesUsed uint64
|
|
||||||
for _, sn := range snapshots {
|
for _, sn := range snapshots {
|
||||||
|
if info, ok := snapshotterFSInfos[sn.Key.Snapshotter]; ok {
|
||||||
// Use the oldest timestamp as the timestamp of imagefs info.
|
// Use the oldest timestamp as the timestamp of imagefs info.
|
||||||
if sn.Timestamp < timestamp {
|
if sn.Timestamp < info.Timestamp {
|
||||||
timestamp = sn.Timestamp
|
info.Timestamp = sn.Timestamp
|
||||||
}
|
}
|
||||||
usedBytes += sn.Size
|
info.Size += sn.Size
|
||||||
inodesUsed += sn.Inodes
|
info.Inodes += sn.Inodes
|
||||||
|
snapshotterFSInfos[sn.Key.Snapshotter] = info
|
||||||
|
} else {
|
||||||
|
snapshotterFSInfos[sn.Key.Snapshotter] = snapshot.Snapshot{
|
||||||
|
Timestamp: sn.Timestamp,
|
||||||
|
Size: sn.Size,
|
||||||
|
Inodes: sn.Inodes,
|
||||||
}
|
}
|
||||||
// TODO(random-liu): Handle content store
|
}
|
||||||
return &runtime.ImageFsInfoResponse{
|
}
|
||||||
ImageFilesystems: []*runtime.FilesystemUsage{
|
|
||||||
{
|
var imageFilesystems []*runtime.FilesystemUsage
|
||||||
Timestamp: timestamp,
|
|
||||||
FsId: &runtime.FilesystemIdentifier{Mountpoint: c.imageFSPath},
|
// Currently kubelet always consumes the first entry of the returned array,
|
||||||
UsedBytes: &runtime.UInt64Value{Value: usedBytes},
|
// so put the default snapshotter as the first entry for compatibility.
|
||||||
InodesUsed: &runtime.UInt64Value{Value: inodesUsed},
|
if info, ok := snapshotterFSInfos[c.config.Snapshotter]; ok {
|
||||||
},
|
imageFilesystems = append(imageFilesystems, &runtime.FilesystemUsage{
|
||||||
},
|
Timestamp: info.Timestamp,
|
||||||
}, nil
|
FsId: &runtime.FilesystemIdentifier{Mountpoint: c.imageFSPaths[c.config.Snapshotter]},
|
||||||
|
UsedBytes: &runtime.UInt64Value{Value: info.Size},
|
||||||
|
InodesUsed: &runtime.UInt64Value{Value: info.Inodes},
|
||||||
|
})
|
||||||
|
delete(snapshotterFSInfos, c.config.Snapshotter)
|
||||||
|
} else {
|
||||||
|
imageFilesystems = append(imageFilesystems, &runtime.FilesystemUsage{
|
||||||
|
Timestamp: time.Now().UnixNano(),
|
||||||
|
FsId: &runtime.FilesystemIdentifier{Mountpoint: c.imageFSPaths[c.config.Snapshotter]},
|
||||||
|
UsedBytes: &runtime.UInt64Value{Value: 0},
|
||||||
|
InodesUsed: &runtime.UInt64Value{Value: 0},
|
||||||
|
})
|
||||||
|
}
|
||||||
|
|
||||||
|
for snapshotter, info := range snapshotterFSInfos {
|
||||||
|
imageFilesystems = append(imageFilesystems, &runtime.FilesystemUsage{
|
||||||
|
Timestamp: info.Timestamp,
|
||||||
|
FsId: &runtime.FilesystemIdentifier{Mountpoint: c.imageFSPaths[snapshotter]},
|
||||||
|
UsedBytes: &runtime.UInt64Value{Value: info.Size},
|
||||||
|
InodesUsed: &runtime.UInt64Value{Value: info.Inodes},
|
||||||
|
})
|
||||||
|
}
|
||||||
|
|
||||||
|
return &runtime.ImageFsInfoResponse{ImageFilesystems: imageFilesystems}, nil
|
||||||
}
|
}
|
||||||
|
@ -32,21 +32,30 @@ func TestImageFsInfo(t *testing.T) {
|
|||||||
c := newTestCRIService()
|
c := newTestCRIService()
|
||||||
snapshots := []snapshotstore.Snapshot{
|
snapshots := []snapshotstore.Snapshot{
|
||||||
{
|
{
|
||||||
|
Key: snapshotstore.Key{
|
||||||
Key: "key1",
|
Key: "key1",
|
||||||
|
Snapshotter: "overlayfs",
|
||||||
|
},
|
||||||
Kind: snapshot.KindActive,
|
Kind: snapshot.KindActive,
|
||||||
Size: 10,
|
Size: 10,
|
||||||
Inodes: 100,
|
Inodes: 100,
|
||||||
Timestamp: 234567,
|
Timestamp: 234567,
|
||||||
},
|
},
|
||||||
{
|
{
|
||||||
|
Key: snapshotstore.Key{
|
||||||
Key: "key2",
|
Key: "key2",
|
||||||
|
Snapshotter: "overlayfs",
|
||||||
|
},
|
||||||
Kind: snapshot.KindCommitted,
|
Kind: snapshot.KindCommitted,
|
||||||
Size: 20,
|
Size: 20,
|
||||||
Inodes: 200,
|
Inodes: 200,
|
||||||
Timestamp: 123456,
|
Timestamp: 123456,
|
||||||
},
|
},
|
||||||
{
|
{
|
||||||
|
Key: snapshotstore.Key{
|
||||||
Key: "key3",
|
Key: "key3",
|
||||||
|
Snapshotter: "overlayfs",
|
||||||
|
},
|
||||||
Kind: snapshot.KindView,
|
Kind: snapshot.KindView,
|
||||||
Size: 0,
|
Size: 0,
|
||||||
Inodes: 0,
|
Inodes: 0,
|
||||||
@ -65,6 +74,7 @@ func TestImageFsInfo(t *testing.T) {
|
|||||||
resp, err := c.ImageFsInfo(context.Background(), &runtime.ImageFsInfoRequest{})
|
resp, err := c.ImageFsInfo(context.Background(), &runtime.ImageFsInfoRequest{})
|
||||||
require.NoError(t, err)
|
require.NoError(t, err)
|
||||||
stats := resp.GetImageFilesystems()
|
stats := resp.GetImageFilesystems()
|
||||||
assert.Len(t, stats, 1)
|
// stats[0] is for default snapshotter, stats[1] is for `overlayfs`
|
||||||
assert.Equal(t, expected, stats[0])
|
assert.Len(t, stats, 2)
|
||||||
|
assert.Equal(t, expected, stats[1])
|
||||||
}
|
}
|
||||||
|
@ -25,8 +25,10 @@ import (
|
|||||||
criconfig "github.com/containerd/containerd/pkg/cri/config"
|
criconfig "github.com/containerd/containerd/pkg/cri/config"
|
||||||
imagestore "github.com/containerd/containerd/pkg/cri/store/image"
|
imagestore "github.com/containerd/containerd/pkg/cri/store/image"
|
||||||
snapshotstore "github.com/containerd/containerd/pkg/cri/store/snapshot"
|
snapshotstore "github.com/containerd/containerd/pkg/cri/store/snapshot"
|
||||||
|
ctrdutil "github.com/containerd/containerd/pkg/cri/util"
|
||||||
"github.com/containerd/containerd/pkg/kmutex"
|
"github.com/containerd/containerd/pkg/kmutex"
|
||||||
"github.com/containerd/containerd/platforms"
|
"github.com/containerd/containerd/platforms"
|
||||||
|
snapshot "github.com/containerd/containerd/snapshots"
|
||||||
"github.com/containerd/log"
|
"github.com/containerd/log"
|
||||||
docker "github.com/distribution/reference"
|
docker "github.com/distribution/reference"
|
||||||
imagedigest "github.com/opencontainers/go-digest"
|
imagedigest "github.com/opencontainers/go-digest"
|
||||||
@ -37,8 +39,8 @@ type CRIImageService struct {
|
|||||||
config criconfig.Config
|
config criconfig.Config
|
||||||
// client is an instance of the containerd client
|
// client is an instance of the containerd client
|
||||||
client *containerd.Client
|
client *containerd.Client
|
||||||
// imageFSPath is the path to image filesystem.
|
// imageFSPaths contains path to image filesystem for snapshotters.
|
||||||
imageFSPath string
|
imageFSPaths map[string]string
|
||||||
// imageStore stores all resources associated with images.
|
// imageStore stores all resources associated with images.
|
||||||
imageStore *imagestore.Store
|
imageStore *imagestore.Store
|
||||||
// snapshotStore stores information of all snapshots.
|
// snapshotStore stores information of all snapshots.
|
||||||
@ -49,25 +51,42 @@ type CRIImageService struct {
|
|||||||
unpackDuplicationSuppressor kmutex.KeyedLocker
|
unpackDuplicationSuppressor kmutex.KeyedLocker
|
||||||
}
|
}
|
||||||
|
|
||||||
func NewService(config criconfig.Config, imageFSPath string, client *containerd.Client) (*CRIImageService, error) {
|
func NewService(config criconfig.Config, imageFSPaths map[string]string, client *containerd.Client) (*CRIImageService, error) {
|
||||||
svc := CRIImageService{
|
svc := CRIImageService{
|
||||||
config: config,
|
config: config,
|
||||||
client: client,
|
client: client,
|
||||||
imageStore: imagestore.NewStore(client.ImageService(), client.ContentStore(), platforms.Default()),
|
imageStore: imagestore.NewStore(client.ImageService(), client.ContentStore(), platforms.Default()),
|
||||||
imageFSPath: imageFSPath,
|
imageFSPaths: imageFSPaths,
|
||||||
snapshotStore: snapshotstore.NewStore(),
|
snapshotStore: snapshotstore.NewStore(),
|
||||||
unpackDuplicationSuppressor: kmutex.New(),
|
unpackDuplicationSuppressor: kmutex.New(),
|
||||||
}
|
}
|
||||||
|
|
||||||
if client.SnapshotService(svc.config.ContainerdConfig.Snapshotter) == nil {
|
snapshotters := map[string]snapshot.Snapshotter{}
|
||||||
return nil, fmt.Errorf("failed to find snapshotter %q", svc.config.ContainerdConfig.Snapshotter)
|
ctx := ctrdutil.NamespacedContext()
|
||||||
|
|
||||||
|
// Add runtime specific snapshotters
|
||||||
|
for _, runtime := range config.ContainerdConfig.Runtimes {
|
||||||
|
snapshotterName := svc.RuntimeSnapshotter(ctx, runtime)
|
||||||
|
if snapshotter := svc.client.SnapshotService(snapshotterName); snapshotter != nil {
|
||||||
|
snapshotters[snapshotterName] = snapshotter
|
||||||
|
} else {
|
||||||
|
return nil, fmt.Errorf("failed to find snapshotter %q", snapshotterName)
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
// Add default snapshotter
|
||||||
|
snapshotterName := svc.config.ContainerdConfig.Snapshotter
|
||||||
|
if snapshotter := svc.client.SnapshotService(snapshotterName); snapshotter != nil {
|
||||||
|
snapshotters[snapshotterName] = snapshotter
|
||||||
|
} else {
|
||||||
|
return nil, fmt.Errorf("failed to find snapshotter %q", snapshotterName)
|
||||||
}
|
}
|
||||||
|
|
||||||
// Start snapshot stats syncer, it doesn't need to be stopped.
|
// Start snapshot stats syncer, it doesn't need to be stopped.
|
||||||
log.L.Info("Start snapshots syncer")
|
log.L.Info("Start snapshots syncer")
|
||||||
snapshotsSyncer := newSnapshotsSyncer(
|
snapshotsSyncer := newSnapshotsSyncer(
|
||||||
svc.snapshotStore,
|
svc.snapshotStore,
|
||||||
svc.client.SnapshotService(svc.config.ContainerdConfig.Snapshotter),
|
snapshotters,
|
||||||
time.Duration(svc.config.StatsCollectPeriod)*time.Second,
|
time.Duration(svc.config.StatsCollectPeriod)*time.Second,
|
||||||
)
|
)
|
||||||
snapshotsSyncer.start()
|
snapshotsSyncer.start()
|
||||||
@ -122,6 +141,10 @@ func (c *CRIImageService) GetImage(id string) (imagestore.Image, error) {
|
|||||||
}
|
}
|
||||||
|
|
||||||
// GetSnapshot returns the snapshot with specified key.
|
// GetSnapshot returns the snapshot with specified key.
|
||||||
func (c *CRIImageService) GetSnapshot(key string) (snapshotstore.Snapshot, error) {
|
func (c *CRIImageService) GetSnapshot(key, snapshotter string) (snapshotstore.Snapshot, error) {
|
||||||
return c.snapshotStore.Get(key)
|
snapshotKey := snapshotstore.Key{
|
||||||
|
Key: key,
|
||||||
|
Snapshotter: snapshotter,
|
||||||
|
}
|
||||||
|
return c.snapshotStore.Get(snapshotKey)
|
||||||
}
|
}
|
||||||
|
@ -42,7 +42,7 @@ const (
|
|||||||
func newTestCRIService() *CRIImageService {
|
func newTestCRIService() *CRIImageService {
|
||||||
return &CRIImageService{
|
return &CRIImageService{
|
||||||
config: testConfig,
|
config: testConfig,
|
||||||
imageFSPath: testImageFSPath,
|
imageFSPaths: map[string]string{"overlayfs": testImageFSPath},
|
||||||
imageStore: imagestore.NewStore(nil, nil, platforms.Default()),
|
imageStore: imagestore.NewStore(nil, nil, platforms.Default()),
|
||||||
snapshotStore: snapshotstore.NewStore(),
|
snapshotStore: snapshotstore.NewStore(),
|
||||||
}
|
}
|
||||||
|
@ -34,16 +34,16 @@ import (
|
|||||||
// benchmark result shows that container cpu/memory stats also need to be cached.
|
// benchmark result shows that container cpu/memory stats also need to be cached.
|
||||||
type snapshotsSyncer struct {
|
type snapshotsSyncer struct {
|
||||||
store *snapshotstore.Store
|
store *snapshotstore.Store
|
||||||
snapshotter snapshot.Snapshotter
|
snapshotters map[string]snapshot.Snapshotter
|
||||||
syncPeriod time.Duration
|
syncPeriod time.Duration
|
||||||
}
|
}
|
||||||
|
|
||||||
// newSnapshotsSyncer creates a snapshot syncer.
|
// newSnapshotsSyncer creates a snapshot syncer.
|
||||||
func newSnapshotsSyncer(store *snapshotstore.Store, snapshotter snapshot.Snapshotter,
|
func newSnapshotsSyncer(store *snapshotstore.Store, snapshotters map[string]snapshot.Snapshotter,
|
||||||
period time.Duration) *snapshotsSyncer {
|
period time.Duration) *snapshotsSyncer {
|
||||||
return &snapshotsSyncer{
|
return &snapshotsSyncer{
|
||||||
store: store,
|
store: store,
|
||||||
snapshotter: snapshotter,
|
snapshotters: snapshotters,
|
||||||
syncPeriod: period,
|
syncPeriod: period,
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
@ -70,19 +70,25 @@ func (s *snapshotsSyncer) start() {
|
|||||||
func (s *snapshotsSyncer) sync() error {
|
func (s *snapshotsSyncer) sync() error {
|
||||||
ctx := ctrdutil.NamespacedContext()
|
ctx := ctrdutil.NamespacedContext()
|
||||||
start := time.Now().UnixNano()
|
start := time.Now().UnixNano()
|
||||||
|
|
||||||
|
for key, snapshotter := range s.snapshotters {
|
||||||
var snapshots []snapshot.Info
|
var snapshots []snapshot.Info
|
||||||
// Do not call `Usage` directly in collect function, because
|
// Do not call `Usage` directly in collect function, because
|
||||||
// `Usage` takes time, we don't want `Walk` to hold read lock
|
// `Usage` takes time, we don't want `Walk` to hold read lock
|
||||||
// of snapshot metadata store for too long time.
|
// of snapshot metadata store for too long time.
|
||||||
// TODO(random-liu): Set timeout for the following 2 contexts.
|
// TODO(random-liu): Set timeout for the following 2 contexts.
|
||||||
if err := s.snapshotter.Walk(ctx, func(ctx context.Context, info snapshot.Info) error {
|
if err := snapshotter.Walk(ctx, func(ctx context.Context, info snapshot.Info) error {
|
||||||
snapshots = append(snapshots, info)
|
snapshots = append(snapshots, info)
|
||||||
return nil
|
return nil
|
||||||
}); err != nil {
|
}); err != nil {
|
||||||
return fmt.Errorf("walk all snapshots failed: %w", err)
|
return fmt.Errorf("walk all snapshots for %q failed: %w", key, err)
|
||||||
}
|
}
|
||||||
for _, info := range snapshots {
|
for _, info := range snapshots {
|
||||||
sn, err := s.store.Get(info.Name)
|
snapshotKey := snapshotstore.Key{
|
||||||
|
Key: info.Name,
|
||||||
|
Snapshotter: key,
|
||||||
|
}
|
||||||
|
sn, err := s.store.Get(snapshotKey)
|
||||||
if err == nil {
|
if err == nil {
|
||||||
// Only update timestamp for non-active snapshot.
|
// Only update timestamp for non-active snapshot.
|
||||||
if sn.Kind == info.Kind && sn.Kind != snapshot.KindActive {
|
if sn.Kind == info.Kind && sn.Kind != snapshot.KindActive {
|
||||||
@ -93,11 +99,14 @@ func (s *snapshotsSyncer) sync() error {
|
|||||||
}
|
}
|
||||||
// Get newest stats if the snapshot is new or active.
|
// Get newest stats if the snapshot is new or active.
|
||||||
sn = snapshotstore.Snapshot{
|
sn = snapshotstore.Snapshot{
|
||||||
|
Key: snapshotstore.Key{
|
||||||
Key: info.Name,
|
Key: info.Name,
|
||||||
|
Snapshotter: key,
|
||||||
|
},
|
||||||
Kind: info.Kind,
|
Kind: info.Kind,
|
||||||
Timestamp: time.Now().UnixNano(),
|
Timestamp: time.Now().UnixNano(),
|
||||||
}
|
}
|
||||||
usage, err := s.snapshotter.Usage(ctx, info.Name)
|
usage, err := snapshotter.Usage(ctx, info.Name)
|
||||||
if err != nil {
|
if err != nil {
|
||||||
if !errdefs.IsNotFound(err) {
|
if !errdefs.IsNotFound(err) {
|
||||||
log.L.WithError(err).Errorf("Failed to get usage for snapshot %q", info.Name)
|
log.L.WithError(err).Errorf("Failed to get usage for snapshot %q", info.Name)
|
||||||
@ -108,6 +117,8 @@ func (s *snapshotsSyncer) sync() error {
|
|||||||
sn.Inodes = uint64(usage.Inodes)
|
sn.Inodes = uint64(usage.Inodes)
|
||||||
s.store.Add(sn)
|
s.store.Add(sn)
|
||||||
}
|
}
|
||||||
|
}
|
||||||
|
|
||||||
for _, sn := range s.store.List() {
|
for _, sn := range s.store.List() {
|
||||||
if sn.Timestamp >= start {
|
if sn.Timestamp >= start {
|
||||||
continue
|
continue
|
||||||
|
@ -30,6 +30,7 @@ import (
|
|||||||
containerstore "github.com/containerd/containerd/pkg/cri/store/container"
|
containerstore "github.com/containerd/containerd/pkg/cri/store/container"
|
||||||
sandboxstore "github.com/containerd/containerd/pkg/cri/store/sandbox"
|
sandboxstore "github.com/containerd/containerd/pkg/cri/store/sandbox"
|
||||||
"github.com/containerd/containerd/pkg/cri/store/stats"
|
"github.com/containerd/containerd/pkg/cri/store/stats"
|
||||||
|
ctrdutil "github.com/containerd/containerd/pkg/cri/util"
|
||||||
"github.com/containerd/containerd/protobuf"
|
"github.com/containerd/containerd/protobuf"
|
||||||
"github.com/containerd/log"
|
"github.com/containerd/log"
|
||||||
"github.com/containerd/typeurl/v2"
|
"github.com/containerd/typeurl/v2"
|
||||||
@ -120,6 +121,12 @@ func (c *criService) toPodSandboxStats(sandbox sandboxstore.Sandbox, statsMap ma
|
|||||||
return nil, nil, fmt.Errorf("failed to find container metric for pod with id %s", sandbox.ID)
|
return nil, nil, fmt.Errorf("failed to find container metric for pod with id %s", sandbox.ID)
|
||||||
}
|
}
|
||||||
|
|
||||||
|
ociRuntime, err := c.getSandboxRuntime(sandbox.Config, sandbox.RuntimeHandler)
|
||||||
|
if err != nil {
|
||||||
|
return nil, nil, fmt.Errorf("failed to get runtimeHandler %q: %w", sandbox.RuntimeHandler, err)
|
||||||
|
}
|
||||||
|
snapshotter := c.RuntimeSnapshotter(ctrdutil.NamespacedContext(), ociRuntime)
|
||||||
|
|
||||||
podRuntimeStats, err := c.convertToCRIStats(podMetric)
|
podRuntimeStats, err := c.convertToCRIStats(podMetric)
|
||||||
if err != nil {
|
if err != nil {
|
||||||
return nil, nil, fmt.Errorf("failed to covert container metrics for sandbox with id %s: %w", sandbox.ID, err)
|
return nil, nil, fmt.Errorf("failed to covert container metrics for sandbox with id %s: %w", sandbox.ID, err)
|
||||||
@ -159,14 +166,14 @@ func (c *criService) toPodSandboxStats(sandbox sandboxstore.Sandbox, statsMap ma
|
|||||||
// If snapshotstore doesn't have cached snapshot information
|
// If snapshotstore doesn't have cached snapshot information
|
||||||
// set WritableLayer usage to zero
|
// set WritableLayer usage to zero
|
||||||
var usedBytes uint64
|
var usedBytes uint64
|
||||||
sn, err := c.GetSnapshot(cntr.ID)
|
sn, err := c.GetSnapshot(cntr.ID, snapshotter)
|
||||||
if err == nil {
|
if err == nil {
|
||||||
usedBytes = sn.Size
|
usedBytes = sn.Size
|
||||||
}
|
}
|
||||||
containerStats.WritableLayer = &runtime.WindowsFilesystemUsage{
|
containerStats.WritableLayer = &runtime.WindowsFilesystemUsage{
|
||||||
Timestamp: sn.Timestamp,
|
Timestamp: sn.Timestamp,
|
||||||
FsId: &runtime.FilesystemIdentifier{
|
FsId: &runtime.FilesystemIdentifier{
|
||||||
Mountpoint: c.imageFSPath,
|
Mountpoint: c.imageFSPaths[snapshotter],
|
||||||
},
|
},
|
||||||
UsedBytes: &runtime.UInt64Value{Value: usedBytes},
|
UsedBytes: &runtime.UInt64Value{Value: usedBytes},
|
||||||
}
|
}
|
||||||
|
@ -392,7 +392,8 @@ func Test_criService_podSandboxStats(t *testing.T) {
|
|||||||
|
|
||||||
func sandboxPod(id string, timestamp time.Time, cachedCPU uint64) sandboxstore.Sandbox {
|
func sandboxPod(id string, timestamp time.Time, cachedCPU uint64) sandboxstore.Sandbox {
|
||||||
return sandboxstore.Sandbox{
|
return sandboxstore.Sandbox{
|
||||||
Metadata: sandboxstore.Metadata{ID: id}, Stats: &stats.ContainerStats{
|
Metadata: sandboxstore.Metadata{ID: id, RuntimeHandler: "runc"},
|
||||||
|
Stats: &stats.ContainerStats{
|
||||||
Timestamp: timestamp,
|
Timestamp: timestamp,
|
||||||
UsageCoreNanoSeconds: cachedCPU,
|
UsageCoreNanoSeconds: cachedCPU,
|
||||||
}}
|
}}
|
||||||
|
@ -77,7 +77,7 @@ type imageService interface {
|
|||||||
UpdateImage(ctx context.Context, r string) error
|
UpdateImage(ctx context.Context, r string) error
|
||||||
|
|
||||||
GetImage(id string) (imagestore.Image, error)
|
GetImage(id string) (imagestore.Image, error)
|
||||||
GetSnapshot(key string) (snapshotstore.Snapshot, error)
|
GetSnapshot(key, snapshotter string) (snapshotstore.Snapshot, error)
|
||||||
|
|
||||||
LocalResolve(refOrID string) (imagestore.Image, error)
|
LocalResolve(refOrID string) (imagestore.Image, error)
|
||||||
}
|
}
|
||||||
@ -87,8 +87,8 @@ type criService struct {
|
|||||||
imageService
|
imageService
|
||||||
// config contains all configurations.
|
// config contains all configurations.
|
||||||
config criconfig.Config
|
config criconfig.Config
|
||||||
// imageFSPath is the path to image filesystem.
|
// imageFSPaths contains path to image filesystem for snapshotters.
|
||||||
imageFSPath string
|
imageFSPaths map[string]string
|
||||||
// os is an interface for all required os operations.
|
// os is an interface for all required os operations.
|
||||||
os osinterface.OS
|
os osinterface.OS
|
||||||
// sandboxStore stores all resources associated with sandboxes.
|
// sandboxStore stores all resources associated with sandboxes.
|
||||||
@ -139,11 +139,21 @@ func NewCRIService(config criconfig.Config, client *containerd.Client, nri *nri.
|
|||||||
return nil, fmt.Errorf("failed to find snapshotter %q", config.ContainerdConfig.Snapshotter)
|
return nil, fmt.Errorf("failed to find snapshotter %q", config.ContainerdConfig.Snapshotter)
|
||||||
}
|
}
|
||||||
|
|
||||||
imageFSPath := imageFSPath(config.ContainerdRootDir, config.ContainerdConfig.Snapshotter)
|
imageFSPaths := map[string]string{}
|
||||||
log.L.Infof("Get image filesystem path %q", imageFSPath)
|
for _, ociRuntime := range config.ContainerdConfig.Runtimes {
|
||||||
|
// Can not use `c.RuntimeSnapshotter() yet, so hard-coding here.`
|
||||||
|
snapshotter := ociRuntime.Snapshotter
|
||||||
|
if snapshotter != "" {
|
||||||
|
imageFSPaths[snapshotter] = imageFSPath(config.ContainerdRootDir, snapshotter)
|
||||||
|
log.L.Infof("Get image filesystem path %q for snapshotter %q", imageFSPaths[snapshotter], snapshotter)
|
||||||
|
}
|
||||||
|
}
|
||||||
|
snapshotter := config.ContainerdConfig.Snapshotter
|
||||||
|
imageFSPaths[snapshotter] = imageFSPath(config.ContainerdRootDir, snapshotter)
|
||||||
|
log.L.Infof("Get image filesystem path %q for snapshotter %q", imageFSPaths[snapshotter], snapshotter)
|
||||||
|
|
||||||
// TODO: expose this as a separate containerd plugin.
|
// TODO: expose this as a separate containerd plugin.
|
||||||
imageService, err := images.NewService(config, imageFSPath, client)
|
imageService, err := images.NewService(config, imageFSPaths, client)
|
||||||
if err != nil {
|
if err != nil {
|
||||||
return nil, fmt.Errorf("unable to create CRI image service: %w", err)
|
return nil, fmt.Errorf("unable to create CRI image service: %w", err)
|
||||||
}
|
}
|
||||||
@ -152,7 +162,7 @@ func NewCRIService(config criconfig.Config, client *containerd.Client, nri *nri.
|
|||||||
imageService: imageService,
|
imageService: imageService,
|
||||||
config: config,
|
config: config,
|
||||||
client: client,
|
client: client,
|
||||||
imageFSPath: imageFSPath,
|
imageFSPaths: imageFSPaths,
|
||||||
os: osinterface.RealOS{},
|
os: osinterface.RealOS{},
|
||||||
sandboxStore: sandboxstore.NewStore(labels),
|
sandboxStore: sandboxstore.NewStore(labels),
|
||||||
containerStore: containerstore.NewStore(labels),
|
containerStore: containerstore.NewStore(labels),
|
||||||
|
@ -33,5 +33,15 @@ var testConfig = criconfig.Config{
|
|||||||
PluginConfig: criconfig.PluginConfig{
|
PluginConfig: criconfig.PluginConfig{
|
||||||
SandboxImage: testSandboxImage,
|
SandboxImage: testSandboxImage,
|
||||||
TolerateMissingHugetlbController: true,
|
TolerateMissingHugetlbController: true,
|
||||||
|
ContainerdConfig: criconfig.ContainerdConfig{
|
||||||
|
Snapshotter: "overlayfs",
|
||||||
|
DefaultRuntimeName: "runc",
|
||||||
|
Runtimes: map[string]criconfig.Runtime{
|
||||||
|
"runc": {
|
||||||
|
Type: "runc",
|
||||||
|
Snapshotter: "overlayfs",
|
||||||
|
},
|
||||||
|
},
|
||||||
|
},
|
||||||
},
|
},
|
||||||
}
|
}
|
||||||
|
@ -23,10 +23,17 @@ import (
|
|||||||
snapshot "github.com/containerd/containerd/snapshots"
|
snapshot "github.com/containerd/containerd/snapshots"
|
||||||
)
|
)
|
||||||
|
|
||||||
|
type Key struct {
|
||||||
|
// Key is the key of the snapshot
|
||||||
|
Key string
|
||||||
|
// Snapshotter is the name of the snapshotter managing the snapshot
|
||||||
|
Snapshotter string
|
||||||
|
}
|
||||||
|
|
||||||
// Snapshot contains the information about the snapshot.
|
// Snapshot contains the information about the snapshot.
|
||||||
type Snapshot struct {
|
type Snapshot struct {
|
||||||
// Key is the key of the snapshot
|
// Key is the key of the snapshot
|
||||||
Key string
|
Key Key
|
||||||
// Kind is the kind of the snapshot (active, committed, view)
|
// Kind is the kind of the snapshot (active, committed, view)
|
||||||
Kind snapshot.Kind
|
Kind snapshot.Kind
|
||||||
// Size is the size of the snapshot in bytes.
|
// Size is the size of the snapshot in bytes.
|
||||||
@ -41,12 +48,12 @@ type Snapshot struct {
|
|||||||
// Store stores all snapshots.
|
// Store stores all snapshots.
|
||||||
type Store struct {
|
type Store struct {
|
||||||
lock sync.RWMutex
|
lock sync.RWMutex
|
||||||
snapshots map[string]Snapshot
|
snapshots map[Key]Snapshot
|
||||||
}
|
}
|
||||||
|
|
||||||
// NewStore creates a snapshot store.
|
// NewStore creates a snapshot store.
|
||||||
func NewStore() *Store {
|
func NewStore() *Store {
|
||||||
return &Store{snapshots: make(map[string]Snapshot)}
|
return &Store{snapshots: make(map[Key]Snapshot)}
|
||||||
}
|
}
|
||||||
|
|
||||||
// Add a snapshot into the store.
|
// Add a snapshot into the store.
|
||||||
@ -58,7 +65,7 @@ func (s *Store) Add(snapshot Snapshot) {
|
|||||||
|
|
||||||
// Get returns the snapshot with specified key. Returns errdefs.ErrNotFound if the
|
// Get returns the snapshot with specified key. Returns errdefs.ErrNotFound if the
|
||||||
// snapshot doesn't exist.
|
// snapshot doesn't exist.
|
||||||
func (s *Store) Get(key string) (Snapshot, error) {
|
func (s *Store) Get(key Key) (Snapshot, error) {
|
||||||
s.lock.RLock()
|
s.lock.RLock()
|
||||||
defer s.lock.RUnlock()
|
defer s.lock.RUnlock()
|
||||||
if sn, ok := s.snapshots[key]; ok {
|
if sn, ok := s.snapshots[key]; ok {
|
||||||
@ -79,7 +86,7 @@ func (s *Store) List() []Snapshot {
|
|||||||
}
|
}
|
||||||
|
|
||||||
// Delete deletes the snapshot with specified key.
|
// Delete deletes the snapshot with specified key.
|
||||||
func (s *Store) Delete(key string) {
|
func (s *Store) Delete(key Key) {
|
||||||
s.lock.Lock()
|
s.lock.Lock()
|
||||||
defer s.lock.Unlock()
|
defer s.lock.Unlock()
|
||||||
delete(s.snapshots, key)
|
delete(s.snapshots, key)
|
||||||
|
@ -27,23 +27,35 @@ import (
|
|||||||
)
|
)
|
||||||
|
|
||||||
func TestSnapshotStore(t *testing.T) {
|
func TestSnapshotStore(t *testing.T) {
|
||||||
snapshots := map[string]Snapshot{
|
key1 := Key{
|
||||||
"key1": {
|
|
||||||
Key: "key1",
|
Key: "key1",
|
||||||
|
Snapshotter: "snapshotter1",
|
||||||
|
}
|
||||||
|
key2 := Key{
|
||||||
|
Key: "key2",
|
||||||
|
Snapshotter: "snapshotter1",
|
||||||
|
}
|
||||||
|
key3 := Key{
|
||||||
|
Key: "key1",
|
||||||
|
Snapshotter: "snapshotter2",
|
||||||
|
}
|
||||||
|
snapshots := map[Key]Snapshot{
|
||||||
|
key1: {
|
||||||
|
Key: key1,
|
||||||
Kind: snapshot.KindActive,
|
Kind: snapshot.KindActive,
|
||||||
Size: 10,
|
Size: 10,
|
||||||
Inodes: 100,
|
Inodes: 100,
|
||||||
Timestamp: time.Now().UnixNano(),
|
Timestamp: time.Now().UnixNano(),
|
||||||
},
|
},
|
||||||
"key2": {
|
key2: {
|
||||||
Key: "key2",
|
Key: key2,
|
||||||
Kind: snapshot.KindCommitted,
|
Kind: snapshot.KindCommitted,
|
||||||
Size: 20,
|
Size: 20,
|
||||||
Inodes: 200,
|
Inodes: 200,
|
||||||
Timestamp: time.Now().UnixNano(),
|
Timestamp: time.Now().UnixNano(),
|
||||||
},
|
},
|
||||||
"key3": {
|
key3: {
|
||||||
Key: "key3",
|
Key: key3,
|
||||||
Kind: snapshot.KindView,
|
Kind: snapshot.KindView,
|
||||||
Size: 0,
|
Size: 0,
|
||||||
Inodes: 0,
|
Inodes: 0,
|
||||||
@ -70,7 +82,19 @@ func TestSnapshotStore(t *testing.T) {
|
|||||||
sns := s.List()
|
sns := s.List()
|
||||||
assert.Len(sns, 3)
|
assert.Len(sns, 3)
|
||||||
|
|
||||||
testKey := "key2"
|
invalidKey := Key{
|
||||||
|
Key: "key2",
|
||||||
|
Snapshotter: "snapshotter2",
|
||||||
|
}
|
||||||
|
t.Logf("should not delete snapshot with invalid key")
|
||||||
|
s.Delete(invalidKey)
|
||||||
|
sns = s.List()
|
||||||
|
assert.Len(sns, 3)
|
||||||
|
|
||||||
|
testKey := Key{
|
||||||
|
Key: "key2",
|
||||||
|
Snapshotter: "snapshotter1",
|
||||||
|
}
|
||||||
|
|
||||||
t.Logf("should be able to delete snapshot")
|
t.Logf("should be able to delete snapshot")
|
||||||
s.Delete(testKey)
|
s.Delete(testKey)
|
||||||
|
Loading…
Reference in New Issue
Block a user